@InterfaceAudience.LimitedPrivate(value="Tools") public class HRegionServer extends Thread implements RegionServerServices, LastSequenceId, ConfigurationObserver
| Modifier and Type | Class and Description | 
|---|---|
| private static class  | HRegionServer.CompactionCheckerInner class that runs on a long period checking if regions need compaction. | 
| private static class  | HRegionServer.MovedRegionInfo | 
| private static class  | HRegionServer.PeriodicMemStoreFlusher | 
| private static class  | HRegionServer.SystemExitWhenAbortTimeoutForce to terminate region server when abort timeout. | 
Thread.State, Thread.UncaughtExceptionHandlerRegionServerServices.PostOpenDeployContext, RegionServerServices.RegionStateTransitionContext| Modifier and Type | Field and Description | 
|---|---|
| (package private) static String | ABORT_TIMEOUT | 
| (package private) static String | ABORT_TIMEOUT_TASK | 
| private Timer | abortMonitor | 
| private AtomicBoolean | abortRequested | 
| private BlockCache | blockCache | 
| private MemStoreFlusher | cacheFlusher | 
| private ChoreService | choreServiceChoreService used to schedule tasks that we want to run periodically | 
| protected ClusterConnection | clusterConnectionCluster connection to be shared by services. | 
| protected String | clusterIdUnique identifier for the cluster we are a part of. | 
| protected ClusterStatusTracker | clusterStatusTracker | 
| (package private) CompactedHFilesDischarger | compactedFileDischarger | 
| private ScheduledChore | compactionCheckerCheck for compactions requests. | 
| private int | compactionCheckFrequency | 
| CompactSplit | compactSplitThread | 
| protected org.apache.hadoop.conf.Configuration | conf | 
| protected ConfigurationManager | configurationManagerConfiguration 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 | 
| private CoordinatedStateManager | csm | 
| private HFileSystem | dataFs | 
| private boolean | dataFsOk | 
| private org.apache.hadoop.fs.Path | dataRootDir | 
| private static long | DEFAULT_ABORT_TIMEOUT | 
| private NettyEventLoopGroupConfig | eventLoopGroupConfig | 
| private org.apache.hbase.thirdparty.com.google.common.cache.Cache<Long,Long> | executedRegionProceduresUsed to cache the open/close region procedures which already executed. | 
| protected ExecutorService | executorService | 
| private int | flushCheckFrequency | 
| private ThroughputController | flushThroughputController | 
| private FileSystemUtilizationChore | fsUtilizationChore | 
| private HealthCheckChore | healthCheckChoreThe health check chore. | 
| private HeapMemoryManager | hMemManager | 
| protected InfoServer | infoServer | 
| private boolean | killed | 
| private LeaseManager | leaseManager | 
| private org.apache.hadoop.hbase.shaded.protobuf.generated.LockServiceProtos.LockService.BlockingInterface | lockStub | 
| private static org.slf4j.Logger | LOG | 
| private MasterAddressTracker | masterAddressTracker | 
| private boolean | masterlessTrue 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 String | MASTERLESS_CONFIG_NAME | 
| private MetricsRegionServer | metricsRegionServer | 
| (package private) MetricsRegionServerWrapperImpl | metricsRegionServerImpl | 
| private MobFileCache | mobFileCache | 
| private org.apache.hbase.thirdparty.com.google.common.cache.Cache<String,HRegionServer.MovedRegionInfo> | movedRegionInfoCacheUsed to cache the moved-out regions | 
| (package private) int | msgInterval | 
| private NamedQueueRecorder | namedQueueRecorderProvide online slow log responses from ringbuffer | 
| (package private) ServerNonceManager | nonceManagerNonce manager. | 
| private ScheduledChore | nonceManagerChoreThe nonce manager chore. | 
| (package private) AtomicBoolean | online | 
| private Map<String,HRegion> | onlineRegionsMap of regions currently being served by this region server. | 
| private ReentrantReadWriteLock | onlineRegionsLockLock for gating access to  onlineRegions. | 
| private int | operationTimeout | 
| private JvmPauseMonitor | pauseMonitor | 
| private static String | PERIOD_COMPACTION | 
| private static String | PERIOD_FLUSH | 
| private ScheduledChore | periodicFlusherCheck for flushes | 
| private RemoteProcedureResultReporter | procedureResultReporter | 
| private Map<String,InetSocketAddress[]> | regionFavoredNodesMapMap 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 | REGIONSERVERregion server process name | 
| private static String | REGIONSERVER_CODECregionserver codec list | 
| private RegionServerAccounting | regionServerAccounting | 
| private ConcurrentMap<byte[],Boolean> | regionsInTransitionInRSA map from RegionName to current action in progress. | 
| private ReplicationSinkService | replicationSinkHandler | 
| private ReplicationSourceService | replicationSourceHandler | 
| private RpcClient | rpcClient | 
| private RpcControllerFactory | rpcControllerFactory | 
| private RpcRetryingCallerFactory | rpcRetryingCallerFactory | 
| protected RSRpcServices | rpcServices | 
| (package private) static String | RS_HOSTNAME_DISABLE_MASTER_REVERSEDNS_KEYDeprecated. 
 since 2.4.0 and will be removed in 4.0.0.
 Use  UNSAFE_RS_HOSTNAME_DISABLE_MASTER_REVERSEDNS_KEYinstead. | 
