Class SplitTableRegionProcedure
java.lang.Object
org.apache.hadoop.hbase.procedure2.Procedure<TEnvironment>
org.apache.hadoop.hbase.procedure2.StateMachineProcedure<MasterProcedureEnv,TState>
org.apache.hadoop.hbase.master.procedure.AbstractStateMachineTableProcedure<TState>
org.apache.hadoop.hbase.master.procedure.AbstractStateMachineRegionProcedure<org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.SplitTableRegionState>
org.apache.hadoop.hbase.master.assignment.SplitTableRegionProcedure
- All Implemented Interfaces:
Comparable<Procedure<MasterProcedureEnv>>
,TableProcedureInterface
@Private
public class SplitTableRegionProcedure
extends AbstractStateMachineRegionProcedure<org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.SplitTableRegionState>
The procedure to split a region in a table. Takes lock on the parent region. It holds the lock
for the life of the procedure.
Throws exception on construction if determines context hostile to spllt (cluster going down or master is shutting down or table is disabled).
-
Nested Class Summary
Modifier and TypeClassDescriptionprivate class
Utility class used to do the file splitting / reference writing in parallel instead of sequentially.Nested classes/interfaces inherited from class org.apache.hadoop.hbase.procedure2.StateMachineProcedure
StateMachineProcedure.Flow
Nested classes/interfaces inherited from class org.apache.hadoop.hbase.procedure2.Procedure
Procedure.LockState
Nested classes/interfaces inherited from interface org.apache.hadoop.hbase.master.procedure.TableProcedureInterface
TableProcedureInterface.TableOperationType
-
Field Summary
Modifier and TypeFieldDescriptionprivate byte[]
private RegionInfo
private RegionInfo
private static final RegionState.State[]
private static final org.slf4j.Logger
private RegionSplitPolicy
Fields inherited from class org.apache.hadoop.hbase.procedure2.StateMachineProcedure
stateCount
Fields inherited from class org.apache.hadoop.hbase.procedure2.Procedure
NO_PROC_ID, NO_TIMEOUT
Fields inherited from interface org.apache.hadoop.hbase.master.procedure.TableProcedureInterface
DUMMY_NAMESPACE_TABLE_NAME
-
Constructor Summary
ConstructorDescriptionSplitTableRegionProcedure
(MasterProcedureEnv env, RegionInfo regionToSplit, byte[] splitRow) -
Method Summary
Modifier and TypeMethodDescriptionprotected boolean
abort
(MasterProcedureEnv env) The abort() call is asynchronous and each procedure must decide how to deal with it, if they want to be abortable.protected Procedure.LockState
The user should override this method if they need a lock on an Entity.private void
assertSplitResultFilesCount
(org.apache.hadoop.fs.FileSystem fs, int expectedSplitResultFileCount, org.apache.hadoop.fs.Path dir) private void
private void
checkSplittable
(MasterProcedureEnv env, RegionInfo regionToSplit) Check whether the region is splittableprivate TransitRegionStateProcedure[]
void
Create daughter regionsprivate TransitRegionStateProcedure[]
private void
protected void
deserializeStateData
(ProcedureStateSerializer serializer) Called on store load to allow the user to decode the previously serialized state.protected StateMachineProcedure.Flow
executeFromState
(MasterProcedureEnv env, org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.SplitTableRegionState state) called to perform a single step of the specified 'state' of the procedureprivate static long
Calculate daughter regionid to use.protected org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.SplitTableRegionState
Return the initial state object that will be used for the first call to executeFromState().private RegionInfo
private ServerName
protected ProcedureMetrics
Override this method to provide procedure specific counters for submitted count, failed count and time histogram.private int
private byte[]
protected org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.SplitTableRegionState
getState
(int stateId) Convert an ordinal (or state id) to an Enum (or more descriptive) state object.protected int
getStateId
(org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.SplitTableRegionState state) Convert the Enum (or more descriptive) state object to an ordinal (or state id).Given an operation type we can take decisions about what to do with pending operations.private boolean
protected boolean
isRollbackSupported
(org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.SplitTableRegionState state) Used by the default implementation of abort() to know if the current state can be aborted and rollback can be triggered.private void
Rollback close parent regionprivate void
Action after rollback a split table region action.private void
Post split region actionsboolean
Prepare to Split region.private void
Action before splitting region in a table.private void
Pre split region actions after the Point-of-No-Return stepprivate void
Post split region actions before the Point-of-No-Return stepprotected void
The user should override this method, and release lock if necessary.private void
private void
Rollback prepare split regionprotected void
rollbackState
(MasterProcedureEnv env, org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.SplitTableRegionState state) To rollbackSplitTableRegionProcedure
, an AssignProcedure is asynchronously submitted for parent region to be split (rollback doesn't wait on the completion of the AssignProcedure) .protected void
serializeStateData
(ProcedureStateSerializer serializer) The user-level code of the procedure may have some state to persist (e.g.private Pair<org.apache.hadoop.fs.Path,
org.apache.hadoop.fs.Path> splitStoreFile
(HRegionFileSystem regionFs, StoreFileTracker tracker, byte[] family, HStoreFile sf) splitStoreFiles
(MasterProcedureEnv env, HRegionFileSystem regionFs) Create Split directoryvoid
Extend the toString() information with the procedure details e.g.private void
Add daughter regions to METAprivate void
Methods inherited from class org.apache.hadoop.hbase.master.procedure.AbstractStateMachineRegionProcedure
getRegion, getTableName, holdLock, setFailure, setRegion
Methods inherited from class org.apache.hadoop.hbase.master.procedure.AbstractStateMachineTableProcedure
checkOnline, checkTableModifiable, getUser, getWALRegionDir, isTableEnabled, preflightChecks, releaseSyncLatch, setUser, waitInitialized
Methods inherited from class org.apache.hadoop.hbase.procedure2.StateMachineProcedure
addChildProcedure, execute, failIfAborted, getCurrentState, getCurrentStateId, getCycles, isEofState, isRollbackSupported, isYieldAfterExecutionStep, isYieldBeforeExecuteFromState, rollback, setNextState, toStringState
Methods inherited from class org.apache.hadoop.hbase.procedure2.Procedure
addStackIndex, afterReplay, beforeReplay, bypass, compareTo, completionCleanup, doExecute, doRollback, elapsedTime, getChildrenLatch, getException, getLastUpdate, getNonceKey, getOwner, getParentProcId, getProcId, getProcIdHashCode, getProcName, getResult, getRootProcedureId, getRootProcId, getStackIndexes, getState, getSubmittedTime, getTimeout, getTimeoutTimestamp, hasChildren, hasException, hasLock, hasOwner, hasParent, hasTimeout, haveSameParent, incChildrenLatch, isBypass, isFailed, isFinished, isInitializing, isLockedWhenLoading, isRunnable, isSuccess, isWaiting, removeStackIndex, setAbortFailure, setChildrenLatch, setExecuted, setFailure, setFailure, setLastUpdate, setNonceKey, setOwner, setOwner, setParentProcId, setProcId, setResult, setRootProcId, setStackIndexes, setState, setSubmittedTime, setTimeout, setTimeoutFailure, shouldWaitClientAck, skipPersistence, suspend, toString, toStringClass, toStringDetails, toStringSimpleSB, updateMetricsOnFinish, updateMetricsOnSubmit, updateTimestamp, wasExecuted
-
Field Details
-
LOG
-
daughterOneRI
-
daughterTwoRI
-
bestSplitRow
-
splitPolicy
-
EXPECTED_SPLIT_STATES
-
-
Constructor Details
-
SplitTableRegionProcedure
public SplitTableRegionProcedure() -
SplitTableRegionProcedure
public SplitTableRegionProcedure(MasterProcedureEnv env, RegionInfo regionToSplit, byte[] splitRow) throws IOException - Throws:
IOException
-
-
Method Details
-
acquireLock
Description copied from class:Procedure
The user should override this method if they need a lock on an Entity. A lock can be anything, and it is up to the implementor. The Procedure Framework will call this method just before it invokesProcedure.execute(Object)
. It callsProcedure.releaseLock(Object)
after the call to execute. If you need to hold the lock for the life of the Procedure -- i.e. you do not want any other Procedure interfering while this Procedure is running, seeProcedure.holdLock(Object)
. Example: in our Master we can execute request in parallel for different tables. We can create t1 and create t2 and these creates can be executed at the same time. Anything else on t1/t2 is queued waiting that specific table create to happen. There are 3 LockState:- LOCK_ACQUIRED should be returned when the proc has the lock and the proc is ready to execute.
- LOCK_YIELD_WAIT should be returned when the proc has not the lock and the framework should take care of readding the procedure back to the runnable set for retry
- LOCK_EVENT_WAIT should be returned when the proc has not the lock and someone will take care of readding the procedure back to the runnable set when the lock is available.
- Overrides:
acquireLock
in classAbstractStateMachineRegionProcedure<org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.SplitTableRegionState>
- Returns:
- the lock state as described above.
-
releaseLock
Description copied from class:Procedure
The user should override this method, and release lock if necessary.- Overrides:
releaseLock
in classAbstractStateMachineRegionProcedure<org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.SplitTableRegionState>
-
getDaughterOneRI
-
getDaughterTwoRI
-
hasBestSplitRow
-
checkSplittable
Check whether the region is splittable- Parameters:
env
- MasterProcedureEnvregionToSplit
- parent Region to be split- Throws:
IOException
-
getDaughterRegionIdTimestamp
Calculate daughter regionid to use.- Parameters:
hri
- ParentRegionInfo
- Returns:
- Daughter region id (timestamp) to use.
-
removeNonDefaultReplicas
- Throws:
IOException
-
checkClosedRegions
- Throws:
IOException
-
executeFromState
protected StateMachineProcedure.Flow executeFromState(MasterProcedureEnv env, org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.SplitTableRegionState state) throws InterruptedException Description copied from class:StateMachineProcedure
called to perform a single step of the specified 'state' of the procedure- Specified by:
executeFromState
in classStateMachineProcedure<MasterProcedureEnv,
org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.SplitTableRegionState> state
- state to execute- Returns:
- Flow.NO_MORE_STATE if the procedure is completed, Flow.HAS_MORE_STATE if there is another step.
- Throws:
InterruptedException
-
rollbackState
protected void rollbackState(MasterProcedureEnv env, org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.SplitTableRegionState state) throws IOException, InterruptedException To rollbackSplitTableRegionProcedure
, an AssignProcedure is asynchronously submitted for parent region to be split (rollback doesn't wait on the completion of the AssignProcedure) . This can be improved by changing rollback() to support sub-procedures. See HBASE-19851 for details.- Specified by:
rollbackState
in classStateMachineProcedure<MasterProcedureEnv,
org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.SplitTableRegionState> state
- state to rollback- Throws:
IOException
- temporary failure, the rollback will retry laterInterruptedException
-
isRollbackSupported
protected boolean isRollbackSupported(org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.SplitTableRegionState state) Description copied from class:StateMachineProcedure
Used by the default implementation of abort() to know if the current state can be aborted and rollback can be triggered.- Overrides:
isRollbackSupported
in classStateMachineProcedure<MasterProcedureEnv,
org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.SplitTableRegionState>
-
getState
protected org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.SplitTableRegionState getState(int stateId) Description copied from class:StateMachineProcedure
Convert an ordinal (or state id) to an Enum (or more descriptive) state object.- Specified by:
getState
in classStateMachineProcedure<MasterProcedureEnv,
org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.SplitTableRegionState> - Parameters:
stateId
- the ordinal() of the state enum (or state id)- Returns:
- the state enum object
-
getStateId
protected int getStateId(org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.SplitTableRegionState state) Description copied from class:StateMachineProcedure
Convert the Enum (or more descriptive) state object to an ordinal (or state id).- Specified by:
getStateId
in classStateMachineProcedure<MasterProcedureEnv,
org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.SplitTableRegionState> - Parameters:
state
- the state enum object- Returns:
- stateId the ordinal() of the state enum (or state id)
-
getInitialState
protected org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.SplitTableRegionState getInitialState()Description copied from class:StateMachineProcedure
Return the initial state object that will be used for the first call to executeFromState().- Specified by:
getInitialState
in classStateMachineProcedure<MasterProcedureEnv,
org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.SplitTableRegionState> - Returns:
- the initial state enum object
-
serializeStateData
Description copied from class:Procedure
The user-level code of the procedure may have some state to persist (e.g. input arguments or current position in the processing state) to be able to resume on failure.- Overrides:
serializeStateData
in classAbstractStateMachineRegionProcedure<org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.SplitTableRegionState>
- Parameters:
serializer
- stores the serializable state- Throws:
IOException
-
deserializeStateData
Description copied from class:Procedure
Called on store load to allow the user to decode the previously serialized state.- Overrides:
deserializeStateData
in classAbstractStateMachineRegionProcedure<org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.SplitTableRegionState>
- Parameters:
serializer
- contains the serialized state- Throws:
IOException
-
toStringClassDetails
Description copied from class:Procedure
Extend the toString() information with the procedure details e.g. className and parameters- Overrides:
toStringClassDetails
in classAbstractStateMachineRegionProcedure<org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.SplitTableRegionState>
- Parameters:
sb
- the string builder to use to append the proc specific information
-
getParentRegion
-
getTableOperationType
Description copied from interface:TableProcedureInterface
Given an operation type we can take decisions about what to do with pending operations. e.g. if we get a delete and we have some table operation pending (e.g. add column) we can abort those operations.- Specified by:
getTableOperationType
in interfaceTableProcedureInterface
- Specified by:
getTableOperationType
in classAbstractStateMachineRegionProcedure<org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.SplitTableRegionState>
- Returns:
- the operation type that the procedure is executing.
-
getProcedureMetrics
Description copied from class:Procedure
Override this method to provide procedure specific counters for submitted count, failed count and time histogram.- Overrides:
getProcedureMetrics
in classProcedure<MasterProcedureEnv>
- Parameters:
env
- The environment passed to the procedure executor- Returns:
- Container object for procedure related metric
-
getSplitRow
-
prepareSplitRegion
Prepare to Split region.- Parameters:
env
- MasterProcedureEnv- Throws:
IOException
-
rollbackPrepareSplit
Rollback prepare split region- Parameters:
env
- MasterProcedureEnv
-
preSplitRegion
Action before splitting region in a table.- Parameters:
env
- MasterProcedureEnv- Throws:
IOException
InterruptedException
-
postRollBackSplitRegion
Action after rollback a split table region action.- Parameters:
env
- MasterProcedureEnv- Throws:
IOException
-
openParentRegion
Rollback close parent region- Throws:
IOException
-
createDaughterRegions
Create daughter regions- Throws:
IOException
-
deleteDaughterRegions
- Throws:
IOException
-
splitStoreFiles
private Pair<List<org.apache.hadoop.fs.Path>,List<org.apache.hadoop.fs.Path>> splitStoreFiles(MasterProcedureEnv env, HRegionFileSystem regionFs) throws IOException Create Split directory- Parameters:
env
- MasterProcedureEnv- Throws:
IOException
-
assertSplitResultFilesCount
private void assertSplitResultFilesCount(org.apache.hadoop.fs.FileSystem fs, int expectedSplitResultFileCount, org.apache.hadoop.fs.Path dir) throws IOException - Throws:
IOException
-
splitStoreFile
private Pair<org.apache.hadoop.fs.Path,org.apache.hadoop.fs.Path> splitStoreFile(HRegionFileSystem regionFs, StoreFileTracker tracker, byte[] family, HStoreFile sf) throws IOException - Throws:
IOException
-
preSplitRegionBeforeMETA
private void preSplitRegionBeforeMETA(MasterProcedureEnv env) throws IOException, InterruptedException Post split region actions before the Point-of-No-Return step- Parameters:
env
- MasterProcedureEnv- Throws:
IOException
InterruptedException
-
updateMeta
Add daughter regions to META- Parameters:
env
- MasterProcedureEnv- Throws:
IOException
-
preSplitRegionAfterMETA
private void preSplitRegionAfterMETA(MasterProcedureEnv env) throws IOException, InterruptedException Pre split region actions after the Point-of-No-Return step- Parameters:
env
- MasterProcedureEnv- Throws:
IOException
InterruptedException
-
postSplitRegion
Post split region actions- Parameters:
env
- MasterProcedureEnv- Throws:
IOException
-
getParentRegionServerName
-
createUnassignProcedures
private TransitRegionStateProcedure[] createUnassignProcedures(MasterProcedureEnv env) throws IOException - Throws:
IOException
-
createAssignProcedures
private TransitRegionStateProcedure[] createAssignProcedures(MasterProcedureEnv env) throws IOException - Throws:
IOException
-
getRegionReplication
- Throws:
IOException
-
writeMaxSequenceIdFile
- Throws:
IOException
-
abort
Description copied from class:Procedure
The abort() call is asynchronous and each procedure must decide how to deal with it, if they want to be abortable. The simplest implementation is to have an AtomicBoolean set in the abort() method and then the execute() will check if the abort flag is set or not. abort() may be called multiple times from the client, so the implementation must be idempotent.NOTE: abort() is not like Thread.interrupt(). It is just a notification that allows the procedure implementor abort.
- Overrides:
abort
in classStateMachineProcedure<MasterProcedureEnv,
org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.SplitTableRegionState>
-