Class HRegionServer
java.lang.Object
java.lang.Thread
org.apache.hadoop.hbase.regionserver.HRegionServer
- All Implemented Interfaces:
Runnable
,Abortable
,ConfigurationObserver
,FavoredNodesForRegion
,LastSequenceId
,MutableOnlineRegions
,OnlineRegions
,RegionServerServices
,Server
,Stoppable
- Direct Known Subclasses:
HMaster
@LimitedPrivate("Tools")
public class HRegionServer
extends Thread
implements RegionServerServices, LastSequenceId, ConfigurationObserver
HRegionServer makes a set of HRegions available to clients. It checks in with the HMaster. There
are many HRegionServers in a single HBase deployment.
-
Nested Class Summary
Modifier and TypeClassDescriptionprivate static class
Inner class that runs on a long period checking if regions need compaction.private static class
private static class
private static class
Force to terminate region server when abort timeout.Nested 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 TypeFieldDescription(package private) static final String
(package private) static final String
private Timer
private AtomicBoolean
private BlockCache
private BootstrapNodeManager
private BrokenStoreFileCleaner
private MemStoreFlusher
private ChoreService
ChoreService used to schedule tasks that we want to run periodicallyprotected ClusterConnection
Cluster connection to be shared by services.protected String
Unique identifier for the cluster we are a part of.protected final ClusterStatusTracker
(package private) CompactedHFilesDischarger
private ScheduledChore
Check for compactions requests.private final int
private CompactSplit
protected final org.apache.hadoop.conf.Configuration
protected final ConfigurationManager
Configuration manager is used to register/deregister and notify the configuration observers when the regionserver is notified that there was a change in the on disk configs.private CoordinatedStateManager
private HFileSystem
private boolean
private org.apache.hadoop.fs.Path
private static final long
private final NettyEventLoopGroupConfig
Used to cache the open/close region procedures which already executed.protected ExecutorService
private final int
private ThroughputController
private FileSystemUtilizationChore
private HealthCheckChore
The health check chore.private HeapMemoryManager
protected InfoServer
private boolean
private boolean
private LeaseManager
private org.apache.hadoop.hbase.shaded.protobuf.generated.LockServiceProtos.LockService.BlockingInterface
private static final org.slf4j.Logger
private final MasterAddressTracker
private final boolean
True if this RegionServer is coming up in a cluster where there is no Master; means it needs to just come up and make do without a Master to talk to: e.g.private static final String
private final MetaRegionLocationCache
Cache for the meta region replica's locations.private MetricsRegionServer
(package private) MetricsRegionServerWrapperImpl
private MobFileCache
private final org.apache.hbase.thirdparty.com.google.common.cache.Cache<String,
HRegionServer.MovedRegionInfo> Used to cache the moved-out regions(package private) final int
private NamedQueueRecorder
Provide online slow log responses from ringbufferprivate NamedQueueServiceChore
(package private) final ServerNonceManager
Nonce manager.private ScheduledChore
The nonce manager chore.(package private) final AtomicBoolean
Map of regions currently being served by this region server.private final ReentrantReadWriteLock
Lock for gating access toonlineRegions
.private final int
private JvmPauseMonitor
private static final String
private static final String
private ScheduledChore
Check for flushesprivate PrefetchExecutorNotifier
private RemoteProcedureResultReporter
Map of encoded region names to the DataNode locations they should be hosted on We store the value as Address since InetSocketAddress is required by the HDFS API (create() that takes favored nodes as hints for placing file blocks).static final String
region server process nameprivate static final String
regionserver codec listprivate final RegionServerAccounting
private final ConcurrentMap<byte[],
Boolean> A map from RegionName to current action in progress.private ReplicationMarkerChore
private ReplicationSinkService
private ReplicationSourceService
private final long
private RpcClient
private RpcControllerFactory
private RpcRetryingCallerFactory
protected final RSRpcServices
(package private) static final String
Deprecated.since 2.4.0 and will be removed in 4.0.0.private RegionServerCoprocessorHost
private RSMobFileCleanerChore
private RegionServerProcedureManagerHost
private RegionServerRpcQuotaManager
private RSSnapshotVerifier
private RegionServerSpaceQuotaManager
private org.apache.hadoop.hbase.shaded.protobuf.generated.RegionServerStatusProtos.RegionServerStatusService.BlockingInterface
private SecureBulkLoadManager
protected ServerName
The server name the Master sees us as.private final int
private boolean
protected final Sleeper
private SplitLogWorker
protected final long
This servers startcode.private boolean
private boolean
private StorefileRefresherChore
private final ConcurrentMap<Long,
Long> Used to cache the open/close region procedures which already submitted.protected TableDescriptors
Go here to get table descriptors.static boolean
For testing only! Set to true to skip notifying region assignment to master .private final int
private static final int
We need a timeout.private Thread.UncaughtExceptionHandler
(package private) int
(package private) int
(package private) static final String
HBASE-18226: This config and hbase.unsafe.regionserver.hostname are mutually exclusive.private UserProvider
protected String
hostname specified by hostname configprivate WALFactory
private HFileSystem
private LogRoller
private org.apache.hadoop.fs.Path
protected final ZKWatcher
Fields inherited from class java.lang.Thread
MAX_PRIORITY, MIN_PRIORITY, NORM_PRIORITY
-
Constructor Summary
ConstructorDescriptionHRegionServer
(org.apache.hadoop.conf.Configuration conf) Starts a HRegionServer at the default location. -
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.void
Add to online regions.private void
addToMovedRegions
(String encodedName, ServerName destination, long closeSeqNum, boolean selfMove) private boolean
private void
blockAndCheckIfStopped
(ZKNodeTracker tracker) Utilty method to wait indefinitely on a znode availability while checking if the region server is shut down(package private) org.apache.hadoop.hbase.shaded.protobuf.generated.RegionServerStatusProtos.RegionSpaceUseReportRequest
buildRegionSpaceUseReportRequest
(RegionSizeStore regionSizes) Builds aRegionServerStatusProtos.RegionSpaceUseReportRequest
protobuf message from the region size map.private void
buildReportAndSend
(org.apache.hadoop.hbase.shaded.protobuf.generated.RegionServerStatusProtos.RegionServerStatusService.BlockingInterface rss, RegionSizeStore regionSizeStore) Builds the region size report and sends it to the master.private org.apache.hadoop.hbase.shaded.protobuf.generated.ClusterStatusProtos.ServerLoad
buildServerLoad
(long reportStartTime, long reportEndTime) protected boolean
protected boolean
protected boolean
private static void
checkCodecs
(org.apache.hadoop.conf.Configuration c) Run test on configured codecs to make sure supporting libs are in place.(package private) boolean
Checks to see if the file system is still accessible.private Throwable
Cleanup after Throwable caught invoking method.(package private) CacheEvictionStats
clearRegionBlockCache
(Region region) private void
closeMetaTableRegions
(boolean abort) Close meta region if we carry itprotected boolean
closeRegion
(String encodedName, boolean abort, ServerName destination) Close asynchronously a region, can be called from the master or internally by the regionserver when stopping.private void
closeRegionIgnoreErrors
(RegionInfo region, boolean abort) Try to close the region, logs a warning on failure but continues.private void
closeUserRegions
(boolean abort) Schedule closes on all user regions.private void
computeIfPersistentBucketCache
(Consumer<BucketCache> computation) protected void
(package private) static HRegionServer
constructRegionServer
(Class<? extends HRegionServer> regionServerClass, org.apache.hadoop.conf.Configuration conf) Utility for constructing an instance of the passed HRegionServer class.private boolean
(package private) org.apache.hadoop.hbase.shaded.protobuf.generated.RegionServerStatusProtos.RegionSpaceUse
convertRegionSize
(RegionInfo regionInfo, Long sizeInBytes) Converts a pair ofRegionInfo
andlong
into aRegionServerStatusProtos.RegionSpaceUse
protobuf message.private IOException
convertThrowableToIOE
(Throwable t, String msg) protected ClusterConnection
createConnection
(org.apache.hadoop.conf.Configuration conf) private void
private static void
createNewReplicationInstance
(org.apache.hadoop.conf.Configuration conf, HRegionServer server, org.apache.hadoop.fs.FileSystem walFs, org.apache.hadoop.fs.Path walDir, org.apache.hadoop.fs.Path oldWALDir, WALFactory walFactory) Load the replication executorService objects, if anyorg.apache.hadoop.hbase.shaded.protobuf.generated.ClusterStatusProtos.RegionLoad
createRegionLoad
(String encodedRegionName) (package private) org.apache.hadoop.hbase.shaded.protobuf.generated.ClusterStatusProtos.RegionLoad
createRegionLoad
(HRegion r, org.apache.hadoop.hbase.shaded.protobuf.generated.ClusterStatusProtos.RegionLoad.Builder regionLoadBldr, org.apache.hadoop.hbase.shaded.protobuf.generated.HBaseProtos.RegionSpecifier.Builder regionSpecifier) private ServerName
Get the current master from ZooKeeper and open the RPC connection to it.protected ServerName
createRegionServerStatusStub
(boolean refresh) Get the current master from ZooKeeper and open the RPC connection to it.private org.apache.hadoop.hbase.shaded.protobuf.generated.RegionServerStatusProtos.ReportRegionStateTransitionRequest
protected RSRpcServices
private org.apache.hadoop.hbase.shaded.protobuf.generated.ClusterStatusProtos.UserLoad
createUserLoad
(String user, MetricsUserSource userSource) private void
void
dumpRowLocks
(PrintWriter out) Used byRSDumpServlet
to generate debugging information.(package private) org.apache.hadoop.hbase.shaded.protobuf.generated.ClientProtos.CoprocessorServiceResponse
execRegionServerService
(org.apache.hbase.thirdparty.com.google.protobuf.RpcController controller, org.apache.hadoop.hbase.shaded.protobuf.generated.ClientProtos.CoprocessorServiceRequest serviceRequest) (package private) void
executeProcedure
(long procId, RSProcedureCallable callable) void
finishRegionProcedure
(long procId) Returns theAccessChecker
May be null if this is a master which not carry table.Returns TheChoreService
instance for this serverReturns a reference to the servers' cluster connection.double
Returns the underlyingCompactSplit
for the serversorg.apache.hadoop.conf.Configuration
Gets the configuration object for this server.Returns : Returns the ConfigurationManager object for testing purposes.Returns a reference to the servers' connection.Get CoordinatedStateManager instance for this server.protected org.apache.hadoop.fs.Path
Returns Return the rootDir.protected Class<? extends javax.servlet.http.HttpServlet>
Returns hbase executor servicegetFavoredNodesForRegion
(String encodedRegionName) Return the favored nodes for a region given its encoded name.org.apache.hadoop.fs.FileSystem
Returns Return the FileSystem object used (can return null!).double
Returns reference to FlushRequesterReturns the controller to avoid flush too fastReturns heap memory manager instanceReturns the info serverorg.apache.hadoop.hbase.shaded.protobuf.generated.ClusterStatusProtos.RegionStoreSequenceIds
getLastSequenceId
(byte[] encodedRegionName) Returns The RegionServer's "Leases" serviceReturns Master address tracker instance.Returns the metrics tracker for the region serverMay be null if this is a master which not carry table.getMovedRegion
(String encodedRegionName) private String
get NamedQueue Provider to add different logs to ringbufferOnly required for "old" log replay; if it's removed, remove this.int
getOnlineRegion
(byte[] regionName) private String
For tests, web ui and metrics.Gets the online tables in this RS.protected String
protected HRegion
getRegion
(byte[] regionName) Protected Utility method for safely obtaining an HRegion handle.ReturnRegion
instance.private HRegion
getRegionByEncodedName
(byte[] regionName, String encodedRegionName) getRegionByEncodedName
(String encodedRegionName) Get all online regions in this RS.getRegions
(TableName tableName) Gets the online regions of the specified table.Returns the RegionServerAccounting for this Region ServerString[]
Returns RegionServer's instance ofRegionServerRpcQuotaManager
Returns RegionServer's instance ofRegionServerSpaceQuotaManager
ConcurrentMap<byte[],
Boolean> Get the regions that are currently being opened or closed in the RS(package private) ReplicationSinkService
Returns Return the object that implements the replication sink executorService.Returns Return the object that implements the replication source executorService.long
Return pause time configured inHConstants.HBASE_RPC_SHORTOPERATION_RETRY_PAUSE_TIME
}Returns a reference to the region server's RPC server(package private) RSSnapshotVerifier
Returns RegionServer's instance ofSecureBulkLoadManager
Returns The unique server name for this server.long
Returns time stamp in millis of when this region server was startedReturns Return table descriptors implementation.protected String
getUseThisHostnameInstead
(org.apache.hadoop.conf.Configuration conf) getWAL
(RegionInfo regionInfo) Returns the WAL for a particular region.private WALActionsListener
getWALEventTrackerListener
(org.apache.hadoop.conf.Configuration conf) (package private) WALFactory
org.apache.hadoop.fs.FileSystem
Returns Return the walFs.org.apache.hadoop.fs.Path
Returns Return the walRootDir.getWALs()
Returns the List of WALs that are used by this server Doesn't include the meta WALprivate long
Returns Current write count for all online regions.ReturnsZKPermissionWatcher
Gets the ZooKeeper instance for this server.protected void
handleReportForDutyResponse
(org.apache.hadoop.hbase.shaded.protobuf.generated.RegionServerStatusProtos.RegionServerStartupResponse c) private void
protected void
private void
private void
private void
Bring up connection to zk ensemble and then wait until a master for this cluster and then after that, wait until cluster 'up' flag has been set.private void
This method is called when HMaster and HRegionServer are started.boolean
Check if the server or client was aborted.boolean
Returns True if the cluster is up.(package private) boolean
Returnstrue
when the data file system is available,false
otherwise.private boolean
private boolean
boolean
isOnline()
Report the status of the server.boolean
boolean
This method is used for testing.boolean
Returns true if a stop has been requested.boolean
Returns True is the server is Stoppingprivate boolean
protected void
kill()
protected void
login
(UserProvider user, String host) static void
int
private static <T extends ReplicationService>
TnewReplicationInstance
(String classname, Class<T> xface, org.apache.hadoop.conf.Configuration conf, HRegionServer server, org.apache.hadoop.fs.FileSystem walFs, org.apache.hadoop.fs.Path logDir, org.apache.hadoop.fs.Path oldLogDir, WALFactory walFactory) void
onConfigurationChange
(org.apache.hadoop.conf.Configuration newConf) This method would be called by theConfigurationManager
object when theConfiguration
object is reloaded from disk.void
Tasks to perform after region open to complete deploy of region on regionserverprotected void
private void
All initialization needed before we go register with Master.
Do bare minimum.protected void
private void
Puts up the webui.regionLock
(List<RegionInfo> regionInfos, String description, Abortable abort) Master based locks on namespaces/tables/regions.private void
boolean
registerService
(com.google.protobuf.Service instance) Registers a new protocol bufferService
subclass as a coprocessor endpoint to be available for handlingvoid
remoteProcedureComplete
(long procId, Throwable error) (package private) void
removeFromMovedRegions
(String encodedName) boolean
removeRegion
(HRegion r, ServerName destination) Removes the given Region from the list of onlineRegions.boolean
reportFileArchivalForQuotas
(TableName tableName, Collection<Map.Entry<String, Long>> archivedFiles) Reports a collection of files, and their sizes, that belonged to the giventable
were just moved to the archive directory.private org.apache.hadoop.hbase.shaded.protobuf.generated.RegionServerStatusProtos.RegionServerStartupResponse
(package private) void
reportProcedureDone
(org.apache.hadoop.hbase.shaded.protobuf.generated.RegionServerStatusProtos.ReportProcedureDoneRequest request) boolean
reportRegionSizesForQuotas
(RegionSizeStore regionSizeStore) Reports the given map of Regions and their size on the filesystem to the active Master.boolean
Notify master that a handler requests to change a region stateprivate static int
roundSize
(long sizeInByte, int sizeUnit) void
run()
The HRegionServer sticks in this loop until closed.private void
protected boolean
Sets the abort state if not already set.protected void
Setup our cluster connection if not already initialized.private static NettyEventLoopGroupConfig
setupNetty
(org.apache.hadoop.conf.Configuration conf) private void
private void
Setup WAL log and replication if enabled.protected final void
shutdownChore
(ScheduledChore chore) private void
shutdownWAL
(boolean close) private boolean
Helper method for use in tests.private static boolean
sleepInterrupted
(long millis) private void
private void
Start up replication source and sink handlers.private void
Start maintenance Threads, Server, Worker and lease checker threads.void
Stop this service.void
Stops the regionserver.protected void
Wait on all threads to finish.(package private) boolean
submitRegionProcedure
(long procId) Will ignore the open/close region procedures which already submitted or executed.toString()
private void
triggerFlushInPrimaryRegion
(HRegion region) Trigger a flush in the primary region replica if this region is a secondary replica.protected void
tryRegionServerReport
(long reportStartTime, long reportEndTime) void
unassign
(byte[] regionName) Unassign the given region from the current regionserver and assign it randomly.(package private) void
Reload the configuration from disk.void
updateRegionFavoredNodesMapping
(String encodedRegionName, List<org.apache.hadoop.hbase.shaded.protobuf.generated.HBaseProtos.ServerName> favoredNodes) Used to update the favored nodes mapping when required.protected void
Wait for an active Master.void
private void
waitOnAllRegionsToClose
(boolean abort) Wait on regions close.boolean
For testingMethods 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
-
Field Details
-
LOG
-
unitMB
int unitMB -
unitKB
int unitKB -
TEST_SKIP_REPORTING_TRANSITION
For testing only! Set to true to skip notifying region assignment to master . -
regionsInTransitionInRS
A map from RegionName to current action in progress. Boolean value indicates: true - if open region action in progress false - if close region action in progress -
submittedRegionProcedures
Used to cache the open/close region procedures which already submitted. SeesubmitRegionProcedure(long)
. -
executedRegionProcedures
private final org.apache.hbase.thirdparty.com.google.common.cache.Cache<Long,Long> executedRegionProceduresUsed to cache the open/close region procedures which already executed. SeesubmitRegionProcedure(long)
. -
movedRegionInfoCache
private final org.apache.hbase.thirdparty.com.google.common.cache.Cache<String,HRegionServer.MovedRegionInfo> movedRegionInfoCacheUsed to cache the moved-out regions -
cacheFlusher
-
hMemManager
-
clusterConnection
Cluster connection to be shared by services. Initialized at server startup and closed when server shuts down. Clients must never close it explicitly. Clients hosted by this Server should make use of this clusterConnection rather than create their own; if they create their own, there is no way for the hosting server to shutdown ongoing client RPCs. -
tableDescriptors
Go here to get table descriptors. -
replicationSourceHandler
-
replicationSinkHandler
-
compactSplitThread
-
onlineRegions
Map of regions currently being served by this region server. Key is the encoded region name. All access should be synchronized. -
onlineRegionsLock
Lock for gating access toonlineRegions
. TODO: If this map is gated by a lock, does it need to be a ConcurrentHashMap? -
regionFavoredNodesMap
Map of encoded region names to the DataNode locations they should be hosted on We store the value as Address since InetSocketAddress is required by the HDFS API (create() that takes favored nodes as hints for placing file blocks). We could have used ServerName here as the value class, but we'd need to convert it to InetSocketAddress at some point before the HDFS API call, and it seems a bit weird to store ServerName since ServerName refers to RegionServers and here we really mean DataNode locations. We don't store it as InetSocketAddress here because the conversion on demand from Address to InetSocketAddress will guarantee the resolution results will be fresh when we need it. -
leaseManager
-
executorService
-
dataFsOk
-
dataFs
-
walFs
-
stopped
-
isShutdownHookInstalled
-
abortRequested
-
ABORT_TIMEOUT
- See Also:
-
DEFAULT_ABORT_TIMEOUT
- See Also:
-
ABORT_TIMEOUT_TASK
- See Also:
-
stopping
-
killed
-
shutDown
-
conf
-
dataRootDir
-
walRootDir
-
threadWakeFrequency
-
msgInterval
-
PERIOD_COMPACTION
- See Also:
-
compactionCheckFrequency
-
PERIOD_FLUSH
- See Also:
-
flushCheckFrequency
-
rssStub
private volatile org.apache.hadoop.hbase.shaded.protobuf.generated.RegionServerStatusProtos.RegionServerStatusService.BlockingInterface rssStub -
lockStub
private volatile org.apache.hadoop.hbase.shaded.protobuf.generated.LockServiceProtos.LockService.BlockingInterface lockStub -
rpcClient
-
rpcRetryingCallerFactory
-
rpcControllerFactory
-
uncaughtExceptionHandler
-
infoServer
-
pauseMonitor
-
rsSnapshotVerifier
-
REGIONSERVER
region server process name- See Also:
-
metricsRegionServer
-
metricsRegionServerImpl
-
choreService
ChoreService used to schedule tasks that we want to run periodically -
compactionChecker
Check for compactions requests. -
periodicFlusher
Check for flushes -
walFactory
-
walRoller
-
procedureResultReporter
-
online
-
zooKeeper
-
masterAddressTracker
-
metaRegionLocationCache
Cache for the meta region replica's locations. Also tracks their changes to avoid stale cache entries. Used for serving ClientMetaService. -
clusterStatusTracker
-
splitLogWorker
-
sleeper
-
operationTimeout
-
shortOperationTimeout
-
retryPauseTime
-
regionServerAccounting
-
namedQueueServiceChore
-
blockCache
-
mobFileCache
-
healthCheckChore
The health check chore. -
nonceManagerChore
The nonce manager chore. -
coprocessorServiceHandlers
-
serverName
The server name the Master sees us as. Its made from the hostname the master passes us, port, and server startcode. Gets set after registration against Master. -
useThisHostnameInstead
hostname specified by hostname config -
RS_HOSTNAME_DISABLE_MASTER_REVERSEDNS_KEY
@Deprecated @LimitedPrivate("Configuration") static final String RS_HOSTNAME_DISABLE_MASTER_REVERSEDNS_KEYDeprecated.since 2.4.0 and will be removed in 4.0.0. UseUNSAFE_RS_HOSTNAME_DISABLE_MASTER_REVERSEDNS_KEY
instead.- See Also:
-
UNSAFE_RS_HOSTNAME_DISABLE_MASTER_REVERSEDNS_KEY
@LimitedPrivate("Configuration") static final String UNSAFE_RS_HOSTNAME_DISABLE_MASTER_REVERSEDNS_KEYHBASE-18226: This config and hbase.unsafe.regionserver.hostname are mutually exclusive. Exception will be thrown if both are used.- See Also:
-
startcode
This servers startcode. -
clusterId
Unique identifier for the cluster we are a part of. -
storefileRefresher
-
rsHost
-
rspmHost
-
rsQuotaManager
-
rsSpaceQuotaManager
-
nonceManager
Nonce manager. Nonces are used to make operations like increment and append idempotent in the case where client doesn't receive the response from a successful operation and retries. We track the successful ops for some time via a nonce sent by client and handle duplicate operations (currently, by failing them; in future we might use MVCC to return result). Nonces are also recovered from WAL during, recovery; however, the caveats (from HBASE-3787) are: - WAL recovery is optimized, and under high load we won't read nearly nonce-timeout worth of past records. If we don't read the records, we don't read and recover the nonces. Some WALs within nonce-timeout at recovery may not even be present due to rolling/cleanup. - There's no WAL recovery during normal region move, so nonces will not be transfered. We can have separate additional "Nonce WAL". It will just contain bunch of numbers and won't be flushed on main path - because WAL itself also contains nonces, if we only flush it before memstore flush, for a given nonce we will either see it in the WAL (if it was never flushed to disk, it will be part of recovery), or we'll see it as part of the nonce log (or both occasionally, which doesn't matter). Nonce log file can be deleted after the latest nonce in it expired. It can also be recovered during move. -
userProvider
-
rpcServices
-
csm
-
configurationManager
Configuration manager is used to register/deregister and notify the configuration observers when the regionserver is notified that there was a change in the on disk configs. -
brokenStoreFileCleaner
-
rsMobFileCleanerChore
-
compactedFileDischarger
-
flushThroughputController
-
secureBulkLoadManager
-
fsUtilizationChore
-
eventLoopGroupConfig
-
namedQueueRecorder
Provide online slow log responses from ringbuffer -
bootstrapNodeManager
-
masterless
True if this RegionServer is coming up in a cluster where there is no Master; means it needs to just come up and make do without a Master to talk to: e.g. in test or HRegionServer is doing other than its usual duties: e.g. as an hollowed-out host whose only purpose is as a Replication-stream sink; see HBASE-18846 for more. TODO: can this replaceTEST_SKIP_REPORTING_TRANSITION
? -
MASTERLESS_CONFIG_NAME
- See Also:
-
REGIONSERVER_CODEC
regionserver codec list- See Also:
-
abortMonitor
-
replicationMarkerChore
-
prefetchExecutorNotifier
-
TIMEOUT_REGION_MOVED
We need a timeout. If not there is a risk of giving a wrong information: this would double the number of network calls instead of reducing them.- See Also:
-
-
Constructor Details
-
HRegionServer
Starts a HRegionServer at the default location. Don't start any services or managers in here in the Constructor. Defer till after we register with the Master as much as possible. SeestartServices()
.- Throws:
IOException
-
-
Method Details
-
getUseThisHostnameInstead
protected String getUseThisHostnameInstead(org.apache.hadoop.conf.Configuration conf) throws IOException - Throws:
IOException
-
setupSignalHandlers
-
setupNetty
-
initializeFileSystem
- Throws:
IOException
-
login
- Throws:
IOException
-
waitForMasterActive
Wait for an active Master. See override in Master superclass for how it is used. -
getProcessName
-
canCreateBaseZNode
-
canUpdateTableDescriptor
-
cacheTableDescriptor
-
createRpcServices
- Throws:
IOException
-
configureInfoServer
-
getDumpServlet
-
dumpRowLocks
Used byRSDumpServlet
to generate debugging information. -
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 interfaceRegionServerServices
- Parameters:
instance
- theService
subclass instance to expose as a coprocessor endpoint- Returns:
true
if the registration was successful,false
-
createClusterConnection
- Throws:
IOException
-
checkCodecs
Run test on configured codecs to make sure supporting libs are in place.- Parameters:
c
- configuration object- Throws:
IOException
- if compression test fails for any regionserver codec
-
getClusterId
-
setupClusterConnection
Setup our cluster connection if not already initialized.- Throws:
IOException
-
preRegistrationInitialization
All initialization needed before we go register with Master.
Do bare minimum. Do bulk of initializations AFTER we've connected to the Master.
In here we just put up the RpcServer, setup Connection, and ZooKeeper. -
initializeZooKeeper
Bring up connection to zk ensemble and then wait until a master for this cluster and then after that, wait until cluster 'up' flag has been set. This is the order in which master does things.Finally open long-living server short-circuit connection.
- Throws:
IOException
InterruptedException
-
blockAndCheckIfStopped
Utilty method to wait indefinitely on a znode availability while checking if the region server is shut down- Parameters:
tracker
- znode tracker to use- Throws:
IOException
- any IO exception, plus if the RS is stoppedInterruptedException
- if the waiting thread is interrupted
-
isClusterUp
Returns True if the cluster is up.- Specified by:
isClusterUp
in interfaceRegionServerServices
-
initializeReplicationMarkerChore
-
run
The HRegionServer sticks in this loop until closed. -
installShutdownHook
This method is called when HMaster and HRegionServer are started. Please see to HBASE-26977 for details. -
isShutdownHookInstalled
This method is used for testing. -
containsMetaTableRegions
-
areAllUserRegionsOffline
-
getWriteRequestCount
Returns Current write count for all online regions. -
tryRegionServerReport
@Private protected void tryRegionServerReport(long reportStartTime, long reportEndTime) throws IOException - Throws:
IOException
-
reportRegionSizesForQuotas
Reports the given map of Regions and their size on the filesystem to the active Master.- Specified by:
reportRegionSizesForQuotas
in interfaceRegionServerServices
- Parameters:
regionSizeStore
- The store containing region sizes- Returns:
- false if FileSystemUtilizationChore should pause reporting to master. true otherwise
-
buildReportAndSend
private void buildReportAndSend(org.apache.hadoop.hbase.shaded.protobuf.generated.RegionServerStatusProtos.RegionServerStatusService.BlockingInterface rss, RegionSizeStore regionSizeStore) throws org.apache.hbase.thirdparty.com.google.protobuf.ServiceException Builds the region size report and sends it to the master. Upon successful sending of the report, the region sizes that were sent are marked as sent.- Parameters:
rss
- The stub to send to the MasterregionSizeStore
- The store containing region sizes- Throws:
org.apache.hbase.thirdparty.com.google.protobuf.ServiceException
-
buildRegionSpaceUseReportRequest
org.apache.hadoop.hbase.shaded.protobuf.generated.RegionServerStatusProtos.RegionSpaceUseReportRequest buildRegionSpaceUseReportRequest(RegionSizeStore regionSizes) Builds aRegionServerStatusProtos.RegionSpaceUseReportRequest
protobuf message from the region size map.- Parameters:
regionSizes
- The size in bytes of regions- Returns:
- The corresponding protocol buffer message.
-
convertRegionSize
org.apache.hadoop.hbase.shaded.protobuf.generated.RegionServerStatusProtos.RegionSpaceUse convertRegionSize(RegionInfo regionInfo, Long sizeInBytes) Converts a pair ofRegionInfo
andlong
into aRegionServerStatusProtos.RegionSpaceUse
protobuf message.- Parameters:
regionInfo
- The RegionInfosizeInBytes
- The size in bytes of the Region- Returns:
- The protocol buffer
-
buildServerLoad
private org.apache.hadoop.hbase.shaded.protobuf.generated.ClusterStatusProtos.ServerLoad buildServerLoad(long reportStartTime, long reportEndTime) throws IOException - Throws:
IOException
-
getOnlineRegionsAsPrintableString
-
waitOnAllRegionsToClose
Wait on regions close. -
sleepInterrupted
-
shutdownWAL
-
getNamedQueueRecorder
get NamedQueue Provider to add different logs to ringbuffer -
handleReportForDutyResponse
protected void handleReportForDutyResponse(org.apache.hadoop.hbase.shaded.protobuf.generated.RegionServerStatusProtos.RegionServerStartupResponse c) throws IOException - Throws:
IOException
-
initializeMemStoreChunkCreator
-
startHeapMemoryManager
-
createMyEphemeralNode
- Throws:
org.apache.zookeeper.KeeperException
-
deleteMyEphemeralNode
- Throws:
org.apache.zookeeper.KeeperException
-
getRegionServerAccounting
Description copied from interface:RegionServerServices
Returns the RegionServerAccounting for this Region Server- Specified by:
getRegionServerAccounting
in interfaceRegionServerServices
-
roundSize
-
computeIfPersistentBucketCache
-
createRegionLoad
org.apache.hadoop.hbase.shaded.protobuf.generated.ClusterStatusProtos.RegionLoad createRegionLoad(HRegion r, org.apache.hadoop.hbase.shaded.protobuf.generated.ClusterStatusProtos.RegionLoad.Builder regionLoadBldr, org.apache.hadoop.hbase.shaded.protobuf.generated.HBaseProtos.RegionSpecifier.Builder regionSpecifier) throws IOException - Parameters:
r
- Region to get RegionLoad for.regionLoadBldr
- the RegionLoad.Builder, can be nullregionSpecifier
- the RegionSpecifier.Builder, can be null- Returns:
- RegionLoad instance.
- Throws:
IOException
-
createUserLoad
private org.apache.hadoop.hbase.shaded.protobuf.generated.ClusterStatusProtos.UserLoad createUserLoad(String user, MetricsUserSource userSource) -
createRegionLoad
public org.apache.hadoop.hbase.shaded.protobuf.generated.ClusterStatusProtos.RegionLoad createRegionLoad(String encodedRegionName) throws IOException - Throws:
IOException
-
isOnline
Report the status of the server. A server is online once all the startup is completed (setting up filesystem, starting executorService threads, etc.). This method is designed mostly to be useful in tests.- Returns:
- true if online, false if not.
-
setupWALAndReplication
Setup WAL log and replication if enabled. Replication setup is done in here because it wants to be hooked up to WAL.- Throws:
IOException
-
getWALEventTrackerListener
-
startReplicationService
Start up replication source and sink handlers.- Throws:
IOException
-
getMasterAddressTracker
Returns Master address tracker instance. -
startServices
Start maintenance Threads, Server, Worker and lease checker threads. Start all threads we need to run. This is called after we've successfully registered with the Master. Install an UncaughtExceptionHandler that calls abort of RegionServer if we get an unhandled exception. We cannot set the handler on all threads. Server's internal Listener thread is off limits. For Server, if an OOME, it waits a while then retries. Meantime, a flush or a compaction that tries to run should trigger same critical condition and the shutdown will run. On its way out, this server will shut down Server. Leases are sort of inbetween. It has an internal thread that while it inherits from Chore, it keeps its own internal stop mechanism so needs to be stopped by this hosting server. Worker logs the exception and exits.- Throws:
IOException
-
initializeThreads
-
registerConfigurationObservers
-
putUpWebUI
Puts up the webui.- Throws:
IOException
-
isHealthy
-
getWALs
Description copied from interface:RegionServerServices
Returns the List of WALs that are used by this server Doesn't include the meta WAL- Specified by:
getWALs
in interfaceRegionServerServices
-
getWAL
Description copied from interface:RegionServerServices
Returns the WAL for a particular region. Pass null for getting the default (common) WAL- Specified by:
getWAL
in interfaceRegionServerServices
- Throws:
IOException
-
getWalRoller
-
getWalFactory
-
getConnection
Description copied from interface:Server
Returns a reference to the servers' connection. Important note: this method returns a reference to Connection which is managed by Server itself, so callers must NOT attempt to close connection obtained.- Specified by:
getConnection
in interfaceServer
-
getClusterConnection
Description copied from interface:Server
Returns a reference to the servers' cluster connection. PreferServer.getConnection()
. Important note: this method returns a reference to Connection which is managed by Server itself, so callers must NOT attempt to close connection obtained.- Specified by:
getClusterConnection
in interfaceServer
-
stop
Description copied from interface:Stoppable
Stop this service. Implementers should favor logging errors over throwing RuntimeExceptions. -
stop
Stops the regionserver.- Parameters:
msg
- Status messageforce
- True if this is a regionserver abortuser
- The user executing the stop request, or null if no user is associated
-
waitForServerOnline
-
postOpenDeployTasks
public void postOpenDeployTasks(RegionServerServices.PostOpenDeployContext context) throws IOException Description copied from interface:RegionServerServices
Tasks to perform after region open to complete deploy of region on regionserver- Specified by:
postOpenDeployTasks
in interfaceRegionServerServices
- Parameters:
context
- the context- Throws:
IOException
-
skipReportingTransition
Helper method for use in tests. Skip the region transition report when there's no master around to receive it. -
createReportRegionStateTransitionRequest
private org.apache.hadoop.hbase.shaded.protobuf.generated.RegionServerStatusProtos.ReportRegionStateTransitionRequest createReportRegionStateTransitionRequest(RegionServerServices.RegionStateTransitionContext context) -
reportRegionStateTransition
public boolean reportRegionStateTransition(RegionServerServices.RegionStateTransitionContext context) Description copied from interface:RegionServerServices
Notify master that a handler requests to change a region state- Specified by:
reportRegionStateTransition
in interfaceRegionServerServices
-
triggerFlushInPrimaryRegion
Trigger a flush in the primary region replica if this region is a secondary replica. Does not block this thread. See RegionReplicaFlushHandler for details. -
getRpcServer
Description copied from interface:RegionServerServices
Returns a reference to the region server's RPC server- Specified by:
getRpcServer
in interfaceRegionServerServices
-
getRSRpcServices
-
abort
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 -
setAbortRequested
Sets the abort state if not already set.- Returns:
- True if abortRequested set to True successfully, false if an abort is already in progress.
-
isAborted
Description copied from interface:Abortable
Check if the server or client was aborted. -
kill
-
scheduleAbortTimer
-
shutdownChore
-
stopServiceThreads
Wait on all threads to finish. Presumption is that all closes and stops have already been called. -
getReplicationSourceService
Returns Return the object that implements the replication source executorService. -
getReplicationSinkService
Returns Return the object that implements the replication sink executorService. -
createRegionServerStatusStub
Get the current master from ZooKeeper and open the RPC connection to it. To get a fresh connection, the current rssStub must be null. Method will block until a master is available. You can break from this block by requesting the server stop.- Returns:
- master + port, or null if server has been stopped
-
createRegionServerStatusStub
Get the current master from ZooKeeper and open the RPC connection to it. To get a fresh connection, the current rssStub must be null. Method will block until a master is available. You can break from this block by requesting the server stop.- Parameters:
refresh
- If true then master address will be read from ZK, otherwise use cached data- Returns:
- master + port, or null if server has been stopped
-
keepLooping
- Returns:
- True if we should break loop because cluster is going down or this server has been stopped or hdfs has gone bad.
-
reportForDuty
private org.apache.hadoop.hbase.shaded.protobuf.generated.RegionServerStatusProtos.RegionServerStartupResponse reportForDuty() throws IOException- Throws:
IOException
-
getLastSequenceId
public org.apache.hadoop.hbase.shaded.protobuf.generated.ClusterStatusProtos.RegionStoreSequenceIds getLastSequenceId(byte[] encodedRegionName) - Specified by:
getLastSequenceId
in interfaceLastSequenceId
- Parameters:
encodedRegionName
- Encoded region name- Returns:
- Last flushed sequence Id for region and its stores. Id will be -1 if it can't be determined
-
closeMetaTableRegions
Close meta region if we carry it- Parameters:
abort
- Whether we're running an abort.
-
closeUserRegions
Schedule closes on all user regions. Should be safe calling multiple times because it wont' close regions that are already closed or that are closing.- Parameters:
abort
- Whether we're running an abort.
-
getInfoServer
Returns the info server -
isStopped
Returns true if a stop has been requested. -
isStopping
Description copied from interface:Server
Returns True is the server is Stopping- Specified by:
isStopping
in interfaceServer
-
getConfiguration
Description copied from interface:Server
Gets the configuration object for this server.- Specified by:
getConfiguration
in interfaceServer
-
getOnlineRegions
-
getNumberOfOnlineRegions
-
getOnlineRegionsLocalContext
For tests, web ui and metrics. This method will only work if HRegionServer is in the same JVM as client; HRegion cannot be serialized to cross an rpc. -
addRegion
Description copied from interface:MutableOnlineRegions
Add to online regions.- Specified by:
addRegion
in interfaceMutableOnlineRegions
-
getCopyOfOnlineRegionsSortedByOffHeapSize
- Returns:
- A new Map of online regions sorted by region off-heap size with the first entry being the biggest. If two regions are the same size, then the last one found wins; i.e. this method may NOT return all regions.
-
getCopyOfOnlineRegionsSortedByOnHeapSize
- Returns:
- A new Map of online regions sorted by region heap size with the first entry being the biggest. If two regions are the same size, then the last one found wins; i.e. this method may NOT return all regions.
-
getStartcode
Returns time stamp in millis of when this region server was started -
getFlushRequester
Returns reference to FlushRequester- Specified by:
getFlushRequester
in interfaceRegionServerServices
- Returns:
- Implementation of
FlushRequester
or null. Usually it will not be null unless during intialization.
-
getCompactionRequestor
- Specified by:
getCompactionRequestor
in interfaceRegionServerServices
- Returns:
- Implementation of
CompactionRequester
or null. Usually it will not be null unless during intialization.
-
getLeaseManager
Description copied from interface:RegionServerServices
Returns The RegionServer's "Leases" service- Specified by:
getLeaseManager
in interfaceRegionServerServices
-
getDataRootDir
Returns Return the rootDir. -
getFileSystem
Description copied from interface:Server
Returns Return the FileSystem object used (can return null!).- Specified by:
getFileSystem
in interfaceServer
-
isDataFileSystemOk
boolean isDataFileSystemOk()Returnstrue
when the data file system is available,false
otherwise. -
getWALRootDir
Returns Return the walRootDir. -
getWALFileSystem
Returns Return the walFs. -
toString
-
getZooKeeper
Description copied from interface:Server
Gets the ZooKeeper instance for this server.- Specified by:
getZooKeeper
in interfaceServer
-
getCoordinatedStateManager
Description copied from interface:Server
Get CoordinatedStateManager instance for this server.- Specified by:
getCoordinatedStateManager
in interfaceServer
-
getServerName
Description copied from interface:Server
Returns The unique server name for this server.- Specified by:
getServerName
in interfaceServer
-
getRegionServerCoprocessorHost
-
getRegionsInTransitionInRS
Description copied from interface:RegionServerServices
Get the regions that are currently being opened or closed in the RS- Specified by:
getRegionsInTransitionInRS
in interfaceRegionServerServices
- Returns:
- map of regions in transition in this RS
-
getExecutorService
Description copied from interface:RegionServerServices
Returns hbase executor service- Specified by:
getExecutorService
in interfaceRegionServerServices
-
getChoreService
Description copied from interface:Server
Returns TheChoreService
instance for this server- Specified by:
getChoreService
in interfaceServer
-
getRegionServerRpcQuotaManager
Description copied from interface:RegionServerServices
Returns RegionServer's instance ofRegionServerRpcQuotaManager
- Specified by:
getRegionServerRpcQuotaManager
in interfaceRegionServerServices
-
createNewReplicationInstance
private static void createNewReplicationInstance(org.apache.hadoop.conf.Configuration conf, HRegionServer server, org.apache.hadoop.fs.FileSystem walFs, org.apache.hadoop.fs.Path walDir, org.apache.hadoop.fs.Path oldWALDir, WALFactory walFactory) throws IOException Load the replication executorService objects, if any- Throws:
IOException
-
newReplicationInstance
private static <T extends ReplicationService> T newReplicationInstance(String classname, Class<T> xface, org.apache.hadoop.conf.Configuration conf, HRegionServer server, org.apache.hadoop.fs.FileSystem walFs, org.apache.hadoop.fs.Path logDir, org.apache.hadoop.fs.Path oldLogDir, WALFactory walFactory) throws IOException - Throws:
IOException
-
getWalGroupsReplicationStatus
-
constructRegionServer
static HRegionServer constructRegionServer(Class<? extends HRegionServer> regionServerClass, org.apache.hadoop.conf.Configuration conf) Utility for constructing an instance of the passed HRegionServer class. -
main
- See Also:
-
getRegions
Gets the online regions of the specified table. This method looks at the in-memory onlineRegions. It does not go tohbase:meta
. Only returns online regions. If a region on this table has been closed during a disable, etc., it will not be included in the returned list. So, the returned list may not necessarily be ALL regions in this table, its all the ONLINE regions in the table.- Specified by:
getRegions
in interfaceOnlineRegions
- Parameters:
tableName
- table to limit the scope of the query- Returns:
- Online regions from
tableName
-
getRegions
Description copied from interface:OnlineRegions
Get all online regions in this RS.- Specified by:
getRegions
in interfaceOnlineRegions
- Returns:
- List of online Region
-
getOnlineTables
Gets the online tables in this RS. This method looks at the in-memory onlineRegions.- Returns:
- all the online tables in this RS
-
getRegionServerCoprocessors
-
closeRegionIgnoreErrors
Try to close the region, logs a warning on failure but continues.- Parameters:
region
- Region to close
-
closeRegion
protected boolean closeRegion(String encodedName, boolean abort, ServerName destination) throws NotServingRegionException Close asynchronously a region, can be called from the master or internally by the regionserver when stopping. If called from the master, the region will update the status.If an opening was in progress, this method will cancel it, but will not start a new close. The coprocessors are not called in this case. A NotServingRegionException exception is thrown.
If a close was in progress, this new request will be ignored, and an exception thrown.
Provides additional flag to indicate if this region blocks should be evicted from the cache.
- Parameters:
encodedName
- Region to closeabort
- True if we are abortingdestination
- Where the Region is being moved too... maybe null if unknown.- Returns:
- True if closed a region.
- Throws:
NotServingRegionException
- if the region is not online
-
getOnlineRegion
- Returns:
- HRegion for the passed binary
regionName
or null if named region is not member of the online regions.
-
getRegion
Description copied from interface:OnlineRegions
ReturnRegion
instance. Only works if caller is in same context, in same JVM. Region is not serializable.- Specified by:
getRegion
in interfaceOnlineRegions
- Returns:
- Region for the passed encoded
encodedRegionName
or null if named region is not member of the online regions.
-
removeRegion
Description copied from interface:MutableOnlineRegions
Removes the given Region from the list of onlineRegions.- Specified by:
removeRegion
in interfaceMutableOnlineRegions
- Parameters:
r
- Region to remove.destination
- Destination, if any, null otherwise.- Returns:
- True if we removed a region from online list.
-
getRegion
Protected Utility method for safely obtaining an HRegion handle.- Parameters:
regionName
- Name of onlineHRegion
to return- Returns:
HRegion
forregionName
- Throws:
NotServingRegionException
-
getRegionByEncodedName
- Throws:
NotServingRegionException
-
getRegionByEncodedName
private HRegion getRegionByEncodedName(byte[] regionName, String encodedRegionName) throws NotServingRegionException - Throws:
NotServingRegionException
-
cleanup
Cleanup after Throwable caught invoking method. Convertst
to IOE if it isn't already.- Parameters:
t
- Throwablemsg
- Message to log in error. Can be null.- Returns:
- Throwable converted to an IOE; methods can only let out IOEs.
-
convertThrowableToIOE
- Parameters:
msg
- Message to put in new IOE if passedt
is not an IOE- Returns:
- Make
t
an IOE if it isn't already.
-
checkFileSystem
boolean checkFileSystem()Checks to see if the file system is still accessible. If not, sets abortRequested and stopRequested- Returns:
- false if file system is not available
-
updateRegionFavoredNodesMapping
public void updateRegionFavoredNodesMapping(String encodedRegionName, List<org.apache.hadoop.hbase.shaded.protobuf.generated.HBaseProtos.ServerName> favoredNodes) Description copied from interface:FavoredNodesForRegion
Used to update the favored nodes mapping when required.- Specified by:
updateRegionFavoredNodesMapping
in interfaceFavoredNodesForRegion
-
getFavoredNodesForRegion
Return the favored nodes for a region given its encoded name. Look at the comment aroundregionFavoredNodesMap
on why we convert to InetSocketAddress[] here.- Specified by:
getFavoredNodesForRegion
in interfaceFavoredNodesForRegion
- Parameters:
encodedRegionName
- the encoded region name.- Returns:
- array of favored locations
-
getNonceManager
Description copied from interface:RegionServerServices
Only required for "old" log replay; if it's removed, remove this.- Specified by:
getNonceManager
in interfaceRegionServerServices
- Returns:
- The RegionServer's NonceManager
-
addToMovedRegions
private void addToMovedRegions(String encodedName, ServerName destination, long closeSeqNum, boolean selfMove) -
removeFromMovedRegions
-
getMovedRegion
-
movedRegionCacheExpiredTime
-
getMyEphemeralNodePath
-
isHealthCheckerConfigured
-
getCompactSplitThread
Returns the underlyingCompactSplit
for the servers -
execRegionServerService
org.apache.hadoop.hbase.shaded.protobuf.generated.ClientProtos.CoprocessorServiceResponse execRegionServerService(org.apache.hbase.thirdparty.com.google.protobuf.RpcController controller, org.apache.hadoop.hbase.shaded.protobuf.generated.ClientProtos.CoprocessorServiceRequest serviceRequest) throws org.apache.hbase.thirdparty.com.google.protobuf.ServiceException - Throws:
org.apache.hbase.thirdparty.com.google.protobuf.ServiceException
-
getBlockCache
May be null if this is a master which not carry table.- Specified by:
getBlockCache
in interfaceRegionServerServices
- Returns:
- The block cache instance used by the regionserver.
-
getMobFileCache
May be null if this is a master which not carry table.- Specified by:
getMobFileCache
in interfaceRegionServerServices
- Returns:
- The cache for mob files used by the regionserver.
-
getAccessChecker
Description copied from interface:RegionServerServices
Returns theAccessChecker
- Specified by:
getAccessChecker
in interfaceRegionServerServices
-
getZKPermissionWatcher
Description copied from interface:RegionServerServices
ReturnsZKPermissionWatcher
- Specified by:
getZKPermissionWatcher
in interfaceRegionServerServices
-
getConfigurationManager
Returns : Returns the ConfigurationManager object for testing purposes. -
getTableDescriptors
Returns Return table descriptors implementation.- Specified by:
getTableDescriptors
in interfaceRegionServerServices
-
updateConfiguration
Reload the configuration from disk.- Throws:
IOException
-
preUpdateConfiguration
- Throws:
IOException
-
postUpdateConfiguration
- Throws:
IOException
-
clearRegionBlockCache
-
getCompactionPressure
- Specified by:
getCompactionPressure
in interfaceRegionServerServices
- Returns:
- the max compaction pressure of all stores on this regionserver. The value should be greater than or equal to 0.0, and any value greater than 1.0 means we enter the emergency state that some stores have too many store files.
- See Also:
-
getHeapMemoryManager
Description copied from interface:RegionServerServices
Returns heap memory manager instance- Specified by:
getHeapMemoryManager
in interfaceRegionServerServices
-
getMemStoreFlusher
-
walRollRequestFinished
For testing- Returns:
- whether all wal roll request finished for this regionserver
-
getFlushThroughputController
Description copied from interface:RegionServerServices
Returns the controller to avoid flush too fast- Specified by:
getFlushThroughputController
in interfaceRegionServerServices
-
getFlushPressure
- Specified by:
getFlushPressure
in interfaceRegionServerServices
- Returns:
- the flush pressure of all stores on this regionserver. The value should be greater than or equal to 0.0, and any value greater than 1.0 means we enter the emergency state that global memstore size already exceeds lower limit.
-
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
-
getMetrics
Description copied from interface:RegionServerServices
Returns the metrics tracker for the region server- Specified by:
getMetrics
in interfaceRegionServerServices
-
getSecureBulkLoadManager
Description copied from interface:RegionServerServices
Returns RegionServer's instance ofSecureBulkLoadManager
- Specified by:
getSecureBulkLoadManager
in interfaceRegionServerServices
-
regionLock
Description copied from interface:RegionServerServices
Master based locks on namespaces/tables/regions.- Specified by:
regionLock
in interfaceRegionServerServices
-
unassign
Description copied from interface:RegionServerServices
Unassign the given region from the current regionserver and assign it randomly. Could still be assigned to us. This is used to solve some tough problems for which you need to reset the state of a region. For example, if you hit FileNotFound exception and want to refresh the store file list.See HBASE-17712 for more details.
- Specified by:
unassign
in interfaceRegionServerServices
- Throws:
IOException
-
getRegionServerSpaceQuotaManager
Description copied from interface:RegionServerServices
Returns RegionServer's instance ofRegionServerSpaceQuotaManager
- Specified by:
getRegionServerSpaceQuotaManager
in interfaceRegionServerServices
-
reportFileArchivalForQuotas
public boolean reportFileArchivalForQuotas(TableName tableName, Collection<Map.Entry<String, Long>> archivedFiles) Description copied from interface:RegionServerServices
Reports a collection of files, and their sizes, that belonged to the giventable
were just moved to the archive directory.- Specified by:
reportFileArchivalForQuotas
in interfaceRegionServerServices
- Parameters:
tableName
- The name of the table that files previously belonged toarchivedFiles
- Files and their sizes that were moved to archive- Returns:
true
if the files were successfully reported,false
otherwise.
-
getEventLoopGroupConfig
-
createConnection
- Specified by:
createConnection
in interfaceServer
- Throws:
IOException
-
executeProcedure
-
remoteProcedureComplete
-
reportProcedureDone
void reportProcedureDone(org.apache.hadoop.hbase.shaded.protobuf.generated.RegionServerStatusProtos.ReportProcedureDoneRequest request) throws IOException - Throws:
IOException
-
submitRegionProcedure
Will ignore the open/close region procedures which already submitted or executed. When master had unfinished open/close region procedure and restarted, new active master may send duplicate open/close region request to regionserver. The open/close request is submitted to a thread pool and execute. So first need a cache for submitted open/close region procedures. After the open/close region request executed and report region transition succeed, cache it in executed region procedures cache. SeefinishRegionProcedure(long)
. After report region transition succeed, master will not send the open/close region request to regionserver again. And we thought that the ongoing duplicate open/close region request should not be delayed more than 600 seconds. So the executed region procedures cache will expire after 600 seconds. See HBASE-22404 for more details.- Parameters:
procId
- the id of the open/close region procedure- Returns:
- true if the procedure can be submitted.
-
finishRegionProcedure
- Parameters:
procId
- the id of the open/close region procedure
-
isShutDown
-
getCompactedHFilesDischarger
-
getRetryPauseTime
Return pause time configured inHConstants.HBASE_RPC_SHORTOPERATION_RETRY_PAUSE_TIME
}- Returns:
- pause time
-
getActiveMaster
-
getBackupMasters
-
getBootstrapNodes
-
getMetaRegionLocationCache
-
getBrokenStoreFileCleaner
-
getRSMobFileCleanerChore
-
getRsSnapshotVerifier
-