| private RegionServerCoprocessorHost | rsHost | 
| private RegionServerProcedureManagerHost | rspmHost | 
| private RegionServerRpcQuotaManager | rsQuotaManager | 
| private RegionServerSpaceQuotaManager | rsSpaceQuotaManager | 
| private org.apache.hadoop.hbase.shaded.protobuf.generated.RegionServerStatusProtos.RegionServerStatusService.BlockingInterface | rssStub | 
| private SecureBulkLoadManager | secureBulkLoadManager | 
| protected ServerName | serverNameThe server name the Master sees us as. | 
| private int | shortOperationTimeout | 
| private boolean | shutDown | 
| protected Sleeper | sleeper | 
| private SlowLogTableOpsChore | slowLogTableOpsChore | 
| private SpanReceiverHost | spanReceiverHost | 
| private SplitLogWorker | splitLogWorker | 
| protected long | startcodeThis servers startcode. | 
| private boolean | stopped | 
| private boolean | stopping | 
| private StorefileRefresherChore | storefileRefresher | 
| private ConcurrentMap<Long,Long> | submittedRegionProceduresUsed to cache the open/close region procedures which already submitted. | 
| protected TableDescriptors | tableDescriptorsGo here to get table descriptors. | 
| static boolean | TEST_SKIP_REPORTING_TRANSITIONFor testing only!  Set to true to skip notifying region assignment to master . | 
| private int | threadWakeFrequency | 
| private static int | TIMEOUT_REGION_MOVEDWe need a timeout. | 
| private Thread.UncaughtExceptionHandler | uncaughtExceptionHandler | 
| (package private) static String | UNSAFE_RS_HOSTNAME_DISABLE_MASTER_REVERSEDNS_KEYHBASE-18226: This config and hbase.unasfe.regionserver.hostname are mutually exclusive. | 
| private UserProvider | userProvider | 
| protected String | useThisHostnameInsteadhostname specified by hostname config | 
| private WALFactory | walFactory | 
| private HFileSystem | walFs | 
| private LogRoller | walRoller | 
| private org.apache.hadoop.fs.Path | walRootDir | 
| protected ZKWatcher | zooKeeper | 
MAX_PRIORITY, MIN_PRIORITY, NORM_PRIORITY| 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. | 
| private void | addToMovedRegions(String encodedName,
                 ServerName destination,
                 long closeSeqNum,
                 boolean selfMove) | 
| 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.RegionSpaceUseReportRequestprotobuf 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 | cacheTableDescriptor() | 
| 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. | 
| (package private) boolean | checkFileSystem()Checks to see if the file system is still accessible. | 
| private Throwable | cleanup(Throwable t,
       String msg)Cleanup after Throwable caught invoking method. | 
