@InterfaceAudience.LimitedPrivate(value="Tools") public class HRegionServer extends HasThread implements RegionServerServices, LastSequenceId, ConfigurationObserver
Modifier and Type | Class and Description |
---|---|
private static class |
HRegionServer.CompactionChecker |
private static class |
HRegionServer.MovedRegionInfo |
protected static class |
HRegionServer.MovedRegionsCleaner
Creates a Chore thread to clean the moved region cache.
|
(package private) static class |
HRegionServer.PeriodicMemStoreFlusher |
private static class |
HRegionServer.SystemExitWhenAbortTimeout
Force to terminate region server when abort timeout.
|
RegionServerServices.PostOpenDeployContext, RegionServerServices.RegionStateTransitionContext
Modifier and Type | Field and Description |
---|---|
static String |
ABORT_TIMEOUT |
static String |
ABORT_TIMEOUT_TASK |
private boolean |
abortRequested |
private BlockCache |
blockCache |
protected MemStoreFlusher |
cacheFlusher |
private ChoreService |
choreService
ChoreService used to schedule tasks that we want to run periodically
|
protected ClusterConnection |
clusterConnection
Cluster connection to be shared by services.
|
protected String |
clusterId
Unique identifier for the cluster we are a part of.
|
protected ClusterStatusTracker |
clusterStatusTracker |
(package private) CompactedHFilesDischarger |
compactedFileDischarger |
(package private) ScheduledChore |
compactionChecker |
private int |
compactionCheckFrequency |
CompactSplit |
compactSplitThread |
protected org.apache.hadoop.conf.Configuration |
conf |
protected ConfigurationManager |
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 Map<String,com.google.protobuf.Service> |
coprocessorServiceHandlers |
protected CoordinatedStateManager |
csm |
private static long |
DEFAULT_ABORT_TIMEOUT |
private NettyEventLoopGroupConfig |
eventLoopGroupConfig |
private org.apache.hbase.thirdparty.com.google.common.cache.Cache<Long,Long> |
executedRegionProcedures
Used to cache the open/close region procedures which already executed.
|
protected ExecutorService |
executorService |
private int |
flushCheckFrequency |
private ThroughputController |
flushThroughputController |
protected HFileSystem |
fs |
protected boolean |
fsOk |
protected FileSystemUtilizationChore |
fsUtilizationChore |
private HealthCheckChore |
healthCheckChore
The health check chore.
|
protected HeapMemoryManager |
hMemManager |
protected InfoServer |
infoServer |
private static int |
INIT_PAUSE_TIME_MS |
(package private) boolean |
killed |
protected Leases |
leases |
protected ReentrantReadWriteLock |
lock |
private org.apache.hadoop.hbase.shaded.protobuf.generated.LockServiceProtos.LockService.BlockingInterface |
lockStub |
private static org.slf4j.Logger |
LOG |
protected static String |
MASTER_HOSTNAME_KEY |
private MasterAddressTracker |
masterAddressTracker |
private boolean |
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.
|
(package private) static String |
MASTERLESS_CONFIG_NAME |
(package private) MetricsRegionServer |
metricsRegionServer |
(package private) MetricsRegionServerWrapperImpl |
metricsRegionServerImpl |
(package private) MetricsTable |
metricsTable |
private MobFileCache |
mobFileCache |
protected Map<String,HRegionServer.MovedRegionInfo> |
movedRegions |
private HRegionServer.MovedRegionsCleaner |
movedRegionsCleaner
Chore to clean periodically the moved region list
|
protected int |
msgInterval |
(package private) ServerNonceManager |
nonceManager
Nonce manager.
|
private ScheduledChore |
nonceManagerChore
The nonce manager chore.
|
protected int |
numRegionsToReport |
(package private) int |
numRetries |
(package private) AtomicBoolean |
online |
protected Map<String,HRegion> |
onlineRegions
Map of regions currently being served by this region server.
|
private int |
operationTimeout |
private JvmPauseMonitor |
pauseMonitor |
private static String |
PERIOD_COMPACTION |
private static String |
PERIOD_FLUSH |
(package private) ScheduledChore |
periodicFlusher |
private RemoteProcedureResultReporter |
procedureResultReporter |
protected Map<String,InetSocketAddress[]> |
regionFavoredNodesMap
Map of encoded region names to the DataNode locations they should be hosted on
We store the value as InetSocketAddress since this is used only in HDFS
API (create() that takes favored nodes as hints for placing file blocks).
|
static String |
REGIONSERVER
region server process name
|
static String |
REGIONSERVER_CODEC
regionserver codec list
|
private RegionServerAccounting |
regionServerAccounting |
protected ConcurrentMap<byte[],Boolean> |
regionsInTransitionInRS |
protected ReplicationSinkService |
replicationSinkHandler |
protected ReplicationSourceService |
replicationSourceHandler |
private org.apache.hadoop.fs.Path |
rootDir |
(package private) ConcurrentMap<String,Integer> |
rowlocks |
(package private) RpcClient |
rpcClient |
private RpcControllerFactory |
rpcControllerFactory |
private RpcRetryingCallerFactory |
rpcRetryingCallerFactory |
protected RSRpcServices |
rpcServices |
(package private) static String |
RS_HOSTNAME_DISABLE_MASTER_REVERSEDNS_KEY |
(package private) static String |
RS_HOSTNAME_KEY |
private RegionServerCoprocessorHost |
rsHost |
private RegionServerProcedureManagerHost |
rspmHost |
private RegionServerRpcQuotaManager |
rsQuotaManager |
private RegionServerSpaceQuotaManager |
rsSpaceQuotaManager |
private org.apache.hadoop.hbase.shaded.protobuf.generated.RegionServerStatusProtos.RegionServerStatusService.BlockingInterface |
rssStub |
protected SecureBulkLoadManager |
secureBulkLoadManager |
protected ServerName |
serverName
The server name the Master sees us as.
|
private int |
shortOperationTimeout |
private boolean |
shutDown |
protected Sleeper |
sleeper |
private SpanReceiverHost |
spanReceiverHost |
private SplitLogWorker |
splitLogWorker |
protected long |
startcode
This servers startcode.
|
private boolean |
stopped |
private boolean |
stopping |
private StorefileRefresherChore |
storefileRefresher |
private ConcurrentMap<Long,Long> |
submittedRegionProcedures
Used to cache the open/close region procedures which already submitted.
|
protected TableDescriptors |
tableDescriptors
Go here to get table descriptors.
|
static boolean |
TEST_SKIP_REPORTING_TRANSITION
For testing only! Set to true to skip notifying region assignment to master .
|
protected int |
threadWakeFrequency |
private static int |
TIMEOUT_REGION_MOVED |
private Thread.UncaughtExceptionHandler |
uncaughtExceptionHandler |
private UserProvider |
userProvider |
protected String |
useThisHostnameInstead |
protected WALFactory |
walFactory |
protected HFileSystem |
walFs |
protected LogRoller |
walRoller |
private org.apache.hadoop.fs.Path |
walRootDir |
protected ZKWatcher |
zooKeeper |
Constructor and Description |
---|
HRegionServer(org.apache.hadoop.conf.Configuration conf)
Starts a HRegionServer at the default location
|
Modifier and Type | Method and Description |
---|---|
void |
abort(String reason) |
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.
|
void |
addRegion(HRegion region)
Add to online regions.
|
protected void |
addToMovedRegions(String encodedName,
ServerName destination,
long closeSeqNum) |
private boolean |
areAllUserRegionsOffline() |
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 a
RegionServerStatusProtos.RegionSpaceUseReportRequest protobuf message from the region size map. |
(package 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.
|
(package private) org.apache.hadoop.hbase.shaded.protobuf.generated.ClusterStatusProtos.ServerLoad |
buildServerLoad(long reportStartTime,
long reportEndTime) |
protected boolean |
canCreateBaseZNode() |
protected boolean |
canUpdateTableDescriptor() |
private static void |
checkCodecs(org.apache.hadoop.conf.Configuration c)
Run test on configured codecs to make sure supporting libs are in place.
|
boolean |
checkFileSystem()
Checks to see if the file system is still accessible.
|
protected void |
cleanMovedRegions()
Remove the expired entries from the moved regions list.
|
private Throwable |
cleanup(Throwable t,
String msg) |
CacheEvictionStats |
clearRegionBlockCache(Region region) |
protected void |
closeAllRegions(boolean abort)
Closes all regions.
|
protected boolean |
closeAndOfflineRegionForSplitOrMerge(List<String> regionEncodedName)
Close and offline the region for split or merge
|
(package private) void |
closeMetaTableRegions(boolean abort)
Close meta region if we carry it
|
protected boolean |
closeRegion(String encodedName,
boolean abort,
ServerName sn)
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.
|
(package private) void |
closeUserRegions(boolean abort)
Schedule closes on all user regions.
|
protected void |
configureInfoServer() |
static HRegionServer |
constructRegionServer(Class<? extends HRegionServer> regionServerClass,
org.apache.hadoop.conf.Configuration conf2)
Utility for constructing an instance of the passed HRegionServer class.
|
private boolean |
containsMetaTableRegions() |
(package private) org.apache.hadoop.hbase.shaded.protobuf.generated.RegionServerStatusProtos.RegionSpaceUse |
convertRegionSize(RegionInfo regionInfo,
Long sizeInBytes)
Converts a pair of
RegionInfo and long into a RegionServerStatusProtos.RegionSpaceUse
protobuf message. |
protected IOException |
convertThrowableToIOE(Throwable t,
String msg) |
protected ClusterConnection |
createClusterConnection()
Create a 'smarter' Connection, one that is capable of by-passing RPC if the request is to
the local server; i.e.
|
Connection |
createConnection(org.apache.hadoop.conf.Configuration conf) |
private void |
createMyEphemeralNode() |
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,
WALProvider walProvider)
Load the replication executorService objects, if any
|
(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) |
org.apache.hadoop.hbase.shaded.protobuf.generated.ClusterStatusProtos.RegionLoad |
createRegionLoad(String encodedRegionName) |
protected ServerName |
createRegionServerStatusStub()
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.
|
protected RSRpcServices |
createRpcServices() |
private void |
deleteMyEphemeralNode() |
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) |
void |
executeProcedure(long procId,
RSProcedureCallable callable) |
void |
finishRegionProcedure(long procId)
|
Optional<BlockCache> |
getBlockCache()
May be null if this is a master which not carry table.
|
ChoreService |
getChoreService() |
ClusterConnection |
getClusterConnection()
Returns a reference to the servers' cluster connection.
|
String |
getClusterId() |
CompactedHFilesDischarger |
getCompactedHFilesDischarger() |
double |
getCompactionPressure() |
CompactionRequester |
getCompactionRequestor() |
CompactSplit |
getCompactSplitThread() |
org.apache.hadoop.conf.Configuration |
getConfiguration()
Gets the configuration object for this server.
|
protected ConfigurationManager |
getConfigurationManager() |
Connection |
getConnection()
Returns a reference to the servers' connection.
|
CoordinatedStateManager |
getCoordinatedStateManager()
Get CoordinatedStateManager instance for this server.
|
(package private) SortedMap<Long,HRegion> |
getCopyOfOnlineRegionsSortedByOffHeapSize() |
(package private) SortedMap<Long,HRegion> |
getCopyOfOnlineRegionsSortedByOnHeapSize() |
protected Class<? extends javax.servlet.http.HttpServlet> |
getDumpServlet() |
NettyEventLoopGroupConfig |
getEventLoopGroupConfig() |
ExecutorService |
getExecutorService() |
InetSocketAddress[] |
getFavoredNodesForRegion(String encodedRegionName)
Return the favored nodes for a region given its encoded name.
|
org.apache.hadoop.fs.FileSystem |
getFileSystem() |
double |
getFlushPressure() |
FlushRequester |
getFlushRequester() |
ThroughputController |
getFlushThroughputController() |
protected TableDescriptors |
getFsTableDescriptors() |
HeapMemoryManager |
getHeapMemoryManager() |
InfoServer |
getInfoServer() |
org.apache.hadoop.hbase.shaded.protobuf.generated.ClusterStatusProtos.RegionStoreSequenceIds |
getLastSequenceId(byte[] encodedRegionName) |
Leases |
getLeases() |
MasterAddressTracker |
getMasterAddressTracker() |
protected Function<TableDescriptorBuilder,TableDescriptorBuilder> |
getMetaTableObserver() |
MetricsRegionServer |
getMetrics() |
Optional<MobFileCache> |
getMobFileCache()
May be null if this is a master which not carry table.
|
private HRegionServer.MovedRegionInfo |
getMovedRegion(String encodedRegionName) |
private String |
getMyEphemeralNodePath() |
ServerNonceManager |
getNonceManager()
Only required for "old" log replay; if it's removed, remove this.
|
int |
getNumberOfOnlineRegions() |
HRegion |
getOnlineRegion(byte[] regionName) |
(package private) String |
getOnlineRegionsAsPrintableString() |
Collection<HRegion> |
getOnlineRegionsLocalContext()
For tests, web ui and metrics.
|
Set<TableName> |
getOnlineTables()
Gets the online tables in this RS.
|
protected String |
getProcessName() |
protected HRegion |
getRegion(byte[] regionName)
Protected Utility method for safely obtaining an HRegion handle.
|
HRegion |
getRegion(String encodedRegionName)
Return
Region instance. |
InetSocketAddress[] |
getRegionBlockLocations(String encodedRegionName) |
protected HRegion |
getRegionByEncodedName(byte[] regionName,
String encodedRegionName) |
HRegion |
getRegionByEncodedName(String encodedRegionName) |
List<HRegion> |
getRegions()
Get all online regions in this RS.
|
List<HRegion> |
getRegions(TableName tableName)
Gets the online regions of the specified table.
|
RegionServerAccounting |
getRegionServerAccounting() |
RegionServerCoprocessorHost |
getRegionServerCoprocessorHost() |
String[] |
getRegionServerCoprocessors() |
MetricsRegionServer |
getRegionServerMetrics() |
RegionServerRpcQuotaManager |
getRegionServerRpcQuotaManager() |
RegionServerSpaceQuotaManager |
getRegionServerSpaceQuotaManager() |
ConcurrentMap<byte[],Boolean> |
getRegionsInTransitionInRS()
Get the regions that are currently being opened or closed in the RS
|
(package private) ReplicationSinkService |
getReplicationSinkService() |
ReplicationSourceService |
getReplicationSourceService() |
protected org.apache.hadoop.fs.Path |
getRootDir() |
RpcServerInterface |
getRpcServer()
Returns a reference to the region server's RPC server
|
RSRpcServices |
getRSRpcServices() |
SecureBulkLoadManager |
getSecureBulkLoadManager() |
ServerName |
getServerName() |
long |
getStartcode() |
TableDescriptors |
getTableDescriptors() |
int |
getThreadWakeFrequency()
Interval at which threads should run
|
protected String |
getUseThisHostnameInstead(org.apache.hadoop.conf.Configuration conf) |
WAL |
getWAL(RegionInfo regionInfo) |
org.apache.hadoop.fs.FileSystem |
getWALFileSystem() |
Map<String,ReplicationStatus> |
getWalGroupsReplicationStatus() |
LogRoller |
getWalRoller() |
org.apache.hadoop.fs.Path |
getWALRootDir() |
List<WAL> |
getWALs() |
(package private) ReentrantReadWriteLock.WriteLock |
getWriteLock() |
private long |
getWriteRequestCount() |
ZKWatcher |
getZooKeeper()
Gets the ZooKeeper instance for this server.
|
protected void |
handleReportForDutyResponse(org.apache.hadoop.hbase.shaded.protobuf.generated.RegionServerStatusProtos.RegionServerStartupResponse c) |
private void |
initializeFileSystem() |
protected void |
initializeMemStoreChunkCreator() |
private void |
initializeThreads() |
private void |
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.
|
boolean |
isAborted()
Check if the server or client was aborted.
|
boolean |
isClusterUp() |
private boolean |
isHealthCheckerConfigured() |
private boolean |
isHealthy() |
boolean |
isOnline()
Report the status of the server.
|
(package private) boolean |
isOnlineRegionsEmpty() |
boolean |
isShutDown() |
boolean |
isStopped() |
boolean |
isStopping() |
private boolean |
keepLooping() |
protected void |
kill() |
protected void |
login(UserProvider user,
String host) |
static void |
main(String[] args) |
protected int |
movedRegionCleanerPeriod() |
private static <T extends ReplicationService> |
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,
WALProvider walProvider) |
void |
onConfigurationChange(org.apache.hadoop.conf.Configuration newConf)
This method would be called by the
ConfigurationManager
object when the Configuration object is reloaded from disk. |
void |
postOpenDeployTasks(RegionServerServices.PostOpenDeployContext context)
Tasks to perform after region open to complete deploy of region on regionserver
|
private void |
preRegistrationInitialization()
All initialization needed before we go register with Master.
Do bare minimum. |
private int |
putUpWebUI()
Puts up the webui.
|
EntityLock |
regionLock(List<RegionInfo> regionInfos,
String description,
Abortable abort)
Master based locks on namespaces/tables/regions.
|
private void |
registerConfigurationObservers() |
boolean |
registerService(com.google.protobuf.Service instance)
Registers a new protocol buffer
Service subclass as a coprocessor endpoint to be
available for handling |
void |
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 given
table were
just moved to the archive directory. |
private org.apache.hadoop.hbase.shaded.protobuf.generated.RegionServerStatusProtos.RegionServerStartupResponse |
reportForDuty() |
(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 |
reportRegionStateTransition(RegionServerServices.RegionStateTransitionContext context)
Notify master that a handler requests to change a region state
|
void |
run()
The HRegionServer sticks in this loop until closed.
|
protected void |
sendShutdownInterrupt()
Called on stop/abort before closing the cluster connection and meta locator.
|
protected void |
setAbortRequested() |
protected void |
setupClusterConnection()
Setup our cluster connection if not already initialized.
|
private static NettyEventLoopGroupConfig |
setupNetty(org.apache.hadoop.conf.Configuration conf) |
private void |
setupWALAndReplication()
Setup WAL log and replication if enabled.
|
private static void |
setupWindows(org.apache.hadoop.conf.Configuration conf,
ConfigurationManager cm)
If running on Windows, do windows-specific setup.
|
private void |
shutdownWAL(boolean close) |
private boolean |
sleep(long millis) |
private void |
startHeapMemoryManager() |
private void |
startReplicationService()
Start up replication source and sink handlers.
|
private void |
startServices() |
void |
stop(String msg)
Stop this service.
|
void |
stop(String msg,
boolean force,
User user)
Stops the regionserver.
|
protected void |
stopServiceThreads()
Wait on all threads to finish.
|
(package private) boolean |
submitRegionProcedure(long procId)
Will ignore the open/close region procedures which already submitted or executed.
|
String |
toString() |
(package 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.
|
void |
updateConfiguration()
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 |
waitForMasterActive()
Wait for an active Master.
|
void |
waitForServerOnline() |
private void |
waitOnAllRegionsToClose(boolean abort)
Wait on regions close.
|
boolean |
walRollRequestFinished()
For testing
|
getName, getThread, interrupt, isAlive, isInterrupted, join, join, join, setDaemon, setName, setPriority, setUncaughtExceptionHandler, start
private static final int INIT_PAUSE_TIME_MS
private static final org.slf4j.Logger LOG
public static boolean TEST_SKIP_REPORTING_TRANSITION
protected final ConcurrentMap<byte[],Boolean> regionsInTransitionInRS
private final ConcurrentMap<Long,Long> submittedRegionProcedures
submitRegionProcedure(long)
.private final org.apache.hbase.thirdparty.com.google.common.cache.Cache<Long,Long> executedRegionProcedures
submitRegionProcedure(long)
.protected MemStoreFlusher cacheFlusher
protected HeapMemoryManager hMemManager
protected ClusterConnection clusterConnection
protected TableDescriptors tableDescriptors
protected ReplicationSourceService replicationSourceHandler
protected ReplicationSinkService replicationSinkHandler
public CompactSplit compactSplitThread
protected final Map<String,HRegion> onlineRegions
protected final Map<String,InetSocketAddress[]> regionFavoredNodesMap
protected ExecutorService executorService
protected volatile boolean fsOk
protected HFileSystem fs
protected HFileSystem walFs
private volatile boolean stopped
private volatile boolean abortRequested
public static final String ABORT_TIMEOUT
private static final long DEFAULT_ABORT_TIMEOUT
public static final String ABORT_TIMEOUT_TASK
ConcurrentMap<String,Integer> rowlocks
private boolean stopping
volatile boolean killed
private volatile boolean shutDown
protected final org.apache.hadoop.conf.Configuration conf
private org.apache.hadoop.fs.Path rootDir
private org.apache.hadoop.fs.Path walRootDir
protected final ReentrantReadWriteLock lock
final int numRetries
protected final int threadWakeFrequency
protected final int msgInterval
private static final String PERIOD_COMPACTION
private final int compactionCheckFrequency
private static final String PERIOD_FLUSH
private final int flushCheckFrequency
protected final int numRegionsToReport
private volatile org.apache.hadoop.hbase.shaded.protobuf.generated.RegionServerStatusProtos.RegionServerStatusService.BlockingInterface rssStub
private volatile org.apache.hadoop.hbase.shaded.protobuf.generated.LockServiceProtos.LockService.BlockingInterface lockStub
private RpcRetryingCallerFactory rpcRetryingCallerFactory
private RpcControllerFactory rpcControllerFactory
private Thread.UncaughtExceptionHandler uncaughtExceptionHandler
protected InfoServer infoServer
private JvmPauseMonitor pauseMonitor
public static final String REGIONSERVER
MetricsRegionServer metricsRegionServer
MetricsRegionServerWrapperImpl metricsRegionServerImpl
MetricsTable metricsTable
private SpanReceiverHost spanReceiverHost
private ChoreService choreService
ScheduledChore compactionChecker
ScheduledChore periodicFlusher
protected volatile WALFactory walFactory
private RemoteProcedureResultReporter procedureResultReporter
final AtomicBoolean online
private final MasterAddressTracker masterAddressTracker
protected final ClusterStatusTracker clusterStatusTracker
private SplitLogWorker splitLogWorker
private final int operationTimeout
private final int shortOperationTimeout
private final RegionServerAccounting regionServerAccounting
private BlockCache blockCache
private MobFileCache mobFileCache
private HealthCheckChore healthCheckChore
private ScheduledChore nonceManagerChore
private Map<String,com.google.protobuf.Service> coprocessorServiceHandlers
protected ServerName serverName
protected String useThisHostnameInstead
@InterfaceAudience.LimitedPrivate(value="Configuration") static final String RS_HOSTNAME_KEY
@InterfaceAudience.LimitedPrivate(value="Configuration") protected static final String MASTER_HOSTNAME_KEY
static final String RS_HOSTNAME_DISABLE_MASTER_REVERSEDNS_KEY
protected final long startcode
private HRegionServer.MovedRegionsCleaner movedRegionsCleaner
private StorefileRefresherChore storefileRefresher
private RegionServerCoprocessorHost rsHost
private RegionServerProcedureManagerHost rspmHost
private RegionServerRpcQuotaManager rsQuotaManager
private RegionServerSpaceQuotaManager rsSpaceQuotaManager
final ServerNonceManager nonceManager
private UserProvider userProvider
protected final RSRpcServices rpcServices
protected CoordinatedStateManager csm
protected final ConfigurationManager configurationManager
CompactedHFilesDischarger compactedFileDischarger
private volatile ThroughputController flushThroughputController
protected SecureBulkLoadManager secureBulkLoadManager
protected FileSystemUtilizationChore fsUtilizationChore
private final NettyEventLoopGroupConfig eventLoopGroupConfig
private final boolean masterless
static final String MASTERLESS_CONFIG_NAME
public static final String REGIONSERVER_CODEC
protected Map<String,HRegionServer.MovedRegionInfo> movedRegions
private static final int TIMEOUT_REGION_MOVED
public HRegionServer(org.apache.hadoop.conf.Configuration conf) throws IOException
IOException
protected String getUseThisHostnameInstead(org.apache.hadoop.conf.Configuration conf) throws IOException
IOException
private static void setupWindows(org.apache.hadoop.conf.Configuration conf, ConfigurationManager cm)
private static NettyEventLoopGroupConfig setupNetty(org.apache.hadoop.conf.Configuration conf)
private void initializeFileSystem() throws IOException
IOException
protected TableDescriptors getFsTableDescriptors() throws IOException
IOException
protected Function<TableDescriptorBuilder,TableDescriptorBuilder> getMetaTableObserver()
protected void login(UserProvider user, String host) throws IOException
IOException
protected void waitForMasterActive()
protected String getProcessName()
protected boolean canCreateBaseZNode()
protected boolean canUpdateTableDescriptor()
protected RSRpcServices createRpcServices() throws IOException
IOException
protected void configureInfoServer()
protected Class<? extends javax.servlet.http.HttpServlet> getDumpServlet()
public boolean registerService(com.google.protobuf.Service instance)
RegionServerServices
Service
subclass as a coprocessor endpoint to be
available for handlingregisterService
in interface RegionServerServices
instance
- the Service
subclass instance to expose as a coprocessor endpointtrue
if the registration was successful, false
protected ClusterConnection createClusterConnection() throws IOException
IOException
private static void checkCodecs(org.apache.hadoop.conf.Configuration c) throws IOException
c
- IOException
public String getClusterId()
protected void setupClusterConnection() throws IOException
IOException
private void preRegistrationInitialization()
private void initializeZooKeeper() throws IOException, InterruptedException
Finally open long-living server short-circuit connection.
IOException
InterruptedException
private void blockAndCheckIfStopped(ZKNodeTracker tracker) throws IOException, InterruptedException
tracker
- znode tracker to useIOException
- any IO exception, plus if the RS is stoppedInterruptedException
public boolean isClusterUp()
isClusterUp
in interface RegionServerServices
public void run()
private boolean containsMetaTableRegions()
private boolean areAllUserRegionsOffline()
private long getWriteRequestCount()
protected void tryRegionServerReport(long reportStartTime, long reportEndTime) throws IOException
IOException
public boolean reportRegionSizesForQuotas(RegionSizeStore regionSizeStore)
reportRegionSizesForQuotas
in interface RegionServerServices
regionSizeStore
- The store containing region sizesvoid buildReportAndSend(org.apache.hadoop.hbase.shaded.protobuf.generated.RegionServerStatusProtos.RegionServerStatusService.BlockingInterface rss, RegionSizeStore regionSizeStore) throws org.apache.hbase.thirdparty.com.google.protobuf.ServiceException
rss
- The stub to send to the MasterregionSizeStore
- The store containing region sizesorg.apache.hbase.thirdparty.com.google.protobuf.ServiceException
org.apache.hadoop.hbase.shaded.protobuf.generated.RegionServerStatusProtos.RegionSpaceUseReportRequest buildRegionSpaceUseReportRequest(RegionSizeStore regionSizes)
RegionServerStatusProtos.RegionSpaceUseReportRequest
protobuf message from the region size map.regionSizeStore
- The size in bytes of regionsorg.apache.hadoop.hbase.shaded.protobuf.generated.RegionServerStatusProtos.RegionSpaceUse convertRegionSize(RegionInfo regionInfo, Long sizeInBytes)
RegionInfo
and long
into a RegionServerStatusProtos.RegionSpaceUse
protobuf message.regionInfo
- The RegionInfosizeInBytes
- The size in bytes of the Regionorg.apache.hadoop.hbase.shaded.protobuf.generated.ClusterStatusProtos.ServerLoad buildServerLoad(long reportStartTime, long reportEndTime) throws IOException
IOException
String getOnlineRegionsAsPrintableString()
private void waitOnAllRegionsToClose(boolean abort)
private boolean sleep(long millis)
private void shutdownWAL(boolean close)
protected void handleReportForDutyResponse(org.apache.hadoop.hbase.shaded.protobuf.generated.RegionServerStatusProtos.RegionServerStartupResponse c) throws IOException
IOException
protected void initializeMemStoreChunkCreator()
private void startHeapMemoryManager()
private void createMyEphemeralNode() throws org.apache.zookeeper.KeeperException, IOException
org.apache.zookeeper.KeeperException
IOException
private void deleteMyEphemeralNode() throws org.apache.zookeeper.KeeperException
org.apache.zookeeper.KeeperException
public RegionServerAccounting getRegionServerAccounting()
getRegionServerAccounting
in interface RegionServerServices
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
IOException
public org.apache.hadoop.hbase.shaded.protobuf.generated.ClusterStatusProtos.RegionLoad createRegionLoad(String encodedRegionName) throws IOException
encodedRegionName
- IOException
public boolean isOnline()
private void setupWALAndReplication() throws IOException
IOException
private void startReplicationService() throws IOException
IOException
public MetricsRegionServer getRegionServerMetrics()
public MasterAddressTracker getMasterAddressTracker()
private void startServices() throws IOException
IOException
private void initializeThreads() throws IOException
IOException
private void registerConfigurationObservers()
private int putUpWebUI() throws IOException
IOException
private boolean isHealthy()
public List<WAL> getWALs() throws IOException
getWALs
in interface RegionServerServices
IOException
public WAL getWAL(RegionInfo regionInfo) throws IOException
getWAL
in interface RegionServerServices
IOException
public LogRoller getWalRoller()
public Connection getConnection()
Server
getConnection
in interface Server
public ClusterConnection getClusterConnection()
Server
Server.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.getClusterConnection
in interface Server
public void stop(String msg)
Stoppable
public void stop(String msg, boolean force, User user)
msg
- Status messageforce
- True if this is a regionserver abortuser
- The user executing the stop request, or null if no user is associatedpublic void waitForServerOnline()
public void postOpenDeployTasks(RegionServerServices.PostOpenDeployContext context) throws IOException
RegionServerServices
postOpenDeployTasks
in interface RegionServerServices
context
- the contextIOException
public boolean reportRegionStateTransition(RegionServerServices.RegionStateTransitionContext context)
RegionServerServices
reportRegionStateTransition
in interface RegionServerServices
void triggerFlushInPrimaryRegion(HRegion region)
public RpcServerInterface getRpcServer()
RegionServerServices
getRpcServer
in interface RegionServerServices
public RSRpcServices getRSRpcServices()
public void abort(String reason, Throwable cause)
protected final void setAbortRequested()
public void abort(String reason)
abort(String, Throwable)
public boolean isAborted()
Abortable
protected void kill()
protected void sendShutdownInterrupt()
protected void stopServiceThreads()
public ReplicationSourceService getReplicationSourceService()
ReplicationSinkService getReplicationSinkService()
protected ServerName createRegionServerStatusStub()
protected ServerName createRegionServerStatusStub(boolean refresh)
refresh
- If true then master address will be read from ZK, otherwise use cached dataprivate boolean keepLooping()
private org.apache.hadoop.hbase.shaded.protobuf.generated.RegionServerStatusProtos.RegionServerStartupResponse reportForDuty() throws IOException
IOException
public org.apache.hadoop.hbase.shaded.protobuf.generated.ClusterStatusProtos.RegionStoreSequenceIds getLastSequenceId(byte[] encodedRegionName)
getLastSequenceId
in interface LastSequenceId
encodedRegionName
- Encoded region nameprotected void closeAllRegions(boolean abort)
void closeMetaTableRegions(boolean abort)
abort
- Whether we're running an abort.void closeUserRegions(boolean abort)
abort
- Whether we're running an abort.public InfoServer getInfoServer()
public boolean isStopped()
public boolean isStopping()
isStopping
in interface Server
public org.apache.hadoop.conf.Configuration getConfiguration()
Server
getConfiguration
in interface Server
ReentrantReadWriteLock.WriteLock getWriteLock()
public int getNumberOfOnlineRegions()
boolean isOnlineRegionsEmpty()
public Collection<HRegion> getOnlineRegionsLocalContext()
public void addRegion(HRegion region)
MutableOnlineRegions
addRegion
in interface MutableOnlineRegions
SortedMap<Long,HRegion> getCopyOfOnlineRegionsSortedByOffHeapSize()
SortedMap<Long,HRegion> getCopyOfOnlineRegionsSortedByOnHeapSize()
public long getStartcode()
public FlushRequester getFlushRequester()
getFlushRequester
in interface RegionServerServices
public CompactionRequester getCompactionRequestor()
getCompactionRequestor
in interface RegionServerServices
CompactionRequester
or null. Usually it will not be null
unless during intialization.public Leases getLeases()
getLeases
in interface RegionServerServices
protected org.apache.hadoop.fs.Path getRootDir()
public org.apache.hadoop.fs.FileSystem getFileSystem()
getFileSystem
in interface Server
public org.apache.hadoop.fs.Path getWALRootDir()
public org.apache.hadoop.fs.FileSystem getWALFileSystem()
public int getThreadWakeFrequency()
public ZKWatcher getZooKeeper()
Server
getZooKeeper
in interface Server
public CoordinatedStateManager getCoordinatedStateManager()
Server
getCoordinatedStateManager
in interface Server
public ServerName getServerName()
getServerName
in interface Server
public RegionServerCoprocessorHost getRegionServerCoprocessorHost()
public ConcurrentMap<byte[],Boolean> getRegionsInTransitionInRS()
RegionServerServices
getRegionsInTransitionInRS
in interface RegionServerServices
public ExecutorService getExecutorService()
getExecutorService
in interface RegionServerServices
public ChoreService getChoreService()
getChoreService
in interface Server
ChoreService
instance for this serverpublic RegionServerRpcQuotaManager getRegionServerRpcQuotaManager()
getRegionServerRpcQuotaManager
in interface RegionServerServices
RegionServerRpcQuotaManager
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, WALProvider walProvider) throws IOException
IOException
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, WALProvider walProvider) throws IOException
IOException
public Map<String,ReplicationStatus> getWalGroupsReplicationStatus()
public static HRegionServer constructRegionServer(Class<? extends HRegionServer> regionServerClass, org.apache.hadoop.conf.Configuration conf2)
regionServerClass
- conf2
- public static void main(String[] args) throws Exception
Exception
HRegionServerCommandLine
public List<HRegion> getRegions(TableName tableName)
hbase: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.getRegions
in interface OnlineRegions
tableName
- tableName
public List<HRegion> getRegions()
OnlineRegions
getRegions
in interface OnlineRegions
public Set<TableName> getOnlineTables()
public String[] getRegionServerCoprocessors()
private void closeRegionIgnoreErrors(RegionInfo region, boolean abort)
region
- Region to closeprotected boolean closeRegion(String encodedName, boolean abort, ServerName sn) throws NotServingRegionException
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.
encodedName
- Region to closeabort
- True if we are abortingNotServingRegionException
- if the region is not onlineprotected boolean closeAndOfflineRegionForSplitOrMerge(List<String> regionEncodedName) throws IOException
regionEncodedName
- the name of the region(s) to closeIOException
public HRegion getOnlineRegion(byte[] regionName)
regionName
or null if
named region is not member of the online regions.public InetSocketAddress[] getRegionBlockLocations(String encodedRegionName)
public HRegion getRegion(String encodedRegionName)
OnlineRegions
Region
instance.
Only works if caller is in same context, in same JVM. Region is not
serializable.getRegion
in interface OnlineRegions
encodedRegionName
or
null if named region is not member of the online regions.public boolean removeRegion(HRegion r, ServerName destination)
MutableOnlineRegions
removeRegion
in interface MutableOnlineRegions
r
- Region to remove.destination
- Destination, if any, null otherwise.protected HRegion getRegion(byte[] regionName) throws NotServingRegionException
regionName
- Name of online HRegion
to returnHRegion
for regionName
NotServingRegionException
public HRegion getRegionByEncodedName(String encodedRegionName) throws NotServingRegionException
NotServingRegionException
protected HRegion getRegionByEncodedName(byte[] regionName, String encodedRegionName) throws NotServingRegionException
NotServingRegionException
protected IOException convertThrowableToIOE(Throwable t, String msg)
public boolean checkFileSystem()
public void updateRegionFavoredNodesMapping(String encodedRegionName, List<org.apache.hadoop.hbase.shaded.protobuf.generated.HBaseProtos.ServerName> favoredNodes)
FavoredNodesForRegion
updateRegionFavoredNodesMapping
in interface FavoredNodesForRegion
public InetSocketAddress[] getFavoredNodesForRegion(String encodedRegionName)
regionFavoredNodesMap
on why it is InetSocketAddress[]getFavoredNodesForRegion
in interface FavoredNodesForRegion
encodedRegionName
- public ServerNonceManager getNonceManager()
RegionServerServices
getNonceManager
in interface RegionServerServices
protected void addToMovedRegions(String encodedName, ServerName destination, long closeSeqNum)
void removeFromMovedRegions(String encodedName)
private HRegionServer.MovedRegionInfo getMovedRegion(String encodedRegionName)
protected void cleanMovedRegions()
protected int movedRegionCleanerPeriod()
private String getMyEphemeralNodePath()
private boolean isHealthCheckerConfigured()
public CompactSplit getCompactSplitThread()
CompactSplit
for the serverspublic 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
org.apache.hbase.thirdparty.com.google.protobuf.ServiceException
public Optional<BlockCache> getBlockCache()
getBlockCache
in interface RegionServerServices
public Optional<MobFileCache> getMobFileCache()
getMobFileCache
in interface RegionServerServices
protected ConfigurationManager getConfigurationManager()
public TableDescriptors getTableDescriptors()
getTableDescriptors
in interface RegionServerServices
public void updateConfiguration()
public CacheEvictionStats clearRegionBlockCache(Region region)
public double getCompactionPressure()
getCompactionPressure
in interface RegionServerServices
Store.getCompactionPressure()
public HeapMemoryManager getHeapMemoryManager()
getHeapMemoryManager
in interface RegionServerServices
public boolean walRollRequestFinished()
public ThroughputController getFlushThroughputController()
getFlushThroughputController
in interface RegionServerServices
public double getFlushPressure()
getFlushPressure
in interface RegionServerServices
public void onConfigurationChange(org.apache.hadoop.conf.Configuration newConf)
ConfigurationObserver
ConfigurationManager
object when the Configuration
object is reloaded from disk.onConfigurationChange
in interface ConfigurationObserver
public MetricsRegionServer getMetrics()
getMetrics
in interface RegionServerServices
public SecureBulkLoadManager getSecureBulkLoadManager()
getSecureBulkLoadManager
in interface RegionServerServices
SecureBulkLoadManager
public EntityLock regionLock(List<RegionInfo> regionInfos, String description, Abortable abort) throws IOException
RegionServerServices
regionLock
in interface RegionServerServices
IOException
public void unassign(byte[] regionName) throws IOException
RegionServerServices
See HBASE-17712 for more details.
unassign
in interface RegionServerServices
IOException
public RegionServerSpaceQuotaManager getRegionServerSpaceQuotaManager()
getRegionServerSpaceQuotaManager
in interface RegionServerServices
RegionServerSpaceQuotaManager
public boolean reportFileArchivalForQuotas(TableName tableName, Collection<Map.Entry<String,Long>> archivedFiles)
RegionServerServices
table
were
just moved to the archive directory.reportFileArchivalForQuotas
in interface RegionServerServices
tableName
- The name of the table that files previously belonged toarchivedFiles
- Files and their sizes that were moved to archivetrue
if the files were successfully reported, false
otherwise.public NettyEventLoopGroupConfig getEventLoopGroupConfig()
public Connection createConnection(org.apache.hadoop.conf.Configuration conf) throws IOException
createConnection
in interface Server
IOException
public void executeProcedure(long procId, RSProcedureCallable callable)
public void remoteProcedureComplete(long procId, Throwable error)
void reportProcedureDone(org.apache.hadoop.hbase.shaded.protobuf.generated.RegionServerStatusProtos.ReportProcedureDoneRequest request) throws IOException
IOException
boolean submitRegionProcedure(long procId)
finishRegionProcedure(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.procId
- the id of the open/close region procedurepublic void finishRegionProcedure(long procId)
procId
- the id of the open/close region procedurepublic boolean isShutDown()
public CompactedHFilesDischarger getCompactedHFilesDischarger()
Copyright © 2007–2020 The Apache Software Foundation. All rights reserved.