@InterfaceAudience.LimitedPrivate(value="Tools") public class HMaster extends HRegionServer implements MasterServices
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().Watcher| Modifier and Type | Class and Description | 
|---|---|
| protected static interface  | HMaster.TableDescriptorGetterImplement to return TableDescriptor after pre-checks | 
Thread.State, Thread.UncaughtExceptionHandlerRegionServerServices.PostOpenDeployContext, RegionServerServices.RegionStateTransitionContextclusterConnection, clusterId, clusterStatusTracker, compactSplitThread, conf, configurationManager, executorService, infoServer, REGIONSERVER, rpcServices, serverName, sleeper, startcode, tableDescriptors, TEST_SKIP_REPORTING_TRANSITION, useThisHostnameInstead, zooKeeperMAX_PRIORITY, MIN_PRIORITY, NORM_PRIORITY| Constructor and Description | 
|---|
| HMaster(org.apache.hadoop.conf.Configuration conf)Initializes the HMaster. | 
| Modifier and Type | Method and Description | 
|---|---|
| void | abort(String reason,
     Throwable cause)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 table | 
| long | addReplicationPeer(String peerId,
                  ReplicationPeerConfig peerConfig,
                  boolean enabled)Add a new replication peer for replicating data to slave cluster | 
| boolean | balance() | 
| boolean | balance(boolean force) | 
| boolean | balanceSwitch(boolean b) | 
| private void | balanceThrottling(long nextBalanceStartTime,
                 int maxRegionsInTransition,
                 long cutoffTime)It first sleep to the next balance plan start time. | 
| protected boolean | cacheTableDescriptor() | 
| protected boolean | canCreateBaseZNode() | 
| protected boolean | canUpdateTableDescriptor() | 
| void | checkIfShouldMoveSystemRegionAsync()Called when a new RegionServer is added to the cluster. | 
| (package private) void | checkInitialized() | 
| protected void | checkServiceStarted() | 
| private void | checkTableExists(TableName tableName) | 
| void | checkTableModifiable(TableName tableName)Check table is modifiable; i.e. | 
| protected void | configureInfoServer() | 
| 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) | 
| (package private) long | createNamespace(NamespaceDescriptor namespaceDescriptor,
               long nonceGroup,
               long nonce)Create a new Namespace. | 
| private void | createProcedureExecutor() | 
| private SpaceQuotaSnapshotNotifier | createQuotaSnapshotNotifier() | 
| protected RSRpcServices | createRpcServices() | 
| protected ServerManager | createServerManager(MasterServices master)
 Create a  ServerManagerinstance. | 
| 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 features | 
| long | 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 table | 
| long | disableReplicationPeer(String peerId)Stop the replication stream to the specified peer | 
| long | disableTable(TableName tableName,
            long nonceGroup,
            long nonce)Disable an existing table | 
| long | enableReplicationPeer(String peerId)Restart the replication stream to the specified peer | 
| long | enableTable(TableName tableName,
           long nonceGroup,
           long nonce)Enable an existing table | 
| private long | executePeerProcedure(ModifyPeerProcedure procedure) | 
| List<RegionPlan> | executeRegionPlansWithThrottling(List<RegionPlan> plans)Execute region plans with throttling | 
| private static void | filterTablesByRegex(Collection<TableDescriptor> descriptors,
                   Pattern pattern)Removes the table descriptors that don't match the pattern. | 
| private void | finishActiveMasterInitialization(MonitoredTask status)Finish initialization of HMaster after becoming the primary master. | 
| Optional<ServerName> | getActiveMaster() | 
| AssignmentManager | getAssignmentManager() | 
| double | getAverageLoad()Compute the average load across all region servers. | 
| (package private) List<ServerName> | getBackupMasters() | 
| CatalogJanitor | getCatalogJanitor() | 
| String | getClientIdAuditPrefix() | 
| String | getClusterId() | 
| ClusterMetrics | getClusterMetrics() | 
| ClusterMetrics | getClusterMetrics(EnumSet<ClusterMetrics.Option> options) | 
| ClusterMetrics | getClusterMetricsWithoutCoprocessor() | 
| ClusterMetrics | getClusterMetricsWithoutCoprocessor(EnumSet<ClusterMetrics.Option> options) | 
| ClusterSchema | getClusterSchema() | 
| CompactionState | getCompactionState(TableName tableName)Get the compaction state of the table | 
| protected Class<? extends javax.servlet.http.HttpServlet> | getDumpServlet() | 
| FavoredNodesManager | getFavoredNodesManager() | 
| HbckChore | getHbckChore() | 
| HFileCleaner | getHFileCleaner() | 
| ProcedureEvent<?> | getInitializedEvent() | 
| long | getLastMajorCompactionTimestamp(TableName table) | 
| long | getLastMajorCompactionTimestampForRegion(byte[] regionName) | 
| LoadBalancer | getLoadBalancer() | 
| String | getLoadBalancerClassName()Fetch the configured  LoadBalancerclass name. | 
| static String | getLoadedCoprocessors()The set of loaded coprocessors is stored in a static set. | 
| LockManager | getLockManager() | 
| List<LockedResource> | getLocks()Get locks | 
| LogCleaner | getLogCleaner() | 
| long | getMasterActiveTime() | 
| MasterCoprocessorHost | getMasterCoprocessorHost() | 
| String[] | getMasterCoprocessors() | 
| MasterFileSystem | getMasterFileSystem() | 
| long | getMasterFinishedInitializationTime() | 
| MetricsMaster | getMasterMetrics() | 
| ProcedureExecutor<MasterProcedureEnv> | getMasterProcedureExecutor() | 
| MasterProcedureManagerHost | getMasterProcedureManagerHost() | 
| MasterQuotaManager | getMasterQuotaManager() | 
| MasterRpcServices | getMasterRpcServices() | 
| long | getMasterStartTime() | 
| MasterWalManager | getMasterWalManager() | 
| private int | getMaxBalancingTime() | 
| private int | getMaxRegionsInTransition() | 
| MetaLocationSyncer | getMetaLocationSyncer()Get the meta location syncer. | 
| MetaRegionLocationCache | getMetaRegionLocationCache() | 
| 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> | getNamespaces()Get all Namespaces | 
| int | getNumWALFiles() | 
| List<Procedure<?>> | getProcedures()Get procedures | 
| ProcedureStore | getProcedureStore() | 
| protected String | getProcessName() | 
| QuotaObserverChore | getQuotaObserverChore() | 
| RegionNormalizerManager | getRegionNormalizerManager() | 
| MemoryBoundedLogMessageBuffer | getRegionServerFatalLogBuffer() | 
| int | getRegionServerInfoPort(ServerName sn) | 
| String | getRegionServerVersion(ServerName sn) | 
| (package private) InetAddress | getRemoteInetAddress(int port,
                    long serverStartCode) | 