| (package private) CacheEvictionStats | clearRegionBlockCache(Region region) | 
| private void | closeMetaTableRegions(boolean abort)Close meta region if we carry it | 
| protected 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. | 
| protected void | configureInfoServer() | 
| (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 | containsMetaTableRegions() | 
| (package private) org.apache.hadoop.hbase.shaded.protobuf.generated.RegionServerStatusProtos.RegionSpaceUse | convertRegionSize(RegionInfo regionInfo,
                 Long sizeInBytes)Converts a pair of  RegionInfoandlonginto aRegionServerStatusProtos.RegionSpaceUseprotobuf message. | 
| private IOException | convertThrowableToIOE(Throwable t,
                     String msg) | 
| protected ClusterConnection | createClusterConnection() | 
| 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,
                            WALFactory walFactory)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) | 
| private 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. | 
| private org.apache.hadoop.hbase.shaded.protobuf.generated.RegionServerStatusProtos.ReportRegionStateTransitionRequest | createReportRegionStateTransitionRequest(RegionServerServices.RegionStateTransitionContext context) | 
| protected RSRpcServices | createRpcServices() | 
| private org.apache.hadoop.hbase.shaded.protobuf.generated.ClusterStatusProtos.UserLoad | createUserLoad(String user,
              MetricsUserSource userSource) | 
| private void | deleteMyEphemeralNode() | 
| (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) | 
| AccessChecker | getAccessChecker() | 
| 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. | 
| (package private) 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 org.apache.hadoop.fs.Path | getDataRootDir() | 
| 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() | 
| HeapMemoryManager | getHeapMemoryManager() | 
| InfoServer | getInfoServer() | 
| org.apache.hadoop.hbase.shaded.protobuf.generated.ClusterStatusProtos.RegionStoreSequenceIds | getLastSequenceId(byte[] encodedRegionName) | 
| LeaseManager | getLeaseManager() | 
| MasterAddressTracker | getMasterAddressTracker() | 
| (package private) MemStoreFlusher | getMemStoreFlusher() | 
| MetricsRegionServer | getMetrics() | 
| Optional<MobFileCache> | getMobFileCache()May be null if this is a master which not carry table. | 
| HRegionServer.MovedRegionInfo | getMovedRegion(String encodedRegionName) | 
| private String | getMyEphemeralNodePath() | 
| NamedQueueRecorder | getNamedQueueRecorder()get NamedQueue Provider to add different logs to ringbuffer | 
| ServerNonceManager | getNonceManager()Only required for "old" log replay; if it's removed, remove this. | 
| int | getNumberOfOnlineRegions() | 
| HRegion | getOnlineRegion(byte[] regionName) | 
| protected Map<String,HRegion> | getOnlineRegions() | 
| 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  Regioninstance. | 
| private 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() | 
| 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() | 
| RpcServerInterface | getRpcServer()Returns a reference to the region server's RPC server | 
| RSRpcServices | getRSRpcServices() | 
| SecureBulkLoadManager | getSecureBulkLoadManager() | 
| ServerName | getServerName() | 
| long | getStartcode() | 
| TableDescriptors | getTableDescriptors() | 
| protected String | getUseThisHostnameInstead(org.apache.hadoop.conf.Configuration conf) | 
| WAL | getWAL(RegionInfo regionInfo) | 
| (package private) WALFactory | getWalFactory() | 
| org.apache.hadoop.fs.FileSystem | getWALFileSystem() | 
| Map<String,ReplicationStatus> | getWalGroupsReplicationStatus() | 
| LogRoller | getWalRoller() | 
| org.apache.hadoop.fs.Path | getWALRootDir() | 
| List<WAL> | getWALs() | 
| private long | getWriteRequestCount() | 
| ZKPermissionWatcher | getZKPermissionWatcher() | 
| 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. | 
| private void | initNamedQueueRecorder(org.apache.hadoop.conf.Configuration conf) | 
| boolean | isAborted()Check if the server or client was aborted. | 
| boolean | isClusterUp() | 
| (package private) boolean | isDataFileSystemOk() | 
| private boolean | isHealthCheckerConfigured() | 
| private boolean | isHealthy() | 
| boolean | isOnline()Report the status of the server. | 
| boolean | isShutDown() | 
| boolean | isStopped() | 
| boolean | isStopping() | 
| private boolean | keepLooping() | 
| protected void | kill() | 
| protected void | login(UserProvider user,
     String host) | 
