Class AssignmentManager
java.lang.Object
org.apache.hadoop.hbase.master.assignment.AssignmentManager
The AssignmentManager is the coordinator for region assign/unassign operations.
- In-memory states of regions and servers are stored in
RegionStates
. - hbase:meta state updates are handled by
RegionStateStore
.
-
Nested Class Summary
Modifier and TypeClassDescriptionprivate static class
private static class
static class
private class
-
Field Summary
Modifier and TypeFieldDescriptionstatic final String
static final String
static final String
static final String
private final int
private final int
private final int
private final Condition
private final ReentrantLock
private final int
private Thread
static final String
private final Object
static final String
private final AssignmentManager.DeadServerMetricRegionChore
private static final int
private static final int
private static final int
private static final int
private static final int
static final boolean
static final int
static final long
private static final String
private static final int
private static final int
static final String
static final String
The number of times to check if the region's previous RS is back online, before giving up and proceeding with assignment on a new RSstatic final String
The wait time in millis before checking again if the region's previous RS is back onlineprivate final boolean
private final int
private final long
private static final org.slf4j.Logger
private final MasterServices
private final MasterRegion
private static final Set<RegionInfo>
private final ProcedureEvent<?>
private final ProcedureEvent<?>
private final MetricsAssignmentManager
static final String
Region in Transition metrics threshold timeprivate static final String
private final String
When the operator uses this configuration option, any version between the current cluster version and the value of "hbase.min.version.move.system.tables" does not trigger any auto-region movement.private final ArrayList<RegionStateNode>
private final RegionStates
private final RegionStateStore
static final String
private final AssignmentManager.RegionInTransitionChore
private final Map<ServerName,
Set<byte[]>> private final AtomicBoolean
private final boolean
private static final RegionState.State[]
private static final RegionState.State[]
private static final RegionState.State[]
private static final RegionState.State[]
private static final RegionState.State[]
static final String
-
Constructor Summary
ConstructorDescriptionAssignmentManager
(MasterServices master, MasterRegion masterRegion) AssignmentManager
(MasterServices master, MasterRegion masterRegion, RegionStateStore stateStore) -
Method Summary
Modifier and TypeMethodDescriptionprivate void
acceptPlan
(HashMap<RegionInfo, RegionStateNode> regions, Map<ServerName, List<RegionInfo>> plan) private void
addToPendingAssignment
(HashMap<RegionInfo, RegionStateNode> regions, Collection<RegionInfo> pendingRegions) long
assign
(RegionInfo regionInfo) long
assign
(RegionInfo regionInfo, ServerName sn) Future<byte[]>
assignAsync
(RegionInfo regionInfo) Submits a procedure that assigns a region without waiting for it to finishFuture<byte[]>
assignAsync
(RegionInfo regionInfo, ServerName sn) Submits a procedure that assigns a region to a target server without waiting for it to finishprivate void
Future<byte[]>
balance
(RegionPlan regionPlan) void
Start a new thread to check if there are region servers whose versions are higher than others.private void
Used to check if the meta loading is done.private void
checkOnlineRegionsReport
(ServerStateNode serverNode, Set<byte[]> regionNames) Check that what the RegionServer reports aligns with the Master's image.private void
closeRegionSilently
(ServerName sn, byte[] regionName) CloseregionName
onsn
silently and immediately without using a Procedure or going via hbase:meta.(package private) static int
private boolean
containsBogusAssignments
(Map<RegionInfo, RegionStateNode> regions, List<RegionInfo> hirs) private TransitRegionStateProcedure
createAssignProcedure
(RegionInfo regionInfo, ServerName sn, boolean override, boolean force) Create an assign TransitRegionStateProcedure.private TransitRegionStateProcedure
createAssignProcedure
(RegionStateNode regionNode, ServerName targetServer) Create an assign TransitRegionStateProcedure.createAssignProcedures
(List<RegionInfo> hris) Create an array of TransitRegionStateProcedure w/o specifying a target server.private TransitRegionStateProcedure[]
createAssignProcedures
(Map<ServerName, List<RegionInfo>> assignments) Tied tocreateAssignProcedures(List)
in that it is called if caller is unable to run this method.createMergeProcedure
(RegionInfo... ris) createMoveRegionProcedure
(RegionInfo regionInfo, ServerName targetServer) createOneAssignProcedure
(RegionInfo ri, boolean override, boolean force) Create one TransitRegionStateProcedure to assign a region w/o specifying a target server.createOneUnassignProcedure
(RegionInfo ri, boolean override, boolean force) Create one TransitRegionStateProcedure to unassign a region.Create round-robin assigns.createRoundRobinAssignProcedures
(List<RegionInfo> hris, List<ServerName> serversToExclude) Create round-robin assigns.createSplitProcedure
(RegionInfo regionToSplit, byte[] splitKey) createTruncateRegionProcedure
(RegionInfo regionToTruncate) createUnassignProceduresForDisabling
(TableName tableName) Called by DisableTableProcedure to unassign all the regions for a table.void
deleteTable
(TableName tableName) Delete the region states.private TransitRegionStateProcedure
forceCreateUnssignProcedure
(RegionStateNode regionNode) (package private) int
(package private) int
private LoadBalancer
org.apache.hadoop.conf.Configuration
For a given cluster with mixed versions of servers, get a list of servers with lower versions, where system table regions should not be assigned to.private FavoredNodesPromoter
getFavoredNodes
(RegionInfo regionInfo) int
long
(package private) MasterServices
private ProcedureEvent<?>
getMetaAssignEvent
(RegionInfo metaRegionInfo) private RegionInfo
getMetaForRegion
(RegionInfo regionInfo) getMetaRegionFromName
(byte[] regionName) int
private MasterProcedureEnv
private MasterProcedureScheduler
getRegionInfo
(byte[] regionName) Resolve a cachedRegionInfo
from the region name as abyte[]
.getRegionInfo
(String encodedRegionName) Resolve a cachedRegionInfo
from the encoded region name as aString
.private RegionInfo
getRegionsOnServer
(ServerName serverName) Returns the regions hosted by the specified server.private Stream<RegionStateNode>
getRegionStateNodes
(TableName tableName, boolean excludeOfflinedSplitParents) getRegionStatesCount
(TableName tableName) Provide regions state count for given table.getReopenStatus
(TableName tableName) Used by the client (via master) to identify if all regions have the schema updatesMap<ServerName,
Set<byte[]>> Returns a snapshot of rsReportsgetSnapShotOfAssignment
(Collection<RegionInfo> regions) private List<RegionInfo>
getSystemTables
(ServerName serverName) getTableRegions
(TableName tableName, boolean excludeOfflinedSplitParents) getTableRegionsAndLocations
(TableName tableName, boolean excludeOfflinedSplitParents) private TableStateManager
private void
handleRegionOverStuckWarningThreshold
(RegionInfo regionInfo) boolean
boolean
isCarryingMeta
(ServerName serverName) private boolean
isCarryingRegion
(ServerName serverName, RegionInfo regionInfo) boolean
boolean
Notice that, this only means the meta region is available on a RS, but the AM may still be loading the region states from meta, so usually you need to checkisMetaLoaded()
first before checking this method, unless you can make sure that your piece of code can only be executed after AM builds the region states.boolean
Return whether AM finishes the meta loading, i.e, the region states rebuilding.boolean
isMetaRegion
(byte[] regionName) private boolean
isMetaRegion
(RegionInfo regionInfo) boolean
boolean
private boolean
isTableDisabled
(TableName tableName) private boolean
isTableEnabled
(TableName tableName) void
private void
loadMeta()
void
markRegionAsMerged
(RegionInfo child, ServerName serverName, RegionInfo[] mergeParents) When called here, the merge has happened.void
markRegionAsSplit
(RegionInfo parent, ServerName serverName, RegionInfo daughterA, RegionInfo daughterB) private void
void
move
(RegionInfo regionInfo) Future<byte[]>
moveAsync
(RegionPlan regionPlan) int
numberOfUnclosedExcessRegionReplicas
(TableName tableName, int newReplicaCount) private int
numberOfUnclosedRegions
(TableName tableName, Function<RegionStateNode, Boolean> shouldSubmit) int
numberOfUnclosedRegionsForDisabling
(TableName tableName) void
offlineRegion
(RegionInfo regionInfo) void
onlineRegion
(RegionInfo regionInfo, ServerName serverName) (package private) CompletableFuture<Void>
persistToMeta
(RegionStateNode regionNode) void
populateRegionStatesFromMeta
(String regionEncodedName) Attempt to loadregionEncodedName
from META, adding any results to theregionStateStore
Is NOT aware of replica regions.void
populateRegionStatesFromMeta
(RegionInfo regionInfo) Attempt to loadregionInfo
from META, adding any results to theregionStateStore
Is NOT aware of replica regions.private void
preTransitCheck
(RegionStateNode regionNode, RegionState.State[] expectedStates) private void
processAssignmentPlans
(HashMap<RegionInfo, RegionStateNode> regions, HashMap<RegionInfo, ServerName> retainMap, List<RegionInfo> hris, List<ServerName> servers) private void
void
Create assign procedure for offline regions.protected void
queueAssign
(RegionStateNode regionNode) Add the assign operation to the assignment queue.regionClosedAbnormally
(RegionStateNode regionNode) (package private) void
regionClosedWithoutPersistingToMeta
(RegionStateNode regionNode) (package private) CompletableFuture<Void>
regionClosing
(RegionStateNode regionNode) (package private) CompletableFuture<Void>
regionFailedOpen
(RegionStateNode regionNode, boolean giveUp) (package private) void
regionOpenedWithoutPersistingToMeta
(RegionStateNode regionNode) (package private) CompletableFuture<Void>
regionOpening
(RegionStateNode regionNode) void
reportOnlineRegions
(ServerName serverName, Set<byte[]> regionNames) The master will call this method when the RS send the regionServerReport().org.apache.hadoop.hbase.shaded.protobuf.generated.RegionServerStatusProtos.ReportRegionStateTransitionResponse
reportRegionStateTransition
(org.apache.hadoop.hbase.shaded.protobuf.generated.RegionServerStatusProtos.ReportRegionStateTransitionRequest req) private void
reportRegionStateTransition
(org.apache.hadoop.hbase.shaded.protobuf.generated.RegionServerStatusProtos.ReportRegionStateTransitionResponse.Builder builder, ServerStateNode serverNode, List<org.apache.hadoop.hbase.shaded.protobuf.generated.RegionServerStatusProtos.RegionStateTransition> transitionList) private boolean
reportTransition
(RegionStateNode regionNode, ServerStateNode serverNode, org.apache.hadoop.hbase.shaded.protobuf.generated.RegionServerStatusProtos.RegionStateTransition.TransitionCode state, long seqId, long procId) private void
setMetaAssigned
(RegionInfo metaRegionInfo, boolean assigned) void
setupRIT
(List<TransitRegionStateProcedure> procs) Create RegionStateNode based on the TRSP list, and attach the TRSP to the RegionStateNode.private boolean
shouldAssignFavoredNodes
(RegionInfo region) void
start()
private void
void
stop()
private void
long
submitServerCrash
(ServerName serverName, boolean shouldSplitWal, boolean force) Usually run by the Master in reaction to server crash during normal processing.private int
submitUnassignProcedure
(TableName tableName, Function<RegionStateNode, Boolean> shouldSubmit, Consumer<RegionStateNode> logRIT, Consumer<TransitRegionStateProcedure> submit) int
submitUnassignProcedureForClosingExcessRegionReplicas
(TableName tableName, int newReplicaCount, Consumer<TransitRegionStateProcedure> submit) Called by ModifyTableProcedure to unassign all the excess region replicas for a table.int
submitUnassignProcedureForDisablingTable
(TableName tableName, Consumer<TransitRegionStateProcedure> submit) Called by DsiableTableProcedure to unassign all regions for a table.private CompletableFuture<Void>
transitStateAndUpdate
(RegionStateNode regionNode, RegionState.State newState, RegionState.State... expectedStates) long
unassign
(RegionInfo regionInfo) private void
updateDeadServerRegionMetrics
(int deadRegions, int unknownRegions) private void
updateRegionMergeTransition
(ServerStateNode serverNode, org.apache.hadoop.hbase.shaded.protobuf.generated.RegionServerStatusProtos.RegionStateTransition.TransitionCode state, RegionInfo merged, RegionInfo hriA, RegionInfo hriB) private void
private void
updateRegionSplitTransition
(ServerStateNode serverNode, org.apache.hadoop.hbase.shaded.protobuf.generated.RegionServerStatusProtos.RegionStateTransition.TransitionCode state, RegionInfo parent, RegionInfo hriA, RegionInfo hriB) private void
updateRegionTransition
(ServerStateNode serverNode, org.apache.hadoop.hbase.shaded.protobuf.generated.RegionServerStatusProtos.RegionStateTransition.TransitionCode state, RegionInfo regionInfo, long seqId, long procId) boolean
waitMetaAssigned
(Procedure<?> proc, RegionInfo regionInfo) Notice that this event does not mean the AM has already finished region state rebuilding.boolean
waitMetaLoaded
(Procedure<?> proc) Wait until AM finishes the meta loading, i.e, the region states rebuilding.private HashMap<RegionInfo,
RegionStateNode> void
This method will be called in master initialization method after callingprocessOfflineRegions()
, as in processOfflineRegions we will generate assign procedures for offline regions, which may be conflict with creating table.
-
Field Details
-
LOG
-
BOOTSTRAP_THREAD_POOL_SIZE_CONF_KEY
- See Also:
-
ASSIGN_DISPATCH_WAIT_MSEC_CONF_KEY
- See Also:
-
DEFAULT_ASSIGN_DISPATCH_WAIT_MSEC
- See Also:
-
ASSIGN_DISPATCH_WAITQ_MAX_CONF_KEY
- See Also:
-
DEFAULT_ASSIGN_DISPATCH_WAITQ_MAX
- See Also:
-
RIT_CHORE_INTERVAL_MSEC_CONF_KEY
- See Also:
-
DEFAULT_RIT_CHORE_INTERVAL_MSEC
- See Also:
-
DEAD_REGION_METRIC_CHORE_INTERVAL_MSEC_CONF_KEY
- See Also:
-
DEFAULT_DEAD_REGION_METRIC_CHORE_INTERVAL_MSEC
- See Also:
-
ASSIGN_MAX_ATTEMPTS
- See Also:
-
DEFAULT_ASSIGN_MAX_ATTEMPTS
- See Also:
-
ASSIGN_RETRY_IMMEDIATELY_MAX_ATTEMPTS
- See Also:
-
DEFAULT_ASSIGN_RETRY_IMMEDIATELY_MAX_ATTEMPTS
- See Also:
-
METRICS_RIT_STUCK_WARNING_THRESHOLD
Region in Transition metrics threshold time- See Also:
-
DEFAULT_RIT_STUCK_WARNING_THRESHOLD
- See Also:
-
UNEXPECTED_STATE_REGION
- See Also:
-
FORCE_REGION_RETAINMENT
- See Also:
-
DEFAULT_FORCE_REGION_RETAINMENT
- See Also:
-
FORCE_REGION_RETAINMENT_WAIT_INTERVAL
The wait time in millis before checking again if the region's previous RS is back online- See Also:
-
DEFAULT_FORCE_REGION_RETAINMENT_WAIT_INTERVAL
- See Also:
-
FORCE_REGION_RETAINMENT_RETRIES
The number of times to check if the region's previous RS is back online, before giving up and proceeding with assignment on a new RS- See Also:
-
DEFAULT_FORCE_REGION_RETAINMENT_RETRIES
- See Also:
-
metaAssignEvent
-
metaLoadEvent
-
metrics
-
ritChore
-
deadMetricChore
-
master
-
running
-
regionStates
-
regionStateStore
-
minVersionToMoveSysTables
When the operator uses this configuration option, any version between the current cluster version and the value of "hbase.min.version.move.system.tables" does not trigger any auto-region movement. Auto-region movement here refers to auto-migration of system table regions to newer server versions. It is assumed that the configured range of versions does not require special handling of moving system table regions to higher versioned RegionServer. This auto-migration is done bycheckIfShouldMoveSystemRegionAsync()
. Example: Let's assume the cluster is on version 1.4.0 and we have set "hbase.min.version.move.system.tables" as "2.0.0". Now if we upgrade one RegionServer on 1.4.0 cluster to 1.6.0 (< 2.0.0), then AssignmentManager will not move hbase:meta, hbase:namespace and other system table regions to newly brought up RegionServer 1.6.0 as part of auto-migration. However, if we upgrade one RegionServer on 1.4.0 cluster to 2.2.0 (> 2.0.0), then AssignmentManager will move all system table regions to newly brought up RegionServer 2.2.0 as part of auto-migration done bycheckIfShouldMoveSystemRegionAsync()
. "hbase.min.version.move.system.tables" is introduced as part of HBASE-22923. -
MIN_VERSION_MOVE_SYS_TABLES_CONFIG
- See Also:
-
DEFAULT_MIN_VERSION_MOVE_SYS_TABLES_CONFIG
- See Also:
-
rsReports
-
shouldAssignRegionsWithFavoredNodes
-
assignDispatchWaitQueueMaxSize
-
assignDispatchWaitMillis
-
assignMaxAttempts
-
assignRetryImmediatelyMaxAttempts
-
masterRegion
-
checkIfShouldMoveSystemRegionLock
-
assignThread
-
forceRegionRetainment
-
forceRegionRetainmentWaitInterval
-
forceRegionRetainmentRetries
-
META_REGION_SET
-
STATES_EXPECTED_ON_OPEN
-
STATES_EXPECTED_ON_CLOSING
-
STATES_EXPECTED_ON_CLOSED
-
STATES_EXPECTED_ON_ASSIGN
-
STATES_EXPECTED_ON_UNASSIGN_OR_MOVE
-
pendingAssignQueue
-
assignQueueLock
-
assignQueueFullCond
-
-
Constructor Details
-
AssignmentManager
-
AssignmentManager
AssignmentManager(MasterServices master, MasterRegion masterRegion, RegionStateStore stateStore)
-
-
Method Details
-
mirrorMetaLocations
- Throws:
IOException
org.apache.zookeeper.KeeperException
-
start
- Throws:
IOException
org.apache.zookeeper.KeeperException
-
setupRIT
Create RegionStateNode based on the TRSP list, and attach the TRSP to the RegionStateNode.This is used to restore the RIT region list, so we do not need to restore it in the loadingMeta method below. And it is also very important as now before submitting a TRSP, we need to attach it to the RegionStateNode, which acts like a guard, so we need to restore this information at the very beginning, before we start processing any procedures.
-
stop
-
isRunning
-
getConfiguration
-
getAssignmentManagerMetrics
-
getBalancer
-
getFavoredNodePromoter
-
getProcedureEnvironment
-
getProcedureScheduler
-
getAssignMaxAttempts
int getAssignMaxAttempts() -
isForceRegionRetainment
-
getForceRegionRetainmentWaitInterval
-
getForceRegionRetainmentRetries
-
getAssignRetryImmediatelyMaxAttempts
-
getRegionStates
-
getRegionsOnServer
Returns the regions hosted by the specified server. Notice that, for SCP, after we submit the SCP, no one can change the region list for the ServerStateNode so we do not need any locks here. And for other usage, this can only give you a snapshot of the current region list for this server, which means, right after you get the region list, new regions may be moved to this server or some regions may be moved out from this server, so you should not use it critically if you need strong consistency. -
getRegionInfo
-
getRegionStateNodes
private Stream<RegionStateNode> getRegionStateNodes(TableName tableName, boolean excludeOfflinedSplitParents) -
getTableRegions
-
getTableRegionsAndLocations
public List<Pair<RegionInfo,ServerName>> getTableRegionsAndLocations(TableName tableName, boolean excludeOfflinedSplitParents) -
getRegionStateStore
-
getFavoredNodes
-
getTableStateManager
-
isTableEnabled
-
isTableDisabled
-
isMetaRegion
-
isMetaRegion
-
getMetaRegionFromName
-
isCarryingMeta
-
isCarryingRegion
-
getMetaForRegion
-
getMetaRegionSet
-
isMetaAssigned
Notice that, this only means the meta region is available on a RS, but the AM may still be loading the region states from meta, so usually you need to checkisMetaLoaded()
first before checking this method, unless you can make sure that your piece of code can only be executed after AM builds the region states.- See Also:
-
isMetaRegionInTransition
-
waitMetaAssigned
Notice that this event does not mean the AM has already finished region state rebuilding. See the comment ofisMetaAssigned()
for more details.- See Also:
-
setMetaAssigned
-
getMetaAssignEvent
-
waitMetaLoaded
Wait until AM finishes the meta loading, i.e, the region states rebuilding. -
wakeMetaLoadedEvent
This method will be called in master initialization method after callingprocessOfflineRegions()
, as in processOfflineRegions we will generate assign procedures for offline regions, which may be conflict with creating table. This is a bit dirty, should be reconsidered after we decide whether to keep theprocessOfflineRegions()
method. -
isMetaLoaded
Return whether AM finishes the meta loading, i.e, the region states rebuilding.- See Also:
-
checkIfShouldMoveSystemRegionAsync
Start a new thread to check if there are region servers whose versions are higher than others. If so, move all system table regions to RS with the highest version to keep compatibility. The reason is, RS in new version may not be able to access RS in old version when there are some incompatible changes.This method is called when a new RegionServer is added to cluster only.
-
getSystemTables
-
preTransitCheck
private void preTransitCheck(RegionStateNode regionNode, RegionState.State[] expectedStates) throws HBaseIOException - Throws:
HBaseIOException
-
createAssignProcedure
private TransitRegionStateProcedure createAssignProcedure(RegionInfo regionInfo, ServerName sn, boolean override, boolean force) throws IOException Create an assign TransitRegionStateProcedure. Makes sure of RegionState. Throws exception if not appropriate UNLESS override is set. Used by hbck2 but also by straightlineassign(RegionInfo, ServerName)
andassignAsync(RegionInfo, ServerName)
.- Parameters:
override
- If false, check RegionState is appropriate for assign; if not throw exception.- Throws:
IOException
- See Also:
-
createAssignProcedure
private TransitRegionStateProcedure createAssignProcedure(RegionStateNode regionNode, ServerName targetServer) Create an assign TransitRegionStateProcedure. Does NO checking of RegionState. Presumes appriopriate state ripe for assign. -
assign
- Throws:
IOException
-
assign
- Throws:
IOException
-
assignAsync
Submits a procedure that assigns a region to a target server without waiting for it to finish- Parameters:
regionInfo
- the region we would like to assignsn
- target server name- Throws:
IOException
-
assignAsync
Submits a procedure that assigns a region without waiting for it to finish- Parameters:
regionInfo
- the region we would like to assign- Throws:
IOException
-
unassign
- Throws:
IOException
-
createMoveRegionProcedure
public TransitRegionStateProcedure createMoveRegionProcedure(RegionInfo regionInfo, ServerName targetServer) throws HBaseIOException - Throws:
HBaseIOException
-
move
- Throws:
IOException
-
moveAsync
- Throws:
HBaseIOException
-
balance
- Throws:
HBaseIOException
-
createRoundRobinAssignProcedures
public TransitRegionStateProcedure[] createRoundRobinAssignProcedures(List<RegionInfo> hris, List<ServerName> serversToExclude) Create round-robin assigns. Use on table creation to distribute out regions across cluster.- Returns:
- AssignProcedures made out of the passed in
hris
and a call to the balancer to populate the assigns with targets chosen using round-robin (default balancer scheme). If at assign-time, the target chosen is no longer up, thats fine, the AssignProcedure will ask the balancer for a new target, and so on.
-
createRoundRobinAssignProcedures
Create round-robin assigns. Use on table creation to distribute out regions across cluster.- Returns:
- AssignProcedures made out of the passed in
hris
and a call to the balancer to populate the assigns with targets chosen using round-robin (default balancer scheme). If at assign-time, the target chosen is no longer up, thats fine, the AssignProcedure will ask the balancer for a new target, and so on.
-
compare
-
createOneAssignProcedure
public TransitRegionStateProcedure createOneAssignProcedure(RegionInfo ri, boolean override, boolean force) Create one TransitRegionStateProcedure to assign a region w/o specifying a target server. This method is called from HBCK2.- Returns:
- an assign or null
-
createOneUnassignProcedure
public TransitRegionStateProcedure createOneUnassignProcedure(RegionInfo ri, boolean override, boolean force) Create one TransitRegionStateProcedure to unassign a region. This method is called from HBCK2.- Returns:
- an unassign or null
-
createAssignProcedures
Create an array of TransitRegionStateProcedure w/o specifying a target server. Used as fallback of caller is unable to docreateAssignProcedures(Map)
. If no target server, at assign time, we will try to use the former location of the region if one exists. This is how we 'retain' the old location across a server restart. Should only be called when you can make sure that no one can touch these regions other than you. For example, when you are creating or enabling table. Presumes all Regions are in appropriate state ripe for assign; no checking of Region state is done in here.- See Also:
-
createAssignProcedures
private TransitRegionStateProcedure[] createAssignProcedures(Map<ServerName, List<RegionInfo>> assignments) Tied tocreateAssignProcedures(List)
in that it is called if caller is unable to run this method. Presumes all Regions are in appropriate state ripe for assign; no checking of Region state is done in here.- Parameters:
assignments
- Map of assignments from which we produce an array of AssignProcedures.- Returns:
- Assignments made from the passed in
assignments
- See Also:
-
forceCreateUnssignProcedure
-
createUnassignProceduresForDisabling
Called by DisableTableProcedure to unassign all the regions for a table. -
submitUnassignProcedure
private int submitUnassignProcedure(TableName tableName, Function<RegionStateNode, Boolean> shouldSubmit, Consumer<RegionStateNode> logRIT, Consumer<TransitRegionStateProcedure> submit) -
submitUnassignProcedureForDisablingTable
public int submitUnassignProcedureForDisablingTable(TableName tableName, Consumer<TransitRegionStateProcedure> submit) Called by DsiableTableProcedure to unassign all regions for a table. Will skip submit unassign procedure if the region is in transition, so you may need to call this method multiple times.- Parameters:
tableName
- the table for closing excess region replicassubmit
- for submitting procedure- Returns:
- the number of regions in transition that we can not schedule unassign procedures
-
submitUnassignProcedureForClosingExcessRegionReplicas
public int submitUnassignProcedureForClosingExcessRegionReplicas(TableName tableName, int newReplicaCount, Consumer<TransitRegionStateProcedure> submit) Called by ModifyTableProcedure to unassign all the excess region replicas for a table. Will skip submit unassign procedure if the region is in transition, so you may need to call this method multiple times.- Parameters:
tableName
- the table for closing excess region replicasnewReplicaCount
- the new replica count, should be less than current replica countsubmit
- for submitting procedure- Returns:
- the number of regions in transition that we can not schedule unassign procedures
-
numberOfUnclosedRegions
private int numberOfUnclosedRegions(TableName tableName, Function<RegionStateNode, Boolean> shouldSubmit) -
numberOfUnclosedRegionsForDisabling
-
numberOfUnclosedExcessRegionReplicas
-
createSplitProcedure
public SplitTableRegionProcedure createSplitProcedure(RegionInfo regionToSplit, byte[] splitKey) throws IOException - Throws:
IOException
-
createTruncateRegionProcedure
public TruncateRegionProcedure createTruncateRegionProcedure(RegionInfo regionToTruncate) throws IOException - Throws:
IOException
-
createMergeProcedure
- Throws:
IOException
-
deleteTable
Delete the region states. This is called by "DeleteTable"- Throws:
IOException
-
reportRegionStateTransition
private void reportRegionStateTransition(org.apache.hadoop.hbase.shaded.protobuf.generated.RegionServerStatusProtos.ReportRegionStateTransitionResponse.Builder builder, ServerStateNode serverNode, List<org.apache.hadoop.hbase.shaded.protobuf.generated.RegionServerStatusProtos.RegionStateTransition> transitionList) throws IOException - Throws:
IOException
-
reportRegionStateTransition
public org.apache.hadoop.hbase.shaded.protobuf.generated.RegionServerStatusProtos.ReportRegionStateTransitionResponse reportRegionStateTransition(org.apache.hadoop.hbase.shaded.protobuf.generated.RegionServerStatusProtos.ReportRegionStateTransitionRequest req) throws PleaseHoldException - Throws:
PleaseHoldException
-
updateRegionTransition
private void updateRegionTransition(ServerStateNode serverNode, org.apache.hadoop.hbase.shaded.protobuf.generated.RegionServerStatusProtos.RegionStateTransition.TransitionCode state, RegionInfo regionInfo, long seqId, long procId) throws IOException - Throws:
IOException
-
reportTransition
private boolean reportTransition(RegionStateNode regionNode, ServerStateNode serverNode, org.apache.hadoop.hbase.shaded.protobuf.generated.RegionServerStatusProtos.RegionStateTransition.TransitionCode state, long seqId, long procId) throws IOException - Throws:
IOException
-
updateRegionSplitTransition
private void updateRegionSplitTransition(ServerStateNode serverNode, org.apache.hadoop.hbase.shaded.protobuf.generated.RegionServerStatusProtos.RegionStateTransition.TransitionCode state, RegionInfo parent, RegionInfo hriA, RegionInfo hriB) throws IOException - Throws:
IOException
-
updateRegionMergeTransition
private void updateRegionMergeTransition(ServerStateNode serverNode, org.apache.hadoop.hbase.shaded.protobuf.generated.RegionServerStatusProtos.RegionStateTransition.TransitionCode state, RegionInfo merged, RegionInfo hriA, RegionInfo hriB) throws IOException - Throws:
IOException
-
reportOnlineRegions
The master will call this method when the RS send the regionServerReport(). The report will contains the "online regions". This method will check the the online regions against the in-memory state of the AM, and we will log a warn message if there is a mismatch. This is because that there is no fencing between the reportRegionStateTransition method and regionServerReport method, so there could be race and introduce inconsistency here, but actually there is no problem. Please see HBASE-21421 and HBASE-21463 for more details. -
closeRegionSilently
CloseregionName
onsn
silently and immediately without using a Procedure or going via hbase:meta. For case where a RegionServer's hosting of a Region is not aligned w/ the Master's accounting of Region state. This is for cleaning up an error in accounting. -
checkOnlineRegionsReport
Check that what the RegionServer reports aligns with the Master's image. If disagreement, we will tell the RegionServer to expediently close a Region we do not think it should have. -
computeRegionInTransitionStat
-
updateRegionsInTransitionMetrics
-
updateDeadServerRegionMetrics
-
handleRegionOverStuckWarningThreshold
-
joinCluster
- Throws:
IOException
-
processOfflineRegions
Create assign procedure for offline regions. Just follow the old processofflineServersWithOnlineRegions method. Since now we do not need to deal with dead server any more, we only deal with the regions in OFFLINE state in this method. And this is a bit strange, that for new regions, we will add it in CLOSED state instead of OFFLINE state, and usually there will be a procedure to track them. The processofflineServersWithOnlineRegions is a legacy from long ago, as things are going really different now, maybe we do not need this method any more. Need to revisit later. -
populateRegionStatesFromMeta
Attempt to loadregionInfo
from META, adding any results to theregionStateStore
Is NOT aware of replica regions.- Parameters:
regionInfo
- the region to be loaded from META.- Throws:
IOException
- If some error occurs while querying META or parsing results.
-
populateRegionStatesFromMeta
Attempt to loadregionEncodedName
from META, adding any results to theregionStateStore
Is NOT aware of replica regions.- Parameters:
regionEncodedName
- encoded name for the region to be loaded from META.- Throws:
IOException
- If some error occurs while querying META or parsing results.
-
loadMeta
- Throws:
IOException
-
checkMetaLoaded
Used to check if the meta loading is done. if not we throw PleaseHoldException since we are rebuilding the RegionStates- Parameters:
hri
- region to check if it is already rebuild- Throws:
PleaseHoldException
- if meta has not been loaded yet
-
getNumRegionsOpened
-
submitServerCrash
Usually run by the Master in reaction to server crash during normal processing. Can also be invoked via external RPC to effect repair; in the latter case, the 'force' flag is set so we push through the SCP though context may indicate already-running-SCP (An old SCP may have exited abnormally, or damaged cluster may still have references in hbase:meta to 'Unknown Servers' -- servers that are not online or in dead servers list, etc.)- Parameters:
force
- Set if the request came in externally over RPC (via hbck2). Force means run the SCP even if it seems as though there might be an outstanding SCP running.- Returns:
- pid of scheduled SCP or
Procedure.NO_PROC_ID
if none scheduled.
-
offlineRegion
-
onlineRegion
-
getSnapShotOfAssignment
-
getReopenStatus
Used by the client (via master) to identify if all regions have the schema updates- Returns:
- Pair indicating the status of the alter command (pending/total)
-
hasRegionsInTransition
-
getRegionsInTransition
-
getAssignedRegions
-
getRegionInfo
Resolve a cachedRegionInfo
from the region name as abyte[]
. -
getRegionInfo
Resolve a cachedRegionInfo
from the encoded region name as aString
. -
transitStateAndUpdate
private CompletableFuture<Void> transitStateAndUpdate(RegionStateNode regionNode, RegionState.State newState, RegionState.State... expectedStates) -
regionOpening
-
regionFailedOpen
-
regionClosing
-
regionOpenedWithoutPersistingToMeta
void regionOpenedWithoutPersistingToMeta(RegionStateNode regionNode) throws UnexpectedStateException - Throws:
UnexpectedStateException
-
regionClosedWithoutPersistingToMeta
void regionClosedWithoutPersistingToMeta(RegionStateNode regionNode) throws UnexpectedStateException - Throws:
UnexpectedStateException
-
persistToMeta
-
regionClosedAbnormally
-
markRegionAsSplit
public void markRegionAsSplit(RegionInfo parent, ServerName serverName, RegionInfo daughterA, RegionInfo daughterB) throws IOException - Throws:
IOException
-
markRegionAsMerged
public void markRegionAsMerged(RegionInfo child, ServerName serverName, RegionInfo[] mergeParents) throws IOException When called here, the merge has happened. The merged regions have been unassigned and the above markRegionClosed has been called on each so they have been disassociated from a hosting Server. The merged region will be open after this call. The merged regions are removed from hbase:meta below. Later they are deleted from the filesystem by the catalog janitor running against hbase:meta. It notices when the merged region no longer holds references to the old regions (References are deleted after a compaction rewrites what the Reference points at but not until the archiver chore runs, are the References removed).- Throws:
IOException
-
shouldAssignFavoredNodes
-
queueAssign
Add the assign operation to the assignment queue. The pending assignment operation will be processed, and each region will be assigned by a server using the balancer. -
startAssignmentThread
-
stopAssignmentThread
-
assignQueueSignal
-
waitOnAssignQueue
-
processAssignQueue
-
containsBogusAssignments
private boolean containsBogusAssignments(Map<RegionInfo, RegionStateNode> regions, List<RegionInfo> hirs) -
processAssignmentPlans
private void processAssignmentPlans(HashMap<RegionInfo, RegionStateNode> regions, HashMap<RegionInfo, ServerName> retainMap, List<RegionInfo> hris, List<ServerName> servers) -
acceptPlan
private void acceptPlan(HashMap<RegionInfo, RegionStateNode> regions, Map<ServerName, throws HBaseIOExceptionList<RegionInfo>> plan) - Throws:
HBaseIOException
-
addToPendingAssignment
private void addToPendingAssignment(HashMap<RegionInfo, RegionStateNode> regions, Collection<RegionInfo> pendingRegions) -
getExcludedServersForSystemTable
For a given cluster with mixed versions of servers, get a list of servers with lower versions, where system table regions should not be assigned to. For system table, we must assign regions to a server with highest version. However, we can disable this exclusion using config: "hbase.min.version.move.system.tables" if checkForMinVersion is true. Detailed explanation available with definition of minVersionToMoveSysTables.- Returns:
- List of Excluded servers for System table regions.
-
getMaster
-
getRSReports
Returns a snapshot of rsReports -
getRegionStatesCount
Provide regions state count for given table. e.g howmany regions of give table are opened/closed/rit etc- Parameters:
tableName
- TableName- Returns:
- region states count
-