| private RemoteProcedureDispatcher.RemoteProcedure<MasterProcedureEnv,?> | getRemoteProcedure(long procId) | 
| HashMap<String,List<Pair<ServerName,ReplicationLoadSource>>> | getReplicationLoad(ServerName[] serverNames) | 
| ReplicationPeerConfig | getReplicationPeerConfig(String peerId)Returns the configured ReplicationPeerConfig for the specified peer | 
| ReplicationPeerManager | getReplicationPeerManager()Returns the  ReplicationPeerManager. | 
| ServerManager | getServerManager() | 
| ServerName | getServerName() | 
| SnapshotManager | getSnapshotManager() | 
| SnapshotQuotaObserverChore | getSnapshotQuotaObserverChore() | 
| SpaceQuotaSnapshotNotifier | getSpaceQuotaSnapshotNotifier() | 
| SplitOrMergeTracker | getSplitOrMergeTracker() | 
| SplitWALManager | getSplitWALManager() | 
| private 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. | 
| TableStateManager | getTableStateManager() | 
| protected String | getUseThisHostnameInstead(org.apache.hadoop.conf.Configuration conf) | 
| Map<String,ReplicationStatus> | getWalGroupsReplicationStatus() | 
| ZKWatcher | getZooKeeper()Gets the ZooKeeper instance for this server. | 
| protected void | initClusterSchemaService() | 
| protected void | initializeZKBasedSystemTrackers()Initialize all ZK based system trackers. | 
| private void | initMobCleaner() | 
| private void | initQuotaManager() | 
| boolean | isActiveMaster()Report whether this master is currently the active master or not. | 
| boolean | isBalancerOn()Queries the state of the  LoadBalancerTracker. | 
| (package private) boolean | isCatalogJanitorEnabled() | 
| private static boolean | isCatalogTable(TableName tableName) | 
| (package private) boolean | isCleanerChoreEnabled() | 
| boolean | isInitialized()Report whether this master has completed with its initialization and is
 ready. | 
| boolean | isInMaintenanceMode()Report whether this master is in maintenance mode. | 
| boolean | isNormalizerOn()Queries the state of the  RegionNormalizerTracker. | 
| private boolean | isRegionOnline(RegionInfo ri) | 
| boolean | isSplitOrMergeEnabled(MasterSwitchType switchType)Queries the state of the  SplitOrMergeTracker. | 
| List<ServerName> | listDecommissionedRegionServers()List region servers marked as decommissioned (previously called 'draining') to not get regions
 assigned to them. | 
| List<String> | listNamespaces()List namespace names | 
| List<ReplicationPeerDescription> | listReplicationPeers(String regex)Return a list of replication peers. | 
| List<TableDescriptor> | listTableDescriptors(String namespace,
                    String regex,
                    List<TableName> tableNameList,
                    boolean includeSysTables)Returns the list of table descriptors that match the specified request | 
| List<TableDescriptor> | listTableDescriptorsByNamespace(String name)Get list of table descriptors by namespace | 
| List<TableName> | listTableNames(String namespace,
              String regex,
              boolean includeSysTables)Returns the list of table names that match the specified request | 
| List<TableName> | listTableNamesByNamespace(String name)Get list of table names by namespace | 
| protected void | login(UserProvider user,
     String host)For compatibility, if failed with regionserver credentials, try the master one | 
| static void | main(String[] args) | 
| 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 table | 
| (package private) long | modifyNamespace(NamespaceDescriptor newNsDescriptor,
               long nonceGroup,
               long nonce)Modify an existing Namespace. | 
