@InterfaceAudience.Private public class TransitRegionStateProcedure extends AbstractStateMachineRegionProcedure<org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.RegionStateTransitionState>
unassign(MasterProcedureEnv, RegionInfo)
and
reopen(MasterProcedureEnv, RegionInfo)
, you do not need to specify a target server, and
for assign(MasterProcedureEnv, RegionInfo, ServerName)
and
move(MasterProcedureEnv, RegionInfo, ServerName)
, if you want to you can provide a
target server. And for move(MasterProcedureEnv, RegionInfo, ServerName)
, if you do not
specify a targetServer, we will select one randomly.
The typical state transition for assigning a region is:
GET_ASSIGN_CANDIDATE ------> OPEN -----> CONFIRM_OPENEDNotice that, if there are failures we may go back to the
GET_ASSIGN_CANDIDATE
state to
try again.
The typical state transition for unassigning a region is:
CLOSE -----> CONFIRM_CLOSEDHere things go a bit different, if there are failures, especially that if there is a server crash, we will go to the
GET_ASSIGN_CANDIDATE
state to bring the region online first, and
then go through the normal way to unassign it.
The typical state transition for reopening/moving a region is:
CLOSE -----> CONFIRM_CLOSED -----> GET_ASSIGN_CANDIDATE ------> OPEN -----> CONFIRM_OPENEDThe retry logic is the same with the above assign/unassign. Notice that, although we allow specify a target server, it just acts as a candidate, we do not guarantee that the region will finally be on the target server. If this is important for you, you should check whether the region is on the target server after the procedure is finished. When you want to schedule a TRSP, please check whether there is still one for this region, and the check should be under the RegionStateNode lock. We will remove the TRSP from a RegionStateNode when we are done, see the code in
reportTransition
method below. There
could be at most one TRSP for a give region.Modifier and Type | Class and Description |
---|---|
static class |
TransitRegionStateProcedure.TransitionType |
StateMachineProcedure.Flow
Procedure.LockState
TableProcedureInterface.TableOperationType
Modifier and Type | Field and Description |
---|---|
private ServerName |
assignCandidate |
private boolean |
forceNewPlan |
private org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.RegionStateTransitionState |
initialState |
private org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.RegionStateTransitionState |
lastState |
private static org.slf4j.Logger |
LOG |
private RegionRemoteProcedureBase |
remoteProc |
private RetryCounter |
retryCounter |
private TransitRegionStateProcedure.TransitionType |
type |
NO_PROC_ID, NO_TIMEOUT
Modifier | Constructor and Description |
---|---|
|
TransitRegionStateProcedure() |
protected |
TransitRegionStateProcedure(MasterProcedureEnv env,
RegionInfo hri,
ServerName assignCandidate,
boolean forceNewPlan,
TransitRegionStateProcedure.TransitionType type) |
Modifier and Type | Method and Description |
---|---|
static TransitRegionStateProcedure |
assign(MasterProcedureEnv env,
RegionInfo region,
boolean forceNewPlan,
ServerName targetServer) |
static TransitRegionStateProcedure |
assign(MasterProcedureEnv env,
RegionInfo region,
ServerName targetServer) |
(package private) void |
attachRemoteProc(RegionRemoteProcedureBase proc) |
private void |
closeRegion(MasterProcedureEnv env,
RegionStateNode regionNode) |
private StateMachineProcedure.Flow |
confirmClosed(MasterProcedureEnv env,
RegionStateNode regionNode) |
private StateMachineProcedure.Flow |
confirmOpened(MasterProcedureEnv env,
RegionStateNode regionNode) |
private static TransitRegionStateProcedure.TransitionType |
convert(org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.RegionTransitionType type) |
private static org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.RegionTransitionType |
convert(TransitRegionStateProcedure.TransitionType type) |
protected void |
deserializeStateData(ProcedureStateSerializer serializer)
Called on store load to allow the user to decode the previously serialized
state.
|
protected Procedure[] |
execute(MasterProcedureEnv env)
The main code of the procedure.
|
protected StateMachineProcedure.Flow |
executeFromState(MasterProcedureEnv env,
org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.RegionStateTransitionState state)
called to perform a single step of the specified 'state' of the procedure
|
protected org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.RegionStateTransitionState |
getInitialState()
Return the initial state object that will be used for the first call to executeFromState().
|
protected ProcedureMetrics |
getProcedureMetrics(MasterProcedureEnv env)
Override this method to provide procedure specific counters for submitted count, failed
count and time histogram.
|
private RegionStateNode |
getRegionStateNode(MasterProcedureEnv env) |
protected org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.RegionStateTransitionState |
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.RegionStateTransitionState state)
Convert the Enum (or more descriptive) state object to an ordinal (or state id).
|
TableProcedureInterface.TableOperationType |
getTableOperationType()
Given an operation type we can take decisions about what to do with pending operations.
|
static TransitRegionStateProcedure |
move(MasterProcedureEnv env,
RegionInfo region,
ServerName targetServer) |
private void |
openRegion(MasterProcedureEnv env,
RegionStateNode regionNode) |
private void |
queueAssign(MasterProcedureEnv env,
RegionStateNode regionNode) |
static TransitRegionStateProcedure |
reopen(MasterProcedureEnv env,
RegionInfo region) |
void |
reportTransition(MasterProcedureEnv env,
RegionStateNode regionNode,
ServerName serverName,
org.apache.hadoop.hbase.shaded.protobuf.generated.RegionServerStatusProtos.RegionStateTransition.TransitionCode code,
long seqId,
long procId) |
protected void |
rollbackState(MasterProcedureEnv env,
org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.RegionStateTransitionState state)
called to perform the rollback of the specified state
|
protected void |
serializeStateData(ProcedureStateSerializer serializer)
The user-level code of the procedure may have some state to
persist (e.g.
|
void |
serverCrashed(MasterProcedureEnv env,
RegionStateNode regionNode,
ServerName serverName) |
private void |
setInitalAndLastState() |
private static TransitRegionStateProcedure |
setOwner(MasterProcedureEnv env,
TransitRegionStateProcedure proc) |
protected boolean |
setTimeoutFailure(MasterProcedureEnv env)
At end of timeout, wake ourselves up so we run again.
|
(package private) void |
stateLoaded(AssignmentManager am,
RegionStateNode regionNode) |
void |
toStringClassDetails(StringBuilder sb)
Extend the toString() information with the procedure details
e.g.
|
static TransitRegionStateProcedure |
unassign(MasterProcedureEnv env,
RegionInfo region) |
(package private) void |
unattachRemoteProc(RegionRemoteProcedureBase proc) |
protected boolean |
waitInitialized(MasterProcedureEnv env)
The
Procedure.doAcquireLock(Object, ProcedureStore) will be split into two steps, first, it will
call us to determine whether we need to wait for initialization, second, it will call
Procedure.acquireLock(Object) to actually handle the lock for this procedure. |
acquireLock, checkTableModifiable, getRegion, getTableName, holdLock, releaseLock, setFailure, setRegion
checkOnline, getUser, getWALRegionDir, preflightChecks, releaseSyncLatch, setUser
abort, addChildProcedure, failIfAborted, getCurrentState, getCurrentStateId, getCycles, isRollbackSupported, isYieldAfterExecutionStep, isYieldBeforeExecuteFromState, rollback, setNextState, toStringState
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, setFailure, setFailure, setLastUpdate, setNonceKey, setOwner, setOwner, setParentProcId, setProcId, setResult, setRootProcId, setStackIndexes, setState, setSubmittedTime, setTimeout, shouldWaitClientAck, skipPersistence, toString, toStringClass, toStringDetails, toStringSimpleSB, updateMetricsOnFinish, updateMetricsOnSubmit, updateTimestamp, wasExecuted
private static final org.slf4j.Logger LOG
private TransitRegionStateProcedure.TransitionType type
private org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.RegionStateTransitionState initialState
private org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.RegionStateTransitionState lastState
private ServerName assignCandidate
private boolean forceNewPlan
private RetryCounter retryCounter
private RegionRemoteProcedureBase remoteProc
public TransitRegionStateProcedure()
protected TransitRegionStateProcedure(MasterProcedureEnv env, RegionInfo hri, ServerName assignCandidate, boolean forceNewPlan, TransitRegionStateProcedure.TransitionType type)
private void setInitalAndLastState()
public TableProcedureInterface.TableOperationType getTableOperationType()
TableProcedureInterface
getTableOperationType
in interface TableProcedureInterface
getTableOperationType
in class AbstractStateMachineRegionProcedure<org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.RegionStateTransitionState>
protected boolean waitInitialized(MasterProcedureEnv env)
Procedure
Procedure.doAcquireLock(Object, ProcedureStore)
will be split into two steps, first, it will
call us to determine whether we need to wait for initialization, second, it will call
Procedure.acquireLock(Object)
to actually handle the lock for this procedure.
This is because that when master restarts, we need to restore the lock state for all the
procedures to not break the semantic if Procedure.holdLock(Object)
is true. But the
ProcedureExecutor
will be started before the master finish initialization(as it is part
of the initialization!), so we need to split the code into two steps, and when restore, we just
restore the lock part and ignore the waitInitialized part. Otherwise there will be dead lock.waitInitialized
in class AbstractStateMachineTableProcedure<org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.RegionStateTransitionState>
private void queueAssign(MasterProcedureEnv env, RegionStateNode regionNode) throws ProcedureSuspendedException
ProcedureSuspendedException
private void openRegion(MasterProcedureEnv env, RegionStateNode regionNode) throws IOException
IOException
private StateMachineProcedure.Flow confirmOpened(MasterProcedureEnv env, RegionStateNode regionNode) throws IOException
IOException
private void closeRegion(MasterProcedureEnv env, RegionStateNode regionNode) throws IOException
IOException
private StateMachineProcedure.Flow confirmClosed(MasterProcedureEnv env, RegionStateNode regionNode) throws IOException
IOException
protected Procedure[] execute(MasterProcedureEnv env) throws ProcedureSuspendedException, ProcedureYieldException, InterruptedException
Procedure
execute
in class StateMachineProcedure<MasterProcedureEnv,org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.RegionStateTransitionState>
env
- the environment passed to the ProcedureExecutorProcedureSuspendedException
- 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.private RegionStateNode getRegionStateNode(MasterProcedureEnv env)
protected StateMachineProcedure.Flow executeFromState(MasterProcedureEnv env, org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.RegionStateTransitionState state) throws ProcedureSuspendedException, ProcedureYieldException, InterruptedException
StateMachineProcedure
executeFromState
in class StateMachineProcedure<MasterProcedureEnv,org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.RegionStateTransitionState>
state
- state to executeProcedureSuspendedException
ProcedureYieldException
InterruptedException
protected boolean setTimeoutFailure(MasterProcedureEnv env)
setTimeoutFailure
in class Procedure<MasterProcedureEnv>
public void reportTransition(MasterProcedureEnv env, RegionStateNode regionNode, ServerName serverName, org.apache.hadoop.hbase.shaded.protobuf.generated.RegionServerStatusProtos.RegionStateTransition.TransitionCode code, long seqId, long procId) throws IOException
IOException
public void serverCrashed(MasterProcedureEnv env, RegionStateNode regionNode, ServerName serverName) throws IOException
IOException
void attachRemoteProc(RegionRemoteProcedureBase proc)
void unattachRemoteProc(RegionRemoteProcedureBase proc)
void stateLoaded(AssignmentManager am, RegionStateNode regionNode)
protected void rollbackState(MasterProcedureEnv env, org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.RegionStateTransitionState state) throws IOException, InterruptedException
StateMachineProcedure
rollbackState
in class StateMachineProcedure<MasterProcedureEnv,org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.RegionStateTransitionState>
state
- state to rollbackIOException
- temporary failure, the rollback will retry laterInterruptedException
protected org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.RegionStateTransitionState getState(int stateId)
StateMachineProcedure
getState
in class StateMachineProcedure<MasterProcedureEnv,org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.RegionStateTransitionState>
stateId
- the ordinal() of the state enum (or state id)protected int getStateId(org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.RegionStateTransitionState state)
StateMachineProcedure
getStateId
in class StateMachineProcedure<MasterProcedureEnv,org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.RegionStateTransitionState>
state
- the state enum objectprotected org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.RegionStateTransitionState getInitialState()
StateMachineProcedure
getInitialState
in class StateMachineProcedure<MasterProcedureEnv,org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.RegionStateTransitionState>
private static TransitRegionStateProcedure.TransitionType convert(org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.RegionTransitionType type)
private static org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.RegionTransitionType convert(TransitRegionStateProcedure.TransitionType type)
protected void serializeStateData(ProcedureStateSerializer serializer) throws IOException
Procedure
serializeStateData
in class AbstractStateMachineRegionProcedure<org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.RegionStateTransitionState>
serializer
- stores the serializable stateIOException
protected void deserializeStateData(ProcedureStateSerializer serializer) throws IOException
Procedure
deserializeStateData
in class AbstractStateMachineRegionProcedure<org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.RegionStateTransitionState>
serializer
- contains the serialized stateIOException
protected ProcedureMetrics getProcedureMetrics(MasterProcedureEnv env)
Procedure
getProcedureMetrics
in class Procedure<MasterProcedureEnv>
env
- The environment passed to the procedure executorpublic void toStringClassDetails(StringBuilder sb)
Procedure
toStringClassDetails
in class AbstractStateMachineRegionProcedure<org.apache.hadoop.hbase.shaded.protobuf.generated.MasterProcedureProtos.RegionStateTransitionState>
sb
- the string builder to use to append the proc specific informationprivate static TransitRegionStateProcedure setOwner(MasterProcedureEnv env, TransitRegionStateProcedure proc)
public static TransitRegionStateProcedure assign(MasterProcedureEnv env, RegionInfo region, @Nullable ServerName targetServer)
public static TransitRegionStateProcedure assign(MasterProcedureEnv env, RegionInfo region, boolean forceNewPlan, @Nullable ServerName targetServer)
public static TransitRegionStateProcedure unassign(MasterProcedureEnv env, RegionInfo region)
public static TransitRegionStateProcedure reopen(MasterProcedureEnv env, RegionInfo region)
public static TransitRegionStateProcedure move(MasterProcedureEnv env, RegionInfo region, @Nullable ServerName targetServer)
Copyright © 2007–2020 The Apache Software Foundation. All rights reserved.