| static void | main(String[] args) | 
| int | movedRegionCacheExpiredTime() | 
| 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,
                      WALFactory walFactory) | 
| void | onConfigurationChange(org.apache.hadoop.conf.Configuration newConf)This method would be called by the  ConfigurationManagerobject when theConfigurationobject 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 void | 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  Servicesubclass 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  tablewere
 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. | 
| private void | scheduleAbortTimer() | 
| protected boolean | setAbortRequested()Sets the abort state if not already set. | 
| 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 | skipReportingTransition(RegionServerServices.RegionStateTransitionContext context)Helper method for use in tests. | 
| private static boolean | sleepInterrupted(long millis) | 
| private void | startHeapMemoryManager() | 
| private void | startReplicationService()Start up replication source and sink handlers. | 
| private void | startServices()Start maintenance Threads, Server, Worker and lease checker threads. | 
| 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() | 
| 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 | 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 | 
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, yieldprivate static final org.slf4j.Logger LOG
@InterfaceAudience.Private public static boolean TEST_SKIP_REPORTING_TRANSITION
private 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).private final org.apache.hbase.thirdparty.com.google.common.cache.Cache<String,HRegionServer.MovedRegionInfo> movedRegionInfoCache
private MemStoreFlusher cacheFlusher
private HeapMemoryManager hMemManager
protected ClusterConnection clusterConnection
protected TableDescriptors tableDescriptors
private ReplicationSourceService replicationSourceHandler
private ReplicationSinkService replicationSinkHandler
public CompactSplit compactSplitThread
private final Map<String,HRegion> onlineRegions
private final ReentrantReadWriteLock onlineRegionsLock
onlineRegions.
 TODO: If this map is gated by a lock, does it need to be a ConcurrentHashMap?private final Map<String,InetSocketAddress[]> regionFavoredNodesMap