| private long | modifyTable(TableName tableName,
           HMaster.TableDescriptorGetter newDescriptorGetter,
           long nonceGroup,
           long nonce,
           boolean shouldCheckDescriptor) | 
| long | modifyTable(TableName tableName,
           TableDescriptor newDescriptor,
           long nonceGroup,
           long nonce)Modify the descriptor of an existing table | 
| void | move(byte[] encodedRegionName,
    byte[] destServerName) | 
| boolean | normalizeRegions(NormalizeTableFilterParams ntfp,
                boolean isHighPriority)Perform normalization of cluster. | 
| private int | putUpJettyServer() | 
| void | recommissionRegionServer(ServerName server,
                        List<byte[]> encodedRegionNames)Remove decommission marker (previously called 'draining') from a region server to allow regions
 assignments. | 
| boolean | registerService(com.google.protobuf.Service instance)Registers a new protocol buffer  Servicesubclass as a coprocessor endpoint to be
 available for handling | 
| void | 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 argument | 
| void | reportMobCompactionEnd(TableName tableName) | 
| void | reportMobCompactionStart(TableName tableName) | 
| void | requestMobCompaction(TableName tableName,
                    List<ColumnFamilyDescriptor> columns,
                    boolean allFiles)Requests mob compaction. | 
| long | restoreSnapshot(org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDescription snapshotDesc,
               long nonceGroup,
               long nonce,
               boolean restoreAcl) | 
| void | run()The HRegionServer sticks in this loop until closed. | 
| void | runReplicationBarrierCleaner()Run the ReplicationBarrierChore. | 
| void | setCatalogJanitorEnabled(boolean b)Switch for the background CatalogJanitor thread. | 
| void | setInitialized(boolean isInitialized) | 
| 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) | 
| private void | startProcedureExecutor() | 
| private void | startServiceThreads() | 
| void | stop(String msg)Stop this service. | 
| private void | stopChores() | 
| void | stopMaster() | 
| private void | stopProcedureExecutor() | 
| protected void | stopServiceThreads()Wait on all threads to finish. | 
| private void | switchSnapshotCleanup(boolean on) | 
| (package private) void | switchSnapshotCleanup(boolean on,
                     boolean synchronous)Turn on/off Snapshot Cleanup Chore | 
| long | truncateTable(TableName tableName,
             boolean preserveSplits,
             long nonceGroup,
             long nonce)Truncate a table | 
| void | updateConfigurationForQuotasObserver(org.apache.hadoop.conf.Configuration conf)Adds the  MasterQuotasObserverto 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 peer | 
| protected void | waitForMasterActive()If configured to put regions on active master,
 wait till a backup master becomes active. | 
