Class BucketCache
- All Implemented Interfaces:
Iterable<CachedBlock>,ConfigurationObserver,HeapSize,BlockCache
BucketAllocator to allocate/free blocks, and uses BucketCache#ramCache
and BucketCache#backingMap in order to determine if a given element is in the cache. The bucket
cache can use off-heap memory ByteBufferIOEngine or mmap
ExclusiveMemoryMmapIOEngine or pmem SharedMemoryMmapIOEngine or local files
FileIOEngine to store/read the block data.
Eviction is via a similar algorithm as used in
LruBlockCache
BucketCache can be used as mainly a block cache (see
CombinedBlockCache), combined with a BlockCache to
decrease CMS GC and heap fragmentation.
It also can be used as a secondary cache (e.g. using a file on ssd/fusionio to store blocks) to enlarge cache space via a victim cache.
-
Nested Class Summary
Nested ClassesModifier and TypeClassDescriptionprivate classUsed to group bucket entries into priority buckets.protected static enumEnum to represent the state of cache(package private) static classWrapped the delegate ConcurrentMap with maintaining its block's reference count.(package private) static classBlock Entry stored in the memory with key,data and so onprivate static class(package private) class -
Field Summary
FieldsModifier and TypeFieldDescription(package private) static final Stringprivate floatAcceptable size of cache (no evictions if size < acceptable)private final AtomicLongCache access count (sequential ID)private StringUseMessageDigestclass's encryption algorithms to check persistent file integrity, default algorithm is MD5private static final intprivate long(package private) static final String(package private) Map<BlockCacheKey,BucketEntry> private AtomicBoolean(package private) static final StringThe cache age of blocks to check if the related file is present on any online regions.(package private) static final longprivate final LongAdderCurrent number of cached elements(package private) NavigableSet<BlockCacheKey>private final longApproximate block sizeprivate BucketAllocatorprivate longprivate final longprivate BucketCachePersisterprivate BucketCache.CacheStateFlag if the cache is enabled or not...private final BucketCacheStatsprivate org.apache.hadoop.conf.Configuration(package private) static final float(package private) static final longprivate static final intstatic final int(package private) static final floatprivate static final String(package private) static final int(package private) static final float(package private) static final float(package private) static final floatprivate static final long(package private) static final floatPriority buckets(package private) static final int(package private) static final int(package private) static final Stringprivate floatFree this floating point factor of extra blocks when evicting.private static final Stringprivate booleanVolatile boolean to track if free space is in process or notprivate final LockMap of hFile -> Region -> File size.private final LongAdder(package private) final IOEngineprivate longprivate final intDuration of IO errors tolerated before we disable cache, 1 min as default(package private) static AtomicBooleanprivate static final org.slf4j.Logger(package private) static final Stringprivate floatIn-memory bucket size(package private) static final Stringprivate floatMinimum threshold of cache (when evicting, evict until size < min)(package private) static final Stringprivate floatMultiple access bucket size(package private) final IdReadWriteLock<Long>A ReentrantReadWriteLock to lock on a particular block identified by offset.private longprivate longprivate final String(package private) static final Stringprivate long(package private) final BucketCache.RAMCacheprivate final LongAdderMap of region -> total size of the region prefetched on this region server.private final ScheduledExecutorServiceStatistics thread schedule pool (for heavy debugging, could remove)(package private) static final StringPriority buckets configprivate floatSingle access bucket sizeprivate static final intStatistics threadprivate static final booleanprivate static final StringUse strong reference for offsetLock or not(package private) final ArrayList<BlockingQueue<BucketCache.RAMQueueEntry>>A list of writer queues.(package private) final BucketCache.WriterThread[] -
Constructor Summary
ConstructorsConstructorDescriptionBucketCache(String ioEngineName, long capacity, int blockSize, int[] bucketSizes, int writerThreadNum, int writerQLen, String persistencePath) BucketCache(String ioEngineName, long capacity, int blockSize, int[] bucketSizes, int writerThreadNum, int writerQLen, String persistencePath, int ioErrorsTolerationDuration, org.apache.hadoop.conf.Configuration conf) BucketCache(String ioEngineName, long capacity, int blockSize, int[] bucketSizes, int writerThreadNum, int writerQLen, String persistencePath, int ioErrorsTolerationDuration, org.apache.hadoop.conf.Configuration conf, Map<String, HRegion> onlineRegions) -
Method Summary
Modifier and TypeMethodDescriptionlong(package private) voidblockEvicted(BlockCacheKey cacheKey, BucketEntry bucketEntry, boolean decrementBlockNumber, boolean evictedByEvictionProcess) This method is invoked after the bucketEntry is removed frombackingMapblockFitsIntoTheCache(HFileBlock block) Checks whether there's enough space left in the cache to accommodate the passed block.private intbucketSizesAboveThresholdCount(float minFactor) Return the count of bucketSizeinfos still need free spacevoidcacheBlock(BlockCacheKey cacheKey, Cacheable buf) Cache the block with the specified name and buffer.voidcacheBlock(BlockCacheKey cacheKey, Cacheable cachedItem, boolean inMemory) Cache the block with the specified name and buffer.voidcacheBlock(BlockCacheKey cacheKey, Cacheable cachedItem, boolean inMemory, boolean waitWhenCache) Cache the block with the specified name and buffer.voidcacheBlockWithWait(BlockCacheKey cacheKey, Cacheable cachedItem, boolean inMemory, boolean wait) Cache the block to ramCacheprotected voidcacheBlockWithWaitInternal(BlockCacheKey cacheKey, Cacheable cachedItem, boolean inMemory, boolean wait) private longcalculateBytesToFree(StringBuilder msgBuffer) private voidCheck whether we tolerate IO error this time.createRecycler(BucketEntry bucketEntry) Create theByteBuffAllocator.RecyclerforBucketEntry.refCnt,which would be used asRefCnt.recyclerofHFileBlock#bufreturned fromgetBlock(org.apache.hadoop.hbase.io.hfile.BlockCacheKey, boolean, boolean, boolean).private voidUsed to shut down the cache -or- turn it off in the case of something broken.(package private) voiddoDrain(List<BucketCache.RAMQueueEntry> entries, ByteBuffer metaBuff) Flush the entries in ramCache to IOEngine and add bucket entry to backingMap.private booleandoEvictBlock(BlockCacheKey cacheKey, BucketEntry bucketEntry, boolean evictedByEvictionProcess) private voidbooleanevictBlock(BlockCacheKey cacheKey) Try to evict the block fromBlockCacheby force.booleanevictBlockIfNoRpcReferenced(BlockCacheKey blockCacheKey) NOTE: This method is only for test.intevictBlocksByHfileName(String hfileName) Evicts all blocks for a specific HFile.intevictBlocksRangeByHfileName(String hfileName, long initOffset, long endOffset) Evict all blocks for the given file name between the passed offset values.(package private) booleanevictBucketEntryIfNoRpcReferenced(BlockCacheKey blockCacheKey, BucketEntry bucketEntry) voidfileCacheCompleted(org.apache.hadoop.fs.Path filePath, long size) private voidfileNotFullyCached(String hfileName) protected voidfinalize()Needed mostly for UTs that might run in the same VM and create different BucketCache instances on different UT methods.(package private) voidfreeBucketEntry(BucketEntry bucketEntry) Free the {BucketEntryactually,which could only be invoked when theBucketEntry.refCntbecoming 0.private voidfreeEntireBuckets(int completelyFreeBucketsNeeded) This method will find the buckets that are minimally occupied and are not reference counted and will free them completely without any constraint on the access times of the elements, and as a process will completely free at most the number of buckets passed, sometimes it might not due to changing refCounts(package private) voidFree the space if the used size reaches acceptableSize() or one size block couldn't be allocated.(package private) floatprotected Stringprivate Set<BlockCacheKey>getAllCacheKeysForFile(String hfileName, long init, long end) private static StringPrepare and return a warning message for Bucket Allocator ExceptiongetBlock(BlockCacheKey key, boolean caching, boolean repeat, boolean updateCacheMetrics) Get the buffer of the block with the specified key.Returns The list of sub blockcaches that make up this one; returns null if no sub caches.longReturns the number of blocks currently cached in the block cache.If the passed cache key relates to a reference (<hfile>.<parentEncRegion>), this method looks for the block from the referred file, in the cache.Returns an Optional containing the size of the block related to the passed key.static Optional<BucketCache>getBucketCacheFromCacheConfig(CacheConfig cacheConf) (package private) longlongReturns the occupied size of data blocks, in bytes.longReturns the occupied size of the block cache, in bytes.longReturns the number of data blocks currently cached in the block cache.(package private) floatlongReturns the free size of the block cache, in bytes.Returns an Optional containing the map of files that have been fully cached (all its blocks are present in the cache.private IOEnginegetIOEngineFromName(String ioEngineName, long capacity, String persistencePath) Get the IOEngine from the IO engine namelongReturns the Max size of the block cache, in bytes.(package private) float(package private) float(package private) float(package private) longgetPartitionSize(float partitionFactor) (package private) long(package private) long(package private) static List<BucketCache.RAMQueueEntry>getRAMQueueEntries(BlockingQueue<BucketCache.RAMQueueEntry> q, List<BucketCache.RAMQueueEntry> receptacle) Blocks until elements available inqthen tries to grab as many as possible before returning.longReturns an Optional containing a map of regions and the percentage of how much of it has been cached so far.intgetRpcRefCount(BlockCacheKey cacheKey) (package private) floatgetStats()Get the statistics for this block cache.longheapSize()Return the approximate 'exclusive deep size' of implementing object.private voidCalled by the constructor to instantiate the writer threads.Checks whether the block for the passed key is already cached.booleanAPI to check whether or not, the cache is enabled.boolean(package private) booleanisCacheInitialized(String api) booleanbooleaniterator()Returns Iterator over the blocks in the cache.private voidjoin()voidlogStats()voidnotifyFileCachingCompleted(org.apache.hadoop.fs.Path fileName, int totalBlockCount, int dataBlockCount, long size) Notifies the cache implementation that the given file has been fully cached (all its blocks made into the cache).voidonConfigurationChange(org.apache.hadoop.conf.Configuration config) Since HBASE-29249, the following properties governin freeSpace behaviour and block priorities were made dynamically configurable: - hbase.bucketcache.acceptfactor - hbase.bucketcache.minfactor - hbase.bucketcache.extrafreefactor - hbase.bucketcache.single.factor - hbase.bucketcache.multi.factor - hbase.bucketcache.multi.factor - hbase.bucketcache.memory.factor The hbase.bucketcache.queue.addition.waittime property allows for introducing a delay in the publishing of blocks for the cache writer threads during prefetch reads only (client reads wouldn't get delayed).private voidparsePB(org.apache.hadoop.hbase.shaded.protobuf.generated.BucketCacheProtos.BucketCacheEntry proto) private void(package private) voidprotected voidputIntoBackingMap(BlockCacheKey key, BucketEntry bucketEntry) Put the new bucket entry into backingMap.protected booleanremoveFromRamCache(BlockCacheKey cacheKey) private voidprivate voidretrieveFromFile(int[] bucketSizes) private voidvoidsetCacheInconsistent(boolean setCacheInconsistent) protected voidshouldCacheBlock(BlockCacheKey key, long maxTimestamp, org.apache.hadoop.conf.Configuration conf) Checks whether the block represented by the given key should be cached or not.shouldCacheFile(HFileInfo hFileInfo, org.apache.hadoop.conf.Configuration conf) Checks whether blocks for the passed file should be cached or not.protected booleanshouldReplaceExistingCacheBlock(BlockCacheKey cacheKey, Cacheable newBlock) voidshutdown()Shutdown the cache.longsize()Returns the total size of the block cache, in bytes.(package private) voidprivate voidstartPersistenceRetriever(int[] bucketSizes, long capacity) protected voidCalled by the constructor to start the writer threads.(package private) voidOnly used in testprivate voidupdateCacheIndex(org.apache.hadoop.hbase.shaded.protobuf.generated.BucketCacheProtos.BackingMap chunk, Map<Integer, String> deserializer) private voidupdateRegionCachedSize(org.apache.hadoop.fs.Path filePath, long cachedSize) private voidprivate voidverifyCapacityAndClasses(long capacitySize, String ioclass, String mapclass) private voidverifyFileIntegrity(org.apache.hadoop.hbase.shaded.protobuf.generated.BucketCacheProtos.BucketCacheEntry proto) booleanwaitForCacheInitialization(long timeout) Wait for the block cache implementation to be completely enabled.Methods inherited from class java.lang.Object
clone, equals, getClass, hashCode, notify, notifyAll, toString, wait, wait, waitMethods inherited from interface org.apache.hadoop.hbase.io.hfile.BlockCache
getBlock, isMetaBlockMethods inherited from interface java.lang.Iterable
forEach, spliterator
-
Field Details
-
LOG
-
SINGLE_FACTOR_CONFIG_NAME
Priority buckets config- See Also:
-
MULTI_FACTOR_CONFIG_NAME
- See Also:
-
MEMORY_FACTOR_CONFIG_NAME
- See Also:
-
EXTRA_FREE_FACTOR_CONFIG_NAME
- See Also:
-
ACCEPT_FACTOR_CONFIG_NAME
- See Also:
-
MIN_FACTOR_CONFIG_NAME
- See Also:
-
BACKING_MAP_PERSISTENCE_CHUNK_SIZE
- See Also:
-
STRONG_REF_KEY
Use strong reference for offsetLock or not- See Also:
-
STRONG_REF_DEFAULT
- See Also:
-
BLOCK_ORPHAN_GRACE_PERIOD
The cache age of blocks to check if the related file is present on any online regions.- See Also:
-
BLOCK_ORPHAN_GRACE_PERIOD_DEFAULT
- See Also:
-
DEFAULT_SINGLE_FACTOR
Priority buckets- See Also:
-
DEFAULT_MULTI_FACTOR
- See Also:
-
DEFAULT_MEMORY_FACTOR
- See Also:
-
DEFAULT_MIN_FACTOR
- See Also:
-
DEFAULT_EXTRA_FREE_FACTOR
- See Also:
-
DEFAULT_ACCEPT_FACTOR
- See Also:
-
DEFAULT_FREE_ENTIRE_BLOCK_FACTOR
- See Also:
-
statThreadPeriod
Statistics thread- See Also:
-
DEFAULT_WRITER_THREADS
- See Also:
-
DEFAULT_WRITER_QUEUE_ITEMS
- See Also:
-
DEFAULT_BACKING_MAP_PERSISTENCE_CHUNK_SIZE
- See Also:
-
ioEngine
-
ramCache
-
backingMap
-
backingMapValidated
-
fullyCachedFiles
Map of hFile -> Region -> File size. This map is used to track all files completed prefetch, together with the region those belong to and the total cached size for the region.TestBlockEvictionOnRegionMovement -
regionCachedSize
Map of region -> total size of the region prefetched on this region server. This is the total size of hFiles for this region prefetched on this region server -
cachePersister
-
cacheState
Flag if the cache is enabled or not... We shut it off if there are IO errors for some time, so that Bucket IO exceptions/errors don't bring down the HBase server. -
writerQueues
A list of writer queues. We have a queue perBucketCache.WriterThreadwe have running. In other words, the work adding blocks to the BucketCache is divided up amongst the running WriterThreads. Its done by taking hash of the cache key modulo queue count. WriterThread when it runs takes whatever has been recently added and 'drains' the entries to the BucketCache. It then updates the ramCache and backingMap accordingly. -
writerThreads
-
freeInProgress
Volatile boolean to track if free space is in process or not -
freeSpaceLock
-
realCacheSize
-
heapSize
-
blockNumber
Current number of cached elements -
accessCount
Cache access count (sequential ID) -
DEFAULT_CACHE_WAIT_TIME
- See Also:
-
cacheStats
-
persistencePath
-
isCacheInconsistent
-
cacheCapacity
-
blockSize
Approximate block size -
ioErrorsTolerationDuration
Duration of IO errors tolerated before we disable cache, 1 min as default -
DEFAULT_ERROR_TOLERATION_DURATION
- See Also:
-
ioErrorStartTime
-
conf
-
offsetLock
A ReentrantReadWriteLock to lock on a particular block identified by offset. The purpose of this is to avoid freeing the block which is being read. -
blocksByHFile
-
scheduleThreadPool
Statistics thread schedule pool (for heavy debugging, could remove) -
bucketAllocator
-
acceptableFactor
Acceptable size of cache (no evictions if size < acceptable) -
minFactor
Minimum threshold of cache (when evicting, evict until size < min) -
extraFreeFactor
Free this floating point factor of extra blocks when evicting. For example free the number of blocks requested * (1 + extraFreeFactor) -
singleFactor
Single access bucket size -
multiFactor
Multiple access bucket size -
memoryFactor
In-memory bucket size -
bucketcachePersistInterval
-
FILE_VERIFY_ALGORITHM
- See Also:
-
DEFAULT_FILE_VERIFY_ALGORITHM
- See Also:
-
QUEUE_ADDITION_WAIT_TIME
- See Also:
-
DEFAULT_QUEUE_ADDITION_WAIT_TIME
- See Also:
-
queueAdditionWaitTime
-
algorithm
UseMessageDigestclass's encryption algorithms to check persistent file integrity, default algorithm is MD5 -
persistenceChunkSize
-
allocFailLogPrevTs
-
ALLOCATION_FAIL_LOG_TIME_PERIOD
- See Also:
-
onlineRegions
-
orphanBlockGracePeriod
-
-
Constructor Details
-
BucketCache
public BucketCache(String ioEngineName, long capacity, int blockSize, int[] bucketSizes, int writerThreadNum, int writerQLen, String persistencePath) throws IOException - Throws:
IOException
-
BucketCache
public BucketCache(String ioEngineName, long capacity, int blockSize, int[] bucketSizes, int writerThreadNum, int writerQLen, String persistencePath, int ioErrorsTolerationDuration, org.apache.hadoop.conf.Configuration conf) throws IOException - Throws:
IOException
-
BucketCache
public BucketCache(String ioEngineName, long capacity, int blockSize, int[] bucketSizes, int writerThreadNum, int writerQLen, String persistencePath, int ioErrorsTolerationDuration, org.apache.hadoop.conf.Configuration conf, Map<String, HRegion> onlineRegions) throws IOException- Throws:
IOException
-
-
Method Details
-
startPersistenceRetriever
-
sanityCheckConfigs
-
instantiateWriterThreads
Called by the constructor to instantiate the writer threads. -
startWriterThreads
Called by the constructor to start the writer threads. Used by tests that need to override starting the threads. -
startBucketCachePersisterThread
-
isCacheEnabled
Description copied from interface:BlockCacheAPI to check whether or not, the cache is enabled.- Specified by:
isCacheEnabledin interfaceBlockCache- Returns:
- returns true if the cache is enabled, false otherwise.
-
getMaxSize
Description copied from interface:BlockCacheReturns the Max size of the block cache, in bytes.- Specified by:
getMaxSizein interfaceBlockCache- Returns:
- size of cache, in bytes
-
getIoEngine
-
getIOEngineFromName
private IOEngine getIOEngineFromName(String ioEngineName, long capacity, String persistencePath) throws IOException Get the IOEngine from the IO engine name- Returns:
- the IOEngine
- Throws:
IOException
-
isCachePersistenceEnabled
-
cacheBlock
Cache the block with the specified name and buffer.- Specified by:
cacheBlockin interfaceBlockCache- Parameters:
cacheKey- block's cache keybuf- block buffer
-
cacheBlock
Cache the block with the specified name and buffer.- Specified by:
cacheBlockin interfaceBlockCache- Parameters:
cacheKey- block's cache keycachedItem- block bufferinMemory- if block is in-memory
-
cacheBlock
public void cacheBlock(BlockCacheKey cacheKey, Cacheable cachedItem, boolean inMemory, boolean waitWhenCache) Cache the block with the specified name and buffer.- Specified by:
cacheBlockin interfaceBlockCache- Parameters:
cacheKey- block's cache keycachedItem- block bufferinMemory- if block is in-memorywaitWhenCache- Whether to wait for the cache to be flushed mainly when BucketCache is configured.
-
cacheBlockWithWait
public void cacheBlockWithWait(BlockCacheKey cacheKey, Cacheable cachedItem, boolean inMemory, boolean wait) Cache the block to ramCache- Parameters:
cacheKey- block's cache keycachedItem- block bufferinMemory- if block is in-memorywait- if true, blocking wait when queue is full
-
shouldReplaceExistingCacheBlock
-
cacheBlockWithWaitInternal
protected void cacheBlockWithWaitInternal(BlockCacheKey cacheKey, Cacheable cachedItem, boolean inMemory, boolean wait) -
getBlockForReference
If the passed cache key relates to a reference (<hfile>.<parentEncRegion>), this method looks for the block from the referred file, in the cache. If present in the cache, the block for the referred file is returned, otherwise, this method returns null. It will also return null if the passed cache key doesn't relate to a reference.- Parameters:
key- the BlockCacheKey instance to look for in the cache.- Returns:
- the cached block from the referred file, null if there's no such block in the cache or the passed key doesn't relate to a reference.
-
getBlock
public Cacheable getBlock(BlockCacheKey key, boolean caching, boolean repeat, boolean updateCacheMetrics) Get the buffer of the block with the specified key.- Specified by:
getBlockin interfaceBlockCache- Parameters:
key- block's cache keycaching- true if the caller caches blocks on cache missesrepeat- Whether this is a repeat lookup for the same blockupdateCacheMetrics- Whether we should update cache metrics or not- Returns:
- buffer of specified cache key, or null if not in cache
-
blockEvicted
void blockEvicted(BlockCacheKey cacheKey, BucketEntry bucketEntry, boolean decrementBlockNumber, boolean evictedByEvictionProcess) This method is invoked after the bucketEntry is removed frombackingMap -
fileNotFullyCached
-
fileCacheCompleted
-
updateRegionCachedSize
-
freeBucketEntry
Free the {BucketEntryactually,which could only be invoked when theBucketEntry.refCntbecoming 0. -
evictBlock
Try to evict the block fromBlockCacheby force. We'll call this in few cases:
1. Close an HFile, and clear all cached blocks.
2. CallAdmin.clearBlockCache(TableName)to clear all blocks for a given table.
Firstly, we'll try to remove the block from RAMCache,and then try to evict from backingMap. Here we evict the block from backingMap immediately, but only free the reference from bucket cache by calling
BucketEntry.markedAsEvicted. If there're still some RPC referring this block, block can only be de-allocated when all of them release the block.NOTICE: we need to grab the write offset lock firstly before releasing the reference from bucket cache. if we don't, we may read an
BucketEntrywith refCnt = 0 whengetBlock(BlockCacheKey, boolean, boolean, boolean), it's a memory leak.- Specified by:
evictBlockin interfaceBlockCache- Parameters:
cacheKey- Block to evict- Returns:
- true to indicate whether we've evicted successfully or not.
-
doEvictBlock
private boolean doEvictBlock(BlockCacheKey cacheKey, BucketEntry bucketEntry, boolean evictedByEvictionProcess) Evict theBlockCacheKeyandBucketEntryfrombackingMapandramCache.
NOTE:When Evict frombackingMap,only the matchedBlockCacheKeyandBucketEntrycould be removed.- Parameters:
cacheKey-BlockCacheKeyto evict.bucketEntry-BucketEntrymatchedBlockCacheKeyto evict.- Returns:
- true to indicate whether we've evicted successfully or not.
-
createRecycler
Create the
ByteBuffAllocator.RecyclerforBucketEntry.refCnt,which would be used asRefCnt.recyclerofHFileBlock#bufreturned fromgetBlock(org.apache.hadoop.hbase.io.hfile.BlockCacheKey, boolean, boolean, boolean). NOTE: forgetBlock(org.apache.hadoop.hbase.io.hfile.BlockCacheKey, boolean, boolean, boolean),theRefCnt.recyclerofHFileBlock#buffrombackingMapandramCacheare different: 1.ForRefCnt.recyclerofHFileBlock#buffrombackingMap, it is the return value of currentcreateRecycler(org.apache.hadoop.hbase.io.hfile.bucket.BucketEntry)method. 2.ForRefCnt.recyclerofHFileBlock#buffromramCache, it isByteBuffAllocator.putbackBuffer(java.nio.ByteBuffer). -
evictBlockIfNoRpcReferenced
NOTE: This method is only for test. -
evictBucketEntryIfNoRpcReferenced
EvictBlockCacheKeyand its correspondingBucketEntryonly ifBucketEntry.isRpcRef()is false.
NOTE:When evict frombackingMap,only the matchedBlockCacheKeyandBucketEntrycould be removed.- Parameters:
blockCacheKey-BlockCacheKeyto evict.bucketEntry-BucketEntrymatchedBlockCacheKeyto evict.- Returns:
- true to indicate whether we've evicted successfully or not.
-
onConfigurationChange
Since HBASE-29249, the following properties governin freeSpace behaviour and block priorities were made dynamically configurable: - hbase.bucketcache.acceptfactor - hbase.bucketcache.minfactor - hbase.bucketcache.extrafreefactor - hbase.bucketcache.single.factor - hbase.bucketcache.multi.factor - hbase.bucketcache.multi.factor - hbase.bucketcache.memory.factor The hbase.bucketcache.queue.addition.waittime property allows for introducing a delay in the publishing of blocks for the cache writer threads during prefetch reads only (client reads wouldn't get delayed). It has also been made dynamic configurable since HBASE-29249. The hbase.bucketcache.persist.intervalinmillis propperty determines the frequency for saving the persistent cache, and it has also been made dynamically configurable since HBASE-29249. The hbase.bucketcache.persistence.chunksize property determines the size of the persistent file splits (due to the limitation of maximum allowed protobuff size), and it has also been made dynamically configurable since HBASE-29249.- Specified by:
onConfigurationChangein interfaceBlockCache- Specified by:
onConfigurationChangein interfaceConfigurationObserver- Parameters:
config- the new configuration to be updated.
-
removeFromRamCache
-
isCacheInconsistent
-
setCacheInconsistent
-
setCacheState
-
logStats
-
getRealCacheSize
-
acceptableSize
-
getPartitionSize
-
bucketSizesAboveThresholdCount
Return the count of bucketSizeinfos still need free space -
freeEntireBuckets
This method will find the buckets that are minimally occupied and are not reference counted and will free them completely without any constraint on the access times of the elements, and as a process will completely free at most the number of buckets passed, sometimes it might not due to changing refCounts- Parameters:
completelyFreeBucketsNeeded- number of buckets to free
-
calculateBytesToFree
-
freeSpace
Free the space if the used size reaches acceptableSize() or one size block couldn't be allocated. When freeing the space, we use the LRU algorithm and ensure there must be some blocks evicted- Parameters:
why- Why we are being called
-
putIntoBackingMap
Put the new bucket entry into backingMap. Notice that we are allowed to replace the existing cache with a new block for the same cache key. there's a corner case: one thread cache a block in ramCache, copy to io-engine and add a bucket entry to backingMap. Caching another new block with the same cache key do the same thing for the same cache key, so if not evict the previous bucket entry, then memory leak happen because the previous bucketEntry is gone but the bucketAllocator do not free its memory.- Parameters:
key- Block cache keybucketEntry- Bucket entry to put into backingMap.- See Also:
-
getAllocationFailWarningMessage
private static String getAllocationFailWarningMessage(BucketAllocatorException fle, BucketCache.RAMQueueEntry re) Prepare and return a warning message for Bucket Allocator Exception- Parameters:
fle- The exceptionre- The RAMQueueEntry for which the exception was thrown.- Returns:
- A warning message created from the input RAMQueueEntry object.
-
doDrain
void doDrain(List<BucketCache.RAMQueueEntry> entries, ByteBuffer metaBuff) throws InterruptedException Flush the entries in ramCache to IOEngine and add bucket entry to backingMap. Process all that are passed in even if failure being sure to remove from ramCache else we'll never undo the references and we'll OOME.- Parameters:
entries- Presumes list passed in here will be processed by this invocation only. No interference expected.- Throws:
InterruptedException
-
getRAMQueueEntries
static List<BucketCache.RAMQueueEntry> getRAMQueueEntries(BlockingQueue<BucketCache.RAMQueueEntry> q, List<BucketCache.RAMQueueEntry> receptacle) throws InterruptedException Blocks until elements available inqthen tries to grab as many as possible before returning.- Parameters:
receptacle- Where to stash the elements taken from queue. We clear before we use it just in case.q- The queue to take from.- Returns:
receptacleladen with elements taken from the queue or empty if none found.- Throws:
InterruptedException
-
persistToFile
- Throws:
IOException- See Also:
-
isCachePersistent
-
getRegionCachedInfo
Description copied from interface:BlockCacheReturns an Optional containing a map of regions and the percentage of how much of it has been cached so far.- Specified by:
getRegionCachedInfoin interfaceBlockCache- Returns:
- empty optional if this method is not supported, otherwise the returned optional contains a map of current regions caching percentage.
-
retrieveFromFile
- Throws:
IOException- See Also:
-
updateRegionSizeMapWhileRetrievingFromFile
-
dumpPrefetchList
-
verifyCapacityAndClasses
private void verifyCapacityAndClasses(long capacitySize, String ioclass, String mapclass) throws IOException - Throws:
IOException
-
verifyFileIntegrity
private void verifyFileIntegrity(org.apache.hadoop.hbase.shaded.protobuf.generated.BucketCacheProtos.BucketCacheEntry proto) -
updateCacheIndex
private void updateCacheIndex(org.apache.hadoop.hbase.shaded.protobuf.generated.BucketCacheProtos.BackingMap chunk, Map<Integer, String> deserializer) throws IOException- Throws:
IOException
-
parsePB
private void parsePB(org.apache.hadoop.hbase.shaded.protobuf.generated.BucketCacheProtos.BucketCacheEntry proto) throws IOException - Throws:
IOException
-
persistChunkedBackingMap
- Throws:
IOException
-
retrieveChunkedBackingMap
- Throws:
IOException
-
checkIOErrorIsTolerated
Check whether we tolerate IO error this time. If the duration of IOEngine throwing errors exceeds ioErrorsDurationTimeTolerated, we will disable the cache -
disableCache
Used to shut down the cache -or- turn it off in the case of something broken. -
join
- Throws:
InterruptedException
-
shutdown
Description copied from interface:BlockCacheShutdown the cache.- Specified by:
shutdownin interfaceBlockCache
-
finalize
Needed mostly for UTs that might run in the same VM and create different BucketCache instances on different UT methods. -
getStats
Description copied from interface:BlockCacheGet the statistics for this block cache.- Specified by:
getStatsin interfaceBlockCache
-
getAllocator
-
heapSize
Description copied from interface:HeapSizeReturn the approximate 'exclusive deep size' of implementing object. Includes count of payload and hosting object sizings. -
size
Description copied from interface:BlockCacheReturns the total size of the block cache, in bytes.- Specified by:
sizein interfaceBlockCache- Returns:
- size of cache, in bytes
-
getCurrentDataSize
Description copied from interface:BlockCacheReturns the occupied size of data blocks, in bytes.- Specified by:
getCurrentDataSizein interfaceBlockCache- Returns:
- occupied space in cache, in bytes
-
getFreeSize
Description copied from interface:BlockCacheReturns the free size of the block cache, in bytes.- Specified by:
getFreeSizein interfaceBlockCache- Returns:
- free space in cache, in bytes
-
getBlockCount
Description copied from interface:BlockCacheReturns the number of blocks currently cached in the block cache.- Specified by:
getBlockCountin interfaceBlockCache- Returns:
- number of blocks in the cache
-
getDataBlockCount
Description copied from interface:BlockCacheReturns the number of data blocks currently cached in the block cache.- Specified by:
getDataBlockCountin interfaceBlockCache- Returns:
- number of blocks in the cache
-
getCurrentSize
Description copied from interface:BlockCacheReturns the occupied size of the block cache, in bytes.- Specified by:
getCurrentSizein interfaceBlockCache- Returns:
- occupied space in cache, in bytes
-
getAlgorithm
-
evictBlocksByHfileName
Evicts all blocks for a specific HFile.This is used for evict-on-close to remove all blocks of a specific HFile.
- Specified by:
evictBlocksByHfileNamein interfaceBlockCache- Returns:
- the number of blocks evicted
-
evictBlocksRangeByHfileName
Description copied from interface:BlockCacheEvict all blocks for the given file name between the passed offset values.- Specified by:
evictBlocksRangeByHfileNamein interfaceBlockCache- Parameters:
hfileName- The file for which blocks should be evicted.initOffset- the initial offset for the range of blocks to be evicted.endOffset- the end offset for the range of blocks to be evicted.- Returns:
- number of blocks evicted.
-
getAllCacheKeysForFile
-
stopWriterThreads
Only used in test- Throws:
InterruptedException
-
iterator
Description copied from interface:BlockCacheReturns Iterator over the blocks in the cache.- Specified by:
iteratorin interfaceBlockCache- Specified by:
iteratorin interfaceIterable<CachedBlock>
-
getBlockCaches
Description copied from interface:BlockCacheReturns The list of sub blockcaches that make up this one; returns null if no sub caches.- Specified by:
getBlockCachesin interfaceBlockCache
-
getRpcRefCount
-
getAcceptableFactor
float getAcceptableFactor() -
getMinFactor
float getMinFactor() -
getExtraFreeFactor
float getExtraFreeFactor() -
getSingleFactor
float getSingleFactor() -
getMultiFactor
float getMultiFactor() -
getMemoryFactor
float getMemoryFactor() -
getQueueAdditionWaitTime
long getQueueAdditionWaitTime() -
getPersistenceChunkSize
long getPersistenceChunkSize() -
getBucketcachePersistInterval
-
getPersistencePath
-
getBackingMap
-
getBackingMapValidated
-
getFullyCachedFiles
Description copied from interface:BlockCacheReturns an Optional containing the map of files that have been fully cached (all its blocks are present in the cache. This method may not be overridden by all implementing classes. In such cases, the returned Optional will be empty.- Specified by:
getFullyCachedFilesin interfaceBlockCache- Returns:
- empty optional if this method is not supported, otherwise the returned optional contains a map of all files that have been fully cached.
-
getBucketCacheFromCacheConfig
-
notifyFileCachingCompleted
public void notifyFileCachingCompleted(org.apache.hadoop.fs.Path fileName, int totalBlockCount, int dataBlockCount, long size) Description copied from interface:BlockCacheNotifies the cache implementation that the given file has been fully cached (all its blocks made into the cache).- Specified by:
notifyFileCachingCompletedin interfaceBlockCache- Parameters:
fileName- the file that has been completely cached.totalBlockCount- the total of blocks cached for this file.dataBlockCount- number of DATA block type cached.size- the size, in bytes, cached.
-
blockFitsIntoTheCache
Description copied from interface:BlockCacheChecks whether there's enough space left in the cache to accommodate the passed block. This method may not be overridden by all implementing classes. In such cases, the returned Optional will be empty. For subclasses implementing this logic, the returned Optional would contain the boolean value reflecting if the passed block fits into the remaining cache space available.- Specified by:
blockFitsIntoTheCachein interfaceBlockCache- Parameters:
block- the block we want to check if fits into the cache.- Returns:
- empty optional if this method is not supported, otherwise the returned optional contains the boolean value informing if the block fits into the cache available space.
-
shouldCacheFile
public Optional<Boolean> shouldCacheFile(HFileInfo hFileInfo, org.apache.hadoop.conf.Configuration conf) Description copied from interface:BlockCacheChecks whether blocks for the passed file should be cached or not. This method may not be overridden by all implementing classes. In such cases, the returned Optional will be empty. For subclasses implementing this logic, the returned Optional would contain the boolean value reflecting if the passed file should indeed be cached.- Specified by:
shouldCacheFilein interfaceBlockCache- Parameters:
hFileInfo- Information about the file to check if it should be cached.conf- The configuration object to use for determining caching behavior.- Returns:
- empty optional if this method is not supported, otherwise the returned optional contains the boolean value informing if the file should be cached.
-
shouldCacheBlock
public Optional<Boolean> shouldCacheBlock(BlockCacheKey key, long maxTimestamp, org.apache.hadoop.conf.Configuration conf) Description copied from interface:BlockCacheChecks whether the block represented by the given key should be cached or not. This method may not be overridden by all implementing classes. In such cases, the returned Optional will be empty. For subclasses implementing this logic, the returned Optional would contain the boolean value reflecting if the passed block should indeed be cached.- Specified by:
shouldCacheBlockin interfaceBlockCache- Parameters:
key- The key representing the block to check if it should be cached.maxTimestamp- The maximum timestamp for the block to check if it should be cached.conf- The configuration object to use for determining caching behavior.- Returns:
- An empty Optional if this method is not supported; otherwise, the returned Optional contains the boolean value indicating if the block should be cached.
-
isAlreadyCached
Description copied from interface:BlockCacheChecks whether the block for the passed key is already cached. This method may not be overridden by all implementing classes. In such cases, the returned Optional will be empty. For subclasses implementing this logic, the returned Optional would contain the boolean value reflecting if the block for the passed key is already cached or not.- Specified by:
isAlreadyCachedin interfaceBlockCache- Parameters:
key- for the block we want to check if it's already in the cache.- Returns:
- empty optional if this method is not supported, otherwise the returned optional contains the boolean value informing if the block is already cached.
-
getBlockSize
Description copied from interface:BlockCacheReturns an Optional containing the size of the block related to the passed key. If the block is not in the cache, returned optional will be empty. Also, this method may not be overridden by all implementing classes. In such cases, the returned Optional will be empty.- Specified by:
getBlockSizein interfaceBlockCache- Parameters:
key- for the block we want to check if it's already in the cache.- Returns:
- empty optional if this method is not supported, otherwise the returned optional contains the boolean value informing if the block is already cached.
-
isCacheInitialized
-
waitForCacheInitialization
Description copied from interface:BlockCacheWait for the block cache implementation to be completely enabled. Some block cache implementations may take longer to initialise, and this initialisation may be asynchronous.- Specified by:
waitForCacheInitializationin interfaceBlockCache- Parameters:
timeout- time to wait for the cache to become enabled.- Returns:
- boolean true if the cache is enabled, false otherwise.
-