private LeaseManager leaseManager
protected ExecutorService executorService
private volatile boolean dataFsOk
private HFileSystem dataFs
private HFileSystem walFs
private volatile boolean stopped
private AtomicBoolean abortRequested
static final String ABORT_TIMEOUT
private static final long DEFAULT_ABORT_TIMEOUT
static final String ABORT_TIMEOUT_TASK
private boolean stopping
private volatile boolean killed
private volatile boolean shutDown
protected final org.apache.hadoop.conf.Configuration conf
private org.apache.hadoop.fs.Path dataRootDir
private org.apache.hadoop.fs.Path walRootDir
private final int threadWakeFrequency
final int msgInterval
private static final String PERIOD_COMPACTION
private final int compactionCheckFrequency
private static final String PERIOD_FLUSH
private final int flushCheckFrequency
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
private MetricsRegionServer metricsRegionServer
MetricsRegionServerWrapperImpl metricsRegionServerImpl
private SpanReceiverHost spanReceiverHost
private ChoreService choreService
private ScheduledChore compactionChecker
private ScheduledChore periodicFlusher
private 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 SlowLogTableOpsChore slowLogTableOpsChore
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
@Deprecated @InterfaceAudience.LimitedPrivate(value="Configuration") static final String RS_HOSTNAME_DISABLE_MASTER_REVERSEDNS_KEY
UNSAFE_RS_HOSTNAME_DISABLE_MASTER_REVERSEDNS_KEY instead.@InterfaceAudience.LimitedPrivate(value="Configuration") static final String UNSAFE_RS_HOSTNAME_DISABLE_MASTER_REVERSEDNS_KEY
protected final long startcode
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
private CoordinatedStateManager csm
protected final ConfigurationManager configurationManager
@InterfaceAudience.Private CompactedHFilesDischarger compactedFileDischarger
private volatile ThroughputController flushThroughputController
private SecureBulkLoadManager secureBulkLoadManager
private FileSystemUtilizationChore fsUtilizationChore
private final NettyEventLoopGroupConfig eventLoopGroupConfig
private NamedQueueRecorder namedQueueRecorder
private final boolean masterless
TEST_SKIP_REPORTING_TRANSITION ?private static final String MASTERLESS_CONFIG_NAME
private static final String REGIONSERVER_CODEC
private Timer abortMonitor
private static final int TIMEOUT_REGION_MOVED
public HRegionServer(org.apache.hadoop.conf.Configuration conf) throws IOException
startServices().IOExceptionprivate void initNamedQueueRecorder(org.apache.hadoop.conf.Configuration conf)
protected String getUseThisHostnameInstead(org.apache.hadoop.conf.Configuration conf) throws IOException
IOExceptionprivate 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
IOExceptionprotected void login(UserProvider user, String host) throws IOException
IOExceptionprotected void waitForMasterActive()
protected String getProcessName()
protected boolean canCreateBaseZNode()
protected boolean canUpdateTableDescriptor()
protected boolean cacheTableDescriptor()
protected RSRpcServices createRpcServices() throws IOException
IOExceptionprotected void configureInfoServer()
protected Class<? extends javax.servlet.http.HttpServlet> getDumpServlet()
public boolean registerService(com.google.protobuf.Service instance)
RegionServerServicesService subclass as a coprocessor endpoint to be
 available for handlingregisterService in interface RegionServerServicesinstance - the Service subclass instance to expose as a coprocessor endpointtrue if the registration was successful, falseprotected ClusterConnection createClusterConnection() throws IOException