| boolean | waitForMetaOnline()Check hbase:meta is up and ready for reading. | 
| boolean | waitForNamespaceOnline()Check hbase:namespace table is assigned. | 
| private void | waitForRegionServers(MonitoredTask status) | 
abort, addRegion, closeRegion, createClusterConnection, createConnection, createRegionLoad, createRegionServerStatusStub, finishRegionProcedure, getAccessChecker, getBlockCache, getChoreService, getClusterConnection, getCompactedHFilesDischarger, getCompactionPressure, getCompactionRequestor, getCompactSplitThread, getConfiguration, getConnection, getCoordinatedStateManager, getDataRootDir, getEventLoopGroupConfig, getExecutorService, getFavoredNodesForRegion, getFileSystem, getFlushPressure, getFlushRequester, getFlushThroughputController, getHeapMemoryManager, getInfoServer, getLastSequenceId, getLeaseManager, getMasterAddressTracker, getMetrics, getMobFileCache, getMovedRegion, getNamedQueueRecorder, getNonceManager, getNumberOfOnlineRegions, getOnlineRegion, getOnlineRegions, getOnlineRegionsLocalContext, getOnlineTables, getRegion, getRegion, getRegionByEncodedName, getRegions, getRegions, getRegionServerAccounting, getRegionServerCoprocessorHost, getRegionServerCoprocessors, getRegionServerRpcQuotaManager, getRegionServerSpaceQuotaManager, getRegionsInTransitionInRS, getReplicationSourceService, getRpcServer, getRSRpcServices, getSecureBulkLoadManager, getStartcode, getTableDescriptors, getWAL, getWALFileSystem, getWalRoller, getWALRootDir, getWALs, getZKPermissionWatcher, handleReportForDutyResponse, initializeMemStoreChunkCreator, isAborted, isClusterUp, isOnline, isShutDown, isStopped, isStopping, kill, movedRegionCacheExpiredTime, onConfigurationChange, postOpenDeployTasks, regionLock, remoteProcedureComplete, removeRegion, reportFileArchivalForQuotas, reportRegionSizesForQuotas, reportRegionStateTransition, setAbortRequested, setupClusterConnection, stop, toString, tryRegionServerReport, unassign, updateRegionFavoredNodesMapping, waitForServerOnline, walRollRequestFinishedactiveCount, 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, yieldequals, finalize, getClass, hashCode, notify, notifyAll, wait, wait, waitgetAccessChecker, getExecutorService, getTableDescriptors, getZKPermissionWatcher, isClusterUpcreateConnection, getChoreService, getClusterConnection, getConfiguration, getConnection, getCoordinatedStateManager, getFileSystem, isStoppingprivate static final org.slf4j.Logger LOG
public static final String MASTER
private final ActiveMasterManager activeMasterManager
private RegionServerTracker regionServerTracker
private DrainingServerTracker drainingServerTracker
LoadBalancerTracker loadBalancerTracker
private MetaLocationSyncer metaLocationSyncer
@InterfaceAudience.Private MasterAddressSyncer masterAddressSyncer
SnapshotCleanupTracker snapshotCleanupTracker
private SplitOrMergeTracker splitOrMergeTracker
private ClusterSchemaService clusterSchemaService
public static final String HBASE_MASTER_WAIT_ON_SERVICE_IN_SECONDS
public static final int DEFAULT_HBASE_MASTER_WAIT_ON_SERVICE_IN_SECONDS
public static final String HBASE_MASTER_CLEANER_INTERVAL
public static final int DEFAULT_HBASE_MASTER_CLEANER_INTERVAL
final MetricsMaster metricsMaster
private MasterFileSystem fileSystemManager
private MasterWalManager walManager
private SplitWALManager splitWALManager
private volatile ServerManager serverManager
private AssignmentManager assignmentManager
private final MetaRegionLocationCache metaRegionLocationCache
private ReplicationPeerManager replicationPeerManager
MemoryBoundedLogMessageBuffer rsFatals
private volatile boolean activeMaster
private final ProcedureEvent<?> initialized
volatile boolean serviceStarted
private final int maxBalancingTime
private final double maxRitPercent
private final LockManager lockManager
private LoadBalancer balancer
private BalancerChore balancerChore
private RegionNormalizerManager regionNormalizerManager
private ClusterStatusChore clusterStatusChore
private ClusterStatusPublisher clusterStatusPublisherChore
private SnapshotCleanerChore snapshotCleanerChore
CatalogJanitor catalogJanitorChore
private DirScanPool cleanerPool
private LogCleaner logCleaner
private HFileCleaner hfileCleaner
private ReplicationBarrierCleaner replicationBarrierCleaner
private ExpiredMobFileCleanerChore expiredMobFileCleanerChore
private MobCompactionChore mobCompactChore
private MasterMobCompactionThread mobCompactThread
private final IdLock mobCompactionLock
private Map<TableName,AtomicInteger> mobCompactionStates
MasterCoprocessorHost cpHost
private final boolean preLoadTableDescriptors
private long masterActiveTime
private long masterFinishedInitializationTime
Map<String,com.google.protobuf.Service> coprocessorServiceHandlers
SnapshotManager snapshotManager
private MasterProcedureManagerHost mpmHost
private RegionsRecoveryChore regionsRecoveryChore
private RegionsRecoveryConfigManager regionsRecoveryConfigManager
private volatile MasterQuotaManager quotaManager
private SpaceQuotaSnapshotNotifier spaceQuotaSnapshotNotifier
private QuotaObserverChore quotaObserverChore
private SnapshotQuotaObserverChore snapshotQuotaChore
private ProcedureExecutor<MasterProcedureEnv> procedureExecutor
private ProcedureStore procedureStore
private MasterRegion masterRegion
private TableStateManager tableStateManager
private FavoredNodesManager favoredNodesManager
private org.apache.hbase.thirdparty.org.eclipse.jetty.server.Server masterJettyServer
private final boolean maintenanceMode
static final String MAINTENANCE_MODE
private final CachedClusterId cachedClusterId
public HMaster(org.apache.hadoop.conf.Configuration conf) throws IOException
 Remaining steps of initialization occur in
 finishActiveMasterInitialization(MonitoredTask) after the master becomes the
 active one.
IOExceptionprotected ActiveMasterManager createActiveMasterManager(ZKWatcher zk, ServerName sn, Server server) throws InterruptedIOException
InterruptedIOExceptionprotected String getUseThisHostnameInstead(org.apache.hadoop.conf.Configuration conf)
getUseThisHostnameInstead in class HRegionServerpublic void run()
HRegionServerrun in interface Runnablerun in class HRegionServerprivate int putUpJettyServer() throws IOException
IOExceptionprotected void login(UserProvider user, String host) throws IOException
login in class HRegionServerIOExceptionprotected void waitForMasterActive()
waitForMasterActive in class HRegionServer@InterfaceAudience.Private public MasterRpcServices getMasterRpcServices()
public boolean balanceSwitch(boolean b) throws IOException
IOExceptionprotected String getProcessName()
getProcessName in class HRegionServerprotected boolean canCreateBaseZNode()
canCreateBaseZNode in class HRegionServerprotected boolean canUpdateTableDescriptor()
canUpdateTableDescriptor in class HRegionServerprotected boolean cacheTableDescriptor()
cacheTableDescriptor in class HRegionServerprotected RSRpcServices createRpcServices() throws IOException
createRpcServices in class HRegionServerIOExceptionprotected void configureInfoServer()
configureInfoServer in class HRegionServerprotected Class<? extends javax.servlet.http.HttpServlet> getDumpServlet()
getDumpServlet in class HRegionServerpublic MetricsMaster getMasterMetrics()
getMasterMetrics in interface MasterServicesMetricsMaster@InterfaceAudience.Private protected void initializeZKBasedSystemTrackers() throws IOException, InterruptedException, org.apache.zookeeper.KeeperException, ReplicationException
RegionServerTracker, it
 should have already been initialized along with ServerManager.IOExceptionInterruptedExceptionorg.apache.zookeeper.KeeperExceptionReplicationException@InterfaceAudience.Private protected AssignmentManager createAssignmentManager(MasterServices master)
