Class CompactingMemStore
java.lang.Object
org.apache.hadoop.hbase.regionserver.AbstractMemStore
org.apache.hadoop.hbase.regionserver.CompactingMemStore
- All Implemented Interfaces:
Closeable,AutoCloseable,MemStore
A memstore implementation which supports in-memory compaction. A compaction pipeline is added
between the active set and the snapshot data structures; it consists of a list of segments that
are subject to compaction. Like the snapshot, all pipeline segments are read-only; updates only
affect the active set. To ensure this property we take advantage of the existing blocking
mechanism -- the active set is pushed to the pipeline while holding the region's updatesLock in
exclusive mode. Periodically, a compaction is applied in the background to all pipeline segments
resulting in a single read-only component. The ``old'' segments are discarded when no scanner is
reading them.
-
Nested Class Summary
Nested ClassesModifier and TypeClassDescriptionstatic enumTypes of indexes (part of immutable segments) to be used after flattening, compaction, or merge are applied.private classThe in-memory-flusher thread performs the flush asynchronously. -
Field Summary
FieldsModifier and TypeFieldDescriptionprotected final AtomicBooleanstatic final Stringstatic final Stringprotected MemStoreCompactorprivate booleanstatic final longstatic final intstatic final Stringprivate static final intstatic final Stringprivate CompactingMemStore.IndexTypeprivate final AtomicBooleanprivate longprivate booleanprivate static final org.slf4j.Loggerprivate CompactionPipelineprivate HStoreFields inherited from class org.apache.hadoop.hbase.regionserver.AbstractMemStore
FIXED_OVERHEAD, regionServices, snapshot, snapshotId -
Constructor Summary
ConstructorsConstructorDescriptionCompactingMemStore(org.apache.hadoop.conf.Configuration conf, CellComparator c, HStore store, RegionServicesForStores regionServices, MemoryCompactionPolicy compactionPolicy) -
Method Summary
Modifier and TypeMethodDescriptionprotected booleancheckAndAddToActiveSize(MutableSegment currActive, Cell cellToAdd, MemStoreSizing memstoreSizing) Check whether anything need to be done based on the current active set size.protected List<KeyValueScanner>createList(int capacity) protected MemStoreCompactorcreateMemStoreCompactor(MemoryCompactionPolicy compactionPolicy) voiddebug()voidflattenOneSegment(long requesterVersion, MemStoreCompactionStrategy.Action action) (package private) voidprotected voidflushInMemory(MutableSegment currActive) private byte[]Flush will first clear out the data in snapshot if any (It will take a second flush invocation to clear the current Cell set).(package private) longprivate Segment(package private) CellgetNextRow(Cell cell) private ThreadPoolExecutorgetPool()private RegionServicesForStoresgetScanners(long readPt) This method is protected underHStore#lockread lock.Returns an ordered list of segments from most recent to oldest in memstorelonggetStore()booleanprotected longheapSize()private voidinitInmemoryFlushSize(org.apache.hadoop.conf.Configuration conf) (package private) void(package private) booleanbooleanisSloppy()protected longkeySize()Returns The total size of cells in this memstore.protected voidpostUpdate(MutableSegment currentActive) Issue any post update synchronization and testslongThis method is called before the flush is executed.protected booleanpreUpdate(MutableSegment currentActive, Cell cell, MemStoreSizing memstoreSizing) Issue any synchronization and test needed before applying the update For compacting memstore this means checking the update can increase the size without overflowprotected voidpushActiveToPipeline(MutableSegment currActive, boolean checkEmpty) NOTE: WhenflushInMemory(MutableSegment)calls this method, due to concurrent writes and because we first add cell size to currActive.getDataSize and then actually add cell to currActive.cellSet, it is possible that currActive.getDataSize could not accommodate cellToAdd but currActive.cellSet is still empty if pending writes which not yet add cells to currActive.cellSet,so forflushInMemory(MutableSegment),checkEmpty parameter is false.private voidprivate voidprivate voidpushToSnapshot(List<ImmutableSegment> segments) voidsetCompositeSnapshot(boolean useCompositeSnapshot) (package private) voidvoidprotected booleansize()protected booleansnapshot()Push the current active memstore segment into the pipeline and create a snapshot of the tail of current compaction pipeline Snapshot must be cleared by call toAbstractMemStore.clearSnapshot(long).voidThis message intends to inform the MemStore that next coming updates are going to be part of the replaying edits from WALprivate voidThe request to cancel the compaction asynchronous task (caused by in-memory flush) The compaction may still happen if the request was sent too late Non-blocking requestvoidThis message intends to inform the MemStore that the replaying edits from WAL are donebooleanswapCompactedSegments(VersionedSegmentsList versionedList, ImmutableSegment result, boolean merge) protected booleanswapPipelineWithNull(VersionedSegmentsList segments) private voidtryFlushInMemoryAndCompactingAsync(MutableSegment currActive) Try to flush the currActive in memory and submit the backgroundCompactingMemStore.InMemoryCompactionRunnabletoRegionServicesForStores.getInMemoryCompactionPool().voidupdateLowestUnflushedSequenceIdInWAL(boolean onlyIfGreater) Updates the wal with the lowest sequence id (oldest entry) that is still in memoryMethods inherited from class org.apache.hadoop.hbase.regionserver.AbstractMemStore
add, add, addToScanners, addToScanners, clearSnapshot, close, doAdd, doClearSnapShot, dump, getActive, getComparator, getConfiguration, getLowest, getNextRow, getSnapshot, getSnapshotSize, resetActive, resetTimeOfOldestEdit, timeOfOldestEdit, toString, upsert
-
Field Details
-
COMPACTING_MEMSTORE_TYPE_KEY
- See Also:
-
COMPACTING_MEMSTORE_TYPE_DEFAULT
-
IN_MEMORY_FLUSH_THRESHOLD_FACTOR_KEY
- See Also:
-
IN_MEMORY_FLUSH_MULTIPLIER
- See Also:
-
IN_MEMORY_CONPACTION_POOL_SIZE_KEY
- See Also:
-
IN_MEMORY_CONPACTION_POOL_SIZE_DEFAULT
- See Also:
-
LOG
-
store
-
pipeline
-
compactor
-
inmemoryFlushSize
-
inMemoryCompactionInProgress
-
inWalReplay
-
allowCompaction
-
compositeSnapshot
-
indexType
-
DEEP_OVERHEAD
-
-
Constructor Details
-
CompactingMemStore
public CompactingMemStore(org.apache.hadoop.conf.Configuration conf, CellComparator c, HStore store, RegionServicesForStores regionServices, MemoryCompactionPolicy compactionPolicy) throws IOException - Throws:
IOException
-
-
Method Details
-
createMemStoreCompactor
protected MemStoreCompactor createMemStoreCompactor(MemoryCompactionPolicy compactionPolicy) throws IllegalArgumentIOException - Throws:
IllegalArgumentIOException
-
initInmemoryFlushSize
-
size
- Returns:
- Total memory occupied by this MemStore. This won't include any size occupied by the snapshot. We assume the snapshot will get cleared soon. This is not thread safe and the memstore may be changed while computing its size. It is the responsibility of the caller to make sure this doesn't happen.
-
preFlushSeqIDEstimation
This method is called before the flush is executed.- Returns:
- an estimation (lower bound) of the unflushed sequence id in memstore after the flush is
executed. if memstore will be cleared returns
HConstants.NO_SEQNUM.
-
isSloppy
-
snapshot
Push the current active memstore segment into the pipeline and create a snapshot of the tail of current compaction pipeline Snapshot must be cleared by call toAbstractMemStore.clearSnapshot(long).AbstractMemStore.clearSnapshot(long).- Returns:
MemStoreSnapshot
-
getFlushableSize
Description copied from interface:MemStoreFlush will first clear out the data in snapshot if any (It will take a second flush invocation to clear the current Cell set). If snapshot is empty, current Cell set will be flushed.- Returns:
- On flush, how much memory we will clear.
-
setInMemoryCompactionCompleted
-
setInMemoryCompactionFlag
-
keySize
Description copied from class:AbstractMemStoreReturns The total size of cells in this memstore. We will not consider cells in the snapshot- Specified by:
keySizein classAbstractMemStore
-
heapSize
- Specified by:
heapSizein classAbstractMemStore- Returns:
- The total heap size of cells in this memstore. We will not consider cells in the snapshot
-
updateLowestUnflushedSequenceIdInWAL
Description copied from class:AbstractMemStoreUpdates the wal with the lowest sequence id (oldest entry) that is still in memory- Specified by:
updateLowestUnflushedSequenceIdInWALin classAbstractMemStore- Parameters:
onlyIfGreater- a flag that marks whether to update the sequence id no matter what or only if it is greater than the previous sequence id
-
startReplayingFromWAL
This message intends to inform the MemStore that next coming updates are going to be part of the replaying edits from WAL -
stopReplayingFromWAL
This message intends to inform the MemStore that the replaying edits from WAL are done -
preUpdate
Issue any synchronization and test needed before applying the update For compacting memstore this means checking the update can increase the size without overflow- Specified by:
preUpdatein classAbstractMemStore- Parameters:
currentActive- the segment to be updatedcell- the cell to be addedmemstoreSizing- object to accumulate region size changes- Returns:
- true iff can proceed with applying the update
-
postUpdate
Description copied from class:AbstractMemStoreIssue any post update synchronization and tests- Specified by:
postUpdatein classAbstractMemStore- Parameters:
currentActive- updated segment
-
sizeAddedPreOperation
- Specified by:
sizeAddedPreOperationin classAbstractMemStore
-
getSegments
Description copied from class:AbstractMemStoreReturns an ordered list of segments from most recent to oldest in memstore- Specified by:
getSegmentsin classAbstractMemStore
-
setCompositeSnapshot
-
swapCompactedSegments
public boolean swapCompactedSegments(VersionedSegmentsList versionedList, ImmutableSegment result, boolean merge) -
flattenOneSegment
- Parameters:
requesterVersion- The caller must hold the VersionedList of the pipeline with version taken earlier. This version must be passed as a parameter here. The flattening happens only if versions match.
-
setIndexType
-
getIndexType
-
hasImmutableSegments
-
getImmutableSegments
-
getSmallestReadPoint
-
getStore
-
getFamilyName
-
getScanners
This method is protected underHStore#lockread lock.- Returns:
- scanner over the memstore. This might include scanner over the snapshot when one is present.
- Throws:
IOException
-
createList
-
checkAndAddToActiveSize
protected boolean checkAndAddToActiveSize(MutableSegment currActive, Cell cellToAdd, MemStoreSizing memstoreSizing) Check whether anything need to be done based on the current active set size. The method is invoked upon every addition to the active set. For CompactingMemStore, flush the active set to the read-only memory if it's size is above threshold- Parameters:
currActive- intended segment to updatecellToAdd- cell to be added to the segmentmemstoreSizing- object to accumulate changed size- Returns:
- true if the cell can be added to the currActive
-
tryFlushInMemoryAndCompactingAsync
Try to flush the currActive in memory and submit the backgroundCompactingMemStore.InMemoryCompactionRunnabletoRegionServicesForStores.getInMemoryCompactionPool(). Just one thread can do the actual flushing in memory.- Parameters:
currActive- current Active Segment to be flush in memory.
-
flushInMemory
void flushInMemory() -
flushInMemory
-
inMemoryCompaction
void inMemoryCompaction() -
getLastSegment
-
getFamilyNameInBytes
-
getPool
-
stopCompaction
The request to cancel the compaction asynchronous task (caused by in-memory flush) The compaction may still happen if the request was sent too late Non-blocking request -
pushActiveToPipeline
NOTE: WhenflushInMemory(MutableSegment)calls this method, due to concurrent writes and because we first add cell size to currActive.getDataSize and then actually add cell to currActive.cellSet, it is possible that currActive.getDataSize could not accommodate cellToAdd but currActive.cellSet is still empty if pending writes which not yet add cells to currActive.cellSet,so forflushInMemory(MutableSegment),checkEmpty parameter is false. But ifsnapshot()called this method,because there is no pending write,checkEmpty parameter could be true. -
pushTailToSnapshot
-
pushPipelineToSnapshot
-
swapPipelineWithNull
-
pushToSnapshot
-
getRegionServices
-
isMemStoreFlushingInMemory
boolean isMemStoreFlushingInMemory() -
getNextRow
- Parameters:
cell- Find the row that comes after this one. If null, we return the first.- Returns:
- Next row or null if none found.
-
getInmemoryFlushSize
long getInmemoryFlushSize() -
debug
-