IOExceptionprivate static void checkCodecs(org.apache.hadoop.conf.Configuration c) throws IOException
c - IOExceptionpublic String getClusterId()
protected void setupClusterConnection() throws IOException
IOExceptionprivate void preRegistrationInitialization()
private void initializeZooKeeper() throws IOException, InterruptedException
Finally open long-living server short-circuit connection.
IOExceptionInterruptedExceptionprivate void blockAndCheckIfStopped(ZKNodeTracker tracker) throws IOException, InterruptedException
tracker - znode tracker to useIOException - any IO exception, plus if the RS is stoppedInterruptedException - if the waiting thread is interruptedpublic boolean isClusterUp()
isClusterUp in interface RegionServerServicespublic void run()
private boolean containsMetaTableRegions()
private boolean areAllUserRegionsOffline()
private long getWriteRequestCount()
@InterfaceAudience.Private protected void tryRegionServerReport(long reportStartTime, long reportEndTime) throws IOException
IOExceptionpublic boolean reportRegionSizesForQuotas(RegionSizeStore regionSizeStore)
reportRegionSizesForQuotas in interface RegionServerServicesregionSizeStore - The store containing region sizesprivate void 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.ServiceExceptionorg.apache.hadoop.hbase.shaded.protobuf.generated.RegionServerStatusProtos.RegionSpaceUseReportRequest buildRegionSpaceUseReportRequest(RegionSizeStore regionSizes)
RegionServerStatusProtos.RegionSpaceUseReportRequest protobuf message from the region size map.regionSizes - 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 Regionprivate org.apache.hadoop.hbase.shaded.protobuf.generated.ClusterStatusProtos.ServerLoad buildServerLoad(long reportStartTime, long reportEndTime) throws IOException
IOExceptionprivate String getOnlineRegionsAsPrintableString()
private void waitOnAllRegionsToClose(boolean abort)
private static boolean sleepInterrupted(long millis)
private void shutdownWAL(boolean close)
public NamedQueueRecorder getNamedQueueRecorder()
protected void handleReportForDutyResponse(org.apache.hadoop.hbase.shaded.protobuf.generated.RegionServerStatusProtos.RegionServerStartupResponse c) throws IOException
IOExceptionprotected void initializeMemStoreChunkCreator()
private void startHeapMemoryManager()
private void createMyEphemeralNode() throws org.apache.zookeeper.KeeperException
org.apache.zookeeper.KeeperExceptionprivate void deleteMyEphemeralNode() throws org.apache.zookeeper.KeeperException
org.apache.zookeeper.KeeperExceptionpublic RegionServerAccounting getRegionServerAccounting()
getRegionServerAccounting in interface RegionServerServicesorg.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
r - Region to get RegionLoad for.regionLoadBldr - the RegionLoad.Builder, can be nullregionSpecifier - the RegionSpecifier.Builder, can be nullIOExceptionprivate org.apache.hadoop.hbase.shaded.protobuf.generated.ClusterStatusProtos.UserLoad createUserLoad(String user, MetricsUserSource userSource)
public org.apache.hadoop.hbase.shaded.protobuf.generated.ClusterStatusProtos.RegionLoad createRegionLoad(String encodedRegionName) throws IOException
IOExceptionpublic boolean isOnline()
private void setupWALAndReplication() throws IOException
IOExceptionprivate void startReplicationService() throws IOException
IOExceptionpublic MasterAddressTracker getMasterAddressTracker()
private void startServices() throws IOException
IOExceptionprivate void initializeThreads()
private void registerConfigurationObservers()
private void putUpWebUI() throws IOException
IOExceptionprivate boolean isHealthy()
public List<WAL> getWALs()
getWALs in interface RegionServerServicespublic WAL getWAL(RegionInfo regionInfo) throws IOException
getWAL in interface RegionServerServicesIOExceptionpublic LogRoller getWalRoller()
WALFactory getWalFactory()
public Connection getConnection()
ServergetConnection in interface Serverpublic ClusterConnection getClusterConnection()
ServerServer.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 Serverpublic void stop(String msg)
Stoppablepublic 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
RegionServerServicespostOpenDeployTasks in interface RegionServerServicescontext - the contextIOExceptionprivate boolean skipReportingTransition(RegionServerServices.RegionStateTransitionContext context)
private org.apache.hadoop.hbase.shaded.protobuf.generated.RegionServerStatusProtos.ReportRegionStateTransitionRequest createReportRegionStateTransitionRequest(RegionServerServices.RegionStateTransitionContext context)
public boolean reportRegionStateTransition(RegionServerServices.RegionStateTransitionContext context)
RegionServerServicesreportRegionStateTransition in interface RegionServerServicesprivate void triggerFlushInPrimaryRegion(HRegion region)
public RpcServerInterface getRpcServer()
RegionServerServicesgetRpcServer in interface RegionServerServices@InterfaceAudience.Private public RSRpcServices getRSRpcServices()
public void abort(String reason, Throwable cause)
protected boolean setAbortRequested()
public void abort(String reason)
abort(String, Throwable)public boolean isAborted()
Abortable@InterfaceAudience.Private protected void kill()
private void scheduleAbortTimer()
protected void stopServiceThreads()
@InterfaceAudience.Private public ReplicationSourceService getReplicationSourceService()
ReplicationSinkService getReplicationSinkService()
private ServerName createRegionServerStatusStub()
@InterfaceAudience.Private 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
IOExceptionpublic org.apache.hadoop.hbase.shaded.protobuf.generated.ClusterStatusProtos.RegionStoreSequenceIds getLastSequenceId(byte[] encodedRegionName)
getLastSequenceId in interface LastSequenceIdencodedRegionName - Encoded region nameprivate void closeMetaTableRegions(boolean abort)
abort - Whether we're running an abort.private void closeUserRegions(boolean abort)
abort - Whether we're running an abort.public InfoServer getInfoServer()
public boolean isStopped()
public boolean isStopping()
isStopping in interface Serverpublic org.apache.hadoop.conf.Configuration getConfiguration()
ServergetConfiguration in interface Serverprotected Map<String,HRegion> getOnlineRegions()
public int getNumberOfOnlineRegions()
public Collection<HRegion> getOnlineRegionsLocalContext()
public void addRegion(HRegion region)
MutableOnlineRegionsaddRegion in interface MutableOnlineRegionsSortedMap<Long,HRegion> getCopyOfOnlineRegionsSortedByOffHeapSize()
SortedMap<Long,HRegion> getCopyOfOnlineRegionsSortedByOnHeapSize()
public long getStartcode()
public FlushRequester getFlushRequester()
getFlushRequester in interface RegionServerServicespublic CompactionRequester getCompactionRequestor()
getCompactionRequestor in interface RegionServerServicesCompactionRequester or null. Usually it will not be null
         unless during intialization.public LeaseManager getLeaseManager()