private void finishActiveMasterInitialization(MonitoredTask status) throws IOException, InterruptedException, org.apache.zookeeper.KeeperException, ReplicationException
IOExceptionInterruptedExceptionorg.apache.zookeeper.KeeperExceptionReplicationException@InterfaceAudience.Private public boolean waitForMetaOnline()
private boolean isRegionOnline(RegionInfo ri)
@InterfaceAudience.Private public boolean waitForNamespaceOnline()
@InterfaceAudience.Private public void updateConfigurationForQuotasObserver(org.apache.hadoop.conf.Configuration conf)
MasterQuotasObserver to the list of configured Master observers to
 automatically remove quotas for a table when that table is deleted.private void initMobCleaner()
@InterfaceAudience.Private protected ServerManager createServerManager(MasterServices master) throws IOException
 Create a ServerManager instance.
 
Will be overridden in tests.
IOExceptionprivate void waitForRegionServers(MonitoredTask status) throws IOException, InterruptedException
IOExceptionInterruptedException@InterfaceAudience.Private protected void initClusterSchemaService() throws IOException, InterruptedException
IOExceptionInterruptedExceptionprivate void initQuotaManager() throws IOException
IOExceptionprivate SpaceQuotaSnapshotNotifier createQuotaSnapshotNotifier()
boolean isCatalogJanitorEnabled()
boolean isCleanerChoreEnabled()
public ServerManager getServerManager()
getServerManager in interface MasterServicesServerManager instance.public MasterFileSystem getMasterFileSystem()
getMasterFileSystem in interface MasterServicesMasterFileSystem utility class.public MasterWalManager getMasterWalManager()
getMasterWalManager in interface MasterServicesMasterWalManager utility class.public SplitWALManager getSplitWALManager()
getSplitWALManager in interface MasterServicespublic TableStateManager getTableStateManager()
getTableStateManager in interface MasterServicesTableStateManagerprivate void startServiceThreads() throws IOException
IOExceptionprotected void stopServiceThreads()
HRegionServerstopServiceThreads in class HRegionServerprivate void createProcedureExecutor() throws IOException
IOExceptionprivate void startProcedureExecutor() throws IOException
IOExceptionvoid switchSnapshotCleanup(boolean on, boolean synchronous)
on - indicates whether Snapshot Cleanup Chore is to be runprivate void switchSnapshotCleanup(boolean on)
private void stopProcedureExecutor()
private void stopChores()
InetAddress getRemoteInetAddress(int port, long serverStartCode) throws UnknownHostException
UnknownHostExceptionprivate int getMaxBalancingTime()
private int getMaxRegionsInTransition()
private void balanceThrottling(long nextBalanceStartTime, int maxRegionsInTransition, long cutoffTime)
nextBalanceStartTime - The next balance plan start timemaxRegionsInTransition - max number of regions in transitioncutoffTime - when to exit balancerpublic boolean balance() throws IOException
IOExceptionpublic boolean skipRegionManagementAction(String action)
skipRegionManagementAction in interface MasterServicesaction - the name of the action under consideration, for logging.true when the caller should exit early, false otherwise.public boolean balance(boolean force) throws IOException
IOExceptionpublic List<RegionPlan> executeRegionPlansWithThrottling(List<RegionPlan> plans)
executeRegionPlansWithThrottling in interface MasterServicesplans - to executepublic RegionNormalizerManager getRegionNormalizerManager()
getRegionNormalizerManager in interface MasterServicesRegionNormalizerManagerpublic boolean normalizeRegions(NormalizeTableFilterParams ntfp, boolean isHighPriority) throws IOException
MasterServicesnormalizeRegions in interface MasterServicesntfp - Selection criteria for identifying which tables to normalize.isHighPriority - true when these requested tables should skip to the front of
   the queue.true when the request was submitted, false otherwise.IOExceptionpublic String getClientIdAuditPrefix()
getClientIdAuditPrefix in interface MasterServicespublic void setCatalogJanitorEnabled(boolean b)
b - If false, the catalog janitor won't do anything.public long mergeRegions(RegionInfo[] regionsToMerge, boolean forcible, long ng, long nonce) throws IOException
MasterServicesmergeRegions in interface MasterServicesregionsToMerge - daughter regions to mergeforcible - whether to force to merge even two regions are not adjacentng - used to detect duplicatenonce - used to detect duplicateIOExceptionpublic long splitRegion(RegionInfo regionInfo, byte[] splitRow, long nonceGroup, long nonce) throws IOException
MasterServicessplitRegion in interface MasterServicesregionInfo - region to splitsplitRow - split pointnonceGroup - used to detect duplicatenonce - used to detect duplicateIOException@InterfaceAudience.Private public void move(byte[] encodedRegionName, byte[] destServerName) throws HBaseIOException
HBaseIOExceptionpublic long createTable(TableDescriptor tableDescriptor, byte[][] splitKeys, long nonceGroup, long nonce) throws IOException
MasterServicescreateTable in interface MasterServicestableDescriptor - The table definitionsplitKeys - Starting row keys for the initial table regions.  If nullnonce - a single region is created.IOExceptionpublic long createSystemTable(TableDescriptor tableDescriptor) throws IOException
MasterServicescreateSystemTable in interface MasterServicestableDescriptor - The system table definition
     a single region is created.IOExceptionprivate void startActiveMasterManager(int infoPort) throws org.apache.zookeeper.KeeperException
