This is an automated email from the ASF dual-hosted git repository.
wchevreuil pushed a commit to branch branch-3
in repository https://gitbox.apache.org/repos/asf/hbase.git
The following commit(s) were added to refs/heads/branch-3 by this push:
new 6527c7ca455 HBASE-29243 Fix BucketCache.notifyFileCachingComplete to
also account for ENCODED_DATA block type (#6885)
6527c7ca455 is described below
commit 6527c7ca45567a6a62b77eb658aae46395612022
Author: Wellington Ramos Chevreuil <[email protected]>
AuthorDate: Thu Apr 10 11:44:34 2025 +0100
HBASE-29243 Fix BucketCache.notifyFileCachingComplete to also account for
ENCODED_DATA block type (#6885)
Signed-off-by: Peter Somogyi <[email protected]>
Reviewed-by: Rahul Agarkar <[email protected]>
---
.../hadoop/hbase/io/hfile/bucket/BucketCache.java | 20 +++++---------
.../hadoop/hbase/io/hfile/CacheTestUtils.java | 24 ++++++++++------
.../hbase/io/hfile/bucket/TestBucketCache.java | 32 +++++++++++++++++++---
3 files changed, 51 insertions(+), 25 deletions(-)
diff --git
a/hbase-server/src/main/java/org/apache/hadoop/hbase/io/hfile/bucket/BucketCache.java
b/hbase-server/src/main/java/org/apache/hadoop/hbase/io/hfile/bucket/BucketCache.java
index 6388b5ea7da..74e6580e7db 100644
---
a/hbase-server/src/main/java/org/apache/hadoop/hbase/io/hfile/bucket/BucketCache.java
+++
b/hbase-server/src/main/java/org/apache/hadoop/hbase/io/hfile/bucket/BucketCache.java
@@ -2296,11 +2296,10 @@ public class BucketCache implements BlockCache,
HeapSize {
ReentrantReadWriteLock lock = offsetLock.getLock(entry.getOffset());
lock.readLock().lock();
locks.add(lock);
- if (backingMap.containsKey(entry) && entry.getBlockType() ==
BlockType.DATA) {
+ if (backingMap.containsKey(entry) && entry.getBlockType().isData()) {
count.increment();
}
});
- int metaCount = totalBlockCount - dataBlockCount;
// BucketCache would only have data blocks
if (dataBlockCount == count.getValue()) {
LOG.debug("File {} has now been fully cached.", fileName);
@@ -2320,18 +2319,13 @@ public class BucketCache implements BlockCache,
HeapSize {
+ "and try the verification again.", fileName);
Thread.sleep(100);
notifyFileCachingCompleted(fileName, totalBlockCount,
dataBlockCount, size);
- } else if (
- (getAllCacheKeysForFile(fileName.getName(), 0,
Long.MAX_VALUE).size() - metaCount)
- == dataBlockCount
- ) {
- LOG.debug("We counted {} data blocks, expected was {}, there was no
more pending in "
- + "the cache write queue but we now found that total cached blocks
for file {} "
- + "is equal to data block count.", count, dataBlockCount,
fileName.getName());
- fileCacheCompleted(fileName, size);
} else {
- LOG.info("We found only {} data blocks cached from a total of {} for
file {}, "
- + "but no blocks pending caching. Maybe cache is full or evictions
"
- + "happened concurrently to cache prefetch.", count,
dataBlockCount, fileName);
+ LOG.info(
+ "The total block count was {}. We found only {} data blocks cached
from "
+ + "a total of {} data blocks for file {}, "
+ + "but no blocks pending caching. Maybe cache is full or
evictions "
+ + "happened concurrently to cache prefetch.",
+ totalBlockCount, count, dataBlockCount, fileName);
}
}
} catch (InterruptedException e) {
diff --git
a/hbase-server/src/test/java/org/apache/hadoop/hbase/io/hfile/CacheTestUtils.java
b/hbase-server/src/test/java/org/apache/hadoop/hbase/io/hfile/CacheTestUtils.java
index 9c40da0db01..e451ec01554 100644
---
a/hbase-server/src/test/java/org/apache/hadoop/hbase/io/hfile/CacheTestUtils.java
+++
b/hbase-server/src/test/java/org/apache/hadoop/hbase/io/hfile/CacheTestUtils.java
@@ -40,6 +40,7 @@ import
org.apache.hadoop.hbase.MultithreadedTestUtil.TestThread;
import org.apache.hadoop.hbase.io.ByteBuffAllocator;
import org.apache.hadoop.hbase.io.HeapSize;
import org.apache.hadoop.hbase.io.compress.Compression;
+import org.apache.hadoop.hbase.io.encoding.DataBlockEncoding;
import org.apache.hadoop.hbase.io.hfile.bucket.BucketCache;
import org.apache.hadoop.hbase.nio.ByteBuff;
import org.apache.hadoop.hbase.util.Bytes;
@@ -282,7 +283,8 @@ public class CacheTestUtils {
return generateBlocksForPath(blockSize, numBlocks, null);
}
- public static HFileBlockPair[] generateBlocksForPath(int blockSize, int
numBlocks, Path path) {
+ public static HFileBlockPair[] generateBlocksForPath(int blockSize, int
numBlocks, Path path,
+ boolean encoded) {
HFileBlockPair[] returnedBlocks = new HFileBlockPair[numBlocks];
Random rand = ThreadLocalRandom.current();
HashSet<String> usedStrings = new HashSet<>();
@@ -301,12 +303,13 @@ public class CacheTestUtils {
HFileContext meta =
new
HFileContextBuilder().withHBaseCheckSum(false).withIncludesMvcc(includesMemstoreTS)
.withIncludesTags(false).withCompression(Compression.Algorithm.NONE)
- .withBytesPerCheckSum(0).withChecksumType(ChecksumType.NULL).build();
- HFileBlock generated =
- new HFileBlock(BlockType.DATA, onDiskSizeWithoutHeader,
uncompressedSizeWithoutHeader,
- prevBlockOffset, ByteBuff.wrap(cachedBuffer),
HFileBlock.DONT_FILL_HEADER, blockSize,
- onDiskSizeWithoutHeader + HConstants.HFILEBLOCK_HEADER_SIZE, -1,
meta,
- ByteBuffAllocator.HEAP);
+
.withDataBlockEncoding(DataBlockEncoding.FAST_DIFF).withBytesPerCheckSum(0)
+ .withChecksumType(ChecksumType.NULL).build();
+ HFileBlock generated = new HFileBlock(encoded ? BlockType.ENCODED_DATA :
BlockType.DATA,
+ onDiskSizeWithoutHeader, uncompressedSizeWithoutHeader,
prevBlockOffset,
+ ByteBuff.wrap(cachedBuffer), HFileBlock.DONT_FILL_HEADER, blockSize,
+ onDiskSizeWithoutHeader + HConstants.HFILEBLOCK_HEADER_SIZE, -1, meta,
+ ByteBuffAllocator.HEAP);
String key = null;
long offset = 0;
if (path != null) {
@@ -320,12 +323,17 @@ public class CacheTestUtils {
}
}
returnedBlocks[i] = new HFileBlockPair();
- returnedBlocks[i].blockName = new BlockCacheKey(key, offset);
+ returnedBlocks[i].blockName =
+ new BlockCacheKey(key, offset, true, encoded ? BlockType.ENCODED_DATA
: BlockType.DATA);
returnedBlocks[i].block = generated;
}
return returnedBlocks;
}
+ public static HFileBlockPair[] generateBlocksForPath(int blockSize, int
numBlocks, Path path) {
+ return generateBlocksForPath(blockSize, numBlocks, path, false);
+ }
+
public static class HFileBlockPair {
BlockCacheKey blockName;
HFileBlock block;
diff --git
a/hbase-server/src/test/java/org/apache/hadoop/hbase/io/hfile/bucket/TestBucketCache.java
b/hbase-server/src/test/java/org/apache/hadoop/hbase/io/hfile/bucket/TestBucketCache.java
index 67e498364ef..6b6191e9f84 100644
---
a/hbase-server/src/test/java/org/apache/hadoop/hbase/io/hfile/bucket/TestBucketCache.java
+++
b/hbase-server/src/test/java/org/apache/hadoop/hbase/io/hfile/bucket/TestBucketCache.java
@@ -927,7 +927,31 @@ public class TestBucketCache {
try {
Path filePath =
new Path(HBASE_TESTING_UTILITY.getDataTestDir(),
"testNotifyFileCachingCompletedSuccess");
- bucketCache = testNotifyFileCachingCompletedForTenBlocks(filePath, 10);
+ bucketCache = testNotifyFileCachingCompletedForTenBlocks(filePath, 10,
false);
+ if (bucketCache.getStats().getFailedInserts() > 0) {
+ LOG.info("There were {} fail inserts, "
+ + "will assert if total blocks in backingMap equals (10 -
failInserts) "
+ + "and file isn't listed as fully cached.",
bucketCache.getStats().getFailedInserts());
+ assertEquals(10 - bucketCache.getStats().getFailedInserts(),
bucketCache.backingMap.size());
+
assertFalse(bucketCache.fullyCachedFiles.containsKey(filePath.getName()));
+ } else {
+
assertTrue(bucketCache.fullyCachedFiles.containsKey(filePath.getName()));
+ }
+ } finally {
+ if (bucketCache != null) {
+ bucketCache.shutdown();
+ }
+ HBASE_TESTING_UTILITY.cleanupTestDir();
+ }
+ }
+
+ @Test
+ public void testNotifyFileCachingCompletedForEncodedDataSuccess() throws
Exception {
+ BucketCache bucketCache = null;
+ try {
+ Path filePath = new Path(HBASE_TESTING_UTILITY.getDataTestDir(),
+ "testNotifyFileCachingCompletedForEncodedDataSuccess");
+ bucketCache = testNotifyFileCachingCompletedForTenBlocks(filePath, 10,
true);
if (bucketCache.getStats().getFailedInserts() > 0) {
LOG.info("There were {} fail inserts, "
+ "will assert if total blocks in backingMap equals (10 -
failInserts) "
@@ -953,7 +977,7 @@ public class TestBucketCache {
"testNotifyFileCachingCompletedNotAllCached");
// Deliberately passing more blocks than we have created to test that
// notifyFileCachingCompleted will not consider the file fully cached
- bucketCache = testNotifyFileCachingCompletedForTenBlocks(filePath, 12);
+ bucketCache = testNotifyFileCachingCompletedForTenBlocks(filePath, 12,
false);
assertFalse(bucketCache.fullyCachedFiles.containsKey(filePath.getName()));
} finally {
if (bucketCache != null) {
@@ -964,7 +988,7 @@ public class TestBucketCache {
}
private BucketCache testNotifyFileCachingCompletedForTenBlocks(Path filePath,
- int totalBlocksToCheck) throws Exception {
+ int totalBlocksToCheck, boolean encoded) throws Exception {
final Path dataTestDir = createAndGetTestDir();
String ioEngineName = "file:" + dataTestDir + "/bucketNoRecycler.cache";
BucketCache bucketCache = new BucketCache(ioEngineName, capacitySize,
constructedBlockSize,
@@ -973,7 +997,7 @@ public class TestBucketCache {
long usedByteSize = bucketCache.getAllocator().getUsedSize();
assertEquals(0, usedByteSize);
HFileBlockPair[] hfileBlockPairs =
- CacheTestUtils.generateBlocksForPath(constructedBlockSize, 10, filePath);
+ CacheTestUtils.generateBlocksForPath(constructedBlockSize, 10, filePath,
encoded);
// Add blocks
for (HFileBlockPair hfileBlockPair : hfileBlockPairs) {
bucketCache.cacheBlock(hfileBlockPair.getBlockName(),
hfileBlockPair.getBlock(), false, true);