getLeaseManager in interface RegionServerServicesprotected org.apache.hadoop.fs.Path getDataRootDir()
public org.apache.hadoop.fs.FileSystem getFileSystem()
getFileSystem in interface Serverboolean isDataFileSystemOk()
true when the data file system is available, false otherwise.public org.apache.hadoop.fs.Path getWALRootDir()
public org.apache.hadoop.fs.FileSystem getWALFileSystem()
public ZKWatcher getZooKeeper()
ServergetZooKeeper in interface Serverpublic CoordinatedStateManager getCoordinatedStateManager()
ServergetCoordinatedStateManager in interface Serverpublic ServerName getServerName()
getServerName in interface Serverpublic RegionServerCoprocessorHost getRegionServerCoprocessorHost()
public ConcurrentMap<byte[],Boolean> getRegionsInTransitionInRS()
RegionServerServicesgetRegionsInTransitionInRS in interface RegionServerServicespublic ExecutorService getExecutorService()
getExecutorService in interface RegionServerServicespublic ChoreService getChoreService()
getChoreService in interface ServerChoreService instance for this serverpublic RegionServerRpcQuotaManager getRegionServerRpcQuotaManager()
getRegionServerRpcQuotaManager in interface RegionServerServicesRegionServerRpcQuotaManagerprivate 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
IOExceptionprivate 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
IOExceptionpublic Map<String,ReplicationStatus> getWalGroupsReplicationStatus()
static HRegionServer constructRegionServer(Class<? extends HRegionServer> regionServerClass, org.apache.hadoop.conf.Configuration conf)
public static void main(String[] args)
HRegionServerCommandLinepublic 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 OnlineRegionstableName - table to limit the scope of the querytableNamepublic List<HRegion> getRegions()
OnlineRegionsgetRegions in interface OnlineRegionspublic Set<TableName> getOnlineTables()
public String[] getRegionServerCoprocessors()
private void closeRegionIgnoreErrors(RegionInfo region, boolean abort)
region - Region to closeprotected boolean closeRegion(String encodedName, boolean abort, ServerName destination) 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 abortingdestination - Where the Region is being moved too... maybe null if unknown.NotServingRegionException - if the region is not onlinepublic HRegion getOnlineRegion(byte[] regionName)
regionName or null if
         named region is not member of the online regions.public HRegion getRegion(String encodedRegionName)
OnlineRegionsRegion instance.
 Only works if caller is in same context, in same JVM. Region is not
 serializable.getRegion in interface OnlineRegionsencodedRegionName or
 null if named region is not member of the online regions.public boolean removeRegion(HRegion r, ServerName destination)
MutableOnlineRegionsremoveRegion in interface MutableOnlineRegionsr - Region to remove.destination - Destination, if any, null otherwise.protected HRegion getRegion(byte[] regionName) throws NotServingRegionException
regionName - Name of online HRegion to returnHRegion for regionNameNotServingRegionExceptionpublic HRegion getRegionByEncodedName(String encodedRegionName) throws NotServingRegionException
NotServingRegionExceptionprivate HRegion getRegionByEncodedName(byte[] regionName, String encodedRegionName) throws NotServingRegionException
NotServingRegionExceptionprivate Throwable cleanup(Throwable t, String msg)
t to
 IOE if it isn't already.t - Throwablemsg - Message to log in error. Can be null.private IOException convertThrowableToIOE(Throwable t, String msg)