org.apache.zookeeper.KeeperExceptionprivate static boolean isCatalogTable(TableName tableName)
public long deleteTable(TableName tableName, long nonceGroup, long nonce) throws IOException
MasterServicesdeleteTable in interface MasterServicestableName - The table nameIOExceptionpublic long truncateTable(TableName tableName, boolean preserveSplits, long nonceGroup, long nonce) throws IOException
MasterServicestruncateTable in interface MasterServicestableName - The table namepreserveSplits - True if the splits should be preservedIOExceptionpublic long addColumn(TableName tableName, ColumnFamilyDescriptor column, long nonceGroup, long nonce) throws IOException
MasterServicesaddColumn in interface MasterServicestableName - The table namecolumn - The column definitionIOExceptionpublic long modifyColumn(TableName tableName, ColumnFamilyDescriptor descriptor, long nonceGroup, long nonce) throws IOException
MasterServicesmodifyColumn in interface MasterServicestableName - The table namedescriptor - The updated column definitionIOExceptionpublic long deleteColumn(TableName tableName, byte[] columnName, long nonceGroup, long nonce) throws IOException
MasterServicesdeleteColumn in interface MasterServicestableName - The table namecolumnName - The column nameIOExceptionpublic long enableTable(TableName tableName, long nonceGroup, long nonce) throws IOException
MasterServicesenableTable in interface MasterServicestableName - The table nameIOExceptionpublic long disableTable(TableName tableName, long nonceGroup, long nonce) throws IOException
MasterServicesdisableTable in interface MasterServicestableName - The table nameIOExceptionprivate long modifyTable(TableName tableName, HMaster.TableDescriptorGetter newDescriptorGetter, long nonceGroup, long nonce, boolean shouldCheckDescriptor) throws IOException
IOExceptionpublic long modifyTable(TableName tableName, TableDescriptor newDescriptor, long nonceGroup, long nonce) throws IOException
MasterServicesmodifyTable in interface MasterServicestableName - The table namenewDescriptor - The updated table descriptorIOExceptionpublic long restoreSnapshot(org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDescription snapshotDesc, long nonceGroup, long nonce, boolean restoreAcl) throws IOException
IOExceptionprivate void checkTableExists(TableName tableName) throws IOException, TableNotFoundException
IOExceptionTableNotFoundExceptionpublic void checkTableModifiable(TableName tableName) throws IOException, TableNotFoundException, TableNotDisabledException
MasterServicescheckTableModifiable in interface MasterServicestableName - Name of table to check.TableNotDisabledExceptionTableNotFoundExceptionIOExceptionpublic ClusterMetrics getClusterMetricsWithoutCoprocessor() throws InterruptedIOException
InterruptedIOExceptionpublic ClusterMetrics getClusterMetricsWithoutCoprocessor(EnumSet<ClusterMetrics.Option> options) throws InterruptedIOException
InterruptedIOExceptionpublic ClusterMetrics getClusterMetrics() throws IOException
IOExceptionpublic ClusterMetrics getClusterMetrics(EnumSet<ClusterMetrics.Option> options) throws IOException
IOExceptionList<ServerName> getBackupMasters()
public static String getLoadedCoprocessors()
public long getMasterStartTime()
public long getMasterActiveTime()
public long getMasterFinishedInitializationTime()
public int getNumWALFiles()
public ProcedureStore getProcedureStore()
public int getRegionServerInfoPort(ServerName sn)
public String getRegionServerVersion(ServerName sn)
getRegionServerVersion in interface MasterServicespublic void checkIfShouldMoveSystemRegionAsync()
MasterServicescheckIfShouldMoveSystemRegionAsync in interface MasterServicespublic String[] getMasterCoprocessors()
public void abort(String reason, Throwable cause)
HRegionServerabort in interface Abortableabort in class HRegionServerreason - the reason we are abortingcause - the exception that caused the abort, or nullpublic ZKWatcher getZooKeeper()
ServergetZooKeeper in interface ServergetZooKeeper in class HRegionServerpublic MasterCoprocessorHost getMasterCoprocessorHost()
getMasterCoprocessorHost in interface MasterServicesMasterCoprocessorHostpublic MasterQuotaManager getMasterQuotaManager()
getMasterQuotaManager in interface MasterServicesMasterQuotaManagerpublic ProcedureExecutor<MasterProcedureEnv> getMasterProcedureExecutor()
getMasterProcedureExecutor in interface MasterServicesProcedureExecutorpublic ServerName getServerName()
getServerName in interface ServergetServerName in class HRegionServerpublic AssignmentManager getAssignmentManager()
getAssignmentManager in interface MasterServicesAssignmentManagerpublic CatalogJanitor getCatalogJanitor()
getCatalogJanitor in interface MasterServicesCatalogJanitorpublic MemoryBoundedLogMessageBuffer getRegionServerFatalLogBuffer()
public void shutdown() throws IOException
IOExceptionpublic void stopMaster() throws IOException
IOExceptionpublic void stop(String msg)
Stoppablestop in interface Stoppablestop in class HRegionServermsg - Why we're stopping.@InterfaceAudience.Private protected void checkServiceStarted() throws ServerNotRunningYetException
ServerNotRunningYetExceptionvoid checkInitialized() throws PleaseHoldException, ServerNotRunningYetException, MasterNotRunningException, MasterStoppedException
public boolean isActiveMaster()
isActiveMaster in interface MasterServicespublic boolean isInitialized()
isInitialized in interface MasterServicespublic boolean isInMaintenanceMode()
isInMaintenanceMode in interface MasterServices@InterfaceAudience.Private public void setInitialized(boolean isInitialized)
public ProcedureEvent<?> getInitializedEvent()
getInitializedEvent in interface MasterServicespublic double getAverageLoad()
public boolean registerService(com.google.protobuf.Service instance)
RegionServerServicesService subclass as a coprocessor endpoint to be
 available for handlingregisterService in interface MasterServicesregisterService in interface RegionServerServicesregisterService in class HRegionServerinstance - the Service subclass instance to expose as a coprocessor endpointtrue if the registration was successful, falsepublic static HMaster constructMaster(Class<? extends HMaster> masterClass, org.apache.hadoop.conf.Configuration conf)
