Class SplitLogManager
SplitLogManager monitors the tasks that it creates using the timeoutMonitor thread. If a task's
progress is slow then SplitLogManagerCoordination.checkTasks() will take away the task from
the owner SplitLogWorker and the task will be up for
grabs again. When the task is done then it is deleted by SplitLogManager.
Clients call splitLogDistributed(Path) to split a region server's log files. The caller
thread waits in this method until all the log files have been split.
All the coordination calls made by this class are asynchronous. This is mainly to help reduce response time seen by the callers.
There is race in this design between the SplitLogManager and the SplitLogWorker. SplitLogManager might re-queue a task that has in reality already been completed by a SplitLogWorker. We rely on the idempotency of the log splitting task for correctness.
It is also assumed that every log splitting task is unique and once completed (either with success or with error) it will be not be submitted again. If a task is resubmitted then there is a risk that old "delete task" can delete the re-submission.
-
Nested Class Summary
Nested ClassesModifier and TypeClassDescriptionstatic enumDeprecated.static classDeprecated.in memory state of an active task.static classDeprecated.Keeps track of the batch of tasks submitted together by a caller in splitLogDistributed().static enumDeprecated.private classDeprecated.Periodically checks all active tasks and resubmits the ones that have timed out -
Field Summary
FieldsModifier and TypeFieldDescriptionprivate final ChoreServiceDeprecated.private final org.apache.hadoop.conf.ConfigurationDeprecated.private Set<ServerName>Deprecated.private final ObjectDeprecated.static final intDeprecated.private longDeprecated.private static final org.slf4j.LoggerDeprecated.private final MasterServicesDeprecated.(package private) final ConcurrentMap<String,SplitLogManager.Task> Deprecated.private SplitLogManager.TimeoutMonitorDeprecated.private longDeprecated. -
Constructor Summary
ConstructorsConstructorDescriptionSplitLogManager(MasterServices master, org.apache.hadoop.conf.Configuration conf) Deprecated.Its OK to construct this object even when region-servers are not online. -
Method Summary
Modifier and TypeMethodDescriptionprivate intDeprecated.private SplitLogManager.TaskcreateTaskIfAbsent(String path, SplitLogManager.TaskBatch batch) Deprecated.Returns null on success, existing task on error(package private) booleanenqueueSplitTask(String taskname, SplitLogManager.TaskBatch batch) Deprecated.Add a task entry to coordination if it is not already there.(package private) static intgetBatchWaitTimeMillis(int remainingTasks) Deprecated.Get the amount of time in milliseconds to wait till next check.private List<org.apache.hadoop.fs.FileStatus>getFileList(List<org.apache.hadoop.fs.Path> logDirs, org.apache.hadoop.fs.PathFilter filter) Deprecated.static List<org.apache.hadoop.fs.FileStatus>getFileList(org.apache.hadoop.conf.Configuration conf, List<org.apache.hadoop.fs.Path> logDirs, org.apache.hadoop.fs.PathFilter filter) Deprecated.Get a list of paths that need to be split given a set of server-specific directories and optionally a filter.private SplitLogManagerCoordinationDeprecated.(package private) ConcurrentMap<String,SplitLogManager.Task> getTasks()Deprecated.(package private) voidhandleDeadWorker(ServerName workerName) Deprecated.(package private) voidhandleDeadWorkers(Set<ServerName> serverNames) Deprecated.longsplitLogDistributed(List<org.apache.hadoop.fs.Path> logDirs) Deprecated.The caller will block until all the log files of the given region server have been processed - successfully split or an error is encountered - by an available worker region server.longsplitLogDistributed(Set<ServerName> serverNames, List<org.apache.hadoop.fs.Path> logDirs, org.apache.hadoop.fs.PathFilter filter) Deprecated.The caller will block until all the hbase:meta log files of the given region server have been processed - successfully split or an error is encountered - by an available worker region server.longsplitLogDistributed(org.apache.hadoop.fs.Path logDir) Deprecated.voidstop()Deprecated.private voidwaitForSplittingCompletion(SplitLogManager.TaskBatch batch, MonitoredTask status) Deprecated.
-
Field Details
-
LOG
Deprecated. -
server
Deprecated. -
conf
Deprecated. -
choreService
Deprecated. -
DEFAULT_UNASSIGNED_TIMEOUT
Deprecated.- See Also:
-
unassignedTimeout
Deprecated. -
lastTaskCreateTime
Deprecated. -
tasks
Deprecated. -
timeoutMonitor
Deprecated. -
deadWorkers
Deprecated. -
deadWorkersLock
Deprecated.
-
-
Constructor Details
-
SplitLogManager
public SplitLogManager(MasterServices master, org.apache.hadoop.conf.Configuration conf) throws IOException Deprecated.Its OK to construct this object even when region-servers are not online. It does lookup the orphan tasks in coordination engine but it doesn't block waiting for them to be done.- Parameters:
master- the master servicesconf- the HBase configuration- Throws:
IOException
-
-
Method Details
-
getSplitLogManagerCoordination
Deprecated. -
getFileList
private List<org.apache.hadoop.fs.FileStatus> getFileList(List<org.apache.hadoop.fs.Path> logDirs, org.apache.hadoop.fs.PathFilter filter) throws IOException Deprecated.- Throws:
IOException
-
getFileList
public static List<org.apache.hadoop.fs.FileStatus> getFileList(org.apache.hadoop.conf.Configuration conf, List<org.apache.hadoop.fs.Path> logDirs, org.apache.hadoop.fs.PathFilter filter) throws IOException Deprecated.Get a list of paths that need to be split given a set of server-specific directories and optionally a filter. SeeAbstractFSWALProvider.getServerNameFromWALDirectoryName(org.apache.hadoop.conf.Configuration, java.lang.String)for more info on directory layout. Should be package-private, but is needed byWALSplitter.split(Path, Path, Path, FileSystem, Configuration, org.apache.hadoop.hbase.wal.WALFactory)for tests.- Throws:
IOException
-
splitLogDistributed
Deprecated.- Parameters:
logDir- one region sever wal dir path in .logs- Returns:
- cumulative size of the logfiles split
- Throws:
IOException- if there was an error while splitting any log file
-
splitLogDistributed
Deprecated.The caller will block until all the log files of the given region server have been processed - successfully split or an error is encountered - by an available worker region server. This method must only be called after the region servers have been brought online.- Parameters:
logDirs- List of log dirs to split- Returns:
- cumulative size of the logfiles split
- Throws:
IOException- If there was an error while splitting any log file
-
splitLogDistributed
public long splitLogDistributed(Set<ServerName> serverNames, List<org.apache.hadoop.fs.Path> logDirs, org.apache.hadoop.fs.PathFilter filter) throws IOException Deprecated.The caller will block until all the hbase:meta log files of the given region server have been processed - successfully split or an error is encountered - by an available worker region server. This method must only be called after the region servers have been brought online.- Parameters:
logDirs- List of log dirs to splitfilter- the Path filter to select specific files for considering- Returns:
- cumulative size of the logfiles split
- Throws:
IOException- If there was an error while splitting any log file
-
enqueueSplitTask
Deprecated.Add a task entry to coordination if it is not already there.- Parameters:
taskname- the path of the log to be splitbatch- the batch this task belongs to- Returns:
- true if a new entry is created, false if it is already there.
-
getBatchWaitTimeMillis
Deprecated.Get the amount of time in milliseconds to wait till next check. Check less frequently if a bunch of work to do still. At a max, check every minute. At a minimum, check every 100ms. This is to alleviate case where perhaps there are a bunch of threads waiting on a completion. For example, if the zk-based implementation, we will scan the '/hbase/splitWAL' dir every time through this loop. If there are lots of WALs to split -- could be tens of thousands if big cluster -- then it will take a while. If the Master has many SCPs waiting on wal splitting -- could be up to 10 x the configured PE thread count (default would be 160) -- then the Master will be putting up a bunch of load on zk. -
waitForSplittingCompletion
Deprecated. -
getTasks
Deprecated. -
activeTasks
Deprecated. -
createTaskIfAbsent
Deprecated.Returns null on success, existing task on error -
stop
Deprecated. -
handleDeadWorker
Deprecated. -
handleDeadWorkers
Deprecated.
-