Class HRegion
- All Implemented Interfaces:
ConfigurationObserver
,PropagatingConfigurationObserver
,HeapSize
,Region
An Region is defined by its table and its key extent.
Locking at the Region level serves only one purpose: preventing the region from being closed (and consequently split) while other operations are ongoing. Each row level operation obtains both a row lock and a region read lock for the duration of the operation. While a scanner is being constructed, getScanner holds a read lock. If the scanner is successfully constructed, it holds a read lock until it is closed. A close takes out a write lock and consequently will block for ongoing operations and will block new operations from starting while the close is in progress.
-
Nested Class Summary
Modifier and TypeClassDescriptionprivate static class
Class that tracks the progress of a batch operations, accumulating status codes and tracking the index at which processing is proceeding.static interface
Listener class to enable callers of bulkLoadHFile() to perform any necessary pre/post processing of a given bulkload callstatic interface
static class
Objects from this class are created when flushing to describe all the different states that that method ends up in.private static class
Batch of mutation operations.(package private) static class
A class that tracks exceptions that have been observed in one batch.(package private) static class
A result object from prepare flush cache stageprivate static final class
Batch of mutations for replay.(package private) class
static class
Class used to represent a lock on a row.(package private) static class
Nested classes/interfaces inherited from interface org.apache.hadoop.hbase.regionserver.Region
Region.Operation, Region.RowLock
-
Field Summary
Modifier and TypeFieldDescriptionprivate final org.apache.hadoop.conf.Configuration
private BlockCache
private final LongAdder
private long
(package private) final long
private final CellComparator
(package private) final LongAdder
(package private) final LongAdder
static final String
static final String
static final String
(package private) final AtomicBoolean
private final Object
(package private) final AtomicBoolean
static final String
Parameter name for compaction after bulkload(package private) final LongAdder
(package private) final LongAdder
(package private) final LongAdder
(package private) final LongAdder
(package private) final LongAdder
protected final org.apache.hadoop.conf.Configuration
private ConfigurationManager
private RegionCoprocessorHost
(package private) final LongAdder
static final long
(package private) static final long
static final int
Default interval for the memstore flushstatic final boolean
static final long
static final long
static final boolean
static final long
static final int
static final int
static final String
(package private) static final long
(package private) static final int
static final boolean
static final boolean
static final String
Conf key for fair locking policy(package private) final LongAdder
static final long
private long
(package private) final LongAdder
private long
private FlushPolicy
private final HRegionFileSystem
static final String
static final String
private TableDescriptor
private boolean
The default setting for whether to enable on-demand CF loading for scan requests to this region.private boolean
private long
Record the sequence id of last flush operation.protected long
protected long
The sequence id of the last replayed open region event from the primary region.private final ConcurrentMap<HStore,
Long> static final String
(package private) final ReentrantReadWriteLock
private final ConcurrentHashMap<HashedBytes,
HRegion.RowLockContext> private static final org.slf4j.Logger
private final AtomicInteger
static final long
The following MAX_FLUSH_PER_CHANGES is large enough because each KeyValue has 20+ bytes overhead.(package private) final long
(package private) final int
(package private) final long
private long
The max sequence id of flushed data on this region.static final String
Conf key to force a flush if there are already enough changes for one region in memstorestatic final String
Conf key for the periodic flush interval(package private) long
private final MemStoreSizing
private final MetricsRegion
private final MetricsRegionWrapperImpl
private MetricsTableRequests
private final int
private final int
private final AtomicInteger
private MobFileCache
private final MultiVersionConcurrencyControl
(package private) final LongAdder
private long
The sequence ID that was enLongAddered when this region was opened.private HRegion.PrepareFlushResult
Saved state from replaying prepare flush cache(package private) final LongAdder
static final String
Mainly used for master local region, where we will replay the WAL file directly without splitting, so it is possible to have WAL files which are not closed cleanly, in this way, hitting EOF is expected so should not consider it as a critical problem.static final String
private final Durability
(package private) final ConcurrentHashMap<Thread,
Boolean> (package private) RegionServicesForStores
private final boolean
private org.apache.hadoop.fs.Path
private final NavigableMap<byte[],
Integer> private final int
(package private) final ExecutorService
(package private) final long
private RegionServerAccounting
protected RegionServerServices
(package private) final ConcurrentHashMap<RegionScanner,
Long> (package private) final ReadPointCalculationLock
static final String
This is for for using HRegion as a local storage, where we may put the recovered edits in a special place.static final String
Config for allow split when file count greater than the configured blocking file countprivate RegionSplitPolicy
private RegionSplitRestriction
private final StoreHotnessProtector
static final int
Default interval for System tables memstore flushprivate Long
(package private) final long
private final ReentrantReadWriteLock
static final String
Whether to useMetaCellComparator
even if we are not meta region.private final WAL
static final String
private org.apache.hadoop.fs.FileSystem
(package private) final LongAdder
(package private) final HRegion.WriteState
-
Constructor Summary
ConstructorDescriptionHRegion
(org.apache.hadoop.fs.Path tableDir, WAL wal, org.apache.hadoop.fs.FileSystem fs, org.apache.hadoop.conf.Configuration confParam, RegionInfo regionInfo, TableDescriptor htd, RegionServerServices rsServices) Deprecated.Use other constructors.HRegion
(HRegionFileSystem fs, WAL wal, org.apache.hadoop.conf.Configuration confParam, TableDescriptor htd, RegionServerServices rsServices) HRegion constructor. -
Method Summary
Modifier and TypeMethodDescriptionvoid
addReadRequestsCount
(long readRequestsCount) void
addRegionToSnapshot
(org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDescription desc, ForeignExceptionSnare exnSnare) Complete taking the snapshot on the region.void
addWriteRequestsCount
(long writeRequestsCount) Perform one or more append operations on a row.private void
applyToMemStore
(HStore store, List<Cell> cells, boolean delta, MemStoreSizing memstoreAccounting) private void
applyToMemStore
(HStore store, Cell cell, MemStoreSizing memstoreAccounting) boolean
batchMutate
(Mutation[] mutations) Perform a batch of mutations.(package private) OperationStatus[]
batchMutate
(Mutation[] mutations, boolean atomic) batchMutate
(Mutation[] mutations, boolean atomic, long nonceGroup, long nonce) private OperationStatus[]
batchMutate
(HRegion.BatchOperation<?> batchOp) Perform a batch of mutations.batchReplay
(WALSplitUtil.MutationReplay[] mutations, long replaySeqId) void
bulkLoadHFiles
(Collection<Pair<byte[], String>> familyPaths, boolean assignSeqId, HRegion.BulkLoadListener bulkLoadListener) Attempts to atomically load a group of hfiles.bulkLoadHFiles
(Collection<Pair<byte[], String>> familyPaths, boolean assignSeqId, HRegion.BulkLoadListener bulkLoadListener, boolean copyFile, List<String> clusterIds, boolean replicate) Attempts to atomically load a group of hfiles.boolean
checkAndMutate
(byte[] row, byte[] family, byte[] qualifier, CompareOperator op, ByteArrayComparable comparator, TimeRange timeRange, Mutation mutation) Deprecated.boolean
checkAndMutate
(byte[] row, Filter filter, TimeRange timeRange, Mutation mutation) Deprecated.checkAndMutate
(CheckAndMutate checkAndMutate) Atomically checks if a row matches the conditions and if it does, it performs the actions.checkAndMutate
(CheckAndMutate checkAndMutate, long nonceGroup, long nonce) private CheckAndMutateResult
checkAndMutateInternal
(CheckAndMutate checkAndMutate, long nonceGroup, long nonce) boolean
checkAndRowMutate
(byte[] row, byte[] family, byte[] qualifier, CompareOperator op, ByteArrayComparable comparator, TimeRange timeRange, RowMutations rm) Deprecated.boolean
checkAndRowMutate
(byte[] row, Filter filter, TimeRange timeRange, RowMutations rm) Deprecated.void
checkFamilies
(Collection<byte[]> families) Check the collection of families for validity.(package private) void
checkFamily
(byte[] family) (package private) void
Check thread interrupt status and throw an exception if interrupted.private void
checkMutationType
(Mutation mutation) private void
checkNegativeMemStoreDataSize
(long memStoreDataSize, long delta) private void
private void
private void
Check if resources to support an update.(package private) void
Make sure this is a valid row for the HRegionprivate void
Optional<byte[]>
Optional<byte[]>
checkSplit
(boolean force) Return the split point.private void
checkTargetRegion
(byte[] encodedRegionName, String exceptionMsg, Object payload) Checks whether the given regionName is either equal to our region, or that the regionName is the primary region to our corresponding range for the secondary replica.void
checkTimestamps
(Map<byte[], List<Cell>> familyMap, long now) Check the collection of families for valid timestampsMap<byte[],
List<HStoreFile>> close()
Close down this HRegion.Map<byte[],
List<HStoreFile>> close
(boolean abort) Map<byte[],
List<HStoreFile>> close
(boolean abort, boolean ignoreStatus) Close this HRegion.Map<byte[],
List<HStoreFile>> close
(boolean abort, boolean ignoreStatus, boolean isGracefulStop) Close down this HRegion.private void
Closes the lock.void
Closes the region operation lock.void
closeRegionOperation
(Region.Operation operation) Closes the region operation lock.void
compact
(boolean majorCompaction) Synchronously compact all stores in the region.boolean
compact
(CompactionContext compaction, HStore store, ThroughputController throughputController) Called by compaction thread and after region is opened to compact the HStores if necessary.boolean
compact
(CompactionContext compaction, HStore store, ThroughputController throughputController, User user) (package private) void
compactStore
(byte[] family, ThroughputController throughputController) This is a helper function that compact the given store.void
This is a helper function that compact all the stores synchronously.static HDFSBlocksDistribution
computeHDFSBlocksDistribution
(org.apache.hadoop.conf.Configuration conf, TableDescriptor tableDescriptor, RegionInfo regionInfo) This is a helper function to compute HDFS block distribution on demandstatic HDFSBlocksDistribution
computeHDFSBlocksDistribution
(org.apache.hadoop.conf.Configuration conf, TableDescriptor tableDescriptor, RegionInfo regionInfo, org.apache.hadoop.fs.Path tablePath) This is a helper function to compute HDFS block distribution on demandstatic HRegion
createHRegion
(org.apache.hadoop.conf.Configuration conf, RegionInfo regionInfo, org.apache.hadoop.fs.FileSystem fs, org.apache.hadoop.fs.Path tableDir, TableDescriptor tableDesc) Create a region under the given table directory.static HRegion
createHRegion
(RegionInfo info, org.apache.hadoop.fs.Path rootDir, org.apache.hadoop.conf.Configuration conf, TableDescriptor hTableDescriptor, WAL wal) static HRegion
createHRegion
(RegionInfo info, org.apache.hadoop.fs.Path rootDir, org.apache.hadoop.conf.Configuration conf, TableDescriptor hTableDescriptor, WAL wal, boolean initialize) Convenience method creating new HRegions.static HRegion
createHRegion
(RegionInfo info, org.apache.hadoop.fs.Path rootDir, org.apache.hadoop.conf.Configuration conf, TableDescriptor hTableDescriptor, WAL wal, boolean initialize, RegionServerServices rsRpcServices) Convenience method creating new HRegions.static HRegionFileSystem
createRegionDir
(org.apache.hadoop.conf.Configuration configuration, RegionInfo ri, org.apache.hadoop.fs.Path rootDir) Create the region directory in the filesystem.(package private) io.opentelemetry.api.trace.Span
createRegionSpan
(String name) private static void
decorateRegionConfiguration
(org.apache.hadoop.conf.Configuration conf) This method modifies the region's configuration in order to inject replication-related featuresvoid
protected void
private void
decrMemStoreSize
(long dataSizeDelta, long heapSizeDelta, long offHeapSizeDelta, int cellsCountDelta) (package private) void
void
Deletes the specified cells/row.private void
deleteRecoveredEdits
(org.apache.hadoop.fs.FileSystem fs, Iterable<org.apache.hadoop.fs.Path> files) void
deregisterChildren
(ConfigurationManager manager) Needs to be called to deregister the children from the manager.(package private) void
If a handler thread is eligible for interrupt, make it ineligible.private void
doAbortFlushToWAL
(WAL wal, long flushOpSeqId, Map<byte[], List<org.apache.hadoop.fs.Path>> committedFiles) private Map<byte[],
List<HStoreFile>> doClose
(boolean abort, MonitoredTask status) private MemStoreSize
doDropStoreMemStoreContentsForSeqId
(HStore s, long currentSeqId) private void
doMiniBatchMutate
(HRegion.BatchOperation<?> batchOp) Called to do a piece of the batch that came in tobatchMutate(Mutation[])
In here we also handle replay of edits on region recover.private void
doProcessRowWithTimeout
(RowProcessor<?, ?> processor, long now, HRegion region, List<Mutation> mutations, WALEdit walEdit, long timeout) protected void
Do preparation for pending compaction.private static void
doSyncOfUnflushedWALChanges
(WAL wal, RegionInfo hri) Sync unflushed WAL changes.doWALAppend
(WALEdit walEdit, Durability durability, List<UUID> clusterIds, long now, long nonceGroup, long nonce) doWALAppend
(WALEdit walEdit, Durability durability, List<UUID> clusterIds, long now, long nonceGroup, long nonce, long origLogSeqNum) Returns writeEntry associated with this appendprivate MemStoreSize
Be careful, this method will drop all data in the memstore of this region.private MemStoreSize
dropMemStoreContentsForSeqId
(long seqId, HStore store) Drops the memstore contents after replaying a flush descriptor or region open event replay if the memstore edits have seqNums smaller than the given seq idprivate void
If all stores ended up dropping their snapshots, we can safely drop the prepareFlushResult(package private) void
If a handler thread was made ineligible for interrupt via {disableInterrupts()
, make it eligible again.boolean
com.google.protobuf.Message
execService
(com.google.protobuf.RpcController controller, org.apache.hadoop.hbase.shaded.protobuf.generated.ClientProtos.CoprocessorServiceCall call) Executes a single protocol buffer coprocessor endpointService
method using the registered protocol handlers.flush
(boolean flushAllStores) Flush the cache.(package private) HRegion.FlushResultImpl
flushcache
(boolean flushAllStores, boolean writeFlushRequestWalMarker, FlushLifeCycleTracker tracker) flushcache
(List<byte[]> families, boolean writeFlushRequestWalMarker, FlushLifeCycleTracker tracker) Flush the cache.Do a get based on the get parameter.Do a get based on the get parameter.(package private) org.apache.hadoop.conf.Configuration
A split takes the config from the parent region & passes it to the daughter region's constructor.long
Returns the number of blocked requestsThe comparator to be used with the regionlong
Returns the number of failed checkAndMutate guardslong
Returns the number of checkAndMutate guards that passedReturns if a given region is in compaction now.int
Returns The priority that this region should have in the compaction queueReturns the coprocessor hostlong
Returns the size of data processed bypassing the WAL, in byteslong
private Durability
Returns effective durability from the passed durability and the table descriptor.org.apache.hadoop.fs.FileSystem
ReturnsFileSystem
being used by this regionlong
Returns filtered read requests count for this regiongetInternal
(Get get, boolean withCoprocessor, long nonceGroup, long nonce) org.apache.hadoop.hbase.shaded.protobuf.generated.ClientProtos.RegionLoadStats
Returns statistics about the current load of the regionlong
long
long
long
long
int
The minimum block size configuration from all relevant column families.getMVCC()
protected long
getNextSequenceId
(WAL wal) Method to safely get the next sequence number.long
Returns the number of mutations processed bypassing the WALlong
getOldestHfileTs
(boolean majorCompactionOnly) This can be used to determine the last time all files of this region were major compacted.long
getOldestSeqIdOfStore
(byte[] familyName) private static ThreadPoolExecutor
getOpenAndCloseThreadPool
(int maxThreads, String threadNamePrefix) long
Returns the latest sequence number that was read from storage when this region was opened(package private) HRegion.PrepareFlushResult
int
org.apache.hadoop.conf.Configuration
long
getReadPoint
(IsolationLevel isolationLevel) Returns readpoint considering given IsolationLevel.long
Returns read requests count for this regionstatic org.apache.hadoop.fs.Path
getRegionDir
(org.apache.hadoop.fs.Path tabledir, String name) Deprecated.For tests only; to be removed.Returns theHRegionFileSystem
used by this regionReturns region information for this region(package private) RegionServerServices
Returns Instance ofRegionServerServices
used by this HRegion.Returns store services for this region, to access services required by store level needs(package private) HRegionWALFileSystem
Returns the WALHRegionFileSystem
used by this regionNavigableMap<byte[],
Integer> getRowLock
(byte[] row) Get an exclusive ( write lock ) lock on a given row.getRowLock
(byte[] row, boolean readLock) Get a row lock for the specified row.private Region.RowLock
getRowLock
(byte[] row, boolean readLock, Region.RowLock prevRowLock) protected Region.RowLock
getRowLockInternal
(byte[] row, boolean readLock, Region.RowLock prevRowLock) getScanner
(Scan scan) Return an iterator that scans over the HRegion, returning the indicated columns and rows specified by theScan
.getScanner
(Scan scan, List<KeyValueScanner> additionalScanners) Return an iterator that scans over the HRegion, returning the indicated columns and rows specified by theScan
.private RegionScannerImpl
getScanner
(Scan scan, List<KeyValueScanner> additionalScanners, long nonceGroup, long nonce) long
private Collection<HStore>
getSpecificStores
(List<byte[]> families) get stores which matches the specified families(package private) RegionSplitPolicy
Returns split policy for this region.getStore
(byte[] column) Return the Store for the given familyprivate HStore
Return HStore instance.getStoreFileList
(byte[][] columns) Returns list of store file names for the given families(package private) ThreadPoolExecutor
getStoreFileOpenAndCloseThreadPool
(String threadNamePrefix) private NavigableMap<byte[],
List<org.apache.hadoop.fs.Path>> Returns Map of StoreFiles by column familyprivate ThreadPoolExecutor
getStoreOpenAndCloseThreadPool
(String threadNamePrefix) Return the list of Stores managed by this regionReturns table descriptor for this regiongetWAL()
Returns WAL in use for this region(package private) org.apache.hadoop.fs.FileSystem
Returns the WALFileSystem
being used by this regionorg.apache.hadoop.fs.Path
long
Returns write request count for this regionprivate void
handleException
(org.apache.hadoop.fs.FileSystem fs, org.apache.hadoop.fs.Path edits, IOException e) int
hashCode()
private static boolean
hasMultipleColumnFamilies
(Collection<Pair<byte[], String>> familyPaths) Determines whether multiple column families are present Precondition: familyPaths is not nullboolean
Returns True if this region has references.long
heapSize()
Return the approximate 'exclusive deep size' of implementing object.(package private) void
incMemStoreSize
(long dataSizeDelta, long heapSizeDelta, long offHeapSizeDelta, int cellsCountDelta) private void
Increase the size of mem store in this region and the size of global mem storePerform one or more increment operations on a row.void
void
long
Deprecated.use HRegion.createHRegion() or HRegion.openHRegion()(package private) long
initialize
(CancelableProgressable reporter) Initialize this region.private long
initializeRegionInternals
(CancelableProgressable reporter, MonitoredTask status) private long
initializeStores
(CancelableProgressable reporter, MonitoredTask status) Open all Stores.private long
initializeStores
(CancelableProgressable reporter, MonitoredTask status, boolean warmup) private void
initializeWarmup
(CancelableProgressable reporter) protected HStore
instantiateHStore
(ColumnFamilyDescriptor family, boolean warmup) protected RegionScannerImpl
instantiateRegionScanner
(Scan scan, List<KeyValueScanner> additionalScanners, long nonceGroup, long nonce) private HRegion.FlushResultImpl
internalFlushcache
(Collection<HStore> storesToFlush, MonitoredTask status, boolean writeFlushWalMarker, FlushLifeCycleTracker tracker) Flushing given stores.private HRegion.FlushResult
internalFlushcache
(MonitoredTask status) Flushing all stores.protected HRegion.FlushResultImpl
internalFlushcache
(WAL wal, long myseqid, Collection<HStore> storesToFlush, MonitoredTask status, boolean writeFlushWalMarker, FlushLifeCycleTracker tracker) Flush the memstore.(package private) HRegion.FlushResultImpl
internalFlushCacheAndCommit
(WAL wal, MonitoredTask status, HRegion.PrepareFlushResult prepareResult, Collection<HStore> storesToFlush) protected HRegion.PrepareFlushResult
internalPrepareFlushCache
(WAL wal, long myseqid, Collection<HStore> storesToFlush, MonitoredTask status, boolean writeFlushWalMarker, FlushLifeCycleTracker tracker) private void
Interrupt any region options that have acquired the region lock viastartRegionOperation(org.apache.hadoop.hbase.regionserver.Region.Operation)
, orstartBulkRegionOperation(boolean)
.private boolean
isAllFamilies
(Collection<HStore> families) Returns True if passed Set is all families in the region.boolean
Returns true if region is available (not closed and not closing)boolean
isClosed()
Returns true if region is closedboolean
Returns True if closing process has startedprivate boolean
isFlushSize
(MemStoreSize size) boolean
boolean
Returns true if region is mergeableboolean
Returns True if region is read only(package private) boolean
boolean
Returns true if region is splittableprivate static boolean
isZeroLengthThenDelete
(org.apache.hadoop.fs.FileSystem fs, org.apache.hadoop.fs.FileStatus stat, org.apache.hadoop.fs.Path p) make sure have been through lease recovery before get file status, so the file length can be trusted.private long
loadRecoveredHFilesIfAny
(Collection<HStore> stores) private void
private void
Try to acquire a lock.private void
logFatLineOnFlush
(Collection<HStore> storesToFlush, long sequenceId) Utility method broken out of internalPrepareFlushCache so that method is smaller.private void
private boolean
matches
(CompareOperator op, int compareResult) (package private) void
metricsUpdateForGet
(List<Cell> results, long before) private OperationStatus
private OperationStatus
private OperationStatus
Performs multiple mutations atomically on a single row.mutateRow
(RowMutations rm, long nonceGroup, long nonce) void
mutateRowsWithLocks
(Collection<Mutation> mutations, Collection<byte[]> rowsToLock, long nonceGroup, long nonce) Perform atomic (all or none) mutations within the region.static HRegion
newHRegion
(org.apache.hadoop.fs.Path tableDir, WAL wal, org.apache.hadoop.fs.FileSystem fs, org.apache.hadoop.conf.Configuration conf, RegionInfo regionInfo, TableDescriptor htd, RegionServerServices rsServices) A utility method to create new instances of HRegion based on theHConstants.REGION_IMPL
configuration property.void
onConfigurationChange
(org.apache.hadoop.conf.Configuration conf) This method would be called by theConfigurationManager
object when theConfiguration
object is reloaded from disk.static HRegion
openHRegion
(org.apache.hadoop.conf.Configuration conf, org.apache.hadoop.fs.FileSystem fs, org.apache.hadoop.fs.Path rootDir, RegionInfo info, TableDescriptor htd, WAL wal) Open a Region.static HRegion
openHRegion
(org.apache.hadoop.conf.Configuration conf, org.apache.hadoop.fs.FileSystem fs, org.apache.hadoop.fs.Path rootDir, RegionInfo info, TableDescriptor htd, WAL wal, RegionServerServices rsServices, CancelableProgressable reporter) Open a Region.static HRegion
openHRegion
(org.apache.hadoop.fs.Path rootDir, RegionInfo info, TableDescriptor htd, WAL wal, org.apache.hadoop.conf.Configuration conf) Open a Region.static HRegion
openHRegion
(org.apache.hadoop.fs.Path rootDir, RegionInfo info, TableDescriptor htd, WAL wal, org.apache.hadoop.conf.Configuration conf, RegionServerServices rsServices, CancelableProgressable reporter) Open a Region.static HRegion
openHRegion
(RegionInfo info, TableDescriptor htd, WAL wal, org.apache.hadoop.conf.Configuration conf) Open a Region.static HRegion
openHRegion
(RegionInfo info, TableDescriptor htd, WAL wal, org.apache.hadoop.conf.Configuration conf, RegionServerServices rsServices, CancelableProgressable reporter) Open a Region.static HRegion
openHRegion
(HRegion other, CancelableProgressable reporter) Useful when reopening a closed region (normally for unit tests)static Region
openHRegion
(Region other, CancelableProgressable reporter) private HRegion
openHRegion
(CancelableProgressable reporter) Open HRegion.static HRegion
openHRegionFromTableDir
(org.apache.hadoop.conf.Configuration conf, org.apache.hadoop.fs.FileSystem fs, org.apache.hadoop.fs.Path tableDir, RegionInfo info, TableDescriptor htd, WAL wal, RegionServerServices rsServices, CancelableProgressable reporter) Open a Region.static HRegion
openReadOnlyFileSystemHRegion
(org.apache.hadoop.conf.Configuration conf, org.apache.hadoop.fs.FileSystem fs, org.apache.hadoop.fs.Path tableDir, RegionInfo info, TableDescriptor htd) Open a Region on a read-only file-system (like hdfs snapshots)private void
prepareDelete
(Delete delete) Prepare a delete for a row mutation processorprivate void
prepareDeleteTimestamps
(Mutation mutation, Map<byte[], List<Cell>> familyMap, byte[] byteNow) Set up correct timestamps in the KVs in Delete object.(package private) void
prepareGet
(Get get) private void
preProcess
(RowProcessor<?, ?> processor, WALEdit walEdit) void
processRowsWithLocks
(RowProcessor<?, ?> processor) Performs atomic multiple reads and writes on a given row.void
processRowsWithLocks
(RowProcessor<?, ?> processor, long nonceGroup, long nonce) Performs atomic multiple reads and writes on a given row.void
processRowsWithLocks
(RowProcessor<?, ?> processor, long timeout, long nonceGroup, long nonce) Performs atomic multiple reads and writes on a given row.void
Puts some data in the table.private void
recordMutationWithoutWal
(Map<byte[], List<Cell>> familyMap) Update LongAdders for number of puts without wal and the size of possible data loss.boolean
Check the region's underlying store files, open the files that have not been opened yet, and remove the store file readers for store files no longer available.protected boolean
refreshStoreFiles
(boolean force) void
registerChildren
(ConfigurationManager manager) Needs to be called to register the children to the manager.boolean
registerService
(com.google.protobuf.Service instance) Registers a new protocol bufferService
subclass as a coprocessor endpoint to be available for handling Region#execService(com.google.protobuf.RpcController, org.apache.hadoop.hbase.protobuf.generated.ClientProtos.CoprocessorServiceCall) calls.private void
releaseRowLocks
(List<Region.RowLock> rowLocks) private void
replayFlushInStores
(org.apache.hadoop.hbase.shaded.protobuf.generated.WALProtos.FlushDescriptor flush, HRegion.PrepareFlushResult prepareFlushResult, boolean dropMemstoreSnapshot) Replays the given flush descriptor by opening the flush files in stores and dropping the memstore snapshots if requested.private long
replayRecoveredEdits
(org.apache.hadoop.fs.Path edits, Map<byte[], Long> maxSeqIdInStores, CancelableProgressable reporter, org.apache.hadoop.fs.FileSystem fs) private long
replayRecoveredEditsForPaths
(long minSeqIdForTheRegion, org.apache.hadoop.fs.FileSystem fs, NavigableSet<org.apache.hadoop.fs.Path> files, CancelableProgressable reporter, org.apache.hadoop.fs.Path regionDir) (package private) long
replayRecoveredEditsIfAny
(Map<byte[], Long> maxSeqIdInStores, CancelableProgressable reporter, MonitoredTask status) Read the edits put under this region by wal splitting process.(package private) void
replayWALBulkLoadEventMarker
(org.apache.hadoop.hbase.shaded.protobuf.generated.WALProtos.BulkLoadDescriptor bulkLoadEvent) (package private) void
replayWALCompactionMarker
(org.apache.hadoop.hbase.shaded.protobuf.generated.WALProtos.CompactionDescriptor compaction, boolean pickCompactionFiles, boolean removeFiles, long replaySeqId) Call to complete a compaction.private void
replayWALFlushAbortMarker
(org.apache.hadoop.hbase.shaded.protobuf.generated.WALProtos.FlushDescriptor flush) private void
replayWALFlushCannotFlushMarker
(org.apache.hadoop.hbase.shaded.protobuf.generated.WALProtos.FlushDescriptor flush, long replaySeqId) (package private) void
replayWALFlushCommitMarker
(org.apache.hadoop.hbase.shaded.protobuf.generated.WALProtos.FlushDescriptor flush) (package private) void
replayWALFlushMarker
(org.apache.hadoop.hbase.shaded.protobuf.generated.WALProtos.FlushDescriptor flush, long replaySeqId) (package private) HRegion.PrepareFlushResult
replayWALFlushStartMarker
(org.apache.hadoop.hbase.shaded.protobuf.generated.WALProtos.FlushDescriptor flush) Replay the flush marker from primary region by creating a corresponding snapshot of the store memstores, only if the memstores do not have a higher seqId from an earlier wal edit (because the events may be coming out of order).(package private) void
replayWALRegionEventMarker
(org.apache.hadoop.hbase.shaded.protobuf.generated.WALProtos.RegionEventDescriptor regionEvent) void
reportCompactionRequestEnd
(boolean isMajor, int numFiles, long filesSizeCompacted) void
void
reportCompactionRequestStart
(boolean isMajor) void
requestCompaction
(byte[] family, String why, int priority, boolean major, CompactionLifeCycleTracker tracker) Request compaction for the given familyvoid
requestCompaction
(String why, int priority, boolean major, CompactionLifeCycleTracker tracker) Request compaction on this region.private void
void
requestFlush
(FlushLifeCycleTracker tracker) Request flush on this region.private void
requestFlush0
(FlushLifeCycleTracker tracker) private void
protected void
restoreEdit
(HStore s, Cell cell, MemStoreSizing memstoreAccounting) Used by testsprivate void
rewriteCellTags
(Map<byte[], List<Cell>> familyMap, Mutation m) Possibly rewrite incoming cell tags.static boolean
rowIsInRange
(RegionInfo info, byte[] row) Determines if the specified row is within the row range specified by the specified RegionInfostatic boolean
rowIsInRange
(RegionInfo info, byte[] row, int offset, short length) void
setBlockCache
(BlockCache blockCache) Only used for unit test which doesn't start region server.void
setClosing
(boolean closing) Exposed for some very specific unit tests.(package private) org.apache.hadoop.hbase.shaded.protobuf.generated.ClusterStatusProtos.RegionLoad.Builder
setCompleteSequenceId
(org.apache.hadoop.hbase.shaded.protobuf.generated.ClusterStatusProtos.RegionLoad.Builder regionLoadBldr) void
setCoprocessorHost
(RegionCoprocessorHost coprocessorHost) private void
void
setMobFileCache
(MobFileCache mobFileCache) Only used for unit test which doesn't start region server.void
setReadsEnabled
(boolean readsEnabled) void
setRestoredRegion
(boolean restoredRegion) void
void
setTimeoutForWriteLock
(long timeoutForWriteLock) ThedoClose(boolean, org.apache.hadoop.hbase.monitoring.MonitoredTask)
will block forever if someone tries proving the dead lock via the unit test.(package private) boolean
shouldFlush
(StringBuilder whyFlush) Should the memstore be flushed now(package private) boolean
shouldFlushStore
(HStore store) Should the store be flushed because it is old enough.private boolean
Check whether we should sync the wal from the table's durability settingsprivate void
startBulkRegionOperation
(boolean writeLockNeeded) This method needs to be called before any public call that reads or modifies stores in bulk.void
This method needs to be called before any public call that reads or modifies data.void
This method needs to be called before any public call that reads or modifies data.private void
sync
(long txid, Durability durability) Calls sync with the given transaction ID(package private) void
throwException
(String title, String regionName) (package private) IOException
Throw the correct exception upon interrupttoString()
void
private static void
updateCellTimestamps
(Iterable<List<Cell>> cellItr, byte[] now) Replace any cell timestamps set toHConstants.LATEST_TIMESTAMP
provided current timestamp.private void
updateDeleteLatestVersionTimestamp
(Cell cell, Get get, int count, byte[] byteNow) private void
updateSequenceId
(Iterable<List<Cell>> cellItr, long sequenceId) void
Wait for all current flushes of the region to completeboolean
waitForFlushes
(long timeout) Wait for all current flushes of the region to completevoid
Wait for all current flushes and compactions of the region to completestatic HRegion
warmupHRegion
(RegionInfo info, TableDescriptor htd, WAL wal, org.apache.hadoop.conf.Configuration conf, RegionServerServices rsServices, CancelableProgressable reporter) private boolean
Returns True if its worth doing a flush before we put up the close flag.private boolean
writeFlushRequestMarkerToWAL
(WAL wal, boolean writeFlushWalMarker) Writes a marker to WAL indicating a flush is requested but cannot be complete due to various reasons.private void
protected void
writeRegionOpenMarker
(WAL wal, long openSeqId) Methods inherited from class java.lang.Object
clone, finalize, getClass, notify, notifyAll, wait, wait, wait
Methods inherited from interface org.apache.hadoop.hbase.regionserver.Region
checkAndMutate, checkAndMutate, checkAndRowMutate, checkAndRowMutate
-
Field Details
-
LOG
-
LOAD_CFS_ON_DEMAND_CONFIG_KEY
- See Also:
-
HBASE_MAX_CELL_SIZE_KEY
- See Also:
-
DEFAULT_MAX_CELL_SIZE
- See Also:
-
HBASE_REGIONSERVER_MINIBATCH_SIZE
- See Also:
-
DEFAULT_HBASE_REGIONSERVER_MINIBATCH_SIZE
- See Also:
-
WAL_HSYNC_CONF_KEY
- See Also:
-
DEFAULT_WAL_HSYNC
- See Also:
-
COMPACTION_AFTER_BULKLOAD_ENABLE
Parameter name for compaction after bulkload- See Also:
-
SPLIT_IGNORE_BLOCKING_ENABLED_KEY
Config for allow split when file count greater than the configured blocking file count- See Also:
-
REGION_STORAGE_POLICY_KEY
- See Also:
-
DEFAULT_REGION_STORAGE_POLICY
- See Also:
-
SPECIAL_RECOVERED_EDITS_DIR
This is for for using HRegion as a local storage, where we may put the recovered edits in a special place. Once this is set, we will only replay the recovered edits under this directory and ignore the original replay directory configs.- See Also:
-
RECOVERED_EDITS_IGNORE_EOF
Mainly used for master local region, where we will replay the WAL file directly without splitting, so it is possible to have WAL files which are not closed cleanly, in this way, hitting EOF is expected so should not consider it as a critical problem.- See Also:
-
USE_META_CELL_COMPARATOR
Whether to useMetaCellComparator
even if we are not meta region. Used when creating master local region.- See Also:
-
DEFAULT_USE_META_CELL_COMPARATOR
- See Also:
-
closed
-
closing
-
maxFlushedSeqId
The max sequence id of flushed data on this region. There is no edit in memory that is less that this sequence id. -
lastFlushOpSeqId
Record the sequence id of last flush operation. Can be in advance ofmaxFlushedSeqId
when flushing a single column family. In this case,maxFlushedSeqId
will be older than the oldest edit in memory. -
lastReplayedOpenRegionSeqId
The sequence id of the last replayed open region event from the primary region. This is used to skip entries before this due to the possibility of replay edits coming out of order from replication. -
lastReplayedCompactionSeqId
-
lockedRows
-
stores
-
coprocessorServiceHandlers
-
memStoreSizing
-
regionServicesForStores
-
numMutationsWithoutWAL
-
dataInMemoryWithoutWAL
-
checkAndMutateChecksPassed
-
checkAndMutateChecksFailed
-
readRequestsCount
-
filteredReadRequestsCount
-
writeRequestsCount
-
blockedRequestsCount
-
compactionsFinished
-
compactionsFailed
-
compactionNumFilesCompacted
-
compactionNumBytesCompacted
-
compactionsQueued
-
flushesQueued
-
blockCache
-
mobFileCache
-
wal
-
fs
-
conf
-
baseConf
-
rowLockWaitDuration
-
DEFAULT_ROWLOCK_WAIT_DURATION
- See Also:
-
regionWalDir
-
walFS
-
isRestoredRegion
-
metricsTableRequests
-
busyWaitDuration
-
DEFAULT_BUSY_WAIT_DURATION
- See Also:
-
maxBusyWaitMultiplier
-
maxBusyWaitDuration
-
maxCellSize
-
miniBatchSize
-
DEFAULT_ROW_PROCESSOR_TIMEOUT
- See Also:
-
rowProcessorExecutor
-
scannerReadPoints
-
smallestReadPointCalcLock
-
openSeqNum
The sequence ID that was enLongAddered when this region was opened. -
isLoadingCfsOnDemandDefault
The default setting for whether to enable on-demand CF loading for scan requests to this region. Requests can override it. -
majorInProgress
-
minorInProgress
-
maxSeqIdInStores
Map<byte[],Long> maxSeqIdInStores -
prepareFlushResult
Saved state from replaying prepare flush cache -
configurationManager
-
timeoutForWriteLock
-
cellComparator
-
minBlockSizeBytes
-
writestate
-
memstoreFlushSize
long memstoreFlushSize -
timestampSlop
-
rowProcessorTimeout
-
lastStoreFlushTimeMap
-
rsServices
-
rsAccounting
-
flushCheckInterval
-
flushPerChanges
-
blockingMemStoreSize
-
lock
-
regionLockHolders
-
updatesLock
-
mvcc
-
coprocessorHost
-
htableDescriptor
-
splitPolicy
-
splitRestriction
-
flushPolicy
-
metricsRegion
-
metricsRegionWrapper
-
regionDurability
-
regionStatsEnabled
-
replicationScope
-
storeHotnessProtector
-
closeLock
-
FAIR_REENTRANT_CLOSE_LOCK
Conf key for fair locking policy- See Also:
-
DEFAULT_FAIR_REENTRANT_CLOSE_LOCK
- See Also:
-
MEMSTORE_PERIODIC_FLUSH_INTERVAL
Conf key for the periodic flush interval- See Also:
-
DEFAULT_CACHE_FLUSH_INTERVAL
Default interval for the memstore flush- See Also:
-
SYSTEM_CACHE_FLUSH_INTERVAL
Default interval for System tables memstore flush- See Also:
-
MEMSTORE_FLUSH_PER_CHANGES
Conf key to force a flush if there are already enough changes for one region in memstore- See Also:
-
DEFAULT_FLUSH_PER_CHANGES
- See Also:
-
MAX_FLUSH_PER_CHANGES
The following MAX_FLUSH_PER_CHANGES is large enough because each KeyValue has 20+ bytes overhead. Therefore, even 1G empty KVs occupy at least 20GB memstore size for a single region- See Also:
-
CLOSE_WAIT_ABORT
- See Also:
-
DEFAULT_CLOSE_WAIT_ABORT
- See Also:
-
CLOSE_WAIT_TIME
- See Also:
-
DEFAULT_CLOSE_WAIT_TIME
- See Also:
-
CLOSE_WAIT_INTERVAL
- See Also:
-
DEFAULT_CLOSE_WAIT_INTERVAL
- See Also:
-
FIXED_OVERHEAD
-
DEEP_OVERHEAD
-
-
Constructor Details
-
HRegion
@Deprecated public HRegion(org.apache.hadoop.fs.Path tableDir, WAL wal, org.apache.hadoop.fs.FileSystem fs, org.apache.hadoop.conf.Configuration confParam, RegionInfo regionInfo, TableDescriptor htd, RegionServerServices rsServices) Deprecated.Use other constructors.HRegion constructor. This constructor should only be used for testing and extensions. Instances of HRegion should be instantiated with thecreateHRegion(org.apache.hadoop.hbase.client.RegionInfo, org.apache.hadoop.fs.Path, org.apache.hadoop.conf.Configuration, org.apache.hadoop.hbase.client.TableDescriptor, org.apache.hadoop.hbase.wal.WAL, boolean)
oropenHRegion(org.apache.hadoop.hbase.client.RegionInfo, org.apache.hadoop.hbase.client.TableDescriptor, org.apache.hadoop.hbase.wal.WAL, org.apache.hadoop.conf.Configuration)
method.- Parameters:
tableDir
- qualified path of directory where region should be located, usually the table directory.wal
- The WAL is the outbound log for any updates to the HRegion The wal file is a logfile from the previous execution that's custom-computed for this HRegion. The HRegionServer computes and sorts the appropriate wal info for this HRegion. If there is a previous wal file (implying that the HRegion has been written-to before), then read it from the supplied path.fs
- is the filesystem.confParam
- is global configuration settings.regionInfo
- - RegionInfo that describes the region is new), then read them from the supplied path.htd
- the table descriptorrsServices
- reference toRegionServerServices
or null
-
HRegion
public HRegion(HRegionFileSystem fs, WAL wal, org.apache.hadoop.conf.Configuration confParam, TableDescriptor htd, RegionServerServices rsServices) HRegion constructor. This constructor should only be used for testing and extensions. Instances of HRegion should be instantiated with thecreateHRegion(org.apache.hadoop.hbase.client.RegionInfo, org.apache.hadoop.fs.Path, org.apache.hadoop.conf.Configuration, org.apache.hadoop.hbase.client.TableDescriptor, org.apache.hadoop.hbase.wal.WAL, boolean)
oropenHRegion(org.apache.hadoop.hbase.client.RegionInfo, org.apache.hadoop.hbase.client.TableDescriptor, org.apache.hadoop.hbase.wal.WAL, org.apache.hadoop.conf.Configuration)
method.- Parameters:
fs
- is the filesystem.wal
- The WAL is the outbound log for any updates to the HRegion The wal file is a logfile from the previous execution that's custom-computed for this HRegion. The HRegionServer computes and sorts the appropriate wal info for this HRegion. If there is a previous wal file (implying that the HRegion has been written-to before), then read it from the supplied path.confParam
- is global configuration settings.htd
- the table descriptorrsServices
- reference toRegionServerServices
or null
-
-
Method Details
-
setRestoredRegion
-
getMetricsTableRequests
-
getSmallestReadPoint
- Returns:
- The smallest mvcc readPoint across all the scanners in this region. Writes older than this readPoint, are included in every read operation.
-
setHTableSpecificConf
-
initialize
Deprecated.use HRegion.createHRegion() or HRegion.openHRegion()Initialize this region. Used only by tests and SplitTransaction to reopen the region. You should use createHRegion() or openHRegion()- Returns:
- What the next sequence (edit) id should be.
- Throws:
IOException
- e
-
initialize
Initialize this region.- Parameters:
reporter
- Tickle every so often if initialize is taking a while.- Returns:
- What the next sequence (edit) id should be.
- Throws:
IOException
-
initializeRegionInternals
private long initializeRegionInternals(CancelableProgressable reporter, MonitoredTask status) throws IOException - Throws:
IOException
-
initializeStores
private long initializeStores(CancelableProgressable reporter, MonitoredTask status) throws IOException Open all Stores.- Returns:
- Highest sequenceId found out in a Store.
- Throws:
IOException
-
initializeStores
private long initializeStores(CancelableProgressable reporter, MonitoredTask status, boolean warmup) throws IOException - Throws:
IOException
-
initializeWarmup
- Throws:
IOException
-
getStoreFiles
Returns Map of StoreFiles by column family -
writeRegionOpenMarker
- Throws:
IOException
-
writeRegionCloseMarker
- Throws:
IOException
-
hasReferences
Returns True if this region has references. -
blockUpdates
-
unblockUpdates
-
getHDFSBlocksDistribution
-
computeHDFSBlocksDistribution
public static HDFSBlocksDistribution computeHDFSBlocksDistribution(org.apache.hadoop.conf.Configuration conf, TableDescriptor tableDescriptor, RegionInfo regionInfo) throws IOException This is a helper function to compute HDFS block distribution on demand- Parameters:
conf
- configurationtableDescriptor
- TableDescriptor of the tableregionInfo
- encoded name of the region- Returns:
- The HDFS blocks distribution for the given region.
- Throws:
IOException
-
computeHDFSBlocksDistribution
public static HDFSBlocksDistribution computeHDFSBlocksDistribution(org.apache.hadoop.conf.Configuration conf, TableDescriptor tableDescriptor, RegionInfo regionInfo, org.apache.hadoop.fs.Path tablePath) throws IOException This is a helper function to compute HDFS block distribution on demand- Parameters:
conf
- configurationtableDescriptor
- TableDescriptor of the tableregionInfo
- encoded name of the regiontablePath
- the table directory- Returns:
- The HDFS blocks distribution for the given region.
- Throws:
IOException
-
incMemStoreSize
Increase the size of mem store in this region and the size of global mem store -
incMemStoreSize
void incMemStoreSize(long dataSizeDelta, long heapSizeDelta, long offHeapSizeDelta, int cellsCountDelta) -
decrMemStoreSize
-
decrMemStoreSize
private void decrMemStoreSize(long dataSizeDelta, long heapSizeDelta, long offHeapSizeDelta, int cellsCountDelta) -
checkNegativeMemStoreDataSize
-
getRegionInfo
Description copied from interface:Region
Returns region information for this region- Specified by:
getRegionInfo
in interfaceRegion
-
getRegionServerServices
Returns Instance ofRegionServerServices
used by this HRegion. Can be null. -
getReadRequestsCount
Description copied from interface:Region
Returns read requests count for this region- Specified by:
getReadRequestsCount
in interfaceRegion
-
getFilteredReadRequestsCount
Description copied from interface:Region
Returns filtered read requests count for this region- Specified by:
getFilteredReadRequestsCount
in interfaceRegion
-
getWriteRequestsCount
Description copied from interface:Region
Returns write request count for this region- Specified by:
getWriteRequestsCount
in interfaceRegion
-
getMemStoreDataSize
- Specified by:
getMemStoreDataSize
in interfaceRegion
- Returns:
- memstore size for this region, in bytes. It just accounts data size of cells added to the memstores of this Region. Means size in bytes for key, value and tags within Cells. It wont consider any java heap overhead for the cell objects or any other.
-
getMemStoreHeapSize
- Specified by:
getMemStoreHeapSize
in interfaceRegion
- Returns:
- memstore heap size for this region, in bytes. It accounts data size of cells added to the memstores of this Region, as well as java heap overhead for the cell objects or any other.
-
getMemStoreOffHeapSize
- Specified by:
getMemStoreOffHeapSize
in interfaceRegion
- Returns:
- memstore off-heap size for this region, in bytes. It accounts data size of cells added to the memstores of this Region, as well as overhead for the cell objects or any other that is allocated off-heap.
-
getRegionServicesForStores
Returns store services for this region, to access services required by store level needs -
getNumMutationsWithoutWAL
Description copied from interface:Region
Returns the number of mutations processed bypassing the WAL- Specified by:
getNumMutationsWithoutWAL
in interfaceRegion
-
getDataInMemoryWithoutWAL
Description copied from interface:Region
Returns the size of data processed bypassing the WAL, in bytes- Specified by:
getDataInMemoryWithoutWAL
in interfaceRegion
-
getBlockedRequestsCount
Description copied from interface:Region
Returns the number of blocked requests- Specified by:
getBlockedRequestsCount
in interfaceRegion
-
getCheckAndMutateChecksPassed
Description copied from interface:Region
Returns the number of checkAndMutate guards that passed- Specified by:
getCheckAndMutateChecksPassed
in interfaceRegion
-
getCheckAndMutateChecksFailed
Description copied from interface:Region
Returns the number of failed checkAndMutate guards- Specified by:
getCheckAndMutateChecksFailed
in interfaceRegion
-
getMetrics
-
isClosed
Description copied from interface:Region
Returns true if region is closed -
isClosing
Description copied from interface:Region
Returns True if closing process has started -
isReadOnly
Description copied from interface:Region
Returns True if region is read only- Specified by:
isReadOnly
in interfaceRegion
-
isAvailable
Description copied from interface:Region
Returns true if region is available (not closed and not closing)- Specified by:
isAvailable
in interfaceRegion
-
isSplittable
Description copied from interface:Region
Returns true if region is splittable- Specified by:
isSplittable
in interfaceRegion
-
isMergeable
Description copied from interface:Region
Returns true if region is mergeable- Specified by:
isMergeable
in interfaceRegion
-
areWritesEnabled
-
getMVCC
-
getMaxFlushedSeqId
- Specified by:
getMaxFlushedSeqId
in interfaceRegion
- Returns:
- the max sequence id of flushed data on this region; no edit in memory will have a sequence id that is less that what is returned here.
-
getReadPoint
Returns readpoint considering given IsolationLevel. Passnull
for default -
isLoadingCfsOnDemandDefault
-
close
Close down this HRegion. Flush the cache, shut down each HStore, don't service any more calls.This method could take some time to execute, so don't call it from a time-sensitive thread.
- Returns:
- Vector of all the storage files that the HRegion's component HStores make use of. It's a list of all StoreFile objects. Returns empty vector if already closed and null if judged that it should not close.
- Throws:
IOException
- eDroppedSnapshotException
- Thrown when replay of wal is required because a Snapshot was not properly persisted. The region is put in closing mode, and the caller MUST abort after this.
-
close
- Throws:
IOException
-
close
Close this HRegion.- Parameters:
abort
- true if server is aborting (only during testing)ignoreStatus
- true if ignore the status (won't be showed on task list)- Returns:
- Vector of all the storage files that the HRegion's component HStores make use of. It's a list of StoreFile objects. Can be null if we are not to close at this time, or we are already closed.
- Throws:
IOException
- eDroppedSnapshotException
- Thrown when replay of wal is required because a Snapshot was not properly persisted. The region is put in closing mode, and the caller MUST abort after this.
-
close
public Map<byte[],List<HStoreFile>> close(boolean abort, boolean ignoreStatus, boolean isGracefulStop) throws IOException Close down this HRegion. Flush the cache unless abort parameter is true, Shut down each HStore, don't service any more calls. This method could take some time to execute, so don't call it from a time-sensitive thread.- Parameters:
abort
- true if server is aborting (only during testing)ignoreStatus
- true if ignore the status (wont be showed on task list)isGracefulStop
- true if region is being closed during graceful stop and the blocks in the BucketCache should not be evicted.- Returns:
- Vector of all the storage files that the HRegion's component HStores make use of. It's a list of StoreFile objects. Can be null if we are not to close at this time or we are already closed.
- Throws:
IOException
- eDroppedSnapshotException
- Thrown when replay of wal is required because a Snapshot was not properly persisted. The region is put in closing mode, and the caller MUST abort after this.
-
setClosing
Exposed for some very specific unit tests. -
setTimeoutForWriteLock
ThedoClose(boolean, org.apache.hadoop.hbase.monitoring.MonitoredTask)
will block forever if someone tries proving the dead lock via the unit test. Instead of blocking, thedoClose(boolean, org.apache.hadoop.hbase.monitoring.MonitoredTask)
will throw exception if you set the timeout.- Parameters:
timeoutForWriteLock
- the second time to wait for the write lock indoClose(boolean, org.apache.hadoop.hbase.monitoring.MonitoredTask)
-
doClose
private Map<byte[],List<HStoreFile>> doClose(boolean abort, MonitoredTask status) throws IOException - Throws:
IOException
-
waitForFlushesAndCompactions
Wait for all current flushes and compactions of the region to complete -
waitForFlushes
Wait for all current flushes of the region to complete -
waitForFlushes
Description copied from interface:Region
Wait for all current flushes of the region to complete- Specified by:
waitForFlushes
in interfaceRegion
- Parameters:
timeout
- The maximum time to wait in milliseconds.- Returns:
- False when timeout elapsed but flushes are not over. True when flushes are over within max wait time period.
-
getReadOnlyConfiguration
- Specified by:
getReadOnlyConfiguration
in interfaceRegion
- Returns:
- a read only configuration of this region; throws
UnsupportedOperationException
if you try to set a configuration.
-
getMinBlockSizeBytes
Description copied from interface:Region
The minimum block size configuration from all relevant column families. This is used when estimating quota consumption.- Specified by:
getMinBlockSizeBytes
in interfaceRegion
-
getStoreOpenAndCloseThreadPool
-
getStoreFileOpenAndCloseThreadPool
-
getOpenAndCloseThreadPool
private static ThreadPoolExecutor getOpenAndCloseThreadPool(int maxThreads, String threadNamePrefix) -
worthPreFlushing
Returns True if its worth doing a flush before we put up the close flag. -
getTableDescriptor
Description copied from interface:Region
Returns table descriptor for this region- Specified by:
getTableDescriptor
in interfaceRegion
-
setTableDescriptor
-
getWAL
Returns WAL in use for this region -
getBlockCache
-
setBlockCache
Only used for unit test which doesn't start region server. -
getMobFileCache
-
setMobFileCache
Only used for unit test which doesn't start region server. -
getSplitPolicy
Returns split policy for this region. -
getBaseConf
org.apache.hadoop.conf.Configuration getBaseConf()A split takes the config from the parent region & passes it to the daughter region's constructor. If 'conf' was passed, you would end up using the HTD of the parent region in addition to the new daughter HTD. Pass 'baseConf' to the daughter regions to avoid this tricky dedupe problem.- Returns:
- Configuration object
-
getFilesystem
ReturnsFileSystem
being used by this region -
getRegionFileSystem
Returns theHRegionFileSystem
used by this region -
getRegionWALFileSystem
Returns the WALHRegionFileSystem
used by this region- Throws:
IOException
-
getWalFileSystem
Returns the WALFileSystem
being used by this region- Throws:
IOException
-
getWALRegionDir
- Returns:
- the Region directory under WALRootDirectory
- Throws:
IOException
- if there is an error getting WALRootDir
-
getEarliestFlushTimeForAllStores
- Specified by:
getEarliestFlushTimeForAllStores
in interfaceRegion
- Returns:
- The earliest time a store in the region was flushed. All other stores in the region would have been flushed either at, or after this time.
-
getOldestHfileTs
Description copied from interface:Region
This can be used to determine the last time all files of this region were major compacted.- Specified by:
getOldestHfileTs
in interfaceRegion
- Parameters:
majorCompactionOnly
- Only consider HFile that are the result of major compaction- Returns:
- the timestamp of the oldest HFile for all stores of this region
- Throws:
IOException
-
setCompleteSequenceId
org.apache.hadoop.hbase.shaded.protobuf.generated.ClusterStatusProtos.RegionLoad.Builder setCompleteSequenceId(org.apache.hadoop.hbase.shaded.protobuf.generated.ClusterStatusProtos.RegionLoad.Builder regionLoadBldr) -
doRegionCompactionPrep
Do preparation for pending compaction.- Throws:
IOException
-
compact
Synchronously compact all stores in the region.This operation could block for a long time, so don't call it from a time-sensitive thread.
Note that no locks are taken to prevent possible conflicts between compaction and splitting activities. The regionserver does not normally compact and split in parallel. However by calling this method you may introduce unexpected and unhandled concurrency. Don't do this unless you know what you are doing.
- Parameters:
majorCompaction
- True to force a major compaction regardless of thresholds- Throws:
IOException
-
compactStores
This is a helper function that compact all the stores synchronously.It is used by utilities and testing
- Throws:
IOException
-
compactStore
This is a helper function that compact the given store.It is used by utilities and testing
- Throws:
IOException
-
compact
public boolean compact(CompactionContext compaction, HStore store, ThroughputController throughputController) throws IOException Called by compaction thread and after region is opened to compact the HStores if necessary.This operation could block for a long time, so don't call it from a time-sensitive thread. Note that no locking is necessary at this level because compaction only conflicts with a region split, and that cannot happen because the region server does them sequentially and not in parallel.
- Parameters:
compaction
- Compaction details, obtained by requestCompaction()- Returns:
- whether the compaction completed
- Throws:
IOException
-
compact
public boolean compact(CompactionContext compaction, HStore store, ThroughputController throughputController, User user) throws IOException - Throws:
IOException
-
flush
Flush the cache.When this method is called the cache will be flushed unless:
- the cache is empty
- the region is closed.
- a flush is already in progress
- writes are disabled
This method may block for some time, so it should not be called from a time-sensitive thread.
- Parameters:
flushAllStores
- whether we want to force a flush of all stores- Returns:
- FlushResult indicating whether the flush was successful or not and if the region needs compacting
- Throws:
IOException
- general io exceptions because a snapshot was not properly persisted.
-
flushcache
HRegion.FlushResultImpl flushcache(boolean flushAllStores, boolean writeFlushRequestWalMarker, FlushLifeCycleTracker tracker) throws IOException - Throws:
IOException
-
flushcache
public HRegion.FlushResultImpl flushcache(List<byte[]> families, boolean writeFlushRequestWalMarker, FlushLifeCycleTracker tracker) throws IOException Flush the cache. When this method is called the cache will be flushed unless:- the cache is empty
- the region is closed.
- a flush is already in progress
- writes are disabled
This method may block for some time, so it should not be called from a time-sensitive thread.
- Parameters:
families
- stores of region to flush.writeFlushRequestWalMarker
- whether to write the flush request marker to WALtracker
- used to track the life cycle of this flush- Returns:
- whether the flush is success and whether the region needs compacting
- Throws:
IOException
- general io exceptionsDroppedSnapshotException
- Thrown when replay of wal is required because a Snapshot was not properly persisted. The region is put in closing mode, and the caller MUST abort after this.
-
getSpecificStores
get stores which matches the specified families- Returns:
- the stores need to be flushed.
-
shouldFlushStore
Should the store be flushed because it is old enough.Every FlushPolicy should call this to determine whether a store is old enough to flush (except that you always flush all stores). Otherwise the method will always returns true which will make a lot of flush requests.
-
shouldFlush
Should the memstore be flushed now -
internalFlushcache
Flushing all stores. -
internalFlushcache
private HRegion.FlushResultImpl internalFlushcache(Collection<HStore> storesToFlush, MonitoredTask status, boolean writeFlushWalMarker, FlushLifeCycleTracker tracker) throws IOException Flushing given stores. -
internalFlushcache
protected HRegion.FlushResultImpl internalFlushcache(WAL wal, long myseqid, Collection<HStore> storesToFlush, MonitoredTask status, boolean writeFlushWalMarker, FlushLifeCycleTracker tracker) throws IOException Flush the memstore. Flushing the memstore is a little tricky. We have a lot of updates in the memstore, all of which have also been written to the wal. We need to write those updates in the memstore out to disk, while being able to process reads/writes as much as possible during the flush operation.This method may block for some time. Every time you call it, we up the regions sequence id even if we don't flush; i.e. the returned region id will be at least one larger than the last edit applied to this region. The returned id does not refer to an actual edit. The returned id can be used for say installing a bulk loaded file just ahead of the last hfile that was the result of this flush, etc.
- Parameters:
wal
- Null if we're NOT to go via wal.myseqid
- The seqid to use ifwal
is null writing out flush file.storesToFlush
- The list of stores to flush.- Returns:
- object describing the flush's state
- Throws:
IOException
- general io exceptionsDroppedSnapshotException
- Thrown when replay of WAL is required.
-
internalPrepareFlushCache
protected HRegion.PrepareFlushResult internalPrepareFlushCache(WAL wal, long myseqid, Collection<HStore> storesToFlush, MonitoredTask status, boolean writeFlushWalMarker, FlushLifeCycleTracker tracker) throws IOException - Throws:
IOException
-
logFatLineOnFlush
Utility method broken out of internalPrepareFlushCache so that method is smaller. -
doAbortFlushToWAL
private void doAbortFlushToWAL(WAL wal, long flushOpSeqId, Map<byte[], List<org.apache.hadoop.fs.Path>> committedFiles) -
doSyncOfUnflushedWALChanges
Sync unflushed WAL changes. See HBASE-8208 for details- Throws:
IOException
-
isAllFamilies
Returns True if passed Set is all families in the region. -
writeFlushRequestMarkerToWAL
Writes a marker to WAL indicating a flush is requested but cannot be complete due to various reasons. Ignores exceptions from WAL. Returns whether the write succeeded.- Returns:
- whether WAL write was successful
-
internalFlushCacheAndCommit
HRegion.FlushResultImpl internalFlushCacheAndCommit(WAL wal, MonitoredTask status, HRegion.PrepareFlushResult prepareResult, Collection<HStore> storesToFlush) throws IOException - Throws:
IOException
-
getNextSequenceId
Method to safely get the next sequence number.- Returns:
- Next sequence number unassociated with any actual edit.
- Throws:
IOException
-
getScanner
Description copied from interface:Region
Return an iterator that scans over the HRegion, returning the indicated columns and rows specified by theScan
.This Iterator must be closed by the caller.
- Specified by:
getScanner
in interfaceRegion
- Parameters:
scan
- configuredScan
- Throws:
IOException
- read exceptions
-
getScanner
public RegionScannerImpl getScanner(Scan scan, List<KeyValueScanner> additionalScanners) throws IOException Description copied from interface:Region
Return an iterator that scans over the HRegion, returning the indicated columns and rows specified by theScan
. The scanner will also include the additional scanners passed along with the scanners for the specified Scan instance. Should be careful with the usage to pass additional scanners only within this RegionThis Iterator must be closed by the caller.
- Specified by:
getScanner
in interfaceRegion
- Parameters:
scan
- configuredScan
additionalScanners
- Any additional scanners to be used- Throws:
IOException
- read exceptions
-
getScanner
private RegionScannerImpl getScanner(Scan scan, List<KeyValueScanner> additionalScanners, long nonceGroup, long nonce) throws IOException - Throws:
IOException
-
instantiateRegionScanner
protected RegionScannerImpl instantiateRegionScanner(Scan scan, List<KeyValueScanner> additionalScanners, long nonceGroup, long nonce) throws IOException - Throws:
IOException
-
prepareDelete
Prepare a delete for a row mutation processor- Parameters:
delete
- The passed delete is modified by this method. WARNING!- Throws:
IOException
-
delete
Description copied from interface:Region
Deletes the specified cells/row.- Specified by:
delete
in interfaceRegion
- Throws:
IOException
-
prepareDeleteTimestamps
private void prepareDeleteTimestamps(Mutation mutation, Map<byte[], List<Cell>> familyMap, byte[] byteNow) throws IOExceptionSet up correct timestamps in the KVs in Delete object. Caller should have the row and region locks.- Throws:
IOException
-
updateDeleteLatestVersionTimestamp
private void updateDeleteLatestVersionTimestamp(Cell cell, Get get, int count, byte[] byteNow) throws IOException - Throws:
IOException
-
put
Description copied from interface:Region
Puts some data in the table.- Specified by:
put
in interfaceRegion
- Throws:
IOException
-
batchMutate
public OperationStatus[] batchMutate(Mutation[] mutations, boolean atomic, long nonceGroup, long nonce) throws IOException - Throws:
IOException
-
batchMutate
Description copied from interface:Region
Perform a batch of mutations.Please do not operate on a same column of a single row in a batch, we will not consider the previous operation in the same batch when performing the operations in the batch.
- Specified by:
batchMutate
in interfaceRegion
- Parameters:
mutations
- the list of mutations- Returns:
- an array of OperationStatus which internally contains the OperationStatusCode and the exceptionMessage if any.
- Throws:
IOException
-
batchMutate
- Throws:
IOException
-
batchReplay
public OperationStatus[] batchReplay(WALSplitUtil.MutationReplay[] mutations, long replaySeqId) throws IOException - Throws:
IOException
-
batchMutate
Perform a batch of mutations. Operations in a batch are stored with highest durability specified of for all operations in a batch, except forDurability.SKIP_WAL
. This function is called frombatchReplay(WALSplitUtil.MutationReplay[], long)
withHRegion.ReplayBatchOperation
instance andbatchMutate(Mutation[])
withHRegion.MutationBatchOperation
instance as an argument. As the processing of replay batch and mutation batch is very similar, lot of code is shared by providing generic methods in base classHRegion.BatchOperation
. The logic for this method anddoMiniBatchMutate(BatchOperation)
is implemented using methods in base class which are overridden by derived classes to implement special behavior.- Parameters:
batchOp
- contains the list of mutations- Returns:
- an array of OperationStatus which internally contains the OperationStatusCode and the exceptionMessage if any.
- Throws:
IOException
- if an IO problem is encountered
-
doMiniBatchMutate
Called to do a piece of the batch that came in tobatchMutate(Mutation[])
In here we also handle replay of edits on region recover. Also gets change in size brought about by applyingbatchOp
.- Throws:
IOException
-
getEffectiveDurability
Returns effective durability from the passed durability and the table descriptor. -
checkAndMutate
@Deprecated public boolean checkAndMutate(byte[] row, byte[] family, byte[] qualifier, CompareOperator op, ByteArrayComparable comparator, TimeRange timeRange, Mutation mutation) throws IOException Deprecated.Description copied from interface:Region
Atomically checks if a row/family/qualifier value matches the expected value and if it does, it performs the mutation. If the passed value is null, the lack of column value (ie: non-existence) is used. See checkAndRowMutate to do many checkAndPuts at a time on a single row.- Specified by:
checkAndMutate
in interfaceRegion
- Parameters:
row
- to checkfamily
- column family to checkqualifier
- column qualifier to checkop
- the comparison operatorcomparator
- the expected valuetimeRange
- time range to checkmutation
- data to put if check succeeds- Returns:
- true if mutation was applied, false otherwise
- Throws:
IOException
-
checkAndMutate
@Deprecated public boolean checkAndMutate(byte[] row, Filter filter, TimeRange timeRange, Mutation mutation) throws IOException Deprecated.Description copied from interface:Region
Atomically checks if a row value matches the filter and if it does, it performs the mutation. See checkAndRowMutate to do many checkAndPuts at a time on a single row.- Specified by:
checkAndMutate
in interfaceRegion
- Parameters:
row
- to checkfilter
- the filtertimeRange
- time range to checkmutation
- data to put if check succeeds- Returns:
- true if mutation was applied, false otherwise
- Throws:
IOException
-
checkAndRowMutate
@Deprecated public boolean checkAndRowMutate(byte[] row, byte[] family, byte[] qualifier, CompareOperator op, ByteArrayComparable comparator, TimeRange timeRange, RowMutations rm) throws IOException Deprecated.Description copied from interface:Region
Atomically checks if a row/family/qualifier value matches the expected values and if it does, it performs the row mutations. If the passed value is null, the lack of column value (ie: non-existence) is used. Use to do many mutations on a single row. Use checkAndMutate to do one checkAndMutate at a time.- Specified by:
checkAndRowMutate
in interfaceRegion
- Parameters:
row
- to checkfamily
- column family to checkqualifier
- column qualifier to checkop
- the comparison operatorcomparator
- the expected valuetimeRange
- time range to checkrm
- data to put if check succeeds- Returns:
- true if mutations were applied, false otherwise
- Throws:
IOException
-
checkAndRowMutate
@Deprecated public boolean checkAndRowMutate(byte[] row, Filter filter, TimeRange timeRange, RowMutations rm) throws IOException Deprecated.Description copied from interface:Region
Atomically checks if a row matches the filter and if it does, it performs the row mutations. Use to do many mutations on a single row. Use checkAndMutate to do one checkAndMutate at a time.- Specified by:
checkAndRowMutate
in interfaceRegion
- Parameters:
row
- to checkfilter
- the filtertimeRange
- time range to checkrm
- data to put if check succeeds- Returns:
- true if mutations were applied, false otherwise
- Throws:
IOException
-
checkAndMutate
Description copied from interface:Region
Atomically checks if a row matches the conditions and if it does, it performs the actions. Use to do many mutations on a single row. Use checkAndMutate to do one checkAndMutate at a time.- Specified by:
checkAndMutate
in interfaceRegion
- Parameters:
checkAndMutate
- the CheckAndMutate object- Returns:
- true if mutations were applied, false otherwise
- Throws:
IOException
- if an error occurred in this method
-
checkAndMutate
public CheckAndMutateResult checkAndMutate(CheckAndMutate checkAndMutate, long nonceGroup, long nonce) throws IOException - Throws:
IOException
-
checkAndMutateInternal
private CheckAndMutateResult checkAndMutateInternal(CheckAndMutate checkAndMutate, long nonceGroup, long nonce) throws IOException - Throws:
IOException
-
checkMutationType
- Throws:
DoNotRetryIOException
-
checkRow
- Throws:
DoNotRetryIOException
-
matches
-
mutate
- Throws:
IOException
-
mutate
- Throws:
IOException
-
mutate
private OperationStatus mutate(Mutation mutation, boolean atomic, long nonceGroup, long nonce) throws IOException - Throws:
IOException
-
addRegionToSnapshot
public void addRegionToSnapshot(org.apache.hadoop.hbase.shaded.protobuf.generated.SnapshotProtos.SnapshotDescription desc, ForeignExceptionSnare exnSnare) throws IOException Complete taking the snapshot on the region. Writes the region info and adds references to the working snapshot directory. TODO for api consistency, consider adding another version with noForeignExceptionSnare
arg. (In the future other cancellable HRegion methods could eventually add aForeignExceptionSnare
, or we could do something fancier).- Parameters:
desc
- snapshot description objectexnSnare
- ForeignExceptionSnare that captures external exceptions in case we need to bail out. This is allowed to be null and will just be ignored in that case.- Throws:
IOException
- if there is an external or internal error causing the snapshot to fail
-
updateSequenceId
- Throws:
IOException
-
updateCellTimestamps
private static void updateCellTimestamps(Iterable<List<Cell>> cellItr, byte[] now) throws IOException Replace any cell timestamps set toHConstants.LATEST_TIMESTAMP
provided current timestamp.- Throws:
IOException
-
rewriteCellTags
Possibly rewrite incoming cell tags. -
checkResources
Check if resources to support an update. We throw RegionTooBusyException if above memstore limit and expect client to retry using some kind of backoff- Throws:
RegionTooBusyException
-
checkReadOnly
- Throws:
IOException
- Throws exception if region is in read-only mode.
-
checkReadsEnabled
- Throws:
IOException
-
setReadsEnabled
-
applyToMemStore
private void applyToMemStore(HStore store, List<Cell> cells, boolean delta, MemStoreSizing memstoreAccounting) - Parameters:
delta
- If we are doing delta changes -- e.g. increment/append -- then this flag will be set; when set we will run operations that make sense in the increment/append scenario but that do not make sense otherwise.- See Also:
-
applyToMemStore
private void applyToMemStore(HStore store, Cell cell, MemStoreSizing memstoreAccounting) throws IOException - Throws:
IOException
- See Also:
-
checkFamilies
Check the collection of families for validity.- Throws:
NoSuchColumnFamilyException
-
checkTimestamps
public void checkTimestamps(Map<byte[], List<Cell>> familyMap, long now) throws FailedSanityCheckExceptionCheck the collection of families for valid timestamps- Parameters:
now
- current timestamp- Throws:
FailedSanityCheckException
-
isFlushSize
-
deleteRecoveredEdits
private void deleteRecoveredEdits(org.apache.hadoop.fs.FileSystem fs, Iterable<org.apache.hadoop.fs.Path> files) throws IOException - Throws:
IOException
-
replayRecoveredEditsIfAny
long replayRecoveredEditsIfAny(Map<byte[], Long> maxSeqIdInStores, CancelableProgressable reporter, MonitoredTask status) throws IOExceptionRead the edits put under this region by wal splitting process. Put the recovered edits back up into this region.We can ignore any wal message that has a sequence ID that's equal to or lower than minSeqId. (Because we know such messages are already reflected in the HFiles.)
While this is running we are putting pressure on memory yet we are outside of our usual accounting because we are not yet an onlined region (this stuff is being run as part of Region initialization). This means that if we're up against global memory limits, we'll not be flagged to flush because we are not online. We can't be flushed by usual mechanisms anyways; we're not yet online so our relative sequenceids are not yet aligned with WAL sequenceids -- not till we come up online, post processing of split edits.
But to help relieve memory pressure, at least manage our own heap size flushing if are in excess of per-region limits. Flushing, though, we have to be careful and avoid using the regionserver/wal sequenceid. Its running on a different line to whats going on in here in this region context so if we crashed replaying these edits, but in the midst had a flush that used the regionserver wal with a sequenceid in excess of whats going on in here in this region and with its split editlogs, then we could miss edits the next time we go to recover. So, we have to flush inline, using seqids that make sense in a this single region context only -- until we online.
- Parameters:
maxSeqIdInStores
- Any edit found in split editlogs needs to be in excess of the maxSeqId for the store to be applied, else its skipped.- Returns:
- the sequence id of the last edit added to this region out of the recovered edits log or
minSeqId
if nothing added from editlogs. - Throws:
IOException
-
handleException
private void handleException(org.apache.hadoop.fs.FileSystem fs, org.apache.hadoop.fs.Path edits, IOException e) throws IOException - Throws:
IOException
-
replayRecoveredEdits
private long replayRecoveredEdits(org.apache.hadoop.fs.Path edits, Map<byte[], Long> maxSeqIdInStores, CancelableProgressable reporter, org.apache.hadoop.fs.FileSystem fs) throws IOException- Parameters:
edits
- File of recovered edits.maxSeqIdInStores
- Maximum sequenceid found in each store. Edits in wal must be larger than this to be replayed for each store.- Returns:
- the sequence id of the last edit added to this region out of the recovered edits log or
minSeqId
if nothing added from editlogs. - Throws:
IOException
-
replayWALCompactionMarker
void replayWALCompactionMarker(org.apache.hadoop.hbase.shaded.protobuf.generated.WALProtos.CompactionDescriptor compaction, boolean pickCompactionFiles, boolean removeFiles, long replaySeqId) throws IOException Call to complete a compaction. Its for the case where we find in the WAL a compaction that was not finished. We could find one recovering a WAL after a regionserver crash. See HBASE-2331.- Throws:
IOException
-
replayWALFlushMarker
void replayWALFlushMarker(org.apache.hadoop.hbase.shaded.protobuf.generated.WALProtos.FlushDescriptor flush, long replaySeqId) throws IOException - Throws:
IOException
-
replayWALFlushStartMarker
HRegion.PrepareFlushResult replayWALFlushStartMarker(org.apache.hadoop.hbase.shaded.protobuf.generated.WALProtos.FlushDescriptor flush) throws IOException Replay the flush marker from primary region by creating a corresponding snapshot of the store memstores, only if the memstores do not have a higher seqId from an earlier wal edit (because the events may be coming out of order).- Throws:
IOException
-
replayWALFlushCommitMarker
void replayWALFlushCommitMarker(org.apache.hadoop.hbase.shaded.protobuf.generated.WALProtos.FlushDescriptor flush) throws IOException - Throws:
IOException
-
replayFlushInStores
private void replayFlushInStores(org.apache.hadoop.hbase.shaded.protobuf.generated.WALProtos.FlushDescriptor flush, HRegion.PrepareFlushResult prepareFlushResult, boolean dropMemstoreSnapshot) throws IOException Replays the given flush descriptor by opening the flush files in stores and dropping the memstore snapshots if requested.- Throws:
IOException
-
loadRecoveredHFilesIfAny
- Throws:
IOException
-
dropMemStoreContents
Be careful, this method will drop all data in the memstore of this region. Currently, this method is used to drop memstore to prevent memory leak when replaying recovered.edits while opening region.- Throws:
IOException
-
dropMemStoreContentsForSeqId
Drops the memstore contents after replaying a flush descriptor or region open event replay if the memstore edits have seqNums smaller than the given seq id- Throws:
IOException
-
doDropStoreMemStoreContentsForSeqId
private MemStoreSize doDropStoreMemStoreContentsForSeqId(HStore s, long currentSeqId) throws IOException - Throws:
IOException
-
replayWALFlushAbortMarker
private void replayWALFlushAbortMarker(org.apache.hadoop.hbase.shaded.protobuf.generated.WALProtos.FlushDescriptor flush) -
replayWALFlushCannotFlushMarker
private void replayWALFlushCannotFlushMarker(org.apache.hadoop.hbase.shaded.protobuf.generated.WALProtos.FlushDescriptor flush, long replaySeqId) -
getPrepareFlushResult
-
replayWALRegionEventMarker
void replayWALRegionEventMarker(org.apache.hadoop.hbase.shaded.protobuf.generated.WALProtos.RegionEventDescriptor regionEvent) throws IOException - Throws:
IOException
-
replayWALBulkLoadEventMarker
void replayWALBulkLoadEventMarker(org.apache.hadoop.hbase.shaded.protobuf.generated.WALProtos.BulkLoadDescriptor bulkLoadEvent) throws IOException - Throws:
IOException
-
dropPrepareFlushIfPossible
If all stores ended up dropping their snapshots, we can safely drop the prepareFlushResult -
refreshStoreFiles
Description copied from interface:Region
Check the region's underlying store files, open the files that have not been opened yet, and remove the store file readers for store files no longer available.- Specified by:
refreshStoreFiles
in interfaceRegion
- Throws:
IOException
-
refreshStoreFiles
- Throws:
IOException
-
logRegionFiles
-
checkTargetRegion
private void checkTargetRegion(byte[] encodedRegionName, String exceptionMsg, Object payload) throws WrongRegionException Checks whether the given regionName is either equal to our region, or that the regionName is the primary region to our corresponding range for the secondary replica.- Throws:
WrongRegionException
-
restoreEdit
Used by tests- Parameters:
s
- Store to add edit too.cell
- Cell to add.
-
isZeroLengthThenDelete
private static boolean isZeroLengthThenDelete(org.apache.hadoop.fs.FileSystem fs, org.apache.hadoop.fs.FileStatus stat, org.apache.hadoop.fs.Path p) throws IOException make sure have been through lease recovery before get file status, so the file length can be trusted.- Parameters:
p
- File to check.- Returns:
- True if file was zero-length (and if so, we'll delete it in here).
- Throws:
IOException
-
instantiateHStore
protected HStore instantiateHStore(ColumnFamilyDescriptor family, boolean warmup) throws IOException - Throws:
IOException
-
getStore
Description copied from interface:Region
Return the Store for the given familyUse with caution. Exposed for use of fixup utilities.
-
getStore
Return HStore instance. Does not do any copy: as the number of store is limited, we iterate on the list. -
getStores
Description copied from interface:Region
Return the list of Stores managed by this regionUse with caution. Exposed for use of fixup utilities.
-
getStoreFileList
Description copied from interface:Region
Returns list of store file names for the given families- Specified by:
getStoreFileList
in interfaceRegion
- Throws:
IllegalArgumentException
-
checkRow
Make sure this is a valid row for the HRegion- Throws:
IOException
-
getRowLock
Get an exclusive ( write lock ) lock on a given row.- Parameters:
row
- Which row to lock.- Returns:
- A locked RowLock. The lock is exclusive and already aqquired.
- Throws:
IOException
-
getRowLock
Description copied from interface:Region
Get a row lock for the specified row. All locks are reentrant. Before calling this function make sure that a region operation has already been started (the calling thread has already acquired the region-close-guard lock).The obtained locks should be released after use by
Region.RowLock.release()
NOTE: the boolean passed here has changed. It used to be a boolean that stated whether or not to wait on the lock. Now it is whether it an exclusive lock is requested.
- Specified by:
getRowLock
in interfaceRegion
- Parameters:
row
- The row actions will be performed againstreadLock
- is the lock reader or writer. True indicates that a non-exclusive lock is requested- Throws:
IOException
- See Also:
-
createRegionSpan
-
getRowLockInternal
protected Region.RowLock getRowLockInternal(byte[] row, boolean readLock, Region.RowLock prevRowLock) throws IOException - Throws:
IOException
-
getRowLock
private Region.RowLock getRowLock(byte[] row, boolean readLock, Region.RowLock prevRowLock) throws IOException - Throws:
IOException
-
releaseRowLocks
-
getReadLockCount
-
getLockedRows
-
hasMultipleColumnFamilies
Determines whether multiple column families are present Precondition: familyPaths is not null- Parameters:
familyPaths
- List of (column family, hfilePath)
-
bulkLoadHFiles
public Map<byte[],List<org.apache.hadoop.fs.Path>> bulkLoadHFiles(Collection<Pair<byte[], String>> familyPaths, boolean assignSeqId, HRegion.BulkLoadListener bulkLoadListener) throws IOExceptionAttempts to atomically load a group of hfiles. This is critical for loading rows with multiple column families atomically.- Parameters:
familyPaths
- List of Pair<byte[] column family, String hfilePath>bulkLoadListener
- Internal hooks enabling massaging/preparation of a file about to be bulk loaded- Returns:
- Map from family to List of store file paths if successful, null if failed recoverably
- Throws:
IOException
- if failed unrecoverably.
-
bulkLoadHFiles
public Map<byte[],List<org.apache.hadoop.fs.Path>> bulkLoadHFiles(Collection<Pair<byte[], String>> familyPaths, boolean assignSeqId, HRegion.BulkLoadListener bulkLoadListener, boolean copyFile, List<String> clusterIds, boolean replicate) throws IOExceptionAttempts to atomically load a group of hfiles. This is critical for loading rows with multiple column families atomically.- Parameters:
familyPaths
- List of Pair<byte[] column family, String hfilePath>bulkLoadListener
- Internal hooks enabling massaging/preparation of a file about to be bulk loadedcopyFile
- always copy hfiles if trueclusterIds
- ids from clusters that had already handled the given bulkload event.- Returns:
- Map from family to List of store file paths if successful, null if failed recoverably
- Throws:
IOException
- if failed unrecoverably.
-
equals
-
hashCode
-
toString
-
newHRegion
public static HRegion newHRegion(org.apache.hadoop.fs.Path tableDir, WAL wal, org.apache.hadoop.fs.FileSystem fs, org.apache.hadoop.conf.Configuration conf, RegionInfo regionInfo, TableDescriptor htd, RegionServerServices rsServices) A utility method to create new instances of HRegion based on theHConstants.REGION_IMPL
configuration property.- Parameters:
tableDir
- qualified path of directory where region should be located, usually the table directory.wal
- The WAL is the outbound log for any updates to the HRegion The wal file is a logfile from the previous execution that's custom-computed for this HRegion. The HRegionServer computes and sorts the appropriate wal info for this HRegion. If there is a previous file (implying that the HRegion has been written-to before), then read it from the supplied path.fs
- is the filesystem.conf
- is global configuration settings.regionInfo
- - RegionInfo that describes the region is new), then read them from the supplied path.htd
- the table descriptor- Returns:
- the new instance
-
createHRegion
public static HRegion createHRegion(RegionInfo info, org.apache.hadoop.fs.Path rootDir, org.apache.hadoop.conf.Configuration conf, TableDescriptor hTableDescriptor, WAL wal, boolean initialize) throws IOException Convenience method creating new HRegions. Used by createTable.- Parameters:
info
- Info for region to create.rootDir
- Root directory for HBase instancewal
- shared WALinitialize
- - true to initialize the region- Returns:
- new HRegion
- Throws:
IOException
-
createHRegion
public static HRegion createHRegion(RegionInfo info, org.apache.hadoop.fs.Path rootDir, org.apache.hadoop.conf.Configuration conf, TableDescriptor hTableDescriptor, WAL wal, boolean initialize, RegionServerServices rsRpcServices) throws IOException Convenience method creating new HRegions. Used by createTable.- Parameters:
info
- Info for region to create.rootDir
- Root directory for HBase instancewal
- shared WALinitialize
- - true to initialize the regionrsRpcServices
- An interface we can request flushes against.- Returns:
- new HRegion
- Throws:
IOException
-
createHRegion
public static HRegion createHRegion(org.apache.hadoop.conf.Configuration conf, RegionInfo regionInfo, org.apache.hadoop.fs.FileSystem fs, org.apache.hadoop.fs.Path tableDir, TableDescriptor tableDesc) throws IOException Create a region under the given table directory.- Throws:
IOException
-
createRegionDir
public static HRegionFileSystem createRegionDir(org.apache.hadoop.conf.Configuration configuration, RegionInfo ri, org.apache.hadoop.fs.Path rootDir) throws IOException Create the region directory in the filesystem.- Throws:
IOException
-
createHRegion
public static HRegion createHRegion(RegionInfo info, org.apache.hadoop.fs.Path rootDir, org.apache.hadoop.conf.Configuration conf, TableDescriptor hTableDescriptor, WAL wal) throws IOException - Throws:
IOException
-
openHRegion
public static HRegion openHRegion(RegionInfo info, TableDescriptor htd, WAL wal, org.apache.hadoop.conf.Configuration conf) throws IOException Open a Region.- Parameters:
info
- Info for region to be opened.wal
- WAL for region to use. This method will call WAL#setSequenceNumber(long) passing the result of the call to HRegion#getMinSequenceId() to ensure the wal id is properly kept up. HRegionStore does this every time it opens a new region.- Returns:
- new HRegion
- Throws:
IOException
-
openHRegion
public static HRegion openHRegion(RegionInfo info, TableDescriptor htd, WAL wal, org.apache.hadoop.conf.Configuration conf, RegionServerServices rsServices, CancelableProgressable reporter) throws IOException Open a Region.- Parameters:
info
- Info for region to be openedhtd
- the table descriptorwal
- WAL for region to use. This method will call WAL#setSequenceNumber(long) passing the result of the call to HRegion#getMinSequenceId() to ensure the wal id is properly kept up. HRegionStore does this every time it opens a new region.conf
- The Configuration object to use.rsServices
- An interface we can request flushes against.reporter
- An interface we can report progress against.- Returns:
- new HRegion
- Throws:
IOException
-
openHRegion
public static HRegion openHRegion(org.apache.hadoop.fs.Path rootDir, RegionInfo info, TableDescriptor htd, WAL wal, org.apache.hadoop.conf.Configuration conf) throws IOException Open a Region.- Parameters:
rootDir
- Root directory for HBase instanceinfo
- Info for region to be opened.htd
- the table descriptorwal
- WAL for region to use. This method will call WAL#setSequenceNumber(long) passing the result of the call to HRegion#getMinSequenceId() to ensure the wal id is properly kept up. HRegionStore does this every time it opens a new region.conf
- The Configuration object to use.- Returns:
- new HRegion
- Throws:
IOException
-
openHRegion
public static HRegion openHRegion(org.apache.hadoop.fs.Path rootDir, RegionInfo info, TableDescriptor htd, WAL wal, org.apache.hadoop.conf.Configuration conf, RegionServerServices rsServices, CancelableProgressable reporter) throws IOException Open a Region.- Parameters:
rootDir
- Root directory for HBase instanceinfo
- Info for region to be opened.htd
- the table descriptorwal
- WAL for region to use. This method will call WAL#setSequenceNumber(long) passing the result of the call to HRegion#getMinSequenceId() to ensure the wal id is properly kept up. HRegionStore does this every time it opens a new region.conf
- The Configuration object to use.rsServices
- An interface we can request flushes against.reporter
- An interface we can report progress against.- Returns:
- new HRegion
- Throws:
IOException
-
openHRegion
public static HRegion openHRegion(org.apache.hadoop.conf.Configuration conf, org.apache.hadoop.fs.FileSystem fs, org.apache.hadoop.fs.Path rootDir, RegionInfo info, TableDescriptor htd, WAL wal) throws IOException Open a Region.- Parameters:
conf
- The Configuration object to use.fs
- Filesystem to userootDir
- Root directory for HBase instanceinfo
- Info for region to be opened.htd
- the table descriptorwal
- WAL for region to use. This method will call WAL#setSequenceNumber(long) passing the result of the call to HRegion#getMinSequenceId() to ensure the wal id is properly kept up. HRegionStore does this every time it opens a new region.- Returns:
- new HRegion
- Throws:
IOException
-
openHRegion
public static HRegion openHRegion(org.apache.hadoop.conf.Configuration conf, org.apache.hadoop.fs.FileSystem fs, org.apache.hadoop.fs.Path rootDir, RegionInfo info, TableDescriptor htd, WAL wal, RegionServerServices rsServices, CancelableProgressable reporter) throws IOException Open a Region.- Parameters:
conf
- The Configuration object to use.fs
- Filesystem to userootDir
- Root directory for HBase instanceinfo
- Info for region to be opened.htd
- the table descriptorwal
- WAL for region to use. This method will call WAL#setSequenceNumber(long) passing the result of the call to HRegion#getMinSequenceId() to ensure the wal id is properly kept up. HRegionStore does this every time it opens a new region.rsServices
- An interface we can request flushes against.reporter
- An interface we can report progress against.- Returns:
- new HRegion
- Throws:
IOException
-
openHRegionFromTableDir
public static HRegion openHRegionFromTableDir(org.apache.hadoop.conf.Configuration conf, org.apache.hadoop.fs.FileSystem fs, org.apache.hadoop.fs.Path tableDir, RegionInfo info, TableDescriptor htd, WAL wal, RegionServerServices rsServices, CancelableProgressable reporter) throws IOException Open a Region.- Parameters:
conf
- The Configuration object to use.fs
- Filesystem to useinfo
- Info for region to be opened.htd
- the table descriptorwal
- WAL for region to use. This method will call WAL#setSequenceNumber(long) passing the result of the call to HRegion#getMinSequenceId() to ensure the wal id is properly kept up. HRegionStore does this every time it opens a new region.rsServices
- An interface we can request flushes against.reporter
- An interface we can report progress against.- Returns:
- new HRegion
- Throws:
IOException
-
getReplicationScope
-
openHRegion
public static HRegion openHRegion(HRegion other, CancelableProgressable reporter) throws IOException Useful when reopening a closed region (normally for unit tests)- Parameters:
other
- original objectreporter
- An interface we can report progress against.- Returns:
- new HRegion
- Throws:
IOException
-
openHRegion
- Throws:
IOException
-
openHRegion
Open HRegion. Calls initialize and sets sequenceId.- Returns:
- Returns
this
- Throws:
IOException
-
openReadOnlyFileSystemHRegion
public static HRegion openReadOnlyFileSystemHRegion(org.apache.hadoop.conf.Configuration conf, org.apache.hadoop.fs.FileSystem fs, org.apache.hadoop.fs.Path tableDir, RegionInfo info, TableDescriptor htd) throws IOException Open a Region on a read-only file-system (like hdfs snapshots)- Parameters:
conf
- The Configuration object to use.fs
- Filesystem to useinfo
- Info for region to be opened.htd
- the table descriptor- Returns:
- new HRegion
- Throws:
IOException
-
warmupHRegion
public static HRegion warmupHRegion(RegionInfo info, TableDescriptor htd, WAL wal, org.apache.hadoop.conf.Configuration conf, RegionServerServices rsServices, CancelableProgressable reporter) throws IOException - Throws:
IOException
-
getRegionDir
@Deprecated public static org.apache.hadoop.fs.Path getRegionDir(org.apache.hadoop.fs.Path tabledir, String name) Deprecated.For tests only; to be removed.Computes the Path of the HRegion- Parameters:
tabledir
- qualified path for tablename
- ENCODED region name- Returns:
- Path of HRegion directory
-
rowIsInRange
Determines if the specified row is within the row range specified by the specified RegionInfo- Parameters:
info
- RegionInfo that specifies the row rangerow
- row to be checked- Returns:
- true if the row is within the range specified by the RegionInfo
-
rowIsInRange
-
get
Description copied from interface:Region
Do a get based on the get parameter.- Specified by:
get
in interfaceRegion
- Parameters:
get
- query parameters- Returns:
- result of the operation
- Throws:
IOException
-
prepareGet
- Throws:
IOException
-
get
Description copied from interface:Region
Do a get based on the get parameter.- Specified by:
get
in interfaceRegion
- Parameters:
get
- query parameterswithCoprocessor
- invoke coprocessor or not. We don't want to always invoke cp.- Returns:
- list of cells resulting from the operation
- Throws:
IOException
-
get
private List<Cell> get(Get get, boolean withCoprocessor, long nonceGroup, long nonce) throws IOException - Throws:
IOException
-
getInternal
private List<Cell> getInternal(Get get, boolean withCoprocessor, long nonceGroup, long nonce) throws IOException - Throws:
IOException
-
metricsUpdateForGet
-
mutateRow
Description copied from interface:Region
Performs multiple mutations atomically on a single row.- Specified by:
mutateRow
in interfaceRegion
- Parameters:
rm
- object that specifies the set of mutations to perform atomically- Returns:
- results of Increment/Append operations. If no Increment/Append operations, it returns null
- Throws:
IOException
-
mutateRow
- Throws:
IOException
-
mutateRowsWithLocks
public void mutateRowsWithLocks(Collection<Mutation> mutations, Collection<byte[]> rowsToLock, long nonceGroup, long nonce) throws IOException Perform atomic (all or none) mutations within the region.- Specified by:
mutateRowsWithLocks
in interfaceRegion
- Parameters:
mutations
- The list of mutations to perform.mutations
can contain operations for multiple rows. Caller has to ensure that all rows are contained in this region.rowsToLock
- Rows to locknonceGroup
- Optional nonce group of the operation (client Id)nonce
- Optional nonce of the operation (unique random id to ensure "more idempotence") If multiple rows are locked care should be taken thatrowsToLock
is sorted in order to avoid deadlocks.- Throws:
IOException
-
getLoadStatistics
public org.apache.hadoop.hbase.shaded.protobuf.generated.ClientProtos.RegionLoadStats getLoadStatistics()Returns statistics about the current load of the region -
processRowsWithLocks
Description copied from interface:Region
Performs atomic multiple reads and writes on a given row.- Specified by:
processRowsWithLocks
in interfaceRegion
- Parameters:
processor
- The object defines the reads and writes to a row.- Throws:
IOException
-
processRowsWithLocks
public void processRowsWithLocks(RowProcessor<?, ?> processor, long nonceGroup, long nonce) throws IOExceptionDescription copied from interface:Region
Performs atomic multiple reads and writes on a given row.- Specified by:
processRowsWithLocks
in interfaceRegion
- Parameters:
processor
- The object defines the reads and writes to a row.nonceGroup
- Optional nonce group of the operation (client Id)nonce
- Optional nonce of the operation (unique random id to ensure "more idempotence")- Throws:
IOException
-
processRowsWithLocks
public void processRowsWithLocks(RowProcessor<?, ?> processor, long timeout, long nonceGroup, long nonce) throws IOExceptionDescription copied from interface:Region
Performs atomic multiple reads and writes on a given row.- Specified by:
processRowsWithLocks
in interfaceRegion
- Parameters:
processor
- The object defines the reads and writes to a row.timeout
- The timeout of the processor.process() execution Use a negative number to switch off the time boundnonceGroup
- Optional nonce group of the operation (client Id)nonce
- Optional nonce of the operation (unique random id to ensure "more idempotence")- Throws:
IOException
-
preProcess
- Throws:
IOException
-
doProcessRowWithTimeout
private void doProcessRowWithTimeout(RowProcessor<?, ?> processor, long now, HRegion region, List<Mutation> mutations, WALEdit walEdit, long timeout) throws IOException- Throws:
IOException
-
append
Description copied from interface:Region
Perform one or more append operations on a row.- Specified by:
append
in interfaceRegion
- Returns:
- result of the operation
- Throws:
IOException
-
append
- Throws:
IOException
-
increment
Description copied from interface:Region
Perform one or more increment operations on a row.- Specified by:
increment
in interfaceRegion
- Returns:
- result of the operation
- Throws:
IOException
-
increment
- Throws:
IOException
-
doWALAppend
private MultiVersionConcurrencyControl.WriteEntry doWALAppend(WALEdit walEdit, Durability durability, List<UUID> clusterIds, long now, long nonceGroup, long nonce) throws IOException - Throws:
IOException
-
doWALAppend
private MultiVersionConcurrencyControl.WriteEntry doWALAppend(WALEdit walEdit, Durability durability, List<UUID> clusterIds, long now, long nonceGroup, long nonce, long origLogSeqNum) throws IOException Returns writeEntry associated with this append- Throws:
IOException
-
checkFamily
- Throws:
NoSuchColumnFamilyException
-
heapSize
Description copied from interface:HeapSize
Return the approximate 'exclusive deep size' of implementing object. Includes count of payload and hosting object sizings. -
registerService
Registers a new protocol bufferService
subclass as a coprocessor endpoint to be available for handling Region#execService(com.google.protobuf.RpcController, org.apache.hadoop.hbase.protobuf.generated.ClientProtos.CoprocessorServiceCall) calls.Only a single instance may be registered per region for a given
Service
subclass (the instances are keyed onDescriptors.ServiceDescriptor.getFullName()
. After the first registration, subsequent calls with the same service name will fail with a return value offalse
.- Parameters:
instance
- theService
subclass instance to expose as a coprocessor endpoint- Returns:
true
if the registration was successful,false
otherwise
-
execService
public com.google.protobuf.Message execService(com.google.protobuf.RpcController controller, org.apache.hadoop.hbase.shaded.protobuf.generated.ClientProtos.CoprocessorServiceCall call) throws IOException Executes a single protocol buffer coprocessor endpointService
method using the registered protocol handlers.Service
implementations must be registered via theregisterService(com.google.protobuf.Service)
method before they are available.- Parameters:
controller
- anRpcContoller
implementation to pass to the invoked servicecall
- aCoprocessorServiceCall
instance identifying the service, method, and parameters for the method invocation- Returns:
- a protocol buffer
Message
instance containing the method's result - Throws:
IOException
- if no registered service handler is found or an error occurs during the invocation- See Also:
-
checkSplit
-
checkSplit
Return the split point. An empty result indicates the region isn't splittable. -
getCompactPriority
Returns The priority that this region should have in the compaction queue -
getCoprocessorHost
Returns the coprocessor host -
setCoprocessorHost
- Parameters:
coprocessorHost
- the new coprocessor host
-
startRegionOperation
Description copied from interface:Region
This method needs to be called before any public call that reads or modifies data. Acquires a read lock and checks if the region is closing or closed.Region.closeRegionOperation()
MUST then always be called after the operation has completed, whether it succeeded or failed.- Specified by:
startRegionOperation
in interfaceRegion
- Throws:
IOException
-
startRegionOperation
Description copied from interface:Region
This method needs to be called before any public call that reads or modifies data. Acquires a read lock and checks if the region is closing or closed.Region.closeRegionOperation()
MUST then always be called after the operation has completed, whether it succeeded or failed.- Specified by:
startRegionOperation
in interfaceRegion
- Parameters:
op
- The operation is about to be taken on the region- Throws:
IOException
-
closeRegionOperation
Description copied from interface:Region
Closes the region operation lock.- Specified by:
closeRegionOperation
in interfaceRegion
- Throws:
IOException
-
closeRegionOperation
Description copied from interface:Region
Closes the region operation lock. This needs to be called in the finally block corresponding to the try block ofRegion.startRegionOperation(Operation)
- Specified by:
closeRegionOperation
in interfaceRegion
- Throws:
IOException
-
startBulkRegionOperation
This method needs to be called before any public call that reads or modifies stores in bulk. It has to be called just before a try. #closeBulkRegionOperation needs to be called in the try's finally block Acquires a writelock and checks if the region is closing or closed.- Throws:
NotServingRegionException
- when the region is closing or closedRegionTooBusyException
- if failed to get the lock in timeInterruptedIOException
- if interrupted while waiting for a lockIOException
-
closeBulkRegionOperation
Closes the lock. This needs to be called in the finally block corresponding to the try block of #startRegionOperation -
recordMutationWithoutWal
Update LongAdders for number of puts without wal and the size of possible data loss. These information are exposed by the region server metrics. -
lock
- Throws:
IOException
-
lock
Try to acquire a lock. Throw RegionTooBusyException if failed to get the lock in time. Throw InterruptedIOException if interrupted while waiting for the lock.- Throws:
IOException
-
sync
Calls sync with the given transaction ID- Parameters:
txid
- should sync up to which transaction- Throws:
IOException
- If anything goes wrong with DFS
-
shouldSyncWAL
Check whether we should sync the wal from the table's durability settings -
getOpenSeqNum
Returns the latest sequence number that was read from storage when this region was opened -
getMaxStoreSeqId
- Specified by:
getMaxStoreSeqId
in interfaceRegion
- Returns:
- map of column family names to max sequence id that was read from storage when this region was opened
-
getOldestSeqIdOfStore
-
getCompactionState
Description copied from interface:Region
Returns if a given region is in compaction now.- Specified by:
getCompactionState
in interfaceRegion
-
reportCompactionRequestStart
-
reportCompactionRequestEnd
-
reportCompactionRequestFailure
-
incrementCompactionsQueuedCount
-
decrementCompactionsQueuedCount
-
incrementFlushesQueuedCount
-
decrementFlushesQueuedCount
-
disableInterrupts
void disableInterrupts()If a handler thread is eligible for interrupt, make it ineligible. Should be paired with {enableInterrupts()
. -
enableInterrupts
void enableInterrupts()If a handler thread was made ineligible for interrupt via {disableInterrupts()
, make it eligible again. No-op if interrupts are already enabled. -
interruptRegionOperations
Interrupt any region options that have acquired the region lock viastartRegionOperation(org.apache.hadoop.hbase.regionserver.Region.Operation)
, orstartBulkRegionOperation(boolean)
. -
checkInterrupt
Check thread interrupt status and throw an exception if interrupted.- Throws:
NotServingRegionException
- if region is closingInterruptedIOException
- if interrupted but region is not closing
-
throwOnInterrupt
Throw the correct exception upon interrupt- Parameters:
t
- cause
-
onConfigurationChange
This method would be called by theConfigurationManager
object when theConfiguration
object is reloaded from disk.- Specified by:
onConfigurationChange
in interfaceConfigurationObserver
-
registerChildren
Needs to be called to register the children to the manager.- Specified by:
registerChildren
in interfacePropagatingConfigurationObserver
- Parameters:
manager
- : to register to
-
deregisterChildren
Needs to be called to deregister the children from the manager.- Specified by:
deregisterChildren
in interfacePropagatingConfigurationObserver
- Parameters:
manager
- : to deregister from
-
getCellComparator
Description copied from interface:Region
The comparator to be used with the region- Specified by:
getCellComparator
in interfaceRegion
-
getMemStoreFlushSize
-
throwException
-
requestCompaction
public void requestCompaction(String why, int priority, boolean major, CompactionLifeCycleTracker tracker) throws IOException Description copied from interface:Region
Request compaction on this region.- Specified by:
requestCompaction
in interfaceRegion
- Throws:
IOException
-
requestCompaction
public void requestCompaction(byte[] family, String why, int priority, boolean major, CompactionLifeCycleTracker tracker) throws IOException Description copied from interface:Region
Request compaction for the given family- Specified by:
requestCompaction
in interfaceRegion
- Throws:
IOException
-
requestFlushIfNeeded
- Throws:
RegionTooBusyException
-
requestFlush
-
requestFlush0
-
requestFlush
Description copied from interface:Region
Request flush on this region.- Specified by:
requestFlush
in interfaceRegion
- Throws:
IOException
-
decorateRegionConfiguration
This method modifies the region's configuration in order to inject replication-related features- Parameters:
conf
- region configurations
-
addReadRequestsCount
-
addWriteRequestsCount
-
isReadsEnabled
boolean isReadsEnabled()
-