Class SnapshotManager
java.lang.Object
org.apache.hadoop.hbase.procedure.ProcedureManager
org.apache.hadoop.hbase.procedure.MasterProcedureManager
org.apache.hadoop.hbase.master.snapshot.SnapshotManager
- All Implemented Interfaces:
Stoppable
@LimitedPrivate("Configuration")
@Unstable
public class SnapshotManager
extends MasterProcedureManager
implements Stoppable
This class manages the procedure of taking and restoring snapshots. There is only one
SnapshotManager for the master.
The class provides methods for monitoring in-progress snapshot actions.
Note: Currently there can only be one snapshot being taken at a time over the cluster. This is a simplification in the current implementation.
-
Field Summary
FieldsModifier and TypeFieldDescriptionprivate ProcedureCoordinatorprivate ExecutorServicestatic final StringEnable or disable snapshot supportstatic final StringWait time before removing a finished sentinel from the in-progress map NOTE: This is used as a safety auto cleanup.private booleanprivate static final org.slf4j.Loggerprivate MasterServicesstatic final StringName of the operation to use in the controllerprivate org.apache.hadoop.fs.Pathprivate final ScheduledExecutorServicestatic final StringConf key for preserving original max file size configsstatic final intnumber of current operations running on the masterstatic final StringConf key for # of threads used by the SnapshotManager thread poolstatic final StringEnable or disable snapshot procedurestatic final booleanstatic final longprivate static final intBy default, check to see if the snapshot is complete every WAKE MILLIS (ms)private static final StringConf key for # of ms elapsed between checks for snapshot errors while waiting for completion.private ScheduledFuture<?>private final Map<TableName,SnapshotSentinel> private final ConcurrentHashMap<org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDescription,Long> private booleanprivate ReentrantReadWriteLockRead write lock between taking snapshot and snapshot HFile cleaner.private WorkerAssigner -
Constructor Summary
ConstructorsConstructorDescriptionSnapshotManager(MasterServices master, ProcedureCoordinator coordinator, ExecutorService pool, int sentinelCleanInterval) Fully specify all necessary components of a snapshot manager. -
Method Summary
Modifier and TypeMethodDescriptionvoidcheckPermissions(org.apache.hadoop.hbase.shaded.protobuf.generated.HBaseProtos.ProcedureDescription desc, AccessChecker accessChecker, User user) Check for required permissions before executing the procedure.voidThrows an exception if snapshot operations (take a snapshot, restore, clone) are not supported.private voidcheckSnapshotSupport(org.apache.hadoop.conf.Configuration conf, MasterFileSystem mfs) Called at startup, to verify if snapshot operation is supported, and to avoid starting the master if there're snapshots present but the cleaners needed are missing.private voidRemove the procedures that are marked as finishedprivate voidRemove the procedures that are marked as finishedprivate voidRemoves "abandoned" snapshot/restore requests.private voidcleanupSentinels(Map<TableName, SnapshotSentinel> sentinels) Remove the sentinels that are marked as finished and the completion time has exceeded the removal timeout.(package private) longcloneSnapshot(org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDescription snapshot, TableDescriptor tableDescriptor, NonceKey nonceKey, boolean restoreAcl, String customSFT) Clone the specified snapshot into a new table.private longcloneSnapshot(org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDescription reqSnapshot, TableName tableName, org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDescription snapshot, TableDescriptor snapshotTableDesc, NonceKey nonceKey, boolean restoreAcl, String customSFT) Clone the specified snapshot.voiddeleteSnapshot(org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDescription snapshot) Delete the specified snapshotvoidexecProcedure(org.apache.hadoop.hbase.shaded.protobuf.generated.HBaseProtos.ProcedureDescription desc) Execute a distributed procedure on clustergetAvailableWorker(ServerName serverName) List<org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDescription>Gets the list of all completed snapshots.private List<org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDescription>getCompletedSnapshots(org.apache.hadoop.fs.Path snapshotDir, boolean withCpCall) Gets the list of all completed snapshots.(package private) ProcedureCoordinatorReturns distributed commit coordinator for all running snapshotsReturn the unique signature of the procedure.voidinitialize(MasterServices master, MetricsMaster metricsMaster) Initialize a globally barriered procedure for master.booleanisProcedureDone(org.apache.hadoop.hbase.shaded.protobuf.generated.HBaseProtos.ProcedureDescription desc) Check if the procedure is finished successfullyprivate booleanisRestoringTable(TableName tableName) Verify if the restore of the specified table is in progress.private booleanisSnapshotCompleted(org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDescription snapshot) Check to see if the snapshot is one of the currently completed snapshots Returns true if the snapshot exists in the "completed snapshots folder".booleanisSnapshotDone(org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDescription expected) Check if the specified snapshot is donebooleanReturns True ifStoppable.stop(String)has been closed.booleanisTableTakingAnySnapshot(TableName tableName) booleanThe snapshot operation processing as following:
1.(package private) booleanisTakingSnapshot(org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDescription snapshot, boolean checkTable) Check to see if there is a snapshot in progress with the same name or on the same table.booleanisTakingSnapshot(TableName tableName) Check to see if the specified table has a snapshot in progress.private booleanisTakingSnapshot(TableName tableName, boolean checkProcedure) Check to see if the specified table has a snapshot in progress.voidprepareWorkingDirectory(org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDescription snapshot) Check to make sure that we are OK to run the passed snapshot.voidregisterSnapshotProcedure(org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDescription snapshot, long procId) voidreleaseSnapshotVerifyWorker(SnapshotVerifyProcedure procedure, ServerName worker, MasterProcedureScheduler scheduler) private SnapshotSentinelremoveSentinelIfFinished(Map<TableName, SnapshotSentinel> sentinels, org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDescription snapshot) Return the handler if it is currently live and has the same snapshot target name.private voidCleans up any zk-coordinated snapshots in the snapshot/.tmp directory that were left from failed snapshot attempts.longrestoreOrCloneSnapshot(org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDescription reqSnapshot, NonceKey nonceKey, boolean restoreAcl, String customSFT) Restore or Clone the specified snapshotprivate longrestoreSnapshot(org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDescription snapshot, TableDescriptor tableDescriptor, NonceKey nonceKey, boolean restoreAcl) Restore the specified snapshot.private longrestoreSnapshot(org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDescription reqSnapshot, TableName tableName, org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDescription snapshot, TableDescriptor snapshotTableDesc, NonceKey nonceKey, boolean restoreAcl) Restore the specified snapshot.private voidprivate voidprivate TableDescriptorsanityCheckBeforeSnapshot(org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDescription snapshot, boolean checkTable) Check if the snapshot can be taken.voidsetSnapshotHandlerForTesting(TableName tableName, SnapshotSentinel handler) Set the handler for the current snapshotprivate voidsnapshotDisabledTable(org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDescription snapshot) Take a snapshot of a disabled table.private voidsnapshotEnabledTable(org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDescription snapshot) Take a snapshot of an enabled table.booleanprivate voidsnapshotTable(org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDescription snapshot, TakeSnapshotHandler handler) Take a snapshot using the specified handler.voidStop this service.private longsubmitSnapshotProcedure(org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDescription snapshot, long nonceGroup, long nonce) voidtakeSnapshot(org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDescription snapshot) Take a snapshot based on the enabled/disabled state of the table.longtakeSnapshot(org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDescription snapshot, long nonceGroup, long nonce) private voidtakeSnapshotInternal(org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDescription snapshot) private org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDescriptiontoSnapshotDescription(org.apache.hadoop.hbase.shaded.protobuf.generated.HBaseProtos.ProcedureDescription desc) voidunregisterSnapshotProcedure(org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDescription snapshot, long procId) private static voidupdateWorkingDirAclsIfRequired(org.apache.hadoop.fs.Path workingDir, org.apache.hadoop.fs.FileSystem workingDirFS) If the parent dir of the snapshot working dir (e.g.Methods inherited from class org.apache.hadoop.hbase.procedure.MasterProcedureManager
execProcedureWithRetMethods inherited from class org.apache.hadoop.hbase.procedure.ProcedureManager
equals, hashCode
-
Field Details
-
LOG
-
SNAPSHOT_WAKE_MILLIS_DEFAULT
By default, check to see if the snapshot is complete every WAKE MILLIS (ms)- See Also:
-
HBASE_SNAPSHOT_SENTINELS_CLEANUP_TIMEOUT_MILLIS
Wait time before removing a finished sentinel from the in-progress map NOTE: This is used as a safety auto cleanup. The snapshot and restore handlers map entries are removed when a user asks if a snapshot or restore is completed. This operation is part of the HBaseAdmin snapshot/restore API flow. In case something fails on the client side and the snapshot/restore state is not reclaimed after a default timeout, the entry is removed from the in-progress map. At this point, if the user asks for the snapshot/restore status, the result will be snapshot done if exists or failed if it doesn't exists.- See Also:
-
SNAPSHOT_SENTINELS_CLEANUP_TIMEOUT_MILLS_DEFAULT
- See Also:
-
HBASE_SNAPSHOT_ENABLED
Enable or disable snapshot support- See Also:
-
SNAPSHOT_WAKE_MILLIS_KEY
Conf key for # of ms elapsed between checks for snapshot errors while waiting for completion.- See Also:
-
ONLINE_SNAPSHOT_CONTROLLER_DESCRIPTION
Name of the operation to use in the controller- See Also:
-
SNAPSHOT_POOL_THREADS_KEY
Conf key for # of threads used by the SnapshotManager thread pool- See Also:
-
SNAPSHOT_POOL_THREADS_DEFAULT
number of current operations running on the master- See Also:
-
SNAPSHOT_MAX_FILE_SIZE_PRESERVE
Conf key for preserving original max file size configs- See Also:
-
SNAPSHOT_PROCEDURE_ENABLED
Enable or disable snapshot procedure- See Also:
-
SNAPSHOT_PROCEDURE_ENABLED_DEFAULT
- See Also:
-
stopped
-
master
-
coordinator
-
isSnapshotSupported
-
snapshotHandlers
-
scheduleThreadPool
-
snapshotHandlerChoreCleanerTask
-
restoreTableToProcIdMap
-
snapshotToProcIdMap
private final ConcurrentHashMap<org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDescription,Long> snapshotToProcIdMap -
verifyWorkerAssigner
-
rootDir
-
executorService
-
takingSnapshotLock
Read write lock between taking snapshot and snapshot HFile cleaner. The cleaner should skip to check the HFiles if any snapshot is in progress, otherwise it may clean a HFile which would belongs to the newly creating snapshot. So we should grab the write lock first when cleaner start to work. (See HBASE-21387)
-
-
Constructor Details
-
SnapshotManager
public SnapshotManager() -
SnapshotManager
@Private SnapshotManager(MasterServices master, ProcedureCoordinator coordinator, ExecutorService pool, int sentinelCleanInterval) throws IOException, UnsupportedOperationException Fully specify all necessary components of a snapshot manager. Exposed for testing.- Parameters:
master- services for the master where the manager is runningcoordinator- procedure coordinator instance. exposed for testing.pool- HBase ExecutorServcie instance, exposed for testing.- Throws:
IOExceptionUnsupportedOperationException
-
-
Method Details
-
getCompletedSnapshots
public List<org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDescription> getCompletedSnapshots() throws IOExceptionGets the list of all completed snapshots.- Returns:
- list of SnapshotDescriptions
- Throws:
IOException- File system exception
-
getCompletedSnapshots
private List<org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDescription> getCompletedSnapshots(org.apache.hadoop.fs.Path snapshotDir, boolean withCpCall) throws IOException Gets the list of all completed snapshots.- Parameters:
snapshotDir- snapshot directorywithCpCall- Whether to call CP hooks- Returns:
- list of SnapshotDescriptions
- Throws:
IOException- File system exception
-
resetTempDir
Cleans up any zk-coordinated snapshots in the snapshot/.tmp directory that were left from failed snapshot attempts. For unfinished procedure2-coordinated snapshots, keep the working directory.- Throws:
IOException- if we can't reach the filesystem
-
deleteSnapshot
public void deleteSnapshot(org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDescription snapshot) throws IOException Delete the specified snapshot- Throws:
SnapshotDoesNotExistException- If the specified snapshot does not exist.IOException- For filesystem IOExceptions
-
isSnapshotDone
public boolean isSnapshotDone(org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDescription expected) throws IOException Check if the specified snapshot is done- Returns:
- true if snapshot is ready to be restored, false if it is still being taken.
- Throws:
IOException- IOException if error from HDFS or RPCUnknownSnapshotException- if snapshot is invalid or does not exist.
-
isTakingSnapshot
boolean isTakingSnapshot(org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDescription snapshot, boolean checkTable) Check to see if there is a snapshot in progress with the same name or on the same table. Currently we have a limitation only allowing a single snapshot per table at a time. Also we don't allow snapshot with the same name.- Parameters:
snapshot- description of the snapshot being checked.checkTable- check if the table is already taking a snapshot.- Returns:
- true if there is a snapshot in progress with the same name or on the same table.
-
isTakingSnapshot
Check to see if the specified table has a snapshot in progress. Currently we have a limitation only allowing a single snapshot per table at a time.- Parameters:
tableName- name of the table being snapshotted.- Returns:
- true if there is a snapshot in progress on the specified table.
-
isTableTakingAnySnapshot
-
isTakingSnapshot
Check to see if the specified table has a snapshot in progress. Since we introduce the SnapshotProcedure, it is a little bit different from before. For zk-coordinated snapshot, we can just consider tables in snapshotHandlers only, but forMergeTableRegionsProcedureandSplitTableRegionProcedure, we need to consider tables in snapshotToProcIdMap also, for the snapshot procedure, we don't need to check if table in snapshot.- Parameters:
tableName- name of the table being snapshotted.checkProcedure- true if we should check tables in snapshotToProcIdMap- Returns:
- true if there is a snapshot in progress on the specified table.
-
prepareWorkingDirectory
public void prepareWorkingDirectory(org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDescription snapshot) throws HBaseSnapshotException Check to make sure that we are OK to run the passed snapshot. Checks to make sure that we aren't already running a snapshot or restore on the requested table.- Parameters:
snapshot- description of the snapshot we want to start- Throws:
HBaseSnapshotException- if the filesystem could not be prepared to start the snapshot
-
updateWorkingDirAclsIfRequired
private static void updateWorkingDirAclsIfRequired(org.apache.hadoop.fs.Path workingDir, org.apache.hadoop.fs.FileSystem workingDirFS) throws IOException If the parent dir of the snapshot working dir (e.g. /hbase/.hbase-snapshot) has non-empty ACLs, use them for the current working dir (e.g. /hbase/.hbase-snapshot/.tmp/{snapshot-name}) so that regardless of whether the snapshot commit phase performs atomic rename or non-atomic copy of the working dir to new snapshot dir, the ACLs are retained.- Parameters:
workingDir- working dir to build the snapshot.workingDirFS- working dir file system.- Throws:
IOException- If ACL read/modify operation fails.
-
snapshotDisabledTable
private void snapshotDisabledTable(org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDescription snapshot) throws IOException Take a snapshot of a disabled table.- Parameters:
snapshot- description of the snapshot to take. Modified to beSnapshotProtos.SnapshotDescription.Type.DISABLED.- Throws:
IOException- if the snapshot could not be started or filesystem for snapshot temporary directory could not be determined
-
snapshotEnabledTable
private void snapshotEnabledTable(org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDescription snapshot) throws IOException Take a snapshot of an enabled table.- Parameters:
snapshot- description of the snapshot to take.- Throws:
IOException- if the snapshot could not be started or filesystem for snapshot temporary directory could not be determined
-
snapshotTable
private void snapshotTable(org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDescription snapshot, TakeSnapshotHandler handler) throws IOException Take a snapshot using the specified handler. On failure the snapshot temporary working directory is removed. NOTE: prepareToTakeSnapshot() called before this one takes care of the rejecting the snapshot request if the table is busy with another snapshot/restore operation.- Parameters:
snapshot- the snapshot descriptionhandler- the snapshot handler- Throws:
IOException
-
getTakingSnapshotLock
-
isTakingAnySnapshot
The snapshot operation processing as following:
1. Create a Snapshot Handler, and do some initialization;
2. Put the handler into snapshotHandlers
So when we consider if any snapshot is taking, we should consider both the takingSnapshotLock and snapshotHandlers;- Returns:
- true to indicate that there're some running snapshots.
-
takeSnapshot
public void takeSnapshot(org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDescription snapshot) throws IOException Take a snapshot based on the enabled/disabled state of the table.- Throws:
HBaseSnapshotException- when a snapshot specific exception occurs.IOException- when some sort of generic IO exception occurs.
-
takeSnapshot
public long takeSnapshot(org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDescription snapshot, long nonceGroup, long nonce) throws IOException - Throws:
IOException
-
submitSnapshotProcedure
private long submitSnapshotProcedure(org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDescription snapshot, long nonceGroup, long nonce) throws IOException - Throws:
IOException
-
takeSnapshotInternal
private void takeSnapshotInternal(org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDescription snapshot) throws IOException - Throws:
IOException
-
sanityCheckBeforeSnapshot
private TableDescriptor sanityCheckBeforeSnapshot(org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDescription snapshot, boolean checkTable) throws IOException Check if the snapshot can be taken. Currently we have some limitations, for zk-coordinated snapshot, we don't allow snapshot with same name or taking multiple snapshots of a table at the same time, for procedure-coordinated snapshot, we don't allow snapshot with same name.- Parameters:
snapshot- description of the snapshot being checked.checkTable- check if the table is already taking a snapshot. For zk-coordinated snapshot, we need to check if another zk-coordinated snapshot is in progress, for the snapshot procedure, this is unnecessary.- Returns:
- the table descriptor of the table
- Throws:
IOException
-
setSnapshotHandlerForTesting
Set the handler for the current snapshotExposed for TESTING
- Parameters:
handler- handler the master should use TODO get rid of this if possible, repackaging, modify tests.
-
getCoordinator
Returns distributed commit coordinator for all running snapshots -
isSnapshotCompleted
private boolean isSnapshotCompleted(org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDescription snapshot) throws IOException Check to see if the snapshot is one of the currently completed snapshots Returns true if the snapshot exists in the "completed snapshots folder".- Parameters:
snapshot- expected snapshot to check- Returns:
- true if the snapshot is stored on the
FileSystem, false if is not stored - Throws:
IOException- if the filesystem throws an unexpected exception,IllegalArgumentException- if snapshot name is invalid.
-
cloneSnapshot
private long cloneSnapshot(org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDescription reqSnapshot, TableName tableName, org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDescription snapshot, TableDescriptor snapshotTableDesc, NonceKey nonceKey, boolean restoreAcl, String customSFT) throws IOException Clone the specified snapshot. The clone will fail if the destination table has a snapshot or restore in progress.- Parameters:
reqSnapshot- Snapshot Descriptor from requesttableName- table to clonesnapshot- Snapshot DescriptorsnapshotTableDesc- Table DescriptornonceKey- unique identifier to prevent duplicated RPC- Returns:
- procId the ID of the clone snapshot procedure
- Throws:
IOException
-
cloneSnapshot
long cloneSnapshot(org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDescription snapshot, TableDescriptor tableDescriptor, NonceKey nonceKey, boolean restoreAcl, String customSFT) throws HBaseSnapshotException Clone the specified snapshot into a new table. The operation will fail if the destination table has a snapshot or restore in progress.- Parameters:
snapshot- Snapshot DescriptortableDescriptor- Table Descriptor of the table to createnonceKey- unique identifier to prevent duplicated RPC- Returns:
- procId the ID of the clone snapshot procedure
- Throws:
HBaseSnapshotException
-
restoreOrCloneSnapshot
public long restoreOrCloneSnapshot(org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDescription reqSnapshot, NonceKey nonceKey, boolean restoreAcl, String customSFT) throws IOException Restore or Clone the specified snapshot- Parameters:
nonceKey- unique identifier to prevent duplicated RPC- Throws:
IOException
-
restoreSnapshot
private long restoreSnapshot(org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDescription reqSnapshot, TableName tableName, org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDescription snapshot, TableDescriptor snapshotTableDesc, NonceKey nonceKey, boolean restoreAcl) throws IOException Restore the specified snapshot. The restore will fail if the destination table has a snapshot or restore in progress.- Parameters:
reqSnapshot- Snapshot Descriptor from requesttableName- table to restoresnapshot- Snapshot DescriptorsnapshotTableDesc- Table DescriptornonceKey- unique identifier to prevent duplicated RPCrestoreAcl- true to restore acl of snapshot- Returns:
- procId the ID of the restore snapshot procedure
- Throws:
IOException
-
restoreSnapshot
private long restoreSnapshot(org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDescription snapshot, TableDescriptor tableDescriptor, NonceKey nonceKey, boolean restoreAcl) throws HBaseSnapshotException Restore the specified snapshot. The restore will fail if the destination table has a snapshot or restore in progress.- Parameters:
snapshot- Snapshot DescriptortableDescriptor- Table DescriptornonceKey- unique identifier to prevent duplicated RPCrestoreAcl- true to restore acl of snapshot- Returns:
- procId the ID of the restore snapshot procedure
- Throws:
HBaseSnapshotException
-
isRestoringTable
Verify if the restore of the specified table is in progress.- Parameters:
tableName- table under restore- Returns:
- true if there is a restore in progress of the specified table.
-
removeSentinelIfFinished
private SnapshotSentinel removeSentinelIfFinished(Map<TableName, SnapshotSentinel> sentinels, org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDescription snapshot) Return the handler if it is currently live and has the same snapshot target name. The handler is removed from the sentinels map if completed.- Parameters:
sentinels- live handlerssnapshot- snapshot description- Returns:
- null if doesn't match, else a live handler.
-
cleanupSentinels
Removes "abandoned" snapshot/restore requests. As part of the HBaseAdmin snapshot/restore API the operation status is checked until completed, and the in-progress maps are cleaned up when the status of a completed task is requested. To avoid having sentinels staying around for long time if something client side is failed, each operation tries to clean up the in-progress maps sentinels finished from a long time. -
cleanupSentinels
Remove the sentinels that are marked as finished and the completion time has exceeded the removal timeout.- Parameters:
sentinels- map of sentinels to clean
-
cleanupCompletedRestoreInMap
Remove the procedures that are marked as finished -
cleanupCompletedSnapshotInMap
Remove the procedures that are marked as finished -
stop
Description copied from interface:StoppableStop this service. Implementers should favor logging errors over throwing RuntimeExceptions. -
isStopped
Description copied from interface:StoppableReturns True ifStoppable.stop(String)has been closed. -
checkSnapshotSupport
Throws an exception if snapshot operations (take a snapshot, restore, clone) are not supported. Called at the beginning of snapshot() and restoreSnapshot() methods.- Throws:
UnsupportedOperationException- if snapshot are not supported
-
checkSnapshotSupport
private void checkSnapshotSupport(org.apache.hadoop.conf.Configuration conf, MasterFileSystem mfs) throws IOException, UnsupportedOperationException Called at startup, to verify if snapshot operation is supported, and to avoid starting the master if there're snapshots present but the cleaners needed are missing. Otherwise we can end up with snapshot data loss.- Parameters:
conf- TheConfigurationobject to usemfs- The MasterFileSystem to use- Throws:
IOException- in case of file-system operation failureUnsupportedOperationException- in case cleaners are missing and there're snapshot in the system
-
initialize
public void initialize(MasterServices master, MetricsMaster metricsMaster) throws org.apache.zookeeper.KeeperException, IOException, UnsupportedOperationException Description copied from class:MasterProcedureManagerInitialize a globally barriered procedure for master.- Specified by:
initializein classMasterProcedureManager- Parameters:
master- Master service interface- Throws:
org.apache.zookeeper.KeeperExceptionIOExceptionUnsupportedOperationException
-
restoreUnfinishedSnapshotProcedure
-
getProcedureSignature
Description copied from class:ProcedureManagerReturn the unique signature of the procedure. This signature uniquely identifies the procedure. By default, this signature is the string used in the procedure controller (i.e., the root ZK node name for the procedure)- Specified by:
getProcedureSignaturein classProcedureManager
-
execProcedure
public void execProcedure(org.apache.hadoop.hbase.shaded.protobuf.generated.HBaseProtos.ProcedureDescription desc) throws IOException Description copied from class:MasterProcedureManagerExecute a distributed procedure on cluster- Overrides:
execProcedurein classMasterProcedureManager- Parameters:
desc- Procedure description- Throws:
IOException
-
checkPermissions
public void checkPermissions(org.apache.hadoop.hbase.shaded.protobuf.generated.HBaseProtos.ProcedureDescription desc, AccessChecker accessChecker, User user) throws IOException Description copied from class:MasterProcedureManagerCheck for required permissions before executing the procedure.- Specified by:
checkPermissionsin classMasterProcedureManager- Throws:
IOException- if permissions requirements are not met.
-
isProcedureDone
public boolean isProcedureDone(org.apache.hadoop.hbase.shaded.protobuf.generated.HBaseProtos.ProcedureDescription desc) throws IOException Description copied from class:MasterProcedureManagerCheck if the procedure is finished successfully- Specified by:
isProcedureDonein classMasterProcedureManager- Parameters:
desc- Procedure description- Returns:
- true if the specified procedure is finished successfully
- Throws:
IOException
-
toSnapshotDescription
private org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDescription toSnapshotDescription(org.apache.hadoop.hbase.shaded.protobuf.generated.HBaseProtos.ProcedureDescription desc) throws IOException - Throws:
IOException
-
registerSnapshotProcedure
public void registerSnapshotProcedure(org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDescription snapshot, long procId) -
unregisterSnapshotProcedure
public void unregisterSnapshotProcedure(org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDescription snapshot, long procId) -
snapshotProcedureEnabled
-
acquireSnapshotVerifyWorker
public ServerName acquireSnapshotVerifyWorker(SnapshotVerifyProcedure procedure) throws ProcedureSuspendedException - Throws:
ProcedureSuspendedException
-
releaseSnapshotVerifyWorker
public void releaseSnapshotVerifyWorker(SnapshotVerifyProcedure procedure, ServerName worker, MasterProcedureScheduler scheduler) -
restoreWorkers
-
getAvailableWorker
-