001/* 002 * Licensed to the Apache Software Foundation (ASF) under one 003 * or more contributor license agreements. See the NOTICE file 004 * distributed with this work for additional information 005 * regarding copyright ownership. The ASF licenses this file 006 * to you under the Apache License, Version 2.0 (the 007 * "License"); you may not use this file except in compliance 008 * with the License. You may obtain a copy of the License at 009 * 010 * http://www.apache.org/licenses/LICENSE-2.0 011 * 012 * Unless required by applicable law or agreed to in writing, software 013 * distributed under the License is distributed on an "AS IS" BASIS, 014 * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. 015 * See the License for the specific language governing permissions and 016 * limitations under the License. 017 */ 018package org.apache.hadoop.hbase.io.hfile; 019 020import static org.apache.hadoop.hbase.HConstants.BUCKET_CACHE_IOENGINE_KEY; 021import static org.apache.hadoop.hbase.HConstants.BUCKET_CACHE_SIZE_KEY; 022import static org.apache.hadoop.hbase.io.ByteBuffAllocator.BUFFER_SIZE_KEY; 023import static org.apache.hadoop.hbase.io.ByteBuffAllocator.MAX_BUFFER_COUNT_KEY; 024import static org.apache.hadoop.hbase.io.ByteBuffAllocator.MIN_ALLOCATE_SIZE_KEY; 025import static org.apache.hadoop.hbase.io.hfile.BlockCacheFactory.BLOCKCACHE_POLICY_KEY; 026import static org.apache.hadoop.hbase.io.hfile.CacheConfig.EVICT_BLOCKS_ON_CLOSE_KEY; 027import static org.junit.Assert.assertEquals; 028import static org.junit.Assert.assertFalse; 029import static org.junit.Assert.assertNull; 030import static org.junit.Assert.assertTrue; 031import static org.junit.Assert.fail; 032 033import java.io.DataInput; 034import java.io.DataOutput; 035import java.io.IOException; 036import java.nio.ByteBuffer; 037import java.util.ArrayList; 038import java.util.Arrays; 039import java.util.List; 040import java.util.Objects; 041import java.util.Random; 042import java.util.concurrent.ThreadLocalRandom; 043import org.apache.hadoop.conf.Configuration; 044import org.apache.hadoop.fs.FSDataInputStream; 045import org.apache.hadoop.fs.FSDataOutputStream; 046import org.apache.hadoop.fs.FileStatus; 047import org.apache.hadoop.fs.FileSystem; 048import org.apache.hadoop.fs.Path; 049import org.apache.hadoop.hbase.ArrayBackedTag; 050import org.apache.hadoop.hbase.ByteBufferKeyValue; 051import org.apache.hadoop.hbase.Cell; 052import org.apache.hadoop.hbase.CellBuilderType; 053import org.apache.hadoop.hbase.CellComparatorImpl; 054import org.apache.hadoop.hbase.CellUtil; 055import org.apache.hadoop.hbase.ExtendedCell; 056import org.apache.hadoop.hbase.ExtendedCellBuilder; 057import org.apache.hadoop.hbase.ExtendedCellBuilderFactory; 058import org.apache.hadoop.hbase.HBaseClassTestRule; 059import org.apache.hadoop.hbase.HBaseCommonTestingUtil; 060import org.apache.hadoop.hbase.HBaseConfiguration; 061import org.apache.hadoop.hbase.HBaseTestingUtil; 062import org.apache.hadoop.hbase.HConstants; 063import org.apache.hadoop.hbase.KeyValue; 064import org.apache.hadoop.hbase.KeyValue.Type; 065import org.apache.hadoop.hbase.KeyValueUtil; 066import org.apache.hadoop.hbase.MetaCellComparator; 067import org.apache.hadoop.hbase.PrivateCellUtil; 068import org.apache.hadoop.hbase.Tag; 069import org.apache.hadoop.hbase.io.ByteBuffAllocator; 070import org.apache.hadoop.hbase.io.compress.Compression; 071import org.apache.hadoop.hbase.io.encoding.DataBlockEncoder; 072import org.apache.hadoop.hbase.io.encoding.DataBlockEncoding; 073import org.apache.hadoop.hbase.io.encoding.IndexBlockEncoding; 074import org.apache.hadoop.hbase.io.hfile.HFile.Reader; 075import org.apache.hadoop.hbase.io.hfile.HFile.Writer; 076import org.apache.hadoop.hbase.io.hfile.ReaderContext.ReaderType; 077import org.apache.hadoop.hbase.nio.ByteBuff; 078import org.apache.hadoop.hbase.regionserver.StoreFileWriter; 079import org.apache.hadoop.hbase.testclassification.IOTests; 080import org.apache.hadoop.hbase.testclassification.SmallTests; 081import org.apache.hadoop.hbase.util.ByteBufferUtils; 082import org.apache.hadoop.hbase.util.Bytes; 083import org.apache.hadoop.io.Writable; 084import org.junit.Assert; 085import org.junit.BeforeClass; 086import org.junit.ClassRule; 087import org.junit.Rule; 088import org.junit.Test; 089import org.junit.experimental.categories.Category; 090import org.junit.rules.TestName; 091import org.mockito.Mockito; 092import org.slf4j.Logger; 093import org.slf4j.LoggerFactory; 094 095/** 096 * test hfile features. 097 */ 098@Category({ IOTests.class, SmallTests.class }) 099public class TestHFile { 100 101 @ClassRule 102 public static final HBaseClassTestRule CLASS_RULE = HBaseClassTestRule.forClass(TestHFile.class); 103 104 @Rule 105 public TestName testName = new TestName(); 106 107 private static final Logger LOG = LoggerFactory.getLogger(TestHFile.class); 108 private static final int NUM_VALID_KEY_TYPES = KeyValue.Type.values().length - 2; 109 private static final HBaseTestingUtil TEST_UTIL = new HBaseTestingUtil(); 110 private static String ROOT_DIR = TEST_UTIL.getDataTestDir("TestHFile").toString(); 111 private final int minBlockSize = 512; 112 private static String localFormatter = "%010d"; 113 private static CacheConfig cacheConf; 114 private static Configuration conf; 115 private static FileSystem fs; 116 117 @BeforeClass 118 public static void setUp() throws Exception { 119 conf = TEST_UTIL.getConfiguration(); 120 cacheConf = new CacheConfig(conf); 121 fs = TEST_UTIL.getTestFileSystem(); 122 } 123 124 public static Reader createReaderFromStream(ReaderContext context, CacheConfig cacheConf, 125 Configuration conf) throws IOException { 126 HFileInfo fileInfo = new HFileInfo(context, conf); 127 Reader preadReader = HFile.createReader(context, fileInfo, cacheConf, conf); 128 fileInfo.initMetaAndIndex(preadReader); 129 preadReader.close(); 130 context = new ReaderContextBuilder() 131 .withFileSystemAndPath(context.getFileSystem(), context.getFilePath()) 132 .withReaderType(ReaderType.STREAM).build(); 133 Reader streamReader = HFile.createReader(context, fileInfo, cacheConf, conf); 134 return streamReader; 135 } 136 137 private ByteBuffAllocator initAllocator(boolean reservoirEnabled, int bufSize, int bufCount, 138 int minAllocSize) { 139 Configuration that = HBaseConfiguration.create(conf); 140 that.setInt(BUFFER_SIZE_KEY, bufSize); 141 that.setInt(MAX_BUFFER_COUNT_KEY, bufCount); 142 // All ByteBuffers will be allocated from the buffers. 143 that.setInt(MIN_ALLOCATE_SIZE_KEY, minAllocSize); 144 return ByteBuffAllocator.create(that, reservoirEnabled); 145 } 146 147 private void fillByteBuffAllocator(ByteBuffAllocator alloc, int bufCount) { 148 // Fill the allocator with bufCount ByteBuffer 149 List<ByteBuff> buffs = new ArrayList<>(); 150 for (int i = 0; i < bufCount; i++) { 151 buffs.add(alloc.allocateOneBuffer()); 152 Assert.assertEquals(alloc.getFreeBufferCount(), 0); 153 } 154 buffs.forEach(ByteBuff::release); 155 Assert.assertEquals(alloc.getFreeBufferCount(), bufCount); 156 } 157 158 @Test 159 public void testReaderWithoutBlockCache() throws Exception { 160 int bufCount = 32; 161 // AllByteBuffers will be allocated from the buffers. 162 ByteBuffAllocator alloc = initAllocator(true, 64 * 1024, bufCount, 0); 163 fillByteBuffAllocator(alloc, bufCount); 164 // start write to store file. 165 Path path = writeStoreFile(); 166 readStoreFile(path, conf, alloc); 167 Assert.assertEquals(bufCount, alloc.getFreeBufferCount()); 168 alloc.clean(); 169 } 170 171 /** 172 * Test case for HBASE-22127 in LruBlockCache. 173 */ 174 @Test 175 public void testReaderWithLRUBlockCache() throws Exception { 176 int bufCount = 1024, blockSize = 64 * 1024; 177 ByteBuffAllocator alloc = initAllocator(true, bufCount, blockSize, 0); 178 fillByteBuffAllocator(alloc, bufCount); 179 Path storeFilePath = writeStoreFile(); 180 // Open the file reader with LRUBlockCache 181 BlockCache lru = new LruBlockCache(1024 * 1024 * 32, blockSize, true, conf); 182 CacheConfig cacheConfig = new CacheConfig(conf, null, lru, alloc); 183 HFile.Reader reader = HFile.createReader(fs, storeFilePath, cacheConfig, true, conf); 184 long offset = 0; 185 while (offset < reader.getTrailer().getLoadOnOpenDataOffset()) { 186 BlockCacheKey key = new BlockCacheKey(storeFilePath.getName(), offset); 187 HFileBlock block = reader.readBlock(offset, -1, true, true, false, true, null, null); 188 offset += block.getOnDiskSizeWithHeader(); 189 // Ensure the block is an heap one. 190 Cacheable cachedBlock = lru.getBlock(key, false, false, true); 191 Assert.assertNotNull(cachedBlock); 192 Assert.assertTrue(cachedBlock instanceof HFileBlock); 193 Assert.assertFalse(((HFileBlock) cachedBlock).isSharedMem()); 194 // Should never allocate off-heap block from allocator because ensure that it's LRU. 195 Assert.assertEquals(bufCount, alloc.getFreeBufferCount()); 196 block.release(); // return back the ByteBuffer back to allocator. 197 } 198 reader.close(); 199 Assert.assertEquals(bufCount, alloc.getFreeBufferCount()); 200 alloc.clean(); 201 lru.shutdown(); 202 } 203 204 private BlockCache initCombinedBlockCache(final String l1CachePolicy) { 205 Configuration that = HBaseConfiguration.create(conf); 206 that.setFloat(BUCKET_CACHE_SIZE_KEY, 32); // 32MB for bucket cache. 207 that.set(BUCKET_CACHE_IOENGINE_KEY, "offheap"); 208 that.set(BLOCKCACHE_POLICY_KEY, l1CachePolicy); 209 BlockCache bc = BlockCacheFactory.createBlockCache(that); 210 Assert.assertNotNull(bc); 211 Assert.assertTrue(bc instanceof CombinedBlockCache); 212 return bc; 213 } 214 215 /** 216 * Test case for HBASE-22127 in CombinedBlockCache 217 */ 218 @Test 219 public void testReaderWithCombinedBlockCache() throws Exception { 220 int bufCount = 1024, blockSize = 64 * 1024; 221 ByteBuffAllocator alloc = initAllocator(true, bufCount, blockSize, 0); 222 fillByteBuffAllocator(alloc, bufCount); 223 Path storeFilePath = writeStoreFile(); 224 // Open the file reader with CombinedBlockCache 225 BlockCache combined = initCombinedBlockCache("LRU"); 226 conf.setBoolean(EVICT_BLOCKS_ON_CLOSE_KEY, true); 227 CacheConfig cacheConfig = new CacheConfig(conf, null, combined, alloc); 228 HFile.Reader reader = HFile.createReader(fs, storeFilePath, cacheConfig, true, conf); 229 long offset = 0; 230 while (offset < reader.getTrailer().getLoadOnOpenDataOffset()) { 231 BlockCacheKey key = new BlockCacheKey(storeFilePath.getName(), offset); 232 HFileBlock block = reader.readBlock(offset, -1, true, true, false, true, null, null); 233 offset += block.getOnDiskSizeWithHeader(); 234 // Read the cached block. 235 Cacheable cachedBlock = combined.getBlock(key, false, false, true); 236 try { 237 Assert.assertNotNull(cachedBlock); 238 Assert.assertTrue(cachedBlock instanceof HFileBlock); 239 HFileBlock hfb = (HFileBlock) cachedBlock; 240 // Data block will be cached in BucketCache, so it should be an off-heap block. 241 if (hfb.getBlockType().isData()) { 242 Assert.assertTrue(hfb.isSharedMem()); 243 } else { 244 // Non-data block will be cached in LRUBlockCache, so it must be an on-heap block. 245 Assert.assertFalse(hfb.isSharedMem()); 246 } 247 } finally { 248 cachedBlock.release(); 249 } 250 block.release(); // return back the ByteBuffer back to allocator. 251 } 252 reader.close(); 253 combined.shutdown(); 254 Assert.assertEquals(bufCount, alloc.getFreeBufferCount()); 255 alloc.clean(); 256 } 257 258 /** 259 * Tests that we properly allocate from the off-heap or on-heap when LRUCache is configured. In 260 * this case, the determining factor is whether we end up caching the block or not. So the below 261 * test cases try different permutations of enabling/disabling via CacheConfig and via user 262 * request (cacheblocks), along with different expected block types. 263 */ 264 @Test 265 public void testReaderBlockAllocationWithLRUCache() throws IOException { 266 // false because caching is fully enabled 267 testReaderBlockAllocationWithLRUCache(true, true, null, false); 268 // false because we only look at cache config when expectedBlockType is non-null 269 testReaderBlockAllocationWithLRUCache(false, true, null, false); 270 // false because cacheBlock is true and even with cache config is disabled, we still cache 271 // important blocks like indexes 272 testReaderBlockAllocationWithLRUCache(false, true, BlockType.INTERMEDIATE_INDEX, false); 273 // true because since it's a DATA block, we honor the cache config 274 testReaderBlockAllocationWithLRUCache(false, true, BlockType.DATA, true); 275 // true for the following 2 because cacheBlock takes precedence over cache config 276 testReaderBlockAllocationWithLRUCache(true, false, null, true); 277 testReaderBlockAllocationWithLRUCache(true, false, BlockType.INTERMEDIATE_INDEX, false); 278 // false for the following 3 because both cache config and cacheBlock are false. 279 // per above, INDEX would supersede cache config, but not cacheBlock 280 testReaderBlockAllocationWithLRUCache(false, false, null, true); 281 testReaderBlockAllocationWithLRUCache(false, false, BlockType.INTERMEDIATE_INDEX, true); 282 testReaderBlockAllocationWithLRUCache(false, false, BlockType.DATA, true); 283 } 284 285 private void testReaderBlockAllocationWithLRUCache(boolean cacheConfigCacheBlockOnRead, 286 boolean cacheBlock, BlockType blockType, boolean expectSharedMem) throws IOException { 287 int bufCount = 1024, blockSize = 64 * 1024; 288 ByteBuffAllocator alloc = initAllocator(true, blockSize, bufCount, 0); 289 fillByteBuffAllocator(alloc, bufCount); 290 Path storeFilePath = writeStoreFile(); 291 Configuration myConf = new Configuration(conf); 292 293 myConf.setBoolean(CacheConfig.CACHE_DATA_ON_READ_KEY, cacheConfigCacheBlockOnRead); 294 // Open the file reader with LRUBlockCache 295 BlockCache lru = new LruBlockCache(1024 * 1024 * 32, blockSize, true, myConf); 296 CacheConfig cacheConfig = new CacheConfig(myConf, null, lru, alloc); 297 HFile.Reader reader = HFile.createReader(fs, storeFilePath, cacheConfig, true, myConf); 298 long offset = 0; 299 while (offset < reader.getTrailer().getLoadOnOpenDataOffset()) { 300 long read = readAtOffsetWithAllocationAsserts(alloc, reader, offset, cacheBlock, blockType, 301 expectSharedMem); 302 if (read < 0) { 303 break; 304 } 305 306 offset += read; 307 } 308 309 reader.close(); 310 Assert.assertEquals(bufCount, alloc.getFreeBufferCount()); 311 alloc.clean(); 312 lru.shutdown(); 313 } 314 315 /** 316 * Tests that we properly allocate from the off-heap or on-heap when CombinedCache is configured. 317 * In this case, we should always use off-heap unless the block is an INDEX (which always goes to 318 * L1 cache which is on-heap) 319 */ 320 @Test 321 public void testReaderBlockAllocationWithCombinedCache() throws IOException { 322 // true because caching is fully enabled and block type null 323 testReaderBlockAllocationWithCombinedCache(true, true, null, true); 324 // false because caching is fully enabled, index block type always goes to on-heap L1 325 testReaderBlockAllocationWithCombinedCache(true, true, BlockType.INTERMEDIATE_INDEX, false); 326 // true because cacheBlocks takes precedence over cache config which block type is null 327 testReaderBlockAllocationWithCombinedCache(false, true, null, true); 328 // false because caching is enabled and block type is index, which always goes to L1 329 testReaderBlockAllocationWithCombinedCache(false, true, BlockType.INTERMEDIATE_INDEX, false); 330 // true because since it's a DATA block, we honor the cache config 331 testReaderBlockAllocationWithCombinedCache(false, true, BlockType.DATA, true); 332 // true for the following 2 because cacheBlock takes precedence over cache config 333 // with caching disabled, we always go to off-heap 334 testReaderBlockAllocationWithCombinedCache(true, false, null, true); 335 testReaderBlockAllocationWithCombinedCache(true, false, BlockType.INTERMEDIATE_INDEX, false); 336 // true for the following 3, because with caching disabled we always go to off-heap 337 testReaderBlockAllocationWithCombinedCache(false, false, null, true); 338 testReaderBlockAllocationWithCombinedCache(false, false, BlockType.INTERMEDIATE_INDEX, true); 339 testReaderBlockAllocationWithCombinedCache(false, false, BlockType.DATA, true); 340 } 341 342 private void testReaderBlockAllocationWithCombinedCache(boolean cacheConfigCacheBlockOnRead, 343 boolean cacheBlock, BlockType blockType, boolean expectSharedMem) throws IOException { 344 int bufCount = 1024, blockSize = 64 * 1024; 345 ByteBuffAllocator alloc = initAllocator(true, blockSize, bufCount, 0); 346 fillByteBuffAllocator(alloc, bufCount); 347 Path storeFilePath = writeStoreFile(); 348 // Open the file reader with CombinedBlockCache 349 BlockCache combined = initCombinedBlockCache("LRU"); 350 Configuration myConf = new Configuration(conf); 351 352 myConf.setBoolean(CacheConfig.CACHE_DATA_ON_READ_KEY, cacheConfigCacheBlockOnRead); 353 myConf.setBoolean(EVICT_BLOCKS_ON_CLOSE_KEY, true); 354 355 CacheConfig cacheConfig = new CacheConfig(myConf, null, combined, alloc); 356 HFile.Reader reader = HFile.createReader(fs, storeFilePath, cacheConfig, true, myConf); 357 long offset = 0; 358 while (offset < reader.getTrailer().getLoadOnOpenDataOffset()) { 359 long read = readAtOffsetWithAllocationAsserts(alloc, reader, offset, cacheBlock, blockType, 360 expectSharedMem); 361 if (read < 0) { 362 break; 363 } 364 365 offset += read; 366 } 367 368 reader.close(); 369 combined.shutdown(); 370 Assert.assertEquals(bufCount, alloc.getFreeBufferCount()); 371 alloc.clean(); 372 } 373 374 private long readAtOffsetWithAllocationAsserts(ByteBuffAllocator alloc, HFile.Reader reader, 375 long offset, boolean cacheBlock, BlockType blockType, boolean expectSharedMem) 376 throws IOException { 377 HFileBlock block; 378 try { 379 block = reader.readBlock(offset, -1, cacheBlock, true, false, true, blockType, null); 380 } catch (IOException e) { 381 if (e.getMessage().contains("Expected block type")) { 382 return -1; 383 } 384 throw e; 385 } 386 387 Assert.assertEquals(expectSharedMem, block.isSharedMem()); 388 389 if (expectSharedMem) { 390 Assert.assertTrue(alloc.getFreeBufferCount() < alloc.getTotalBufferCount()); 391 } else { 392 // Should never allocate off-heap block from allocator because ensure that it's LRU. 393 Assert.assertEquals(alloc.getTotalBufferCount(), alloc.getFreeBufferCount()); 394 } 395 396 try { 397 return block.getOnDiskSizeWithHeader(); 398 } finally { 399 block.release(); // return back the ByteBuffer back to allocator. 400 } 401 } 402 403 private void readStoreFile(Path storeFilePath, Configuration conf, ByteBuffAllocator alloc) 404 throws Exception { 405 // Open the file reader with block cache disabled. 406 CacheConfig cache = new CacheConfig(conf, null, null, alloc); 407 HFile.Reader reader = HFile.createReader(fs, storeFilePath, cache, true, conf); 408 long offset = 0; 409 while (offset < reader.getTrailer().getLoadOnOpenDataOffset()) { 410 HFileBlock block = reader.readBlock(offset, -1, false, true, false, true, null, null); 411 offset += block.getOnDiskSizeWithHeader(); 412 block.release(); // return back the ByteBuffer back to allocator. 413 } 414 reader.close(); 415 } 416 417 private Path writeStoreFile() throws IOException { 418 Path storeFileParentDir = new Path(TEST_UTIL.getDataTestDir(), "TestHFile"); 419 HFileContext meta = new HFileContextBuilder().withBlockSize(64 * 1024).build(); 420 StoreFileWriter sfw = new StoreFileWriter.Builder(conf, fs).withOutputDir(storeFileParentDir) 421 .withFileContext(meta).build(); 422 final int rowLen = 32; 423 Random rand = ThreadLocalRandom.current(); 424 for (int i = 0; i < 1000; ++i) { 425 byte[] k = RandomKeyValueUtil.randomOrderedKey(rand, i); 426 byte[] v = RandomKeyValueUtil.randomValue(rand); 427 int cfLen = rand.nextInt(k.length - rowLen + 1); 428 KeyValue kv = new KeyValue(k, 0, rowLen, k, rowLen, cfLen, k, rowLen + cfLen, 429 k.length - rowLen - cfLen, rand.nextLong(), generateKeyType(rand), v, 0, v.length); 430 sfw.append(kv); 431 } 432 433 sfw.close(); 434 return sfw.getPath(); 435 } 436 437 public static KeyValue.Type generateKeyType(Random rand) { 438 if (rand.nextBoolean()) { 439 // Let's make half of KVs puts. 440 return KeyValue.Type.Put; 441 } else { 442 KeyValue.Type keyType = KeyValue.Type.values()[1 + rand.nextInt(NUM_VALID_KEY_TYPES)]; 443 if (keyType == KeyValue.Type.Minimum || keyType == KeyValue.Type.Maximum) { 444 throw new RuntimeException("Generated an invalid key type: " + keyType + ". " 445 + "Probably the layout of KeyValue.Type has changed."); 446 } 447 return keyType; 448 } 449 } 450 451 /** 452 * Test empty HFile. Test all features work reasonably when hfile is empty of entries. 453 */ 454 @Test 455 public void testEmptyHFile() throws IOException { 456 Path f = new Path(ROOT_DIR, testName.getMethodName()); 457 HFileContext context = new HFileContextBuilder().withIncludesTags(false).build(); 458 Writer w = 459 HFile.getWriterFactory(conf, cacheConf).withPath(fs, f).withFileContext(context).create(); 460 w.close(); 461 Reader r = HFile.createReader(fs, f, cacheConf, true, conf); 462 assertFalse(r.getFirstKey().isPresent()); 463 assertFalse(r.getLastKey().isPresent()); 464 } 465 466 /** 467 * Create 0-length hfile and show that it fails 468 */ 469 @Test 470 public void testCorrupt0LengthHFile() throws IOException { 471 Path f = new Path(ROOT_DIR, testName.getMethodName()); 472 FSDataOutputStream fsos = fs.create(f); 473 fsos.close(); 474 475 try { 476 Reader r = HFile.createReader(fs, f, cacheConf, true, conf); 477 } catch (CorruptHFileException | IllegalArgumentException che) { 478 // Expected failure 479 return; 480 } 481 fail("Should have thrown exception"); 482 } 483 484 @Test 485 public void testCorruptOutOfOrderHFileWrite() throws IOException { 486 Path path = new Path(ROOT_DIR, testName.getMethodName()); 487 FSDataOutputStream mockedOutputStream = Mockito.mock(FSDataOutputStream.class); 488 String columnFamily = "MyColumnFamily"; 489 String tableName = "MyTableName"; 490 HFileContext fileContext = 491 new HFileContextBuilder().withHFileName(testName.getMethodName() + "HFile") 492 .withBlockSize(minBlockSize).withColumnFamily(Bytes.toBytes(columnFamily)) 493 .withTableName(Bytes.toBytes(tableName)).withHBaseCheckSum(false) 494 .withCompression(Compression.Algorithm.NONE).withCompressTags(false).build(); 495 HFileWriterImpl writer = 496 new HFileWriterImpl(conf, cacheConf, path, mockedOutputStream, fileContext); 497 ExtendedCellBuilder cellBuilder = 498 ExtendedCellBuilderFactory.create(CellBuilderType.SHALLOW_COPY); 499 byte[] row = Bytes.toBytes("foo"); 500 byte[] qualifier = Bytes.toBytes("qualifier"); 501 byte[] cf = Bytes.toBytes(columnFamily); 502 byte[] val = Bytes.toBytes("fooVal"); 503 long firstTS = 100L; 504 long secondTS = 101L; 505 ExtendedCell firstCell = cellBuilder.setRow(row).setValue(val).setTimestamp(firstTS) 506 .setQualifier(qualifier).setFamily(cf).setType(Cell.Type.Put).build(); 507 ExtendedCell secondCell = cellBuilder.setRow(row).setValue(val).setTimestamp(secondTS) 508 .setQualifier(qualifier).setFamily(cf).setType(Cell.Type.Put).build(); 509 // second Cell will sort "higher" than the first because later timestamps should come first 510 writer.append(firstCell); 511 try { 512 writer.append(secondCell); 513 } catch (IOException ie) { 514 String message = ie.getMessage(); 515 Assert.assertTrue(message.contains("not lexically larger")); 516 Assert.assertTrue(message.contains(tableName)); 517 Assert.assertTrue(message.contains(columnFamily)); 518 return; 519 } 520 Assert.fail("Exception wasn't thrown even though Cells were appended in the wrong order!"); 521 } 522 523 public static void truncateFile(FileSystem fs, Path src, Path dst) throws IOException { 524 FileStatus fst = fs.getFileStatus(src); 525 long len = fst.getLen(); 526 len = len / 2; 527 528 // create a truncated hfile 529 FSDataOutputStream fdos = fs.create(dst); 530 byte[] buf = new byte[(int) len]; 531 FSDataInputStream fdis = fs.open(src); 532 fdis.read(buf); 533 fdos.write(buf); 534 fdis.close(); 535 fdos.close(); 536 } 537 538 /** 539 * Create a truncated hfile and verify that exception thrown. 540 */ 541 @Test 542 public void testCorruptTruncatedHFile() throws IOException { 543 Path f = new Path(ROOT_DIR, testName.getMethodName()); 544 HFileContext context = new HFileContextBuilder().build(); 545 Writer w = HFile.getWriterFactory(conf, cacheConf).withPath(this.fs, f).withFileContext(context) 546 .create(); 547 writeSomeRecords(w, 0, 100, false); 548 w.close(); 549 550 Path trunc = new Path(f.getParent(), "trucated"); 551 truncateFile(fs, w.getPath(), trunc); 552 553 try { 554 HFile.createReader(fs, trunc, cacheConf, true, conf); 555 } catch (CorruptHFileException | IllegalArgumentException che) { 556 // Expected failure 557 return; 558 } 559 fail("Should have thrown exception"); 560 } 561 562 // write some records into the hfile 563 // write them twice 564 private int writeSomeRecords(Writer writer, int start, int n, boolean useTags) 565 throws IOException { 566 String value = "value"; 567 KeyValue kv; 568 for (int i = start; i < (start + n); i++) { 569 String key = String.format(localFormatter, Integer.valueOf(i)); 570 if (useTags) { 571 Tag t = new ArrayBackedTag((byte) 1, "myTag1"); 572 Tag[] tags = new Tag[1]; 573 tags[0] = t; 574 kv = new KeyValue(Bytes.toBytes(key), Bytes.toBytes("family"), Bytes.toBytes("qual"), 575 HConstants.LATEST_TIMESTAMP, Bytes.toBytes(value + key), tags); 576 writer.append(kv); 577 } else { 578 kv = new KeyValue(Bytes.toBytes(key), Bytes.toBytes("family"), Bytes.toBytes("qual"), 579 Bytes.toBytes(value + key)); 580 writer.append(kv); 581 } 582 } 583 return (start + n); 584 } 585 586 private void readAllRecords(HFileScanner scanner) throws IOException { 587 readAndCheckbytes(scanner, 0, 100); 588 } 589 590 // read the records and check 591 private int readAndCheckbytes(HFileScanner scanner, int start, int n) throws IOException { 592 String value = "value"; 593 int i = start; 594 for (; i < (start + n); i++) { 595 ByteBuffer key = ByteBuffer.wrap(((KeyValue) scanner.getKey()).getKey()); 596 ByteBuffer val = scanner.getValue(); 597 String keyStr = String.format(localFormatter, Integer.valueOf(i)); 598 String valStr = value + keyStr; 599 KeyValue kv = new KeyValue(Bytes.toBytes(keyStr), Bytes.toBytes("family"), 600 Bytes.toBytes("qual"), Bytes.toBytes(valStr)); 601 byte[] keyBytes = 602 new KeyValue.KeyOnlyKeyValue(Bytes.toBytes(key), 0, Bytes.toBytes(key).length).getKey(); 603 assertTrue("bytes for keys do not match " + keyStr + " " + Bytes.toString(Bytes.toBytes(key)), 604 Arrays.equals(kv.getKey(), keyBytes)); 605 byte[] valBytes = Bytes.toBytes(val); 606 assertTrue("bytes for vals do not match " + valStr + " " + Bytes.toString(valBytes), 607 Arrays.equals(Bytes.toBytes(valStr), valBytes)); 608 if (!scanner.next()) { 609 break; 610 } 611 } 612 assertEquals(i, start + n - 1); 613 return (start + n); 614 } 615 616 private byte[] getSomeKey(int rowId) { 617 KeyValue kv = new KeyValue(Bytes.toBytes(String.format(localFormatter, Integer.valueOf(rowId))), 618 Bytes.toBytes("family"), Bytes.toBytes("qual"), HConstants.LATEST_TIMESTAMP, Type.Put); 619 return kv.getKey(); 620 } 621 622 private void writeRecords(Writer writer, boolean useTags) throws IOException { 623 writeSomeRecords(writer, 0, 100, useTags); 624 writer.close(); 625 } 626 627 private FSDataOutputStream createFSOutput(Path name) throws IOException { 628 // if (fs.exists(name)) fs.delete(name, true); 629 FSDataOutputStream fout = fs.create(name); 630 return fout; 631 } 632 633 /** 634 * test none codecs 635 */ 636 void basicWithSomeCodec(String codec, boolean useTags) throws IOException { 637 if (useTags) { 638 conf.setInt("hfile.format.version", 3); 639 } 640 Path ncHFile = new Path(ROOT_DIR, "basic.hfile." + codec.toString() + useTags); 641 FSDataOutputStream fout = createFSOutput(ncHFile); 642 HFileContext meta = new HFileContextBuilder().withBlockSize(minBlockSize) 643 .withCompression(HFileWriterImpl.compressionByName(codec)).build(); 644 Writer writer = 645 HFile.getWriterFactory(conf, cacheConf).withOutputStream(fout).withFileContext(meta).create(); 646 LOG.info(Objects.toString(writer)); 647 writeRecords(writer, useTags); 648 fout.close(); 649 FSDataInputStream fin = fs.open(ncHFile); 650 ReaderContext context = new ReaderContextBuilder().withFileSystemAndPath(fs, ncHFile).build(); 651 Reader reader = createReaderFromStream(context, cacheConf, conf); 652 System.out.println(cacheConf.toString()); 653 // Load up the index. 654 // Get a scanner that caches and that does not use pread. 655 HFileScanner scanner = reader.getScanner(conf, true, false); 656 // Align scanner at start of the file. 657 scanner.seekTo(); 658 readAllRecords(scanner); 659 int seekTo = scanner.seekTo(KeyValueUtil.createKeyValueFromKey(getSomeKey(50))); 660 System.out.println(seekTo); 661 assertTrue("location lookup failed", 662 scanner.seekTo(KeyValueUtil.createKeyValueFromKey(getSomeKey(50))) == 0); 663 // read the key and see if it matches 664 ByteBuffer readKey = ByteBuffer.wrap(((KeyValue) scanner.getKey()).getKey()); 665 assertTrue("seeked key does not match", Arrays.equals(getSomeKey(50), Bytes.toBytes(readKey))); 666 667 scanner.seekTo(KeyValueUtil.createKeyValueFromKey(getSomeKey(0))); 668 ByteBuffer val1 = scanner.getValue(); 669 scanner.seekTo(KeyValueUtil.createKeyValueFromKey(getSomeKey(0))); 670 ByteBuffer val2 = scanner.getValue(); 671 assertTrue(Arrays.equals(Bytes.toBytes(val1), Bytes.toBytes(val2))); 672 673 reader.close(); 674 fin.close(); 675 fs.delete(ncHFile, true); 676 } 677 678 @Test 679 public void testTFileFeatures() throws IOException { 680 testHFilefeaturesInternals(false); 681 testHFilefeaturesInternals(true); 682 } 683 684 protected void testHFilefeaturesInternals(boolean useTags) throws IOException { 685 basicWithSomeCodec("none", useTags); 686 basicWithSomeCodec("gz", useTags); 687 } 688 689 private void writeNumMetablocks(Writer writer, int n) { 690 for (int i = 0; i < n; i++) { 691 writer.appendMetaBlock("HFileMeta" + i, new Writable() { 692 private int val; 693 694 public Writable setVal(int val) { 695 this.val = val; 696 return this; 697 } 698 699 @Override 700 public void write(DataOutput out) throws IOException { 701 out.write(Bytes.toBytes("something to test" + val)); 702 } 703 704 @Override 705 public void readFields(DataInput in) throws IOException { 706 } 707 }.setVal(i)); 708 } 709 } 710 711 private void someTestingWithMetaBlock(Writer writer) { 712 writeNumMetablocks(writer, 10); 713 } 714 715 private void readNumMetablocks(Reader reader, int n) throws IOException { 716 for (int i = 0; i < n; i++) { 717 ByteBuff actual = reader.getMetaBlock("HFileMeta" + i, false).getBufferWithoutHeader(); 718 ByteBuffer expected = ByteBuffer.wrap(Bytes.toBytes("something to test" + i)); 719 assertEquals("failed to match metadata", Bytes.toStringBinary(expected), Bytes.toStringBinary( 720 actual.array(), actual.arrayOffset() + actual.position(), actual.capacity())); 721 } 722 } 723 724 private void someReadingWithMetaBlock(Reader reader) throws IOException { 725 readNumMetablocks(reader, 10); 726 } 727 728 private void metablocks(final String compress) throws Exception { 729 Path mFile = new Path(ROOT_DIR, "meta.hfile"); 730 FSDataOutputStream fout = createFSOutput(mFile); 731 HFileContext meta = 732 new HFileContextBuilder().withCompression(HFileWriterImpl.compressionByName(compress)) 733 .withBlockSize(minBlockSize).build(); 734 Writer writer = 735 HFile.getWriterFactory(conf, cacheConf).withOutputStream(fout).withFileContext(meta).create(); 736 someTestingWithMetaBlock(writer); 737 writer.close(); 738 fout.close(); 739 ReaderContext context = new ReaderContextBuilder().withFileSystemAndPath(fs, mFile).build(); 740 Reader reader = createReaderFromStream(context, cacheConf, conf); 741 // No data -- this should return false. 742 assertFalse(reader.getScanner(conf, false, false).seekTo()); 743 someReadingWithMetaBlock(reader); 744 fs.delete(mFile, true); 745 reader.close(); 746 } 747 748 // test meta blocks for hfiles 749 @Test 750 public void testMetaBlocks() throws Exception { 751 metablocks("none"); 752 metablocks("gz"); 753 } 754 755 @Test 756 public void testNullMetaBlocks() throws Exception { 757 for (Compression.Algorithm compressAlgo : HBaseCommonTestingUtil.COMPRESSION_ALGORITHMS) { 758 Path mFile = new Path(ROOT_DIR, "nometa_" + compressAlgo + ".hfile"); 759 FSDataOutputStream fout = createFSOutput(mFile); 760 HFileContext meta = 761 new HFileContextBuilder().withCompression(compressAlgo).withBlockSize(minBlockSize).build(); 762 Writer writer = HFile.getWriterFactory(conf, cacheConf).withOutputStream(fout) 763 .withFileContext(meta).create(); 764 KeyValue kv = 765 new KeyValue(Bytes.toBytes("foo"), Bytes.toBytes("f1"), null, Bytes.toBytes("value")); 766 writer.append(kv); 767 writer.close(); 768 fout.close(); 769 Reader reader = HFile.createReader(fs, mFile, cacheConf, true, conf); 770 assertNull(reader.getMetaBlock("non-existant", false)); 771 } 772 } 773 774 /** 775 * Make sure the ordinals for our compression algorithms do not change on us. 776 */ 777 @Test 778 public void testCompressionOrdinance() { 779 assertTrue(Compression.Algorithm.LZO.ordinal() == 0); 780 assertTrue(Compression.Algorithm.GZ.ordinal() == 1); 781 assertTrue(Compression.Algorithm.NONE.ordinal() == 2); 782 assertTrue(Compression.Algorithm.SNAPPY.ordinal() == 3); 783 assertTrue(Compression.Algorithm.LZ4.ordinal() == 4); 784 } 785 786 @Test 787 public void testShortMidpointSameQual() { 788 ExtendedCell left = 789 ExtendedCellBuilderFactory.create(CellBuilderType.DEEP_COPY).setRow(Bytes.toBytes("a")) 790 .setFamily(Bytes.toBytes("a")).setQualifier(Bytes.toBytes("a")).setTimestamp(11) 791 .setType(Type.Maximum.getCode()).setValue(HConstants.EMPTY_BYTE_ARRAY).build(); 792 ExtendedCell right = 793 ExtendedCellBuilderFactory.create(CellBuilderType.DEEP_COPY).setRow(Bytes.toBytes("a")) 794 .setFamily(Bytes.toBytes("a")).setQualifier(Bytes.toBytes("a")).setTimestamp(9) 795 .setType(Type.Maximum.getCode()).setValue(HConstants.EMPTY_BYTE_ARRAY).build(); 796 ExtendedCell mid = HFileWriterImpl.getMidpoint(CellComparatorImpl.COMPARATOR, left, right); 797 assertTrue( 798 PrivateCellUtil.compareKeyIgnoresMvcc(CellComparatorImpl.COMPARATOR, left, mid) <= 0); 799 assertTrue( 800 PrivateCellUtil.compareKeyIgnoresMvcc(CellComparatorImpl.COMPARATOR, mid, right) == 0); 801 } 802 803 private ExtendedCell getCell(byte[] row, byte[] family, byte[] qualifier) { 804 return ExtendedCellBuilderFactory.create(CellBuilderType.DEEP_COPY).setRow(row) 805 .setFamily(family).setQualifier(qualifier).setTimestamp(HConstants.LATEST_TIMESTAMP) 806 .setType(KeyValue.Type.Maximum.getCode()).setValue(HConstants.EMPTY_BYTE_ARRAY).build(); 807 } 808 809 @Test 810 public void testGetShortMidpoint() { 811 ExtendedCell left = getCell(Bytes.toBytes("a"), Bytes.toBytes("a"), Bytes.toBytes("a")); 812 ExtendedCell right = getCell(Bytes.toBytes("a"), Bytes.toBytes("a"), Bytes.toBytes("a")); 813 ExtendedCell mid = HFileWriterImpl.getMidpoint(CellComparatorImpl.COMPARATOR, left, right); 814 assertTrue( 815 PrivateCellUtil.compareKeyIgnoresMvcc(CellComparatorImpl.COMPARATOR, left, mid) <= 0); 816 assertTrue( 817 PrivateCellUtil.compareKeyIgnoresMvcc(CellComparatorImpl.COMPARATOR, mid, right) <= 0); 818 left = getCell(Bytes.toBytes("a"), Bytes.toBytes("a"), Bytes.toBytes("a")); 819 right = getCell(Bytes.toBytes("b"), Bytes.toBytes("a"), Bytes.toBytes("a")); 820 mid = HFileWriterImpl.getMidpoint(CellComparatorImpl.COMPARATOR, left, right); 821 assertTrue(PrivateCellUtil.compareKeyIgnoresMvcc(CellComparatorImpl.COMPARATOR, left, mid) < 0); 822 assertTrue( 823 PrivateCellUtil.compareKeyIgnoresMvcc(CellComparatorImpl.COMPARATOR, mid, right) <= 0); 824 left = getCell(Bytes.toBytes("g"), Bytes.toBytes("a"), Bytes.toBytes("a")); 825 right = getCell(Bytes.toBytes("i"), Bytes.toBytes("a"), Bytes.toBytes("a")); 826 mid = HFileWriterImpl.getMidpoint(CellComparatorImpl.COMPARATOR, left, right); 827 assertTrue(PrivateCellUtil.compareKeyIgnoresMvcc(CellComparatorImpl.COMPARATOR, left, mid) < 0); 828 assertTrue( 829 PrivateCellUtil.compareKeyIgnoresMvcc(CellComparatorImpl.COMPARATOR, mid, right) <= 0); 830 left = getCell(Bytes.toBytes("a"), Bytes.toBytes("a"), Bytes.toBytes("a")); 831 right = getCell(Bytes.toBytes("bbbbbbb"), Bytes.toBytes("a"), Bytes.toBytes("a")); 832 mid = HFileWriterImpl.getMidpoint(CellComparatorImpl.COMPARATOR, left, right); 833 assertTrue(PrivateCellUtil.compareKeyIgnoresMvcc(CellComparatorImpl.COMPARATOR, left, mid) < 0); 834 assertTrue( 835 PrivateCellUtil.compareKeyIgnoresMvcc(CellComparatorImpl.COMPARATOR, mid, right) < 0); 836 assertEquals(1, mid.getRowLength()); 837 left = getCell(Bytes.toBytes("a"), Bytes.toBytes("a"), Bytes.toBytes("a")); 838 right = getCell(Bytes.toBytes("b"), Bytes.toBytes("a"), Bytes.toBytes("a")); 839 mid = HFileWriterImpl.getMidpoint(CellComparatorImpl.COMPARATOR, left, right); 840 assertTrue(PrivateCellUtil.compareKeyIgnoresMvcc(CellComparatorImpl.COMPARATOR, left, mid) < 0); 841 assertTrue( 842 PrivateCellUtil.compareKeyIgnoresMvcc(CellComparatorImpl.COMPARATOR, mid, right) <= 0); 843 left = getCell(Bytes.toBytes("a"), Bytes.toBytes("a"), Bytes.toBytes("a")); 844 right = getCell(Bytes.toBytes("a"), Bytes.toBytes("aaaaaaaa"), Bytes.toBytes("b")); 845 mid = HFileWriterImpl.getMidpoint(CellComparatorImpl.COMPARATOR, left, right); 846 assertTrue(PrivateCellUtil.compareKeyIgnoresMvcc(CellComparatorImpl.COMPARATOR, left, mid) < 0); 847 assertTrue( 848 PrivateCellUtil.compareKeyIgnoresMvcc(CellComparatorImpl.COMPARATOR, mid, right) < 0); 849 assertEquals(2, mid.getFamilyLength()); 850 left = getCell(Bytes.toBytes("a"), Bytes.toBytes("a"), Bytes.toBytes("a")); 851 right = getCell(Bytes.toBytes("a"), Bytes.toBytes("a"), Bytes.toBytes("aaaaaaaaa")); 852 mid = HFileWriterImpl.getMidpoint(CellComparatorImpl.COMPARATOR, left, right); 853 assertTrue(PrivateCellUtil.compareKeyIgnoresMvcc(CellComparatorImpl.COMPARATOR, left, mid) < 0); 854 assertTrue( 855 PrivateCellUtil.compareKeyIgnoresMvcc(CellComparatorImpl.COMPARATOR, mid, right) < 0); 856 assertEquals(2, mid.getQualifierLength()); 857 left = getCell(Bytes.toBytes("a"), Bytes.toBytes("a"), Bytes.toBytes("a")); 858 right = getCell(Bytes.toBytes("a"), Bytes.toBytes("a"), Bytes.toBytes("b")); 859 mid = HFileWriterImpl.getMidpoint(CellComparatorImpl.COMPARATOR, left, right); 860 assertTrue(PrivateCellUtil.compareKeyIgnoresMvcc(CellComparatorImpl.COMPARATOR, left, mid) < 0); 861 assertTrue( 862 PrivateCellUtil.compareKeyIgnoresMvcc(CellComparatorImpl.COMPARATOR, mid, right) <= 0); 863 assertEquals(1, mid.getQualifierLength()); 864 865 // Verify boundary conditions 866 left = getCell(Bytes.toBytes("a"), Bytes.toBytes("a"), new byte[] { 0x00, (byte) 0xFE }); 867 right = getCell(Bytes.toBytes("a"), Bytes.toBytes("a"), new byte[] { 0x00, (byte) 0xFF }); 868 mid = HFileWriterImpl.getMidpoint(CellComparatorImpl.COMPARATOR, left, right); 869 assertTrue(PrivateCellUtil.compareKeyIgnoresMvcc(CellComparatorImpl.COMPARATOR, left, mid) < 0); 870 assertTrue( 871 PrivateCellUtil.compareKeyIgnoresMvcc(CellComparatorImpl.COMPARATOR, mid, right) == 0); 872 left = getCell(Bytes.toBytes("a"), Bytes.toBytes("a"), new byte[] { 0x00, 0x12 }); 873 right = getCell(Bytes.toBytes("a"), Bytes.toBytes("a"), new byte[] { 0x00, 0x12, 0x00 }); 874 mid = HFileWriterImpl.getMidpoint(CellComparatorImpl.COMPARATOR, left, right); 875 assertTrue(PrivateCellUtil.compareKeyIgnoresMvcc(CellComparatorImpl.COMPARATOR, left, mid) < 0); 876 assertTrue( 877 PrivateCellUtil.compareKeyIgnoresMvcc(CellComparatorImpl.COMPARATOR, mid, right) == 0); 878 879 // Assert that if meta comparator, it returns the right cell -- i.e. no 880 // optimization done. 881 left = getCell(Bytes.toBytes("g"), Bytes.toBytes("a"), Bytes.toBytes("a")); 882 right = getCell(Bytes.toBytes("i"), Bytes.toBytes("a"), Bytes.toBytes("a")); 883 mid = HFileWriterImpl.getMidpoint(MetaCellComparator.META_COMPARATOR, left, right); 884 assertTrue(PrivateCellUtil.compareKeyIgnoresMvcc(CellComparatorImpl.COMPARATOR, left, mid) < 0); 885 assertTrue( 886 PrivateCellUtil.compareKeyIgnoresMvcc(CellComparatorImpl.COMPARATOR, mid, right) == 0); 887 byte[] family = Bytes.toBytes("family"); 888 byte[] qualA = Bytes.toBytes("qfA"); 889 byte[] qualB = Bytes.toBytes("qfB"); 890 final CellComparatorImpl keyComparator = CellComparatorImpl.COMPARATOR; 891 // verify that faked shorter rowkey could be generated 892 long ts = 5; 893 KeyValue kv1 = new KeyValue(Bytes.toBytes("the quick brown fox"), family, qualA, ts, Type.Put); 894 KeyValue kv2 = new KeyValue(Bytes.toBytes("the who test text"), family, qualA, ts, Type.Put); 895 ExtendedCell newKey = HFileWriterImpl.getMidpoint(keyComparator, kv1, kv2); 896 assertTrue(keyComparator.compare(kv1, newKey) < 0); 897 assertTrue((keyComparator.compare(kv2, newKey)) > 0); 898 byte[] expectedArray = Bytes.toBytes("the r"); 899 Bytes.equals(newKey.getRowArray(), newKey.getRowOffset(), newKey.getRowLength(), expectedArray, 900 0, expectedArray.length); 901 902 // verify: same with "row + family + qualifier", return rightKey directly 903 kv1 = new KeyValue(Bytes.toBytes("ilovehbase"), family, qualA, 5, Type.Put); 904 kv2 = new KeyValue(Bytes.toBytes("ilovehbase"), family, qualA, 0, Type.Put); 905 assertTrue(keyComparator.compare(kv1, kv2) < 0); 906 newKey = HFileWriterImpl.getMidpoint(keyComparator, kv1, kv2); 907 assertTrue(keyComparator.compare(kv1, newKey) < 0); 908 assertTrue((keyComparator.compare(kv2, newKey)) == 0); 909 kv1 = new KeyValue(Bytes.toBytes("ilovehbase"), family, qualA, -5, Type.Put); 910 kv2 = new KeyValue(Bytes.toBytes("ilovehbase"), family, qualA, -10, Type.Put); 911 assertTrue(keyComparator.compare(kv1, kv2) < 0); 912 newKey = HFileWriterImpl.getMidpoint(keyComparator, kv1, kv2); 913 assertTrue(keyComparator.compare(kv1, newKey) < 0); 914 assertTrue((keyComparator.compare(kv2, newKey)) == 0); 915 916 // verify: same with row, different with qualifier 917 kv1 = new KeyValue(Bytes.toBytes("ilovehbase"), family, qualA, 5, Type.Put); 918 kv2 = new KeyValue(Bytes.toBytes("ilovehbase"), family, qualB, 5, Type.Put); 919 assertTrue(keyComparator.compare(kv1, kv2) < 0); 920 newKey = HFileWriterImpl.getMidpoint(keyComparator, kv1, kv2); 921 assertTrue(keyComparator.compare(kv1, newKey) < 0); 922 assertTrue((keyComparator.compare(kv2, newKey)) > 0); 923 assertTrue(Arrays.equals(CellUtil.cloneFamily(newKey), family)); 924 assertTrue(Arrays.equals(CellUtil.cloneQualifier(newKey), qualB)); 925 assertTrue(newKey.getTimestamp() == HConstants.LATEST_TIMESTAMP); 926 assertTrue(newKey.getTypeByte() == Type.Maximum.getCode()); 927 928 // verify metaKeyComparator's getShortMidpointKey output 929 final CellComparatorImpl metaKeyComparator = MetaCellComparator.META_COMPARATOR; 930 kv1 = new KeyValue(Bytes.toBytes("ilovehbase123"), family, qualA, 5, Type.Put); 931 kv2 = new KeyValue(Bytes.toBytes("ilovehbase234"), family, qualA, 0, Type.Put); 932 newKey = HFileWriterImpl.getMidpoint(metaKeyComparator, kv1, kv2); 933 assertTrue(metaKeyComparator.compare(kv1, newKey) < 0); 934 assertTrue((metaKeyComparator.compare(kv2, newKey) == 0)); 935 936 // verify common fix scenario 937 kv1 = new KeyValue(Bytes.toBytes("ilovehbase"), family, qualA, ts, Type.Put); 938 kv2 = new KeyValue(Bytes.toBytes("ilovehbaseandhdfs"), family, qualA, ts, Type.Put); 939 assertTrue(keyComparator.compare(kv1, kv2) < 0); 940 newKey = HFileWriterImpl.getMidpoint(keyComparator, kv1, kv2); 941 assertTrue(keyComparator.compare(kv1, newKey) < 0); 942 assertTrue((keyComparator.compare(kv2, newKey)) > 0); 943 expectedArray = Bytes.toBytes("ilovehbasea"); 944 Bytes.equals(newKey.getRowArray(), newKey.getRowOffset(), newKey.getRowLength(), expectedArray, 945 0, expectedArray.length); 946 // verify only 1 offset scenario 947 kv1 = new KeyValue(Bytes.toBytes("100abcdefg"), family, qualA, ts, Type.Put); 948 kv2 = new KeyValue(Bytes.toBytes("101abcdefg"), family, qualA, ts, Type.Put); 949 assertTrue(keyComparator.compare(kv1, kv2) < 0); 950 newKey = HFileWriterImpl.getMidpoint(keyComparator, kv1, kv2); 951 assertTrue(keyComparator.compare(kv1, newKey) < 0); 952 assertTrue((keyComparator.compare(kv2, newKey)) > 0); 953 expectedArray = Bytes.toBytes("101"); 954 Bytes.equals(newKey.getRowArray(), newKey.getRowOffset(), newKey.getRowLength(), expectedArray, 955 0, expectedArray.length); 956 } 957 958 @Test 959 public void testDBEShipped() throws IOException { 960 for (DataBlockEncoding encoding : DataBlockEncoding.values()) { 961 DataBlockEncoder encoder = encoding.getEncoder(); 962 if (encoder == null) { 963 continue; 964 } 965 Path f = new Path(ROOT_DIR, testName.getMethodName() + "_" + encoding); 966 HFileContext context = 967 new HFileContextBuilder().withIncludesTags(false).withDataBlockEncoding(encoding).build(); 968 HFileWriterImpl writer = (HFileWriterImpl) HFile.getWriterFactory(conf, cacheConf) 969 .withPath(fs, f).withFileContext(context).create(); 970 971 KeyValue kv = new KeyValue(Bytes.toBytes("testkey1"), Bytes.toBytes("family"), 972 Bytes.toBytes("qual"), Bytes.toBytes("testvalue")); 973 KeyValue kv2 = new KeyValue(Bytes.toBytes("testkey2"), Bytes.toBytes("family"), 974 Bytes.toBytes("qual"), Bytes.toBytes("testvalue")); 975 KeyValue kv3 = new KeyValue(Bytes.toBytes("testkey3"), Bytes.toBytes("family"), 976 Bytes.toBytes("qual"), Bytes.toBytes("testvalue")); 977 978 ByteBuffer buffer = ByteBuffer.wrap(kv.getBuffer()); 979 ByteBuffer buffer2 = ByteBuffer.wrap(kv2.getBuffer()); 980 ByteBuffer buffer3 = ByteBuffer.wrap(kv3.getBuffer()); 981 982 writer.append(new ByteBufferKeyValue(buffer, 0, buffer.remaining())); 983 writer.beforeShipped(); 984 985 // pollute first cell's backing ByteBuffer 986 ByteBufferUtils.copyFromBufferToBuffer(buffer3, buffer); 987 988 // write another cell, if DBE not Shipped, test will fail 989 writer.append(new ByteBufferKeyValue(buffer2, 0, buffer2.remaining())); 990 writer.close(); 991 } 992 } 993 994 /** 995 * Test case for CombinedBlockCache with TinyLfu as L1 cache 996 */ 997 @Test 998 public void testReaderWithTinyLfuCombinedBlockCache() throws Exception { 999 testReaderCombinedCache("TinyLfu"); 1000 } 1001 1002 /** 1003 * Test case for CombinedBlockCache with AdaptiveLRU as L1 cache 1004 */ 1005 @Test 1006 public void testReaderWithAdaptiveLruCombinedBlockCache() throws Exception { 1007 testReaderCombinedCache("AdaptiveLRU"); 1008 } 1009 1010 /** 1011 * Test case for CombinedBlockCache with AdaptiveLRU as L1 cache 1012 */ 1013 @Test 1014 public void testReaderWithLruCombinedBlockCache() throws Exception { 1015 testReaderCombinedCache("LRU"); 1016 } 1017 1018 private void testReaderCombinedCache(final String l1CachePolicy) throws Exception { 1019 int bufCount = 1024; 1020 int blockSize = 64 * 1024; 1021 ByteBuffAllocator alloc = initAllocator(true, bufCount, blockSize, 0); 1022 fillByteBuffAllocator(alloc, bufCount); 1023 Path storeFilePath = writeStoreFile(); 1024 // Open the file reader with CombinedBlockCache 1025 BlockCache combined = initCombinedBlockCache(l1CachePolicy); 1026 conf.setBoolean(EVICT_BLOCKS_ON_CLOSE_KEY, true); 1027 CacheConfig cacheConfig = new CacheConfig(conf, null, combined, alloc); 1028 HFile.Reader reader = HFile.createReader(fs, storeFilePath, cacheConfig, true, conf); 1029 long offset = 0; 1030 Cacheable cachedBlock = null; 1031 while (offset < reader.getTrailer().getLoadOnOpenDataOffset()) { 1032 BlockCacheKey key = new BlockCacheKey(storeFilePath.getName(), offset); 1033 HFileBlock block = reader.readBlock(offset, -1, true, true, false, true, null, null); 1034 offset += block.getOnDiskSizeWithHeader(); 1035 // Read the cached block. 1036 cachedBlock = combined.getBlock(key, false, false, true); 1037 try { 1038 Assert.assertNotNull(cachedBlock); 1039 Assert.assertTrue(cachedBlock instanceof HFileBlock); 1040 HFileBlock hfb = (HFileBlock) cachedBlock; 1041 // Data block will be cached in BucketCache, so it should be an off-heap block. 1042 if (hfb.getBlockType().isData()) { 1043 Assert.assertTrue(hfb.isSharedMem()); 1044 } else if (!l1CachePolicy.equals("TinyLfu")) { 1045 Assert.assertFalse(hfb.isSharedMem()); 1046 } 1047 } finally { 1048 cachedBlock.release(); 1049 } 1050 block.release(); // return back the ByteBuffer back to allocator. 1051 } 1052 reader.close(); 1053 combined.shutdown(); 1054 if (cachedBlock != null) { 1055 Assert.assertEquals(0, cachedBlock.refCnt()); 1056 } 1057 Assert.assertEquals(bufCount, alloc.getFreeBufferCount()); 1058 alloc.clean(); 1059 } 1060 1061 @Test 1062 public void testHFileContextBuilderWithIndexEncoding() throws IOException { 1063 HFileContext context = 1064 new HFileContextBuilder().withIndexBlockEncoding(IndexBlockEncoding.PREFIX_TREE).build(); 1065 HFileContext newContext = new HFileContextBuilder(context).build(); 1066 assertTrue(newContext.getIndexBlockEncoding() == IndexBlockEncoding.PREFIX_TREE); 1067 } 1068}