Package org.apache.hadoop.hbase.master
Class HMaster
java.lang.Object
java.lang.Thread
org.apache.hadoop.hbase.regionserver.HRegionServer
org.apache.hadoop.hbase.master.HMaster
- All Implemented Interfaces:
Runnable
,Abortable
,ConfigurationObserver
,MasterServices
,FavoredNodesForRegion
,LastSequenceId
,MutableOnlineRegions
,OnlineRegions
,RegionServerServices
,Server
,Stoppable
- Direct Known Subclasses:
HMasterCommandLine.LocalHMaster
HMaster is the "master server" for HBase. An HBase cluster has one active master. If many masters
are started, all compete. Whichever wins goes on to run the cluster. All others park themselves
in their constructor until master or cluster shutdown or until the active master loses its lease
in zookeeper. Thereafter, all running master jostle to take over master role.
The Master can be asked shutdown the cluster. See
shutdown()
. In this case it will tell
all regionservers to go down and then wait on them all reporting in that they are down. This
master will then shut itself down.
You can also shutdown just this master. Call stopMaster()
.- See Also:
-
Watcher
-
Nested Class Summary
Modifier and TypeClassDescriptionprotected static interface
Implement to return TableDescriptor after pre-checksNested classes/interfaces inherited from class java.lang.Thread
Thread.State, Thread.UncaughtExceptionHandler
Nested classes/interfaces inherited from interface org.apache.hadoop.hbase.regionserver.RegionServerServices
RegionServerServices.PostOpenDeployContext, RegionServerServices.RegionStateTransitionContext
-
Field Summary
Modifier and TypeFieldDescriptionprivate boolean
private final ActiveMasterManager
private AssignmentManager
private LoadBalancer
private BalancerChore
private final CachedClusterId
(package private) CatalogJanitor
private ClusterSchemaService
private ClusterStatusChore
private ClusterStatusPublisher
(package private) MasterCoprocessorHost
static final int
static final int
private static final boolean
private static boolean
private DrainingServerTracker
private DirScanPool
private FavoredNodesManager
private MasterFileSystem
static final String
static final String
private HbckChore
private List<org.apache.hadoop.fs.Path>
private List<HFileCleaner>
private final ProcedureEvent<?>
(package private) LoadBalancerStateStore
private final LockManager
private static final org.slf4j.Logger
private LogCleaner
private DirScanPool
(package private) static final String
private final boolean
static final String
private long
(package private) MasterAddressSyncer
private long
private org.apache.hbase.thirdparty.org.eclipse.jetty.server.Server
jetty server for master to redirect requests to regionserver infoServerprivate MasterRegion
private final int
private final double
private MetaLocationSyncer
(package private) final MetricsMaster
private final IdLock
private Map<TableName,
AtomicInteger> private MobFileCleanerChore
private MobFileCompactionChore
private MasterProcedureManagerHost
private OldWALsDirSizeChore
private final boolean
private ProcedureExecutor<MasterProcedureEnv>
private ProcedureStore
private MasterQuotaManager
private QuotaObserverChore
private RegionNormalizerManager
private final RegionServerTracker
private RegionsRecoveryChore
private RegionsRecoveryConfigManager
private ReplicationBarrierCleaner
private ReplicationPeerManager
Store whether we allow replication peer modification operations.private RollingUpgradeChore
(package private) MemoryBoundedLogMessageBuffer
private RegionServerList
private ServerManager
(package private) boolean
private DirScanPool
private SnapshotCleanerChore
(package private) SnapshotCleanupStateStore
(package private) SnapshotManager
private SnapshotQuotaObserverChore
private SpaceQuotaSnapshotNotifier
private SplitOrMergeStateStore
private SplitWALManager
private TaskGroup
private TableStateManager
private MasterWalManager
static final String
Fields inherited from class org.apache.hadoop.hbase.regionserver.HRegionServer
clusterConnection, clusterId, clusterStatusTracker, conf, configurationManager, executorService, infoServer, REGIONSERVER, rpcServices, serverName, sleeper, startcode, tableDescriptors, TEST_SKIP_REPORTING_TRANSITION, useThisHostnameInstead, zooKeeper
Fields inherited from class java.lang.Thread
MAX_PRIORITY, MIN_PRIORITY, NORM_PRIORITY
-
Constructor Summary
-
Method Summary
Modifier and TypeMethodDescriptionvoid
Cause the server to exit without closing the regions it is serving, the log it is using and without notifying the master.boolean
abortProcedure
(long procId, boolean mayInterruptIfRunning) Abort a procedure.long
addColumn
(TableName tableName, ColumnFamilyDescriptor column, long nonceGroup, long nonce) Add a new column to an existing tablelong
addReplicationPeer
(String peerId, ReplicationPeerConfig peerConfig, boolean enabled) Add a new replication peer for replicating data to slave clusterbalance()
balance
(BalanceRequest request) Trigger a normal balance, seebalance()
.boolean
balanceSwitch
(boolean b) private void
balanceThrottling
(long nextBalanceStartTime, int maxRegionsInTransition, long cutoffTime) It first sleep to the next balance plan start time.private org.apache.hbase.thirdparty.org.eclipse.jetty.servlet.ServletHolder
private org.apache.hbase.thirdparty.org.eclipse.jetty.servlet.ServletHolder
protected boolean
protected boolean
protected boolean
void
Called when a new RegionServer is added to the cluster.(package private) void
protected void
private void
checkTableExists
(TableName tableName) void
checkTableModifiable
(TableName tableName) Check table is modifiable; i.e.protected void
static HMaster
constructMaster
(Class<? extends HMaster> masterClass, org.apache.hadoop.conf.Configuration conf) Utility for constructing an instance of the passed HMaster class.protected ActiveMasterManager
createActiveMasterManager
(ZKWatcher zk, ServerName sn, Server server) Protected to have custom implementations in tests override the default ActiveMaster implementation.protected AssignmentManager
createAssignmentManager
(MasterServices master, MasterRegion masterRegion) private void
createMissingCFsInMetaDuringUpgrade
(TableDescriptor metaDescriptor) (package private) long
createNamespace
(NamespaceDescriptor namespaceDescriptor, long nonceGroup, long nonce) Create a new Namespace.private void
private SpaceQuotaSnapshotNotifier
protected RSRpcServices
protected ServerManager
createServerManager
(MasterServices master, RegionServerList storage) Create aServerManager
instance.long
createSystemTable
(TableDescriptor tableDescriptor) Create a system table using the given table definition.long
createTable
(TableDescriptor tableDescriptor, byte[][] splitKeys, long nonceGroup, long nonce) Create a table using the given table definition.void
decommissionRegionServers
(List<ServerName> servers, boolean offload) Mark region server(s) as decommissioned (previously called 'draining') to prevent additional regions from getting assigned to them.static void
decorateMasterConfiguration
(org.apache.hadoop.conf.Configuration conf) This method modifies the master's configuration in order to inject replication-related featureslong
deleteColumn
(TableName tableName, byte[] columnName, long nonceGroup, long nonce) Delete a column from an existing table(package private) long
deleteNamespace
(String name, long nonceGroup, long nonce) Delete an existing Namespace.long
deleteTable
(TableName tableName, long nonceGroup, long nonce) Delete a tablelong
disableReplicationPeer
(String peerId) Stop the replication stream to the specified peerlong
disableTable
(TableName tableName, long nonceGroup, long nonce) Disable an existing tablelong
enableReplicationPeer
(String peerId) Restart the replication stream to the specified peerlong
enableTable
(TableName tableName, long nonceGroup, long nonce) Enable an existing tableprivate long
executePeerProcedure
(ModifyPeerProcedure procedure) Execute region plans with throttlingprivate static void
filterTablesByRegex
(Collection<TableDescriptor> descriptors, Pattern pattern) Removes the table descriptors that don't match the pattern.private void
Finish initialization of HMaster after becoming the primary master.void
Flush master local regionlong
flushTable
(TableName tableName, List<byte[]> columnFamilies, long nonceGroup, long nonce) Flush an existing tableint
Returns info port of active master or 0 if any exception occurs.Returns Master's instance of theAssignmentManager
double
Compute the average load across all region servers.int
Returns Master's instance ofCatalogJanitor
Returns Client info for use as prefix on an audit log string; who did an actionReturns cluster statusgetClusterMetrics
(EnumSet<ClusterMetrics.Option> options) Returns Master's instance ofClusterSchema
getCompactionState
(TableName tableName) Get the compaction state of the tableprotected Class<? extends javax.servlet.http.HttpServlet>
Returns Favored Nodes ManagerReturns Master's instance ofHbckChore
Returns Tripped when Master has finished initialization.long
long
getLastMajorCompactionTimestampForRegion
(byte[] regionName) Returns the timestamp of the last successful major compaction for the passed region or 0 if no HFile resulting from a major compaction existsReturns load balancerFetch the configuredLoadBalancer
class name.static String
The set of loaded coprocessors is stored in a static set.ReturnsLockManager
to lock namespaces/tables/regions.getLocks()
Get lockslong
Returns timestamp in millis when HMaster became the active master.Returns Master's instance ofMasterCoprocessorHost
String[]
Returns array of coprocessor SimpleNames.Returns Master's filesystemMasterFileSystem
utility class.long
Returns timestamp in millis when HMaster finished becoming the active masterReturns Master's instance ofMetricsMaster
Returns Master's instance ofProcedureExecutor
Returns the underlying MasterProcedureManagerHostReturns Master's instance ofMasterQuotaManager
long
Returns timestamp in millis when HMaster was started.Returns Master's WALsMasterWalManager
utility class.private int
Returns Maximum time we should run balancer forprivate int
Returns Maximum number of regions in transitionGet the meta location syncer.org.apache.hadoop.hbase.shaded.protobuf.generated.AdminProtos.GetRegionInfoResponse.CompactionState
getMobCompactionState
(TableName tableName) Gets the mob file compaction state for a specific table.(package private) NamespaceDescriptor
getNamespace
(String name) Get a Namespace(package private) List<NamespaceDescriptor>
Get all Namespacesint
private Map<ServerName,
ServerMetrics> Get proceduresprotected String
Returns Master's instance ofRegionNormalizerManager
int
(package private) InetAddress
getRemoteInetAddress
(int port, long serverStartCode) Returns Get remote side's InetAddressgetRemoteProcedure
(long procId) getReplicationLoad
(ServerName[] serverNames) getReplicationPeerConfig
(String peerId) Returns the configured ReplicationPeerConfig for the specified peerReturns theReplicationPeerManager
.Returns Master'sServerManager
instance.Returns The unique server name for this server.Returns the underlying snapshot managerReturns return null if current is zk-based WAL splittingprivate List<TableDescriptor>
getTableDescriptors
(List<TableDescriptor> htds, String namespace, String regex, List<TableName> tableNameList, boolean includeSysTables) Return a list of table table descriptors after applying any provided filter parameters.Returns Master's instance ofTableStateManager
private List<ServerName>
protected String
getUseThisHostnameInstead
(org.apache.hadoop.conf.Configuration conf) Gets the ZooKeeper instance for this server.protected void
private void
initializeCoprocessorHost
(org.apache.hadoop.conf.Configuration conf) private void
Initialize all ZK based system trackers.private void
private void
boolean
Report whether this master is currently the active master or not.boolean
Queries the state of theLoadBalancerStateStore
.boolean
private static boolean
isCatalogTable
(TableName tableName) (package private) boolean
boolean
Report whether this master has completed with its initialization and is ready.boolean
Report whether this master is in maintenance mode.boolean
Queries the state of theRegionNormalizerStateStore
.boolean
isOnline()
Report whether this master is started This method is used for testing.private boolean
boolean
boolean
isSplitOrMergeEnabled
(MasterSwitchType switchType) Queries the state of theSplitOrMergeStateStore
.List region servers marked as decommissioned (previously called 'draining') to not get regions assigned to them.List namespace nameslistReplicationPeers
(String regex) Return a list of replication peers.listTableDescriptors
(String namespace, String regex, List<TableName> tableNameList, boolean includeSysTables) Returns the list of table descriptors that match the specified requestGet list of table descriptors by namespacelistTableNames
(String namespace, String regex, boolean includeSysTables) Returns the list of table names that match the specified requestGet list of table names by namespaceprotected void
login
(UserProvider user, String host) For compatibility, if failed with regionserver credentials, try the master onestatic void
long
mergeRegions
(RegionInfo[] regionsToMerge, boolean forcible, long ng, long nonce) Merge regions in a table.long
modifyColumn
(TableName tableName, ColumnFamilyDescriptor descriptor, long nonceGroup, long nonce) Modify the column descriptor of an existing column in an existing tablelong
modifyColumnStoreFileTracker
(TableName tableName, byte[] family, String dstSFT, long nonceGroup, long nonce) Modify the store file tracker of an existing column in an existing table(package private) long
modifyNamespace
(NamespaceDescriptor newNsDescriptor, long nonceGroup, long nonce) Modify an existing Namespace.long
modifyTable
(TableName tableName, TableDescriptor newDescriptor, long nonceGroup, long nonce, boolean reopenRegions) Modify the descriptor of an existing tableprivate long
modifyTable
(TableName tableName, HMaster.TableDescriptorGetter newDescriptorGetter, long nonceGroup, long nonce, boolean shouldCheckDescriptor) private long
modifyTable
(TableName tableName, HMaster.TableDescriptorGetter newDescriptorGetter, long nonceGroup, long nonce, boolean shouldCheckDescriptor, boolean reopenRegions) long
modifyTableStoreFileTracker
(TableName tableName, String dstSFT, long nonceGroup, long nonce) Modify the store file tracker of an existing tablevoid
move
(byte[] encodedRegionName, byte[] destServerName) boolean
normalizeRegions
(NormalizeTableFilterParams ntfp, boolean isHighPriority) Perform normalization of cluster.void
onConfigurationChange
(org.apache.hadoop.conf.Configuration newConf) This method would be called by theConfigurationManager
object when theConfiguration
object is reloaded from disk.protected void
protected void
private int
void
recommissionRegionServer
(ServerName server, List<byte[]> encodedRegionNames) Remove decommission marker (previously called 'draining') from a region server to allow regions assignments.private void
boolean
registerService
(com.google.protobuf.Service instance) Registers a new protocol bufferService
subclass as a coprocessor endpoint to be available for handlingvoid
remoteProcedureCompleted
(long procId) void
remoteProcedureFailed
(long procId, RemoteProcedureException error) long
removeReplicationPeer
(String peerId) Removes a peer and stops the replication(package private) long
reopenRegions
(TableName tableName, List<byte[]> regionNames, long nonceGroup, long nonce) Reopen regions provided in the argumentboolean
replicationPeerModificationSwitch
(boolean on) void
reportMobCompactionEnd
(TableName tableName) void
reportMobCompactionStart
(TableName tableName) long
restoreSnapshot
(org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDescription snapshotDesc, long nonceGroup, long nonce, boolean restoreAcl, String customSFT) void
run()
The HRegionServer sticks in this loop until closed.void
Run the ReplicationBarrierChore.(package private) void
setAssignmentManager
(AssignmentManager assignmentManager) void
setCatalogJanitorChoreForTesting
(CatalogJanitor catalogJanitorChore) Used for testing only to set Mock objects.void
setCatalogJanitorEnabled
(boolean b) Switch for the background CatalogJanitor thread.(package private) static void
setDisableBalancerChoreForTest
(boolean disable) void
setHbckChoreForTesting
(HbckChore hbckChore) Used for testing only to set Mock objects.void
setInitialized
(boolean isInitialized) (package private) void
setLoadBalancer
(LoadBalancer loadBalancer) private void
setQuotasObserver
(org.apache.hadoop.conf.Configuration conf) void
setServiceStarted
(boolean started) Mainly used in procedure related tests, where we will restart ProcedureExecutor and AssignmentManager, but we do not want to restart master(to speed up the test), so we need to disable rpc for a while otherwise some critical rpc requests such as reportRegionStateTransition could fail and cause region server to abort.void
shutdown()
Shutdown the cluster.boolean
skipRegionManagementAction
(String action) Checks master state before initiating action over region topology.long
splitRegion
(RegionInfo regionInfo, byte[] splitRow, long nonceGroup, long nonce) Split a region.private void
startActiveMasterManager
(int infoPort) protected void
private void
void
Stop this service.private void
void
private void
protected void
Wait on all threads to finish.private void
switchSnapshotCleanup
(boolean on) (package private) void
switchSnapshotCleanup
(boolean on, boolean synchronous) Turn on/off Snapshot Cleanup Chorelong
truncateRegion
(RegionInfo regionInfo, long nonceGroup, long nonce) Truncate regionlong
truncateTable
(TableName tableName, boolean preserveSplits, long nonceGroup, long nonce) Truncate a tableprivate void
void
updateConfigurationForQuotasObserver
(org.apache.hadoop.conf.Configuration conf) Adds theMasterQuotasObserver
to the list of configured Master observers to automatically remove quotas for a table when that table is deleted.long
updateReplicationPeerConfig
(String peerId, ReplicationPeerConfig peerConfig) Update the peerConfig for the specified peerprotected void
If configured to put regions on active master, wait till a backup master becomes active.boolean
Check hbase:meta is up and ready for reading.boolean
Check hbase:namespace table is assigned.private void
waitForRegionServers
(MonitoredTask status) Methods inherited from class org.apache.hadoop.hbase.regionserver.HRegionServer
addRegion, closeRegion, createClusterConnection, createConnection, createRegionLoad, createRegionServerStatusStub, dumpRowLocks, finishRegionProcedure, getAccessChecker, getBlockCache, getBrokenStoreFileCleaner, getChoreService, getClusterConnection, getCompactedHFilesDischarger, getCompactionPressure, getCompactionRequestor, getCompactSplitThread, getConfiguration, getConfigurationManager, getConnection, getCoordinatedStateManager, getDataRootDir, getEventLoopGroupConfig, getExecutorService, getFavoredNodesForRegion, getFileSystem, getFlushPressure, getFlushRequester, getFlushThroughputController, getHeapMemoryManager, getInfoServer, getLastSequenceId, getLeaseManager, getMasterAddressTracker, getMemStoreFlusher, getMetaRegionLocationCache, getMetrics, getMobFileCache, getMovedRegion, getNamedQueueRecorder, getNonceManager, getNumberOfOnlineRegions, getOnlineRegion, getOnlineRegions, getOnlineRegionsLocalContext, getOnlineTables, getRegion, getRegion, getRegionByEncodedName, getRegions, getRegions, getRegionServerAccounting, getRegionServerCoprocessorHost, getRegionServerCoprocessors, getRegionServerRpcQuotaManager, getRegionServerSpaceQuotaManager, getRegionsInTransitionInRS, getReplicationSourceService, getRetryPauseTime, getRpcServer, getRSMobFileCleanerChore, getRSRpcServices, getSecureBulkLoadManager, getStartcode, getTableDescriptors, getWAL, getWALFileSystem, getWalRoller, getWALRootDir, getWALs, getZKPermissionWatcher, handleReportForDutyResponse, initializeMemStoreChunkCreator, isAborted, isClusterUp, isShutDown, isShutdownHookInstalled, isStopped, isStopping, kill, movedRegionCacheExpiredTime, postOpenDeployTasks, regionLock, remoteProcedureComplete, removeRegion, reportFileArchivalForQuotas, reportRegionSizesForQuotas, reportRegionStateTransition, setAbortRequested, setupClusterConnection, shutdownChore, stop, toString, tryRegionServerReport, unassign, updateRegionFavoredNodesMapping, waitForServerOnline, walRollRequestFinished
Methods inherited from class java.lang.Thread
activeCount, checkAccess, clone, countStackFrames, currentThread, destroy, dumpStack, enumerate, getAllStackTraces, getContextClassLoader, getDefaultUncaughtExceptionHandler, getId, getName, getPriority, getStackTrace, getState, getThreadGroup, getUncaughtExceptionHandler, holdsLock, interrupt, interrupted, isAlive, isDaemon, isInterrupted, join, join, join, resume, setContextClassLoader, setDaemon, setDefaultUncaughtExceptionHandler, setName, setPriority, setUncaughtExceptionHandler, sleep, sleep, start, stop, stop, suspend, yield
Methods inherited from class java.lang.Object
equals, finalize, getClass, hashCode, notify, notifyAll, wait, wait, wait
Methods inherited from interface org.apache.hadoop.hbase.master.MasterServices
getAccessChecker, getExecutorService, getTableDescriptors, getZKPermissionWatcher, isClusterUp, modifyTable
Methods inherited from interface org.apache.hadoop.hbase.Server
createConnection, getChoreService, getClusterConnection, getConfiguration, getConnection, getCoordinatedStateManager, getFileSystem, isStopping
-
Field Details
-
LOG
-
MASTER
- See Also:
-
activeMasterManager
-
regionServerTracker
-
drainingServerTracker
-
loadBalancerStateStore
-
metaLocationSyncer
-
masterAddressSyncer
-
snapshotCleanupStateStore
-
splitOrMergeStateStore
-
clusterSchemaService
-
HBASE_MASTER_WAIT_ON_SERVICE_IN_SECONDS
- See Also:
-
DEFAULT_HBASE_MASTER_WAIT_ON_SERVICE_IN_SECONDS
- See Also:
-
HBASE_MASTER_CLEANER_INTERVAL
- See Also:
-
DEFAULT_HBASE_MASTER_CLEANER_INTERVAL
- See Also:
-
metricsMaster
-
fileSystemManager
-
walManager
-
splitWALManager
-
serverManager
-
assignmentManager
-
replicationPeerManager
-
rsFatals
-
activeMaster
-
initialized
-
serviceStarted
-
maxBalancingTime
-
maxRitPercent
-
lockManager
-
balancer
-
balancerChore
-
disableBalancerChoreForTest
-
regionNormalizerManager
-
clusterStatusChore
-
clusterStatusPublisherChore
-
snapshotCleanerChore
-
hbckChore
-
catalogJanitorChore
-
logCleanerPool
-
logCleaner
-
hfileCleaners
-
hfileCleanerPaths
-
exclusiveHFileCleanerPool
-
replicationBarrierCleaner
-
mobFileCleanerChore
-
mobFileCompactionChore
-
rollingUpgradeChore
-
mobCompactionLock
-
mobCompactionStates
-
cpHost
-
preLoadTableDescriptors
-
masterActiveTime
-
masterFinishedInitializationTime
-
coprocessorServiceHandlers
Map<String,com.google.protobuf.Service> coprocessorServiceHandlers -
snapshotManager
-
mpmHost
-
regionsRecoveryChore
-
regionsRecoveryConfigManager
-
quotaManager
-
spaceQuotaSnapshotNotifier
-
quotaObserverChore
-
snapshotQuotaChore
-
oldWALsDirSizeChore
-
procedureExecutor
-
procedureStore
-
masterRegion
-
rsListStorage
-
tableStateManager
-
favoredNodesManager
-
masterJettyServer
jetty server for master to redirect requests to regionserver infoServer -
maintenanceMode
-
MAINTENANCE_MODE
- See Also:
-
cachedClusterId
-
WARMUP_BEFORE_MOVE
- See Also:
-
DEFAULT_WARMUP_BEFORE_MOVE
- See Also:
-
startupTaskGroup
-
replicationPeerModificationStateStore
Store whether we allow replication peer modification operations.
-
-
Constructor Details
-
HMaster
Initializes the HMaster. The steps are as follows:- Initialize the local HRegionServer
- Start the ActiveMasterManager.
Remaining steps of initialization occur in
finishActiveMasterInitialization()
after the master becomes the active one.- Throws:
IOException
-
-
Method Details
-
createActiveMasterManager
protected ActiveMasterManager createActiveMasterManager(ZKWatcher zk, ServerName sn, Server server) throws InterruptedIOException Protected to have custom implementations in tests override the default ActiveMaster implementation.- Throws:
InterruptedIOException
-
getUseThisHostnameInstead
- Overrides:
getUseThisHostnameInstead
in classHRegionServer
-
registerConfigurationObservers
-
preUpdateConfiguration
- Overrides:
preUpdateConfiguration
in classHRegionServer
- Throws:
IOException
-
postUpdateConfiguration
- Overrides:
postUpdateConfiguration
in classHRegionServer
- Throws:
IOException
-
run
Description copied from class:HRegionServer
The HRegionServer sticks in this loop until closed.- Specified by:
run
in interfaceRunnable
- Overrides:
run
in classHRegionServer
-
putUpJettyServer
- Throws:
IOException
-
login
For compatibility, if failed with regionserver credentials, try the master one- Overrides:
login
in classHRegionServer
- Throws:
IOException
-
waitForMasterActive
If configured to put regions on active master, wait till a backup master becomes active. Otherwise, loop till the server is stopped or aborted.- Overrides:
waitForMasterActive
in classHRegionServer
-
getMasterRpcServices
-
balanceSwitch
- Throws:
IOException
-
getProcessName
- Overrides:
getProcessName
in classHRegionServer
-
canCreateBaseZNode
- Overrides:
canCreateBaseZNode
in classHRegionServer
-
canUpdateTableDescriptor
- Overrides:
canUpdateTableDescriptor
in classHRegionServer
-
cacheTableDescriptor
- Overrides:
cacheTableDescriptor
in classHRegionServer
-
createRpcServices
- Overrides:
createRpcServices
in classHRegionServer
- Throws:
IOException
-
configureInfoServer
- Overrides:
configureInfoServer
in classHRegionServer
-
buildApiV1Servlet
-
buildHbckServlet
-
getDumpServlet
- Overrides:
getDumpServlet
in classHRegionServer
-
getMasterMetrics
Description copied from interface:MasterServices
Returns Master's instance ofMetricsMaster
- Specified by:
getMasterMetrics
in interfaceMasterServices
-
initializeZKBasedSystemTrackers
private void initializeZKBasedSystemTrackers() throws IOException, org.apache.zookeeper.KeeperException, ReplicationException, DeserializationExceptionInitialize all ZK based system trackers. But do not includeRegionServerTracker
, it should have already been initialized along withServerManager
.- Throws:
IOException
org.apache.zookeeper.KeeperException
ReplicationException
DeserializationException
-
createAssignmentManager
@Private protected AssignmentManager createAssignmentManager(MasterServices master, MasterRegion masterRegion) -
tryMigrateMetaLocationsFromZooKeeper
private void tryMigrateMetaLocationsFromZooKeeper() throws IOException, org.apache.zookeeper.KeeperException- Throws:
IOException
org.apache.zookeeper.KeeperException
-
finishActiveMasterInitialization
private void finishActiveMasterInitialization() throws IOException, InterruptedException, org.apache.zookeeper.KeeperException, ReplicationException, DeserializationExceptionFinish initialization of HMaster after becoming the primary master. The startup order is a bit complicated but very important, do not change it unless you know what you are doing.- Initialize file system based components - file system manager, wal manager, table descriptors, etc
- Publish cluster id
- Here comes the most complicated part - initialize server manager, assignment manager and
region server tracker
- Create server manager
- Create master local region
- Create procedure executor, load the procedures, but do not start workers. We will start it later after we finish scheduling SCPs to avoid scheduling duplicated SCPs for the same server
- Create assignment manager and start it, load the meta region state, but do not load data from meta region
- Start region server tracker, construct the online servers set and find out dead servers and schedule SCP for them. The online servers will be constructed by scanning zk, and we will also scan the wal directory to find out possible live region servers, and the differences between these two sets are the dead servers
- If this is a new deploy, schedule a InitMetaProcedure to initialize meta
- Start necessary service threads - balancer, catalog janitor, executor services, and also the procedure executor, etc. Notice that the balancer must be created first as assignment manager may use it when assigning regions.
- Wait for meta to be initialized if necessary, start table state manager.
- Wait for enough region servers to check-in
- Let assignment manager load data from meta and construct region states
- Start all other things such as chore services, etc
- Throws:
IOException
InterruptedException
org.apache.zookeeper.KeeperException
ReplicationException
DeserializationException
-
setHbckChoreForTesting
Used for testing only to set Mock objects.- Parameters:
hbckChore
- hbckChore
-
setCatalogJanitorChoreForTesting
Used for testing only to set Mock objects.- Parameters:
catalogJanitorChore
- catalogJanitorChore
-
createMissingCFsInMetaDuringUpgrade
- Throws:
IOException
-
waitForMetaOnline
Check hbase:meta is up and ready for reading. For use during Master startup only.- Returns:
- True if meta is UP and online and startup can progress. Otherwise, meta is not online and we will hold here until operator intervention.
-
isRegionOnline
- Returns:
- True if region is online and scannable else false if an error or shutdown (Otherwise we just block in here holding up all forward-progess).
-
waitForNamespaceOnline
Check hbase:namespace table is assigned. If not, startup will hang looking for the ns table (TODO: Fix this! NS should not hold-up startup).- Returns:
- True if namespace table is up/online.
-
updateConfigurationForQuotasObserver
@Private public void updateConfigurationForQuotasObserver(org.apache.hadoop.conf.Configuration conf) Adds theMasterQuotasObserver
to the list of configured Master observers to automatically remove quotas for a table when that table is deleted. -
initMobCleaner
-
createServerManager
@Private protected ServerManager createServerManager(MasterServices master, RegionServerList storage) throws IOException Create a
ServerManager
instance.Will be overridden in tests.
- Throws:
IOException
-
waitForRegionServers
- Throws:
IOException
InterruptedException
-
initClusterSchemaService
- Throws:
IOException
InterruptedException
-
initQuotaManager
- Throws:
IOException
-
createQuotaSnapshotNotifier
-
isCatalogJanitorEnabled
-
isCleanerChoreEnabled
boolean isCleanerChoreEnabled() -
getServerManager
Description copied from interface:MasterServices
Returns Master'sServerManager
instance.- Specified by:
getServerManager
in interfaceMasterServices
-
getMasterFileSystem
Description copied from interface:MasterServices
Returns Master's filesystemMasterFileSystem
utility class.- Specified by:
getMasterFileSystem
in interfaceMasterServices
-
getMasterWalManager
Description copied from interface:MasterServices
Returns Master's WALsMasterWalManager
utility class.- Specified by:
getMasterWalManager
in interfaceMasterServices
-
getSplitWALManager
Description copied from interface:MasterServices
Returns return null if current is zk-based WAL splitting- Specified by:
getSplitWALManager
in interfaceMasterServices
-
getTableStateManager
Description copied from interface:MasterServices
Returns Master's instance ofTableStateManager
- Specified by:
getTableStateManager
in interfaceMasterServices
-
startServiceThreads
- Throws:
IOException
-
stopServiceThreads
Description copied from class:HRegionServer
Wait on all threads to finish. Presumption is that all closes and stops have already been called.- Overrides:
stopServiceThreads
in classHRegionServer
-
createProcedureExecutor
- Throws:
IOException
-
startProcedureExecutor
- Throws:
IOException
-
switchSnapshotCleanup
Turn on/off Snapshot Cleanup Chore- Parameters:
on
- indicates whether Snapshot Cleanup Chore is to be run- Throws:
IOException
-
switchSnapshotCleanup
- Throws:
IOException
-
stopProcedureExecutor
-
stopChores
-
getRemoteInetAddress
Returns Get remote side's InetAddress- Throws:
UnknownHostException
-
getMaxBalancingTime
Returns Maximum time we should run balancer for -
getMaxRegionsInTransition
Returns Maximum number of regions in transition -
balanceThrottling
private void balanceThrottling(long nextBalanceStartTime, int maxRegionsInTransition, long cutoffTime) It first sleep to the next balance plan start time. Meanwhile, throttling by the max number regions in transition to protect availability.- Parameters:
nextBalanceStartTime
- The next balance plan start timemaxRegionsInTransition
- max number of regions in transitioncutoffTime
- when to exit balancer
-
balance
- Throws:
IOException
-
balanceOrUpdateMetrics
Trigger a normal balance, seebalance()
. If the balance is not executed this time, the metrics related to the balance will be updated. When balance is running, related metrics will be updated at the same time. But if some checking logic failed and cause the balancer exit early, we lost the chance to update balancer metrics. This will lead to user missing the latest balancer info.- Throws:
IOException
-
skipRegionManagementAction
Checks master state before initiating action over region topology.- Specified by:
skipRegionManagementAction
in interfaceMasterServices
- Parameters:
action
- the name of the action under consideration, for logging.- Returns:
true
when the caller should exit early,false
otherwise.
-
balance
- Throws:
IOException
-
executeRegionPlansWithThrottling
Execute region plans with throttling- Specified by:
executeRegionPlansWithThrottling
in interfaceMasterServices
- Parameters:
plans
- to execute- Returns:
- succeeded plans
-
getRegionNormalizerManager
Description copied from interface:MasterServices
Returns Master's instance ofRegionNormalizerManager
- Specified by:
getRegionNormalizerManager
in interfaceMasterServices
-
normalizeRegions
public boolean normalizeRegions(NormalizeTableFilterParams ntfp, boolean isHighPriority) throws IOException Description copied from interface:MasterServices
Perform normalization of cluster.- Specified by:
normalizeRegions
in interfaceMasterServices
- Parameters:
ntfp
- Selection criteria for identifying which tables to normalize.isHighPriority
-true
when these requested tables should skip to the front of the queue.- Returns:
true
when the request was submitted,false
otherwise.- Throws:
IOException
-
getClientIdAuditPrefix
Returns Client info for use as prefix on an audit log string; who did an action- Specified by:
getClientIdAuditPrefix
in interfaceMasterServices
-
setCatalogJanitorEnabled
Switch for the background CatalogJanitor thread. Used for testing. The thread will continue to run. It will just be a noop if disabled.- Parameters:
b
- If false, the catalog janitor won't do anything.
-
mergeRegions
public long mergeRegions(RegionInfo[] regionsToMerge, boolean forcible, long ng, long nonce) throws IOException Description copied from interface:MasterServices
Merge regions in a table.- Specified by:
mergeRegions
in interfaceMasterServices
- Parameters:
regionsToMerge
- daughter regions to mergeforcible
- whether to force to merge even two regions are not adjacentng
- used to detect duplicatenonce
- used to detect duplicate- Returns:
- procedure Id
- Throws:
IOException
-
splitRegion
public long splitRegion(RegionInfo regionInfo, byte[] splitRow, long nonceGroup, long nonce) throws IOException Description copied from interface:MasterServices
Split a region.- Specified by:
splitRegion
in interfaceMasterServices
- Parameters:
regionInfo
- region to splitsplitRow
- split pointnonceGroup
- used to detect duplicatenonce
- used to detect duplicate- Returns:
- procedure Id
- Throws:
IOException
-
move
- Throws:
HBaseIOException
-
createTable
public long createTable(TableDescriptor tableDescriptor, byte[][] splitKeys, long nonceGroup, long nonce) throws IOException Description copied from interface:MasterServices
Create a table using the given table definition.- Specified by:
createTable
in interfaceMasterServices
- Parameters:
tableDescriptor
- The table definitionsplitKeys
- Starting row keys for the initial table regions. If null a single region is created.- Throws:
IOException
-
createSystemTable
Description copied from interface:MasterServices
Create a system table using the given table definition.- Specified by:
createSystemTable
in interfaceMasterServices
- Parameters:
tableDescriptor
- The system table definition a single region is created.- Throws:
IOException
-
startActiveMasterManager
- Throws:
org.apache.zookeeper.KeeperException
-
isCatalogTable
-
deleteTable
Description copied from interface:MasterServices
Delete a table- Specified by:
deleteTable
in interfaceMasterServices
- Parameters:
tableName
- The table name- Throws:
IOException
-
truncateTable
public long truncateTable(TableName tableName, boolean preserveSplits, long nonceGroup, long nonce) throws IOException Description copied from interface:MasterServices
Truncate a table- Specified by:
truncateTable
in interfaceMasterServices
- Parameters:
tableName
- The table namepreserveSplits
- True if the splits should be preserved- Throws:
IOException
-
truncateRegion
Description copied from interface:MasterServices
Truncate region- Specified by:
truncateRegion
in interfaceMasterServices
- Parameters:
regionInfo
- region to be truncatednonceGroup
- the nonce groupnonce
- the nonce- Returns:
- procedure Id
- Throws:
IOException
-
addColumn
public long addColumn(TableName tableName, ColumnFamilyDescriptor column, long nonceGroup, long nonce) throws IOException Description copied from interface:MasterServices
Add a new column to an existing table- Specified by:
addColumn
in interfaceMasterServices
- Parameters:
tableName
- The table namecolumn
- The column definition- Throws:
IOException
-
modifyColumn
public long modifyColumn(TableName tableName, ColumnFamilyDescriptor descriptor, long nonceGroup, long nonce) throws IOException Description copied from interface:MasterServices
Modify the column descriptor of an existing column in an existing table- Specified by:
modifyColumn
in interfaceMasterServices
- Parameters:
tableName
- The table namedescriptor
- The updated column definition- Throws:
IOException
-
modifyColumnStoreFileTracker
public long modifyColumnStoreFileTracker(TableName tableName, byte[] family, String dstSFT, long nonceGroup, long nonce) throws IOException Description copied from interface:MasterServices
Modify the store file tracker of an existing column in an existing table- Specified by:
modifyColumnStoreFileTracker
in interfaceMasterServices
- Throws:
IOException
-
deleteColumn
public long deleteColumn(TableName tableName, byte[] columnName, long nonceGroup, long nonce) throws IOException Description copied from interface:MasterServices
Delete a column from an existing table- Specified by:
deleteColumn
in interfaceMasterServices
- Parameters:
tableName
- The table namecolumnName
- The column name- Throws:
IOException
-
enableTable
Description copied from interface:MasterServices
Enable an existing table- Specified by:
enableTable
in interfaceMasterServices
- Parameters:
tableName
- The table name- Throws:
IOException
-
disableTable
Description copied from interface:MasterServices
Disable an existing table- Specified by:
disableTable
in interfaceMasterServices
- Parameters:
tableName
- The table name- Throws:
IOException
-
modifyTable
private long modifyTable(TableName tableName, HMaster.TableDescriptorGetter newDescriptorGetter, long nonceGroup, long nonce, boolean shouldCheckDescriptor) throws IOException - Throws:
IOException
-
modifyTable
private long modifyTable(TableName tableName, HMaster.TableDescriptorGetter newDescriptorGetter, long nonceGroup, long nonce, boolean shouldCheckDescriptor, boolean reopenRegions) throws IOException - Throws:
IOException
-
modifyTable
public long modifyTable(TableName tableName, TableDescriptor newDescriptor, long nonceGroup, long nonce, boolean reopenRegions) throws IOException Description copied from interface:MasterServices
Modify the descriptor of an existing table- Specified by:
modifyTable
in interfaceMasterServices
- Parameters:
tableName
- The table namenewDescriptor
- The updated table descriptorreopenRegions
- Whether to reopen regions after modifying the table descriptor- Throws:
IOException
-
modifyTableStoreFileTracker
public long modifyTableStoreFileTracker(TableName tableName, String dstSFT, long nonceGroup, long nonce) throws IOException Description copied from interface:MasterServices
Modify the store file tracker of an existing table- Specified by:
modifyTableStoreFileTracker
in interfaceMasterServices
- Throws:
IOException
-
restoreSnapshot
public long restoreSnapshot(org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDescription snapshotDesc, long nonceGroup, long nonce, boolean restoreAcl, String customSFT) throws IOException - Throws:
IOException
-
checkTableExists
- Throws:
IOException
TableNotFoundException
-
checkTableModifiable
public void checkTableModifiable(TableName tableName) throws IOException, TableNotFoundException, TableNotDisabledException Description copied from interface:MasterServices
Check table is modifiable; i.e. exists and is offline.- Specified by:
checkTableModifiable
in interfaceMasterServices
- Parameters:
tableName
- Name of table to check.- Throws:
IOException
TableNotFoundException
TableNotDisabledException
-
getClusterMetricsWithoutCoprocessor
- Throws:
InterruptedIOException
-
getClusterMetricsWithoutCoprocessor
public ClusterMetrics getClusterMetricsWithoutCoprocessor(EnumSet<ClusterMetrics.Option> options) throws InterruptedIOException - Throws:
InterruptedIOException
-
getUnknownServers
-
getOnlineServers
-
getClusterMetrics
Returns cluster status- Throws:
IOException
-
getClusterMetrics
- Throws:
IOException
-
getActiveMaster
- Overrides:
getActiveMaster
in classHRegionServer
-
getBackupMasters
- Overrides:
getBackupMasters
in classHRegionServer
-
getActiveMasterInfoPort
Returns info port of active master or 0 if any exception occurs. -
getBackupMasterInfoPort
- Parameters:
sn
- is ServerName of the backup master- Returns:
- info port of backup master or 0 if any exception occurs.
-
getBootstrapNodes
- Overrides:
getBootstrapNodes
in classHRegionServer
-
getLoadedCoprocessors
The set of loaded coprocessors is stored in a static set. Since it's statically allocated, it does not require that HMaster's cpHost be initialized prior to accessing it.- Returns:
- a String representation of the set of names of the loaded coprocessors.
-
getMasterStartTime
Returns timestamp in millis when HMaster was started. -
getMasterActiveTime
Returns timestamp in millis when HMaster became the active master. -
getMasterFinishedInitializationTime
Returns timestamp in millis when HMaster finished becoming the active master -
getNumWALFiles
-
getProcedureStore
-
getRegionServerInfoPort
-
getRegionServerVersion
- Specified by:
getRegionServerVersion
in interfaceMasterServices
-
checkIfShouldMoveSystemRegionAsync
Description copied from interface:MasterServices
Called when a new RegionServer is added to the cluster. Checks if new server has a newer version than any existing server and will move system tables there if so.- Specified by:
checkIfShouldMoveSystemRegionAsync
in interfaceMasterServices
-
getMasterCoprocessors
Returns array of coprocessor SimpleNames. -
abort
Description copied from class:HRegionServer
Cause the server to exit without closing the regions it is serving, the log it is using and without notifying the master. Used unit testing and on catastrophic events such as HDFS is yanked out from under hbase or we OOME. the reason we are aborting the exception that caused the abort, or null- Specified by:
abort
in interfaceAbortable
- Overrides:
abort
in classHRegionServer
- Parameters:
reason
- Why we're aborting.cause
- Throwable that caused abort. Can be null.
-
getZooKeeper
Description copied from interface:Server
Gets the ZooKeeper instance for this server.- Specified by:
getZooKeeper
in interfaceServer
- Overrides:
getZooKeeper
in classHRegionServer
-
getMasterCoprocessorHost
Description copied from interface:MasterServices
Returns Master's instance ofMasterCoprocessorHost
- Specified by:
getMasterCoprocessorHost
in interfaceMasterServices
-
getMasterQuotaManager
Description copied from interface:MasterServices
Returns Master's instance ofMasterQuotaManager
- Specified by:
getMasterQuotaManager
in interfaceMasterServices
-
getMasterProcedureExecutor
Description copied from interface:MasterServices
Returns Master's instance ofProcedureExecutor
- Specified by:
getMasterProcedureExecutor
in interfaceMasterServices
-
getServerName
Description copied from interface:Server
Returns The unique server name for this server.- Specified by:
getServerName
in interfaceServer
- Overrides:
getServerName
in classHRegionServer
-
getAssignmentManager
Description copied from interface:MasterServices
Returns Master's instance of theAssignmentManager
- Specified by:
getAssignmentManager
in interfaceMasterServices
-
getCatalogJanitor
Description copied from interface:MasterServices
Returns Master's instance ofCatalogJanitor
- Specified by:
getCatalogJanitor
in interfaceMasterServices
-
getRegionServerFatalLogBuffer
-
getStartupProgress
-
shutdown
Shutdown the cluster. Master runs a coordinated stop of all RegionServers and then itself.- Throws:
IOException
-
stopMaster
- Throws:
IOException
-
stop
Description copied from interface:Stoppable
Stop this service. Implementers should favor logging errors over throwing RuntimeExceptions.- Specified by:
stop
in interfaceStoppable
- Overrides:
stop
in classHRegionServer
- Parameters:
msg
- Why we're stopping.
-
checkServiceStarted
- Throws:
ServerNotRunningYetException
-
checkInitialized
-
isActiveMaster
Report whether this master is currently the active master or not. If not active master, we are parked on ZK waiting to become active. This method is used for testing.- Specified by:
isActiveMaster
in interfaceMasterServices
- Returns:
- true if active master, false if not.
-
isInitialized
Report whether this master has completed with its initialization and is ready. If ready, the master is also the active master. A standby master is never ready. This method is used for testing.- Specified by:
isInitialized
in interfaceMasterServices
- Returns:
- true if master is ready to go, false if not.
-
isOnline
Report whether this master is started This method is used for testing.- Overrides:
isOnline
in classHRegionServer
- Returns:
- true if master is ready to go, false if not.
-
isInMaintenanceMode
Report whether this master is in maintenance mode.- Specified by:
isInMaintenanceMode
in interfaceMasterServices
- Returns:
- true if master is in maintenanceMode
-
setInitialized
-
setServiceStarted
Mainly used in procedure related tests, where we will restart ProcedureExecutor and AssignmentManager, but we do not want to restart master(to speed up the test), so we need to disable rpc for a while otherwise some critical rpc requests such as reportRegionStateTransition could fail and cause region server to abort. -
getInitializedEvent
Description copied from interface:MasterServices
Returns Tripped when Master has finished initialization.- Specified by:
getInitializedEvent
in interfaceMasterServices
-
getAverageLoad
Compute the average load across all region servers. Currently, this uses a very naive computation - just uses the number of regions being served, ignoring stats about number of requests.- Returns:
- the average load
-
registerService
Description copied from interface:RegionServerServices
Registers a new protocol bufferService
subclass as a coprocessor endpoint to be available for handling- Specified by:
registerService
in interfaceMasterServices
- Specified by:
registerService
in interfaceRegionServerServices
- Overrides:
registerService
in classHRegionServer
- Parameters:
instance
- theService
subclass instance to expose as a coprocessor endpoint- Returns:
true
if the registration was successful,false
-
constructMaster
public static HMaster constructMaster(Class<? extends HMaster> masterClass, org.apache.hadoop.conf.Configuration conf) Utility for constructing an instance of the passed HMaster class.- Returns:
- HMaster instance.
-
main
- See Also:
-
getHFileCleaner
-
getHFileCleaners
-
getLogCleaner
-
getSnapshotManager
Returns the underlying snapshot manager- Specified by:
getSnapshotManager
in interfaceMasterServices
-
getMasterProcedureManagerHost
Returns the underlying MasterProcedureManagerHost- Specified by:
getMasterProcedureManagerHost
in interfaceMasterServices
-
getClusterSchema
Description copied from interface:MasterServices
Returns Master's instance ofClusterSchema
- Specified by:
getClusterSchema
in interfaceMasterServices
-
createNamespace
long createNamespace(NamespaceDescriptor namespaceDescriptor, long nonceGroup, long nonce) throws IOException Create a new Namespace.- Parameters:
namespaceDescriptor
- descriptor for new NamespacenonceGroup
- Identifier for the source of the request, a client or process.nonce
- A unique identifier for this operation from the client or process identified bynonceGroup
(the source must ensure each operation gets a unique id).- Returns:
- procedure id
- Throws:
IOException
-
modifyNamespace
long modifyNamespace(NamespaceDescriptor newNsDescriptor, long nonceGroup, long nonce) throws IOException Modify an existing Namespace.- Parameters:
nonceGroup
- Identifier for the source of the request, a client or process.nonce
- A unique identifier for this operation from the client or process identified bynonceGroup
(the source must ensure each operation gets a unique id).- Returns:
- procedure id
- Throws:
IOException
-
deleteNamespace
Delete an existing Namespace. Only empty Namespaces (no tables) can be removed.- Parameters:
nonceGroup
- Identifier for the source of the request, a client or process.nonce
- A unique identifier for this operation from the client or process identified bynonceGroup
(the source must ensure each operation gets a unique id).- Returns:
- procedure id
- Throws:
IOException
-
getNamespace
Get a Namespace- Parameters:
name
- Name of the Namespace- Returns:
- Namespace descriptor for
name
- Throws:
IOException
-
getNamespaces
Get all Namespaces- Returns:
- All Namespace descriptors
- Throws:
IOException
-
listNamespaces
List namespace names- Returns:
- All namespace names
- Throws:
IOException
-
listTableNamesByNamespace
Description copied from interface:MasterServices
Get list of table names by namespace- Specified by:
listTableNamesByNamespace
in interfaceMasterServices
- Parameters:
name
- namespace name- Returns:
- table names
- Throws:
IOException
-
listTableDescriptorsByNamespace
Description copied from interface:MasterServices
Get list of table descriptors by namespace- Specified by:
listTableDescriptorsByNamespace
in interfaceMasterServices
- Parameters:
name
- namespace name- Throws:
IOException
-
abortProcedure
Description copied from interface:MasterServices
Abort a procedure.- Specified by:
abortProcedure
in interfaceMasterServices
- Parameters:
procId
- ID of the proceduremayInterruptIfRunning
- if the proc completed at least one step, should it be aborted?- Returns:
- true if aborted, false if procedure already completed or does not exist
- Throws:
IOException
-
getProcedures
Description copied from interface:MasterServices
Get procedures- Specified by:
getProcedures
in interfaceMasterServices
- Returns:
- procedure list
- Throws:
IOException
-
getLocks
Description copied from interface:MasterServices
Get locks- Specified by:
getLocks
in interfaceMasterServices
- Returns:
- lock list
- Throws:
IOException
-
listTableDescriptors
public List<TableDescriptor> listTableDescriptors(String namespace, String regex, List<TableName> tableNameList, boolean includeSysTables) throws IOException Returns the list of table descriptors that match the specified request- Parameters:
namespace
- the namespace to query, or null if querying for allregex
- The regular expression to match against, or null if querying for alltableNameList
- the list of table names, or null if querying for allincludeSysTables
- False to match only against userspace tables- Returns:
- the list of table descriptors
- Throws:
IOException
-
listTableNames
public List<TableName> listTableNames(String namespace, String regex, boolean includeSysTables) throws IOException Returns the list of table names that match the specified request- Parameters:
regex
- The regular expression to match against, or null if querying for allnamespace
- the namespace to query, or null if querying for allincludeSysTables
- False to match only against userspace tables- Returns:
- the list of table names
- Throws:
IOException
-
getTableDescriptors
private List<TableDescriptor> getTableDescriptors(List<TableDescriptor> htds, String namespace, String regex, List<TableName> tableNameList, boolean includeSysTables) throws IOException Return a list of table table descriptors after applying any provided filter parameters. Note that the user-facing description of this filter logic is presented on the class-level javadoc ofNormalizeTableFilterParams
.- Throws:
IOException
-
filterTablesByRegex
Removes the table descriptors that don't match the pattern.- Parameters:
descriptors
- list of table descriptors to filterpattern
- the regex to use
-
getLastMajorCompactionTimestamp
- Specified by:
getLastMajorCompactionTimestamp
in interfaceMasterServices
- Parameters:
table
- the table for which last successful major compaction time is queried- Returns:
- the timestamp of the last successful major compaction for the passed table, or 0 if no HFile resulting from a major compaction exists
- Throws:
IOException
-
getLastMajorCompactionTimestampForRegion
Description copied from interface:MasterServices
Returns the timestamp of the last successful major compaction for the passed region or 0 if no HFile resulting from a major compaction exists- Specified by:
getLastMajorCompactionTimestampForRegion
in interfaceMasterServices
- Throws:
IOException
-
getMobCompactionState
public org.apache.hadoop.hbase.shaded.protobuf.generated.AdminProtos.GetRegionInfoResponse.CompactionState getMobCompactionState(TableName tableName) Gets the mob file compaction state for a specific table. Whether all the mob files are selected is known during the compaction execution, but the statistic is done just before compaction starts, it is hard to know the compaction type at that time, so the rough statistics are chosen for the mob file compaction. Only two compaction states are available, CompactionState.MAJOR_AND_MINOR and CompactionState.NONE.- Parameters:
tableName
- The current table name.- Returns:
- If a given table is in mob file compaction now.
-
reportMobCompactionStart
- Throws:
IOException
-
reportMobCompactionEnd
- Throws:
IOException
-
isBalancerOn
Queries the state of theLoadBalancerStateStore
. If the balancer is not initialized, false is returned.- Returns:
- The state of the load balancer, or false if the load balancer isn't defined.
-
isNormalizerOn
Queries the state of theRegionNormalizerStateStore
. If it's not initialized, false is returned. -
isSplitOrMergeEnabled
Queries the state of theSplitOrMergeStateStore
. If it is not initialized, false is returned. If switchType is illegal, false will return.- Specified by:
isSplitOrMergeEnabled
in interfaceMasterServices
- Parameters:
switchType
- seeMasterSwitchType
- Returns:
- The state of the switch
-
getLoadBalancerClassName
Fetch the configuredLoadBalancer
class name. If none is set, a default is returned.- Returns:
- The name of the
LoadBalancer
in use.
-
getSplitOrMergeStateStore
-
getLoadBalancer
Description copied from interface:MasterServices
Returns load balancer- Specified by:
getLoadBalancer
in interfaceMasterServices
-
getFavoredNodesManager
Description copied from interface:MasterServices
Returns Favored Nodes Manager- Specified by:
getFavoredNodesManager
in interfaceMasterServices
-
executePeerProcedure
- Throws:
IOException
-
addReplicationPeer
public long addReplicationPeer(String peerId, ReplicationPeerConfig peerConfig, boolean enabled) throws ReplicationException, IOException Description copied from interface:MasterServices
Add a new replication peer for replicating data to slave cluster- Specified by:
addReplicationPeer
in interfaceMasterServices
- Parameters:
peerId
- a short name that identifies the peerpeerConfig
- configuration for the replication slave clusterenabled
- peer state, true if ENABLED and false if DISABLED- Throws:
ReplicationException
IOException
-
removeReplicationPeer
Description copied from interface:MasterServices
Removes a peer and stops the replication- Specified by:
removeReplicationPeer
in interfaceMasterServices
- Parameters:
peerId
- a short name that identifies the peer- Throws:
ReplicationException
IOException
-
enableReplicationPeer
Description copied from interface:MasterServices
Restart the replication stream to the specified peer- Specified by:
enableReplicationPeer
in interfaceMasterServices
- Parameters:
peerId
- a short name that identifies the peer- Throws:
ReplicationException
IOException
-
disableReplicationPeer
Description copied from interface:MasterServices
Stop the replication stream to the specified peer- Specified by:
disableReplicationPeer
in interfaceMasterServices
- Parameters:
peerId
- a short name that identifies the peer- Throws:
ReplicationException
IOException
-
getReplicationPeerConfig
public ReplicationPeerConfig getReplicationPeerConfig(String peerId) throws ReplicationException, IOException Description copied from interface:MasterServices
Returns the configured ReplicationPeerConfig for the specified peer- Specified by:
getReplicationPeerConfig
in interfaceMasterServices
- Parameters:
peerId
- a short name that identifies the peer- Returns:
- ReplicationPeerConfig for the peer
- Throws:
ReplicationException
IOException
-
updateReplicationPeerConfig
public long updateReplicationPeerConfig(String peerId, ReplicationPeerConfig peerConfig) throws ReplicationException, IOException Description copied from interface:MasterServices
Update the peerConfig for the specified peer- Specified by:
updateReplicationPeerConfig
in interfaceMasterServices
- Parameters:
peerId
- a short name that identifies the peerpeerConfig
- new config for the peer- Throws:
ReplicationException
IOException
-
listReplicationPeers
public List<ReplicationPeerDescription> listReplicationPeers(String regex) throws ReplicationException, IOException Description copied from interface:MasterServices
Return a list of replication peers.- Specified by:
listReplicationPeers
in interfaceMasterServices
- Parameters:
regex
- The regular expression to match peer id- Returns:
- a list of replication peers description
- Throws:
ReplicationException
IOException
-
replicationPeerModificationSwitch
- Specified by:
replicationPeerModificationSwitch
in interfaceMasterServices
- Throws:
IOException
-
isReplicationPeerModificationEnabled
- Specified by:
isReplicationPeerModificationEnabled
in interfaceMasterServices
-
decommissionRegionServers
public void decommissionRegionServers(List<ServerName> servers, boolean offload) throws HBaseIOException Mark region server(s) as decommissioned (previously called 'draining') to prevent additional regions from getting assigned to them. Also unload the regions on the servers asynchronously.0- Parameters:
servers
- Region servers to decommission.- Throws:
HBaseIOException
-
listDecommissionedRegionServers
List region servers marked as decommissioned (previously called 'draining') to not get regions assigned to them.- Returns:
- List of decommissioned servers.
-
recommissionRegionServer
public void recommissionRegionServer(ServerName server, List<byte[]> encodedRegionNames) throws IOException Remove decommission marker (previously called 'draining') from a region server to allow regions assignments. Load regions onto the server asynchronously if a list of regions is given- Parameters:
server
- Region server to remove decommission marker from.- Throws:
IOException
-
getLockManager
Description copied from interface:MasterServices
ReturnsLockManager
to lock namespaces/tables/regions.- Specified by:
getLockManager
in interfaceMasterServices
-
getQuotaObserverChore
-
getSpaceQuotaSnapshotNotifier
-
getRemoteProcedure
private RemoteProcedureDispatcher.RemoteProcedure<MasterProcedureEnv,?> getRemoteProcedure(long procId) -
remoteProcedureCompleted
-
remoteProcedureFailed
-
reopenRegions
long reopenRegions(TableName tableName, List<byte[]> regionNames, long nonceGroup, long nonce) throws IOException Reopen regions provided in the argument- Parameters:
tableName
- The current table nameregionNames
- The region names of the regions to reopennonceGroup
- Identifier for the source of the request, a client or processnonce
- A unique identifier for this operation from the client or process identified bynonceGroup
(the source must ensure each operation gets a unique id).- Returns:
- procedure Id
- Throws:
IOException
- if reopening region fails while running procedure
-
getReplicationPeerManager
Description copied from interface:MasterServices
Returns theReplicationPeerManager
.- Specified by:
getReplicationPeerManager
in interfaceMasterServices
-
getReplicationLoad
public HashMap<String,List<Pair<ServerName, getReplicationLoadReplicationLoadSource>>> (ServerName[] serverNames) -
decorateMasterConfiguration
This method modifies the master's configuration in order to inject replication-related features -
getWalGroupsReplicationStatus
- Overrides:
getWalGroupsReplicationStatus
in classHRegionServer
-
getHbckChore
Description copied from interface:MasterServices
Returns Master's instance ofHbckChore
- Specified by:
getHbckChore
in interfaceMasterServices
-
getClusterId
- Overrides:
getClusterId
in classHRegionServer
-
runReplicationBarrierCleaner
Description copied from interface:MasterServices
Run the ReplicationBarrierChore.- Specified by:
runReplicationBarrierCleaner
in interfaceMasterServices
-
getSnapshotQuotaObserverChore
-
getCompactionState
Get the compaction state of the table- Parameters:
tableName
- The table name- Returns:
- CompactionState Compaction state of the table
-
getMetaLocationSyncer
Description copied from interface:MasterServices
Get the meta location syncer. We need to get this in MTP to tell the syncer the new meta replica count.- Specified by:
getMetaLocationSyncer
in interfaceMasterServices
-
flushMasterStore
Description copied from interface:MasterServices
Flush master local region- Specified by:
flushMasterStore
in interfaceMasterServices
- Throws:
IOException
-
getMasterRegion
-
getLiveRegionServers
-
setLoadBalancer
-
setAssignmentManager
-
setDisableBalancerChoreForTest
-
onConfigurationChange
Description copied from interface:ConfigurationObserver
This method would be called by theConfigurationManager
object when theConfiguration
object is reloaded from disk.- Specified by:
onConfigurationChange
in interfaceConfigurationObserver
- Overrides:
onConfigurationChange
in classHRegionServer
-
setQuotasObserver
-
initializeCoprocessorHost
-
flushTable
public long flushTable(TableName tableName, List<byte[]> columnFamilies, long nonceGroup, long nonce) throws IOException Description copied from interface:MasterServices
Flush an existing table- Specified by:
flushTable
in interfaceMasterServices
- Parameters:
tableName
- The table namecolumnFamilies
- The column families to flushnonceGroup
- the nonce groupnonce
- the nonce- Returns:
- the flush procedure id
- Throws:
IOException
-