masterClass - public static void main(String[] args)
HMasterCommandLinepublic HFileCleaner getHFileCleaner()
public LogCleaner getLogCleaner()
public SnapshotManager getSnapshotManager()
getSnapshotManager in interface MasterServicespublic MasterProcedureManagerHost getMasterProcedureManagerHost()
getMasterProcedureManagerHost in interface MasterServicespublic ClusterSchema getClusterSchema()
getClusterSchema in interface MasterServicesClusterSchemalong createNamespace(NamespaceDescriptor namespaceDescriptor, long nonceGroup, long nonce) throws IOException
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 by
 nonceGroup (the source must ensure each operation gets a unique id).IOExceptionlong modifyNamespace(NamespaceDescriptor newNsDescriptor, long nonceGroup, long nonce) throws IOException
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 by
 nonceGroup (the source must ensure each operation gets a unique id).IOExceptionlong deleteNamespace(String name, long nonceGroup, long nonce) throws IOException
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 by
 nonceGroup (the source must ensure each operation gets a unique id).IOExceptionNamespaceDescriptor getNamespace(String name) throws IOException
name - Name of the NamespacenameIOExceptionList<NamespaceDescriptor> getNamespaces() throws IOException
IOExceptionpublic List<String> listNamespaces() throws IOException
IOExceptionpublic List<TableName> listTableNamesByNamespace(String name) throws IOException
MasterServiceslistTableNamesByNamespace in interface MasterServicesname - namespace nameIOExceptionpublic List<TableDescriptor> listTableDescriptorsByNamespace(String name) throws IOException
MasterServiceslistTableDescriptorsByNamespace in interface MasterServicesname - namespace nameIOExceptionpublic boolean abortProcedure(long procId, boolean mayInterruptIfRunning) throws IOException
MasterServicesabortProcedure in interface MasterServicesprocId - ID of the proceduremayInterruptIfRunning - if the proc completed at least one step, should it be aborted?IOExceptionpublic List<Procedure<?>> getProcedures() throws IOException
MasterServicesgetProcedures in interface MasterServicesIOExceptionpublic List<LockedResource> getLocks() throws IOException
MasterServicesgetLocks in interface MasterServicesIOExceptionpublic List<TableDescriptor> listTableDescriptors(String namespace, String regex, List<TableName> tableNameList, boolean includeSysTables) throws IOException
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 tablesIOExceptionpublic List<TableName> listTableNames(String namespace, String regex, boolean includeSysTables) throws IOException
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 tablesIOExceptionprivate List<TableDescriptor> getTableDescriptors(List<TableDescriptor> htds, String namespace, String regex, List<TableName> tableNameList, boolean includeSysTables) throws IOException
NormalizeTableFilterParams.IOExceptionprivate static void filterTablesByRegex(Collection<TableDescriptor> descriptors, Pattern pattern)
descriptors - list of table descriptors to filterpattern - the regex to usepublic long getLastMajorCompactionTimestamp(TableName table) throws IOException
getLastMajorCompactionTimestamp in interface MasterServicestable - the table for which last successful major compaction time is queriedIOExceptionpublic long getLastMajorCompactionTimestampForRegion(byte[] regionName) throws IOException
getLastMajorCompactionTimestampForRegion in interface MasterServicesIOExceptionpublic org.apache.hadoop.hbase.shaded.protobuf.generated.AdminProtos.GetRegionInfoResponse.CompactionState getMobCompactionState(TableName tableName)
tableName - The current table name.public void reportMobCompactionStart(TableName tableName) throws IOException
IOExceptionpublic void reportMobCompactionEnd(TableName tableName) throws IOException
IOExceptionpublic void requestMobCompaction(TableName tableName, List<ColumnFamilyDescriptor> columns, boolean allFiles) throws IOException
tableName - The table the compact.columns - The compacted columns.allFiles - Whether add all mob files into the compaction.IOExceptionpublic boolean isBalancerOn()
LoadBalancerTracker. If the balancer is not initialized,
 false is returned.public boolean isNormalizerOn()