msg - Message to put in new IOE if passed t is not an IOEt an IOE if it isn't already.boolean checkFileSystem()
public void updateRegionFavoredNodesMapping(String encodedRegionName, List<org.apache.hadoop.hbase.shaded.protobuf.generated.HBaseProtos.ServerName> favoredNodes)
FavoredNodesForRegionupdateRegionFavoredNodesMapping in interface FavoredNodesForRegionpublic InetSocketAddress[] getFavoredNodesForRegion(String encodedRegionName)
regionFavoredNodesMap on why it is InetSocketAddress[]getFavoredNodesForRegion in interface FavoredNodesForRegionpublic ServerNonceManager getNonceManager()
RegionServerServicesgetNonceManager in interface RegionServerServicesprivate void addToMovedRegions(String encodedName, ServerName destination, long closeSeqNum, boolean selfMove)
void removeFromMovedRegions(String encodedName)
@InterfaceAudience.Private public HRegionServer.MovedRegionInfo getMovedRegion(String encodedRegionName)
@InterfaceAudience.Private public int movedRegionCacheExpiredTime()
private String getMyEphemeralNodePath()
private boolean isHealthCheckerConfigured()
public CompactSplit getCompactSplitThread()
CompactSplit for the serversorg.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.ServiceExceptionpublic Optional<BlockCache> getBlockCache()
getBlockCache in interface RegionServerServicespublic Optional<MobFileCache> getMobFileCache()
getMobFileCache in interface RegionServerServicespublic AccessChecker getAccessChecker()
getAccessChecker in interface RegionServerServicesAccessCheckerpublic ZKPermissionWatcher getZKPermissionWatcher()
getZKPermissionWatcher in interface RegionServerServicesZKPermissionWatcher@InterfaceAudience.Private ConfigurationManager getConfigurationManager()
public TableDescriptors getTableDescriptors()
getTableDescriptors in interface RegionServerServicesvoid updateConfiguration()
CacheEvictionStats clearRegionBlockCache(Region region)
public double getCompactionPressure()
getCompactionPressure in interface RegionServerServicesStore.getCompactionPressure()public HeapMemoryManager getHeapMemoryManager()
getHeapMemoryManager in interface RegionServerServicesMemStoreFlusher getMemStoreFlusher()
@InterfaceAudience.Private public boolean walRollRequestFinished()
public ThroughputController getFlushThroughputController()
getFlushThroughputController in interface RegionServerServicespublic double getFlushPressure()
getFlushPressure in interface RegionServerServicespublic void onConfigurationChange(org.apache.hadoop.conf.Configuration newConf)
ConfigurationObserverConfigurationManager
 object when the Configuration object is reloaded from disk.onConfigurationChange in interface ConfigurationObserverpublic MetricsRegionServer getMetrics()
getMetrics in interface RegionServerServicespublic SecureBulkLoadManager getSecureBulkLoadManager()
getSecureBulkLoadManager in interface RegionServerServicesSecureBulkLoadManagerpublic EntityLock regionLock(List<RegionInfo> regionInfos, String description, Abortable abort)
RegionServerServicesregionLock in interface RegionServerServicespublic void unassign(byte[] regionName) throws IOException
RegionServerServicesSee HBASE-17712 for more details.
unassign in interface RegionServerServicesIOExceptionpublic RegionServerSpaceQuotaManager getRegionServerSpaceQuotaManager()
getRegionServerSpaceQuotaManager in interface RegionServerServicesRegionServerSpaceQuotaManagerpublic boolean reportFileArchivalForQuotas(TableName tableName, Collection<Map.Entry<String,Long>> archivedFiles)
RegionServerServicestable were
 just moved to the archive directory.reportFileArchivalForQuotas in interface RegionServerServicestableName - 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 ServerIOExceptionvoid 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
IOExceptionboolean 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()
@InterfaceAudience.Private public CompactedHFilesDischarger getCompactedHFilesDischarger()
Copyright © 2007–2021 The Apache Software Foundation. All rights reserved.