Class StateMachineProcedure<TEnvironment,TState>
java.lang.Object
org.apache.hadoop.hbase.procedure2.Procedure<TEnvironment>
org.apache.hadoop.hbase.procedure2.StateMachineProcedure<TEnvironment,TState>
- All Implemented Interfaces:
Comparable<Procedure<TEnvironment>>
- Direct Known Subclasses:
AbstractPeerNoLockProcedure
,AbstractStateMachineNamespaceProcedure
,AbstractStateMachineTableProcedure
,AssignReplicationQueuesProcedure
,MigrateNamespaceTableProcedure
,MigrateReplicationQueueFromZkToTableProcedure
,RecoverMetaProcedure
,ServerCrashProcedure
,SplitWALProcedure
,SwitchRpcThrottleProcedure
@Private
@Evolving
public abstract class StateMachineProcedure<TEnvironment,TState>
extends Procedure<TEnvironment>
Procedure described by a series of steps.
The procedure implementor must have an enum of 'states', describing the various step of the procedure. Once the procedure is running, the procedure-framework will call executeFromState() using the 'state' provided by the user. The first call to executeFromState() will be performed with 'state = null'. The implementor can jump between states using setNextState(MyStateEnum.ordinal()). The rollback will call rollbackState() for each state that was executed, in reverse order.
-
Nested Class Summary
Nested classes/interfaces inherited from class org.apache.hadoop.hbase.procedure2.Procedure
Procedure.LockState
-
Field Summary
Modifier and TypeFieldDescriptionprivate final AtomicBoolean
private int
Cycles on same state.private static final int
private static final org.slf4j.Logger
private int
Ordinal of the previous state.protected int
private StateMachineProcedure.Flow
private int[]
private List<Procedure<TEnvironment>>
Fields inherited from class org.apache.hadoop.hbase.procedure2.Procedure
NO_PROC_ID, NO_TIMEOUT
-
Constructor Summary
-
Method Summary
Modifier and TypeMethodDescriptionprotected boolean
abort
(TEnvironment env) The abort() call is asynchronous and each procedure must decide how to deal with it, if they want to be abortable.protected <T extends Procedure<TEnvironment>>
voidaddChildProcedure
(T... subProcedure) Add a child procedure to executeprotected void
deserializeStateData
(ProcedureStateSerializer serializer) Called on store load to allow the user to decode the previously serialized state.protected Procedure[]
execute
(TEnvironment env) The main code of the procedure.protected abstract StateMachineProcedure.Flow
executeFromState
(TEnvironment env, TState state) called to perform a single step of the specified 'state' of the procedureprotected final void
If procedure has more states then abort it otherwise procedure is finished and abort can be ignored.protected TState
int
This method is used from test code as it cannot be assumed that state transition will happen sequentially.protected final int
protected abstract TState
Return the initial state object that will be used for the first call to executeFromState().protected abstract TState
getState
(int stateId) Convert an ordinal (or state id) to an Enum (or more descriptive) state object.protected abstract int
getStateId
(TState state) Convert the Enum (or more descriptive) state object to an ordinal (or state id).private boolean
protected boolean
protected final boolean
Return whether the procedure supports rollback.protected boolean
isRollbackSupported
(TState state) Used by the default implementation of abort() to know if the current state can be aborted and rollback can be triggered.protected boolean
By default, the procedure framework/executor will try to run procedures start to finish.protected boolean
isYieldBeforeExecuteFromState
(TEnvironment env, TState state) By default, the executor will try ro run all the steps of the procedure start to finish.protected void
rollback
(TEnvironment env) The code to undo what was done by the execute() code.protected abstract void
rollbackState
(TEnvironment env, TState state) called to perform the rollback of the specified stateprotected void
serializeStateData
(ProcedureStateSerializer serializer) The user-level code of the procedure may have some state to persist (e.g.private void
setNextState
(int stateId) Set the next state for the procedure.protected void
setNextState
(TState state) Set the next state for the procedure.protected void
toStringState
(StringBuilder builder) Called fromProcedure.toString()
when interpolatingProcedure
State.Methods inherited from class org.apache.hadoop.hbase.procedure2.Procedure
acquireLock, addStackIndex, afterReplay, beforeReplay, bypass, compareTo, completionCleanup, doAcquireLock, doExecute, doReleaseLock, doRollback, elapsedTime, getChildrenLatch, getException, getLastUpdate, getNonceKey, getOwner, getParentProcId, getProcedureMetrics, getProcId, getProcIdHashCode, getProcName, getResult, getRootProcedureId, getRootProcId, getStackIndexes, getState, getSubmittedTime, getTimeout, getTimeoutTimestamp, hasChildren, hasException, hasLock, hasOwner, hasParent, hasTimeout, haveSameParent, holdLock, incChildrenLatch, isBypass, isFailed, isFinished, isInitializing, isLockedWhenLoading, isRunnable, isSuccess, isWaiting, lockedWhenLoading, needPersistence, releaseLock, removeStackIndex, resetPersistence, restoreLock, setAbortFailure, setChildrenLatch, setExecuted, setFailure, setFailure, setLastUpdate, setNonceKey, setOwner, setOwner, setParentProcId, setProcId, setResult, setRootProcId, setStackIndexes, setState, setSubmittedTime, setTimeout, setTimeoutFailure, shouldWaitClientAck, skipPersistence, suspend, toString, toStringClass, toStringClassDetails, toStringDetails, toStringSimpleSB, tryRunnable, updateMetricsOnFinish, updateMetricsOnSubmit, updateTimestamp, waitInitialized, wasExecuted
-
Field Details
-
LOG
-
EOF_STATE
- See Also:
-
aborted
-
stateFlow
-
stateCount
-
states
-
subProcList
-
cycles
Cycles on same state. Good for figuring if we are stuck. -
previousState
Ordinal of the previous state. So we can tell if we are progressing or not.
-
-
Constructor Details
-
StateMachineProcedure
public StateMachineProcedure()
-
-
Method Details
-
getCycles
-
executeFromState
protected abstract StateMachineProcedure.Flow executeFromState(TEnvironment env, TState state) throws ProcedureSuspendedException, ProcedureYieldException, InterruptedException called to perform a single step of the specified 'state' of the procedure- Parameters:
state
- state to execute- Returns:
- Flow.NO_MORE_STATE if the procedure is completed, Flow.HAS_MORE_STATE if there is another step.
- Throws:
ProcedureSuspendedException
ProcedureYieldException
InterruptedException
-
rollbackState
protected abstract void rollbackState(TEnvironment env, TState state) throws IOException, InterruptedException called to perform the rollback of the specified state- Parameters:
state
- state to rollback- Throws:
IOException
- temporary failure, the rollback will retry laterInterruptedException
-
getState
Convert an ordinal (or state id) to an Enum (or more descriptive) state object.- Parameters:
stateId
- the ordinal() of the state enum (or state id)- Returns:
- the state enum object
-
getStateId
Convert the Enum (or more descriptive) state object to an ordinal (or state id).- Parameters:
state
- the state enum object- Returns:
- stateId the ordinal() of the state enum (or state id)
-
getInitialState
Return the initial state object that will be used for the first call to executeFromState().- Returns:
- the initial state enum object
-
setNextState
Set the next state for the procedure.- Parameters:
state
- the state enum object
-
isYieldBeforeExecuteFromState
By default, the executor will try ro run all the steps of the procedure start to finish. Return true to make the executor yield between execution steps to give other procedures time to run their steps.- Parameters:
state
- the state we are going to execute next.- Returns:
- Return true if the executor should yield before the execution of the specified step. Defaults to return false.
-
addChildProcedure
Add a child procedure to execute- Parameters:
subProcedure
- the child procedure
-
execute
protected Procedure[] execute(TEnvironment env) throws ProcedureSuspendedException, ProcedureYieldException, InterruptedException Description copied from class:Procedure
The main code of the procedure. It must be idempotent since execute() may be called multiple times in case of machine failure in the middle of the execution.- Specified by:
execute
in classProcedure<TEnvironment>
- Parameters:
env
- the environment passed to the ProcedureExecutor- Returns:
- a set of sub-procedures to run or ourselves if there is more work to do or null if the procedure is done.
- Throws:
ProcedureSuspendedException
- Signal to the executor that Procedure has suspended itself and has set itself up waiting for an external event to wake it back up again.ProcedureYieldException
- the procedure will be added back to the queue and retried later.InterruptedException
- the procedure will be added back to the queue and retried later.
-
rollback
Description copied from class:Procedure
The code to undo what was done by the execute() code. It is called when the procedure or one of the sub-procedures failed or an abort was requested. It should cleanup all the resources created by the execute() call. The implementation must be idempotent since rollback() may be called multiple time in case of machine failure in the middle of the execution.- Specified by:
rollback
in classProcedure<TEnvironment>
- Parameters:
env
- the environment passed to the ProcedureExecutor- Throws:
IOException
- temporary failure, the rollback will retry laterInterruptedException
- the procedure will be added back to the queue and retried later
-
isEofState
-
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.
- Specified by:
abort
in classProcedure<TEnvironment>
-
failIfAborted
If procedure has more states then abort it otherwise procedure is finished and abort can be ignored. -
isRollbackSupported
Description copied from class:Procedure
Return whether the procedure supports rollback. If the procedure does not support rollback, we can skip the rollback state management which could increase the performance. See HBASE-28210 and HBASE-28212.- Overrides:
isRollbackSupported
in classProcedure<TEnvironment>
-
isRollbackSupported
Used by the default implementation of abort() to know if the current state can be aborted and rollback can be triggered. -
isYieldAfterExecutionStep
Description copied from class:Procedure
By default, the procedure framework/executor will try to run procedures start to finish. Return true to make the executor yield between each execution step to give other procedures a chance to run.- Overrides:
isYieldAfterExecutionStep
in classProcedure<TEnvironment>
- Parameters:
env
- the environment passed to the ProcedureExecutor- Returns:
- Return true if the executor should yield on completion of an execution step. Defaults to return false.
-
hasMoreState
-
getCurrentState
-
getCurrentStateId
This method is used from test code as it cannot be assumed that state transition will happen sequentially. Some procedures may skip steps/ states, some may add intermediate steps in future. -
setNextState
Set the next state for the procedure.- Parameters:
stateId
- the ordinal() of the state enum (or state id)
-
toStringState
Description copied from class:Procedure
Called fromProcedure.toString()
when interpolatingProcedure
State. Allows decorating generic Procedure State with Procedure particulars.- Overrides:
toStringState
in classProcedure<TEnvironment>
- Parameters:
builder
- Append currentProcedureProtos.ProcedureState
-
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.- Specified by:
serializeStateData
in classProcedure<TEnvironment>
- 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.- Specified by:
deserializeStateData
in classProcedure<TEnvironment>
- Parameters:
serializer
- contains the serialized state- Throws:
IOException
-