RegionNormalizerTracker. If it's not initialized,
 false is returned.public boolean isSplitOrMergeEnabled(MasterSwitchType switchType)
SplitOrMergeTracker. If it is not initialized,
 false is returned. If switchType is illegal, false will return.isSplitOrMergeEnabled in interface MasterServicesswitchType - see MasterSwitchTypepublic String getLoadBalancerClassName()
LoadBalancer class name. If none is set, a default is returned.LoadBalancer in use.public SplitOrMergeTracker getSplitOrMergeTracker()
public LoadBalancer getLoadBalancer()
getLoadBalancer in interface MasterServicespublic FavoredNodesManager getFavoredNodesManager()
getFavoredNodesManager in interface MasterServicesprivate long executePeerProcedure(ModifyPeerProcedure procedure) throws IOException
IOExceptionpublic long addReplicationPeer(String peerId, ReplicationPeerConfig peerConfig, boolean enabled) throws ReplicationException, IOException
MasterServicesaddReplicationPeer in interface MasterServicespeerId - a short name that identifies the peerpeerConfig - configuration for the replication slave clusterenabled - peer state, true if ENABLED and false if DISABLEDReplicationExceptionIOExceptionpublic long removeReplicationPeer(String peerId) throws ReplicationException, IOException
MasterServicesremoveReplicationPeer in interface MasterServicespeerId - a short name that identifies the peerReplicationExceptionIOExceptionpublic long enableReplicationPeer(String peerId) throws ReplicationException, IOException
MasterServicesenableReplicationPeer in interface MasterServicespeerId - a short name that identifies the peerReplicationExceptionIOExceptionpublic long disableReplicationPeer(String peerId) throws ReplicationException, IOException
MasterServicesdisableReplicationPeer in interface MasterServicespeerId - a short name that identifies the peerReplicationExceptionIOExceptionpublic ReplicationPeerConfig getReplicationPeerConfig(String peerId) throws ReplicationException, IOException
MasterServicesgetReplicationPeerConfig in interface MasterServicespeerId - a short name that identifies the peerReplicationExceptionIOExceptionpublic long updateReplicationPeerConfig(String peerId, ReplicationPeerConfig peerConfig) throws ReplicationException, IOException
MasterServicesupdateReplicationPeerConfig in interface MasterServicespeerId - a short name that identifies the peerpeerConfig - new config for the peerReplicationExceptionIOExceptionpublic List<ReplicationPeerDescription> listReplicationPeers(String regex) throws ReplicationException, IOException
MasterServiceslistReplicationPeers in interface MasterServicesregex - The regular expression to match peer idReplicationExceptionIOExceptionpublic void decommissionRegionServers(List<ServerName> servers, boolean offload) throws HBaseIOException
servers - Region servers to decommission.HBaseIOExceptionpublic List<ServerName> listDecommissionedRegionServers()
public void recommissionRegionServer(ServerName server, List<byte[]> encodedRegionNames) throws IOException
server - Region server to remove decommission marker from.IOExceptionpublic LockManager getLockManager()
getLockManager in interface MasterServicesLockManager to lock namespaces/tables/regions.public QuotaObserverChore getQuotaObserverChore()
public SpaceQuotaSnapshotNotifier getSpaceQuotaSnapshotNotifier()
private RemoteProcedureDispatcher.RemoteProcedure<MasterProcedureEnv,?> getRemoteProcedure(long procId)
public void remoteProcedureCompleted(long procId)
public void remoteProcedureFailed(long procId, RemoteProcedureException error)
long reopenRegions(TableName tableName, List<byte[]> regionNames, long nonceGroup, long nonce) throws IOException
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 by
   nonceGroup (the source must ensure each operation gets a unique id).IOException - if reopening region fails while running procedurepublic ReplicationPeerManager getReplicationPeerManager()
MasterServicesReplicationPeerManager.getReplicationPeerManager in interface MasterServicespublic HashMap<String,List<Pair<ServerName,ReplicationLoadSource>>> getReplicationLoad(ServerName[] serverNames)
@InterfaceAudience.Private public static void decorateMasterConfiguration(org.apache.hadoop.conf.Configuration conf)
public Map<String,ReplicationStatus> getWalGroupsReplicationStatus()
getWalGroupsReplicationStatus in class HRegionServerpublic HbckChore getHbckChore()
public Optional<ServerName> getActiveMaster()
public void runReplicationBarrierCleaner()
MasterServicesrunReplicationBarrierCleaner in interface MasterServicespublic SnapshotQuotaObserverChore getSnapshotQuotaObserverChore()
public String getClusterId()
getClusterId in class HRegionServerpublic MetaRegionLocationCache getMetaRegionLocationCache()
public CompactionState getCompactionState(TableName tableName)
tableName - The table namepublic MetaLocationSyncer getMetaLocationSyncer()
MasterServicesgetMetaLocationSyncer in interface MasterServicesCopyright © 2007–2021 The Apache Software Foundation. All rights reserved.