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.CacheConfig.EVICT_BLOCKS_ON_CLOSE_KEY; 026import static org.junit.Assert.assertEquals; 027import static org.junit.Assert.assertFalse; 028import static org.junit.Assert.assertNull; 029import static org.junit.Assert.assertTrue; 030import static org.junit.Assert.fail; 031 032import java.io.DataInput; 033import java.io.DataOutput; 034import java.io.IOException; 035import java.nio.ByteBuffer; 036import java.util.ArrayList; 037import java.util.Arrays; 038import java.util.List; 039import java.util.Objects; 040import java.util.Random; 041import org.apache.hadoop.conf.Configuration; 042import org.apache.hadoop.fs.FSDataInputStream; 043import org.apache.hadoop.fs.FSDataOutputStream; 044import org.apache.hadoop.fs.FileStatus; 045import org.apache.hadoop.fs.FileSystem; 046import org.apache.hadoop.fs.Path; 047import org.apache.hadoop.hbase.ArrayBackedTag; 048import org.apache.hadoop.hbase.ByteBufferKeyValue; 049import org.apache.hadoop.hbase.Cell; 050import org.apache.hadoop.hbase.CellBuilder; 051import org.apache.hadoop.hbase.CellBuilderFactory; 052import org.apache.hadoop.hbase.CellBuilderType; 053import org.apache.hadoop.hbase.CellComparatorImpl; 054import org.apache.hadoop.hbase.CellUtil; 055import org.apache.hadoop.hbase.HBaseClassTestRule; 056import org.apache.hadoop.hbase.HBaseCommonTestingUtility; 057import org.apache.hadoop.hbase.HBaseConfiguration; 058import org.apache.hadoop.hbase.HBaseTestingUtility; 059import org.apache.hadoop.hbase.HConstants; 060import org.apache.hadoop.hbase.KeyValue; 061import org.apache.hadoop.hbase.KeyValue.Type; 062import org.apache.hadoop.hbase.KeyValueUtil; 063import org.apache.hadoop.hbase.MetaCellComparator; 064import org.apache.hadoop.hbase.PrivateCellUtil; 065import org.apache.hadoop.hbase.Tag; 066import org.apache.hadoop.hbase.io.ByteBuffAllocator; 067import org.apache.hadoop.hbase.io.compress.Compression; 068import org.apache.hadoop.hbase.io.encoding.DataBlockEncoder; 069import org.apache.hadoop.hbase.io.encoding.DataBlockEncoding; 070import org.apache.hadoop.hbase.io.hfile.HFile.Reader; 071import org.apache.hadoop.hbase.io.hfile.HFile.Writer; 072import org.apache.hadoop.hbase.io.hfile.ReaderContext.ReaderType; 073import org.apache.hadoop.hbase.nio.ByteBuff; 074import org.apache.hadoop.hbase.regionserver.StoreFileWriter; 075import org.apache.hadoop.hbase.testclassification.IOTests; 076import org.apache.hadoop.hbase.testclassification.SmallTests; 077import org.apache.hadoop.hbase.util.ByteBufferUtils; 078import org.apache.hadoop.hbase.util.Bytes; 079import org.apache.hadoop.io.Writable; 080import org.junit.Assert; 081import org.junit.BeforeClass; 082import org.junit.ClassRule; 083import org.junit.Rule; 084import org.junit.Test; 085import org.junit.experimental.categories.Category; 086import org.junit.rules.TestName; 087import org.mockito.Mockito; 088import org.slf4j.Logger; 089import org.slf4j.LoggerFactory; 090 091/** 092 * test hfile features. 093 */ 094@Category({IOTests.class, SmallTests.class}) 095public class TestHFile { 096 097 @ClassRule 098 public static final HBaseClassTestRule CLASS_RULE = 099 HBaseClassTestRule.forClass(TestHFile.class); 100 101 @Rule public TestName testName = new TestName(); 102 103 private static final Logger LOG = LoggerFactory.getLogger(TestHFile.class); 104 private static final int NUM_VALID_KEY_TYPES = KeyValue.Type.values().length - 2; 105 private static final HBaseTestingUtility TEST_UTIL = new HBaseTestingUtility(); 106 private static String ROOT_DIR = 107 TEST_UTIL.getDataTestDir("TestHFile").toString(); 108 private final int minBlockSize = 512; 109 private static String localFormatter = "%010d"; 110 private static CacheConfig cacheConf; 111 private static Configuration conf ; 112 private static FileSystem fs; 113 114 @BeforeClass 115 public static void setUp() throws Exception { 116 conf = TEST_UTIL.getConfiguration(); 117 cacheConf = new CacheConfig(conf); 118 fs = TEST_UTIL.getTestFileSystem(); 119 } 120 121 public static Reader createReaderFromStream(ReaderContext context, CacheConfig cacheConf, 122 Configuration conf) throws IOException { 123 HFileInfo fileInfo = new HFileInfo(context, conf); 124 Reader preadReader = HFile.createReader(context, fileInfo, cacheConf, conf); 125 fileInfo.initMetaAndIndex(preadReader); 126 preadReader.close(); 127 context = new ReaderContextBuilder() 128 .withFileSystemAndPath(context.getFileSystem(), context.getFilePath()) 129 .withReaderType(ReaderType.STREAM) 130 .build(); 131 Reader streamReader = HFile.createReader(context, fileInfo, cacheConf, conf); 132 return streamReader; 133 } 134 135 private ByteBuffAllocator initAllocator(boolean reservoirEnabled, int bufSize, int bufCount, 136 int minAllocSize) { 137 Configuration that = HBaseConfiguration.create(conf); 138 that.setInt(BUFFER_SIZE_KEY, bufSize); 139 that.setInt(MAX_BUFFER_COUNT_KEY, bufCount); 140 // All ByteBuffers will be allocated from the buffers. 141 that.setInt(MIN_ALLOCATE_SIZE_KEY, minAllocSize); 142 return ByteBuffAllocator.create(that, reservoirEnabled); 143 } 144 145 private void fillByteBuffAllocator(ByteBuffAllocator alloc, int bufCount) { 146 // Fill the allocator with bufCount ByteBuffer 147 List<ByteBuff> buffs = new ArrayList<>(); 148 for (int i = 0; i < bufCount; i++) { 149 buffs.add(alloc.allocateOneBuffer()); 150 Assert.assertEquals(alloc.getFreeBufferCount(), 0); 151 } 152 buffs.forEach(ByteBuff::release); 153 Assert.assertEquals(alloc.getFreeBufferCount(), bufCount); 154 } 155 156 @Test 157 public void testReaderWithoutBlockCache() throws Exception { 158 int bufCount = 32; 159 // AllByteBuffers will be allocated from the buffers. 160 ByteBuffAllocator alloc = initAllocator(true, 64 * 1024, bufCount, 0); 161 fillByteBuffAllocator(alloc, bufCount); 162 // start write to store file. 163 Path path = writeStoreFile(); 164 try { 165 readStoreFile(path, conf, alloc); 166 } catch (Exception e) { 167 // fail test 168 assertTrue(false); 169 } 170 Assert.assertEquals(bufCount, alloc.getFreeBufferCount()); 171 alloc.clean(); 172 } 173 174 /** 175 * Test case for HBASE-22127 in LruBlockCache. 176 */ 177 @Test 178 public void testReaderWithLRUBlockCache() throws Exception { 179 int bufCount = 1024, blockSize = 64 * 1024; 180 ByteBuffAllocator alloc = initAllocator(true, bufCount, blockSize, 0); 181 fillByteBuffAllocator(alloc, bufCount); 182 Path storeFilePath = writeStoreFile(); 183 // Open the file reader with LRUBlockCache 184 BlockCache lru = new LruBlockCache(1024 * 1024 * 32, blockSize, true, conf); 185 CacheConfig cacheConfig = new CacheConfig(conf, null, lru, alloc); 186 HFile.Reader reader = HFile.createReader(fs, storeFilePath, cacheConfig, true, conf); 187 long offset = 0; 188 while (offset < reader.getTrailer().getLoadOnOpenDataOffset()) { 189 BlockCacheKey key = new BlockCacheKey(storeFilePath.getName(), offset); 190 HFileBlock block = reader.readBlock(offset, -1, true, true, false, true, null, null); 191 offset += block.getOnDiskSizeWithHeader(); 192 // Ensure the block is an heap one. 193 Cacheable cachedBlock = lru.getBlock(key, false, false, true); 194 Assert.assertNotNull(cachedBlock); 195 Assert.assertTrue(cachedBlock instanceof HFileBlock); 196 Assert.assertFalse(((HFileBlock) cachedBlock).isSharedMem()); 197 // Should never allocate off-heap block from allocator because ensure that it's LRU. 198 Assert.assertEquals(bufCount, alloc.getFreeBufferCount()); 199 block.release(); // return back the ByteBuffer back to allocator. 200 } 201 reader.close(); 202 Assert.assertEquals(bufCount, alloc.getFreeBufferCount()); 203 alloc.clean(); 204 lru.shutdown(); 205 } 206 207 private BlockCache initCombinedBlockCache() { 208 Configuration that = HBaseConfiguration.create(conf); 209 that.setFloat(BUCKET_CACHE_SIZE_KEY, 32); // 32MB for bucket cache. 210 that.set(BUCKET_CACHE_IOENGINE_KEY, "offheap"); 211 BlockCache bc = BlockCacheFactory.createBlockCache(that); 212 Assert.assertNotNull(bc); 213 Assert.assertTrue(bc instanceof CombinedBlockCache); 214 return bc; 215 } 216 217 /** 218 * Test case for HBASE-22127 in CombinedBlockCache 219 */ 220 @Test 221 public void testReaderWithCombinedBlockCache() throws Exception { 222 int bufCount = 1024, blockSize = 64 * 1024; 223 ByteBuffAllocator alloc = initAllocator(true, bufCount, blockSize, 0); 224 fillByteBuffAllocator(alloc, bufCount); 225 Path storeFilePath = writeStoreFile(); 226 // Open the file reader with CombinedBlockCache 227 BlockCache combined = initCombinedBlockCache(); 228 conf.setBoolean(EVICT_BLOCKS_ON_CLOSE_KEY, true); 229 CacheConfig cacheConfig = new CacheConfig(conf, null, combined, alloc); 230 HFile.Reader reader = HFile.createReader(fs, storeFilePath, cacheConfig, true, conf); 231 long offset = 0; 232 while (offset < reader.getTrailer().getLoadOnOpenDataOffset()) { 233 BlockCacheKey key = new BlockCacheKey(storeFilePath.getName(), offset); 234 HFileBlock block = reader.readBlock(offset, -1, true, true, false, true, null, null); 235 offset += block.getOnDiskSizeWithHeader(); 236 // Read the cached block. 237 Cacheable cachedBlock = combined.getBlock(key, false, false, true); 238 try { 239 Assert.assertNotNull(cachedBlock); 240 Assert.assertTrue(cachedBlock instanceof HFileBlock); 241 HFileBlock hfb = (HFileBlock) cachedBlock; 242 // Data block will be cached in BucketCache, so it should be an off-heap block. 243 if (hfb.getBlockType().isData()) { 244 Assert.assertTrue(hfb.isSharedMem()); 245 } else { 246 // Non-data block will be cached in LRUBlockCache, so it must be an on-heap block. 247 Assert.assertFalse(hfb.isSharedMem()); 248 } 249 } finally { 250 cachedBlock.release(); 251 } 252 block.release(); // return back the ByteBuffer back to allocator. 253 } 254 reader.close(); 255 combined.shutdown(); 256 Assert.assertEquals(bufCount, alloc.getFreeBufferCount()); 257 alloc.clean(); 258 } 259 260 private void readStoreFile(Path storeFilePath, Configuration conf, ByteBuffAllocator alloc) 261 throws Exception { 262 // Open the file reader with block cache disabled. 263 CacheConfig cache = new CacheConfig(conf, null, null, alloc); 264 HFile.Reader reader = HFile.createReader(fs, storeFilePath, cache, true, conf); 265 long offset = 0; 266 while (offset < reader.getTrailer().getLoadOnOpenDataOffset()) { 267 HFileBlock block = reader.readBlock(offset, -1, false, true, false, true, null, null); 268 offset += block.getOnDiskSizeWithHeader(); 269 block.release(); // return back the ByteBuffer back to allocator. 270 } 271 reader.close(); 272 } 273 274 private Path writeStoreFile() throws IOException { 275 Path storeFileParentDir = new Path(TEST_UTIL.getDataTestDir(), "TestHFile"); 276 HFileContext meta = new HFileContextBuilder().withBlockSize(64 * 1024).build(); 277 StoreFileWriter sfw = 278 new StoreFileWriter.Builder(conf, fs).withOutputDir(storeFileParentDir) 279 .withFileContext(meta).build(); 280 281 final int rowLen = 32; 282 Random RNG = new Random(); 283 for (int i = 0; i < 1000; ++i) { 284 byte[] k = RandomKeyValueUtil.randomOrderedKey(RNG, i); 285 byte[] v = RandomKeyValueUtil.randomValue(RNG); 286 int cfLen = RNG.nextInt(k.length - rowLen + 1); 287 KeyValue kv = 288 new KeyValue(k, 0, rowLen, k, rowLen, cfLen, k, rowLen + cfLen, 289 k.length - rowLen - cfLen, RNG.nextLong(), generateKeyType(RNG), v, 0, v.length); 290 sfw.append(kv); 291 } 292 293 sfw.close(); 294 return sfw.getPath(); 295 } 296 297 public static KeyValue.Type generateKeyType(Random rand) { 298 if (rand.nextBoolean()) { 299 // Let's make half of KVs puts. 300 return KeyValue.Type.Put; 301 } else { 302 KeyValue.Type keyType = KeyValue.Type.values()[1 + rand.nextInt(NUM_VALID_KEY_TYPES)]; 303 if (keyType == KeyValue.Type.Minimum || keyType == KeyValue.Type.Maximum) { 304 throw new RuntimeException("Generated an invalid key type: " + keyType + ". " 305 + "Probably the layout of KeyValue.Type has changed."); 306 } 307 return keyType; 308 } 309 } 310 311 /** 312 * Test empty HFile. 313 * Test all features work reasonably when hfile is empty of entries. 314 * @throws IOException 315 */ 316 @Test 317 public void testEmptyHFile() throws IOException { 318 Path f = new Path(ROOT_DIR, testName.getMethodName()); 319 HFileContext context = new HFileContextBuilder().withIncludesTags(false).build(); 320 Writer w = 321 HFile.getWriterFactory(conf, cacheConf).withPath(fs, f).withFileContext(context).create(); 322 w.close(); 323 Reader r = HFile.createReader(fs, f, cacheConf, true, conf); 324 assertFalse(r.getFirstKey().isPresent()); 325 assertFalse(r.getLastKey().isPresent()); 326 } 327 328 /** 329 * Create 0-length hfile and show that it fails 330 */ 331 @Test 332 public void testCorrupt0LengthHFile() throws IOException { 333 Path f = new Path(ROOT_DIR, testName.getMethodName()); 334 FSDataOutputStream fsos = fs.create(f); 335 fsos.close(); 336 337 try { 338 Reader r = HFile.createReader(fs, f, cacheConf, true, conf); 339 } catch (CorruptHFileException | IllegalArgumentException che) { 340 // Expected failure 341 return; 342 } 343 fail("Should have thrown exception"); 344 } 345 346 @Test 347 public void testCorruptOutOfOrderHFileWrite() throws IOException { 348 Path path = new Path(ROOT_DIR, testName.getMethodName()); 349 FSDataOutputStream mockedOutputStream = Mockito.mock(FSDataOutputStream.class); 350 String columnFamily = "MyColumnFamily"; 351 String tableName = "MyTableName"; 352 HFileContext fileContext = new HFileContextBuilder() 353 .withHFileName(testName.getMethodName() + "HFile") 354 .withBlockSize(minBlockSize) 355 .withColumnFamily(Bytes.toBytes(columnFamily)) 356 .withTableName(Bytes.toBytes(tableName)) 357 .withHBaseCheckSum(false) 358 .withCompression(Compression.Algorithm.NONE) 359 .withCompressTags(false) 360 .build(); 361 HFileWriterImpl writer = new HFileWriterImpl(conf, cacheConf, path, mockedOutputStream, 362 fileContext); 363 CellBuilder cellBuilder = CellBuilderFactory.create(CellBuilderType.SHALLOW_COPY); 364 byte[] row = Bytes.toBytes("foo"); 365 byte[] qualifier = Bytes.toBytes("qualifier"); 366 byte[] cf = Bytes.toBytes(columnFamily); 367 byte[] val = Bytes.toBytes("fooVal"); 368 long firstTS = 100L; 369 long secondTS = 101L; 370 Cell firstCell = cellBuilder.setRow(row).setValue(val).setTimestamp(firstTS) 371 .setQualifier(qualifier).setFamily(cf).setType(Cell.Type.Put).build(); 372 Cell secondCell= cellBuilder.setRow(row).setValue(val).setTimestamp(secondTS) 373 .setQualifier(qualifier).setFamily(cf).setType(Cell.Type.Put).build(); 374 //second Cell will sort "higher" than the first because later timestamps should come first 375 writer.append(firstCell); 376 try { 377 writer.append(secondCell); 378 } catch(IOException ie){ 379 String message = ie.getMessage(); 380 Assert.assertTrue(message.contains("not lexically larger")); 381 Assert.assertTrue(message.contains(tableName)); 382 Assert.assertTrue(message.contains(columnFamily)); 383 return; 384 } 385 Assert.fail("Exception wasn't thrown even though Cells were appended in the wrong order!"); 386 } 387 388 public static void truncateFile(FileSystem fs, Path src, Path dst) throws IOException { 389 FileStatus fst = fs.getFileStatus(src); 390 long len = fst.getLen(); 391 len = len / 2 ; 392 393 // create a truncated hfile 394 FSDataOutputStream fdos = fs.create(dst); 395 byte[] buf = new byte[(int)len]; 396 FSDataInputStream fdis = fs.open(src); 397 fdis.read(buf); 398 fdos.write(buf); 399 fdis.close(); 400 fdos.close(); 401 } 402 403 /** 404 * Create a truncated hfile and verify that exception thrown. 405 */ 406 @Test 407 public void testCorruptTruncatedHFile() throws IOException { 408 Path f = new Path(ROOT_DIR, testName.getMethodName()); 409 HFileContext context = new HFileContextBuilder().build(); 410 Writer w = HFile.getWriterFactory(conf, cacheConf).withPath(this.fs, f) 411 .withFileContext(context).create(); 412 writeSomeRecords(w, 0, 100, false); 413 w.close(); 414 415 Path trunc = new Path(f.getParent(), "trucated"); 416 truncateFile(fs, w.getPath(), trunc); 417 418 try { 419 HFile.createReader(fs, trunc, cacheConf, true, conf); 420 } catch (CorruptHFileException | IllegalArgumentException che) { 421 // Expected failure 422 return; 423 } 424 fail("Should have thrown exception"); 425 } 426 427 // write some records into the hfile 428 // write them twice 429 private int writeSomeRecords(Writer writer, int start, int n, boolean useTags) 430 throws IOException { 431 String value = "value"; 432 KeyValue kv; 433 for (int i = start; i < (start + n); i++) { 434 String key = String.format(localFormatter, Integer.valueOf(i)); 435 if (useTags) { 436 Tag t = new ArrayBackedTag((byte) 1, "myTag1"); 437 Tag[] tags = new Tag[1]; 438 tags[0] = t; 439 kv = new KeyValue(Bytes.toBytes(key), Bytes.toBytes("family"), Bytes.toBytes("qual"), 440 HConstants.LATEST_TIMESTAMP, Bytes.toBytes(value + key), tags); 441 writer.append(kv); 442 } else { 443 kv = new KeyValue(Bytes.toBytes(key), Bytes.toBytes("family"), Bytes.toBytes("qual"), 444 Bytes.toBytes(value + key)); 445 writer.append(kv); 446 } 447 } 448 return (start + n); 449 } 450 451 private void readAllRecords(HFileScanner scanner) throws IOException { 452 readAndCheckbytes(scanner, 0, 100); 453 } 454 455 // read the records and check 456 private int readAndCheckbytes(HFileScanner scanner, int start, int n) 457 throws IOException { 458 String value = "value"; 459 int i = start; 460 for (; i < (start + n); i++) { 461 ByteBuffer key = ByteBuffer.wrap(((KeyValue)scanner.getKey()).getKey()); 462 ByteBuffer val = scanner.getValue(); 463 String keyStr = String.format(localFormatter, Integer.valueOf(i)); 464 String valStr = value + keyStr; 465 KeyValue kv = new KeyValue(Bytes.toBytes(keyStr), Bytes.toBytes("family"), 466 Bytes.toBytes("qual"), Bytes.toBytes(valStr)); 467 byte[] keyBytes = new KeyValue.KeyOnlyKeyValue(Bytes.toBytes(key), 0, 468 Bytes.toBytes(key).length).getKey(); 469 assertTrue("bytes for keys do not match " + keyStr + " " + 470 Bytes.toString(Bytes.toBytes(key)), 471 Arrays.equals(kv.getKey(), keyBytes)); 472 byte [] valBytes = Bytes.toBytes(val); 473 assertTrue("bytes for vals do not match " + valStr + " " + 474 Bytes.toString(valBytes), 475 Arrays.equals(Bytes.toBytes(valStr), valBytes)); 476 if (!scanner.next()) { 477 break; 478 } 479 } 480 assertEquals(i, start + n - 1); 481 return (start + n); 482 } 483 484 private byte[] getSomeKey(int rowId) { 485 KeyValue kv = new KeyValue(String.format(localFormatter, Integer.valueOf(rowId)).getBytes(), 486 Bytes.toBytes("family"), Bytes.toBytes("qual"), HConstants.LATEST_TIMESTAMP, Type.Put); 487 return kv.getKey(); 488 } 489 490 private void writeRecords(Writer writer, boolean useTags) throws IOException { 491 writeSomeRecords(writer, 0, 100, useTags); 492 writer.close(); 493 } 494 495 private FSDataOutputStream createFSOutput(Path name) throws IOException { 496 //if (fs.exists(name)) fs.delete(name, true); 497 FSDataOutputStream fout = fs.create(name); 498 return fout; 499 } 500 501 /** 502 * test none codecs 503 * @param useTags 504 */ 505 void basicWithSomeCodec(String codec, boolean useTags) throws IOException { 506 if (useTags) { 507 conf.setInt("hfile.format.version", 3); 508 } 509 Path ncHFile = new Path(ROOT_DIR, "basic.hfile." + codec.toString() + useTags); 510 FSDataOutputStream fout = createFSOutput(ncHFile); 511 HFileContext meta = new HFileContextBuilder() 512 .withBlockSize(minBlockSize) 513 .withCompression(HFileWriterImpl.compressionByName(codec)) 514 .build(); 515 Writer writer = HFile.getWriterFactory(conf, cacheConf) 516 .withOutputStream(fout) 517 .withFileContext(meta) 518 .create(); 519 LOG.info(Objects.toString(writer)); 520 writeRecords(writer, useTags); 521 fout.close(); 522 FSDataInputStream fin = fs.open(ncHFile); 523 ReaderContext context = new ReaderContextBuilder().withFileSystemAndPath(fs, ncHFile).build(); 524 Reader reader = createReaderFromStream(context, cacheConf, conf); 525 System.out.println(cacheConf.toString()); 526 // Load up the index. 527 // Get a scanner that caches and that does not use pread. 528 HFileScanner scanner = reader.getScanner(true, false); 529 // Align scanner at start of the file. 530 scanner.seekTo(); 531 readAllRecords(scanner); 532 int seekTo = scanner.seekTo(KeyValueUtil.createKeyValueFromKey(getSomeKey(50))); 533 System.out.println(seekTo); 534 assertTrue("location lookup failed", 535 scanner.seekTo(KeyValueUtil.createKeyValueFromKey(getSomeKey(50))) == 0); 536 // read the key and see if it matches 537 ByteBuffer readKey = ByteBuffer.wrap(((KeyValue)scanner.getKey()).getKey()); 538 assertTrue("seeked key does not match", Arrays.equals(getSomeKey(50), 539 Bytes.toBytes(readKey))); 540 541 scanner.seekTo(KeyValueUtil.createKeyValueFromKey(getSomeKey(0))); 542 ByteBuffer val1 = scanner.getValue(); 543 scanner.seekTo(KeyValueUtil.createKeyValueFromKey(getSomeKey(0))); 544 ByteBuffer val2 = scanner.getValue(); 545 assertTrue(Arrays.equals(Bytes.toBytes(val1), Bytes.toBytes(val2))); 546 547 reader.close(); 548 fin.close(); 549 fs.delete(ncHFile, true); 550 } 551 552 @Test 553 public void testTFileFeatures() throws IOException { 554 testHFilefeaturesInternals(false); 555 testHFilefeaturesInternals(true); 556 } 557 558 protected void testHFilefeaturesInternals(boolean useTags) throws IOException { 559 basicWithSomeCodec("none", useTags); 560 basicWithSomeCodec("gz", useTags); 561 } 562 563 private void writeNumMetablocks(Writer writer, int n) { 564 for (int i = 0; i < n; i++) { 565 writer.appendMetaBlock("HFileMeta" + i, new Writable() { 566 private int val; 567 public Writable setVal(int val) { this.val = val; return this; } 568 569 @Override 570 public void write(DataOutput out) throws IOException { 571 out.write(("something to test" + val).getBytes()); 572 } 573 574 @Override 575 public void readFields(DataInput in) throws IOException { } 576 }.setVal(i)); 577 } 578 } 579 580 private void someTestingWithMetaBlock(Writer writer) { 581 writeNumMetablocks(writer, 10); 582 } 583 584 private void readNumMetablocks(Reader reader, int n) throws IOException { 585 for (int i = 0; i < n; i++) { 586 ByteBuff actual = reader.getMetaBlock("HFileMeta" + i, false).getBufferWithoutHeader(); 587 ByteBuffer expected = 588 ByteBuffer.wrap(("something to test" + i).getBytes()); 589 assertEquals( 590 "failed to match metadata", 591 Bytes.toStringBinary(expected), 592 Bytes.toStringBinary(actual.array(), actual.arrayOffset() + actual.position(), 593 actual.capacity())); 594 } 595 } 596 597 private void someReadingWithMetaBlock(Reader reader) throws IOException { 598 readNumMetablocks(reader, 10); 599 } 600 601 private void metablocks(final String compress) throws Exception { 602 Path mFile = new Path(ROOT_DIR, "meta.hfile"); 603 FSDataOutputStream fout = createFSOutput(mFile); 604 HFileContext meta = new HFileContextBuilder() 605 .withCompression(HFileWriterImpl.compressionByName(compress)) 606 .withBlockSize(minBlockSize).build(); 607 Writer writer = HFile.getWriterFactory(conf, cacheConf) 608 .withOutputStream(fout) 609 .withFileContext(meta) 610 .create(); 611 someTestingWithMetaBlock(writer); 612 writer.close(); 613 fout.close(); 614 ReaderContext context = new ReaderContextBuilder().withFileSystemAndPath(fs, mFile).build(); 615 Reader reader = createReaderFromStream(context, cacheConf, conf); 616 // No data -- this should return false. 617 assertFalse(reader.getScanner(false, false).seekTo()); 618 someReadingWithMetaBlock(reader); 619 fs.delete(mFile, true); 620 reader.close(); 621 } 622 623 // test meta blocks for hfiles 624 @Test 625 public void testMetaBlocks() throws Exception { 626 metablocks("none"); 627 metablocks("gz"); 628 } 629 630 @Test 631 public void testNullMetaBlocks() throws Exception { 632 for (Compression.Algorithm compressAlgo : 633 HBaseCommonTestingUtility.COMPRESSION_ALGORITHMS) { 634 Path mFile = new Path(ROOT_DIR, "nometa_" + compressAlgo + ".hfile"); 635 FSDataOutputStream fout = createFSOutput(mFile); 636 HFileContext meta = new HFileContextBuilder().withCompression(compressAlgo) 637 .withBlockSize(minBlockSize).build(); 638 Writer writer = HFile.getWriterFactory(conf, cacheConf) 639 .withOutputStream(fout) 640 .withFileContext(meta) 641 .create(); 642 KeyValue kv = new KeyValue("foo".getBytes(), "f1".getBytes(), null, "value".getBytes()); 643 writer.append(kv); 644 writer.close(); 645 fout.close(); 646 Reader reader = HFile.createReader(fs, mFile, cacheConf, true, conf); 647 assertNull(reader.getMetaBlock("non-existant", false)); 648 } 649 } 650 651 /** 652 * Make sure the ordinals for our compression algorithms do not change on us. 653 */ 654 @Test 655 public void testCompressionOrdinance() { 656 assertTrue(Compression.Algorithm.LZO.ordinal() == 0); 657 assertTrue(Compression.Algorithm.GZ.ordinal() == 1); 658 assertTrue(Compression.Algorithm.NONE.ordinal() == 2); 659 assertTrue(Compression.Algorithm.SNAPPY.ordinal() == 3); 660 assertTrue(Compression.Algorithm.LZ4.ordinal() == 4); 661 } 662 663 @Test 664 public void testShortMidpointSameQual() { 665 Cell left = CellUtil.createCell(Bytes.toBytes("a"), 666 Bytes.toBytes("a"), 667 Bytes.toBytes("a"), 668 11, 669 KeyValue.Type.Maximum.getCode(), 670 HConstants.EMPTY_BYTE_ARRAY); 671 Cell right = CellUtil.createCell(Bytes.toBytes("a"), 672 Bytes.toBytes("a"), 673 Bytes.toBytes("a"), 674 9, 675 KeyValue.Type.Maximum.getCode(), 676 HConstants.EMPTY_BYTE_ARRAY); 677 Cell mid = HFileWriterImpl.getMidpoint(CellComparatorImpl.COMPARATOR, left, right); 678 assertTrue(PrivateCellUtil.compareKeyIgnoresMvcc(CellComparatorImpl.COMPARATOR, left, mid) <= 0); 679 assertTrue(PrivateCellUtil.compareKeyIgnoresMvcc(CellComparatorImpl.COMPARATOR, mid, right) == 0); 680 } 681 682 @Test 683 public void testGetShortMidpoint() { 684 Cell left = CellUtil.createCell(Bytes.toBytes("a"), Bytes.toBytes("a"), Bytes.toBytes("a")); 685 Cell right = CellUtil.createCell(Bytes.toBytes("a"), Bytes.toBytes("a"), Bytes.toBytes("a")); 686 Cell mid = HFileWriterImpl.getMidpoint(CellComparatorImpl.COMPARATOR, left, right); 687 assertTrue(PrivateCellUtil.compareKeyIgnoresMvcc(CellComparatorImpl.COMPARATOR, left, mid) <= 0); 688 assertTrue(PrivateCellUtil.compareKeyIgnoresMvcc(CellComparatorImpl.COMPARATOR, mid, right) <= 0); 689 690 left = CellUtil.createCell(Bytes.toBytes("a"), Bytes.toBytes("a"), Bytes.toBytes("a")); 691 right = CellUtil.createCell(Bytes.toBytes("b"), Bytes.toBytes("a"), Bytes.toBytes("a")); 692 mid = HFileWriterImpl.getMidpoint(CellComparatorImpl.COMPARATOR, left, right); 693 assertTrue(PrivateCellUtil.compareKeyIgnoresMvcc(CellComparatorImpl.COMPARATOR, left, mid) < 0); 694 assertTrue(PrivateCellUtil.compareKeyIgnoresMvcc(CellComparatorImpl.COMPARATOR, mid, right) <= 0); 695 696 left = CellUtil.createCell(Bytes.toBytes("g"), Bytes.toBytes("a"), Bytes.toBytes("a")); 697 right = CellUtil.createCell(Bytes.toBytes("i"), Bytes.toBytes("a"), Bytes.toBytes("a")); 698 mid = HFileWriterImpl.getMidpoint(CellComparatorImpl.COMPARATOR, left, right); 699 assertTrue(PrivateCellUtil.compareKeyIgnoresMvcc(CellComparatorImpl.COMPARATOR, left, mid) < 0); 700 assertTrue(PrivateCellUtil.compareKeyIgnoresMvcc(CellComparatorImpl.COMPARATOR, mid, right) <= 0); 701 702 left = CellUtil.createCell(Bytes.toBytes("a"), Bytes.toBytes("a"), Bytes.toBytes("a")); 703 right = CellUtil.createCell(Bytes.toBytes("bbbbbbb"), Bytes.toBytes("a"), Bytes.toBytes("a")); 704 mid = HFileWriterImpl.getMidpoint(CellComparatorImpl.COMPARATOR, left, right); 705 assertTrue(PrivateCellUtil.compareKeyIgnoresMvcc(CellComparatorImpl.COMPARATOR, left, mid) < 0); 706 assertTrue(PrivateCellUtil.compareKeyIgnoresMvcc(CellComparatorImpl.COMPARATOR, mid, right) < 0); 707 assertEquals(1, mid.getRowLength()); 708 709 left = CellUtil.createCell(Bytes.toBytes("a"), Bytes.toBytes("a"), Bytes.toBytes("a")); 710 right = CellUtil.createCell(Bytes.toBytes("a"), Bytes.toBytes("b"), Bytes.toBytes("a")); 711 mid = HFileWriterImpl.getMidpoint(CellComparatorImpl.COMPARATOR, left, right); 712 assertTrue(PrivateCellUtil.compareKeyIgnoresMvcc(CellComparatorImpl.COMPARATOR, left, mid) < 0); 713 assertTrue(PrivateCellUtil.compareKeyIgnoresMvcc(CellComparatorImpl.COMPARATOR, mid, right) <= 0); 714 715 left = CellUtil.createCell(Bytes.toBytes("a"), Bytes.toBytes("a"), Bytes.toBytes("a")); 716 right = CellUtil.createCell(Bytes.toBytes("a"), Bytes.toBytes("aaaaaaaa"), Bytes.toBytes("b")); 717 mid = HFileWriterImpl.getMidpoint(CellComparatorImpl.COMPARATOR, left, right); 718 assertTrue(PrivateCellUtil.compareKeyIgnoresMvcc(CellComparatorImpl.COMPARATOR, left, mid) < 0); 719 assertTrue(PrivateCellUtil.compareKeyIgnoresMvcc(CellComparatorImpl.COMPARATOR, mid, right) < 0); 720 assertEquals(2, mid.getFamilyLength()); 721 722 left = CellUtil.createCell(Bytes.toBytes("a"), Bytes.toBytes("a"), Bytes.toBytes("a")); 723 right = CellUtil.createCell(Bytes.toBytes("a"), Bytes.toBytes("a"), Bytes.toBytes("aaaaaaaaa")); 724 mid = HFileWriterImpl.getMidpoint(CellComparatorImpl.COMPARATOR, left, right); 725 assertTrue(PrivateCellUtil.compareKeyIgnoresMvcc(CellComparatorImpl.COMPARATOR, left, mid) < 0); 726 assertTrue(PrivateCellUtil.compareKeyIgnoresMvcc(CellComparatorImpl.COMPARATOR, mid, right) < 0); 727 assertEquals(2, mid.getQualifierLength()); 728 729 left = CellUtil.createCell(Bytes.toBytes("a"), Bytes.toBytes("a"), Bytes.toBytes("a")); 730 right = CellUtil.createCell(Bytes.toBytes("a"), Bytes.toBytes("a"), Bytes.toBytes("b")); 731 mid = HFileWriterImpl.getMidpoint(CellComparatorImpl.COMPARATOR, left, right); 732 assertTrue(PrivateCellUtil.compareKeyIgnoresMvcc(CellComparatorImpl.COMPARATOR, left, mid) < 0); 733 assertTrue(PrivateCellUtil.compareKeyIgnoresMvcc(CellComparatorImpl.COMPARATOR, mid, right) <= 0); 734 assertEquals(1, mid.getQualifierLength()); 735 736 // Assert that if meta comparator, it returns the right cell -- i.e. no 737 // optimization done. 738 left = CellUtil.createCell(Bytes.toBytes("g"), Bytes.toBytes("a"), Bytes.toBytes("a")); 739 right = CellUtil.createCell(Bytes.toBytes("i"), Bytes.toBytes("a"), Bytes.toBytes("a")); 740 mid = HFileWriterImpl.getMidpoint(MetaCellComparator.META_COMPARATOR, left, right); 741 assertTrue(PrivateCellUtil.compareKeyIgnoresMvcc(CellComparatorImpl.COMPARATOR, left, mid) < 0); 742 assertTrue(PrivateCellUtil.compareKeyIgnoresMvcc(CellComparatorImpl.COMPARATOR, mid, right) == 0); 743 744 /** 745 * See HBASE-7845 746 */ 747 byte[] rowA = Bytes.toBytes("rowA"); 748 byte[] rowB = Bytes.toBytes("rowB"); 749 750 byte[] family = Bytes.toBytes("family"); 751 byte[] qualA = Bytes.toBytes("qfA"); 752 byte[] qualB = Bytes.toBytes("qfB"); 753 final CellComparatorImpl keyComparator = CellComparatorImpl.COMPARATOR; 754 // verify that faked shorter rowkey could be generated 755 long ts = 5; 756 KeyValue kv1 = new KeyValue(Bytes.toBytes("the quick brown fox"), family, qualA, ts, Type.Put); 757 KeyValue kv2 = new KeyValue(Bytes.toBytes("the who test text"), family, qualA, ts, Type.Put); 758 Cell newKey = HFileWriterImpl.getMidpoint(keyComparator, kv1, kv2); 759 assertTrue(keyComparator.compare(kv1, newKey) < 0); 760 assertTrue((keyComparator.compare(kv2, newKey)) > 0); 761 byte[] expectedArray = Bytes.toBytes("the r"); 762 Bytes.equals(newKey.getRowArray(), newKey.getRowOffset(), newKey.getRowLength(), expectedArray, 763 0, expectedArray.length); 764 765 // verify: same with "row + family + qualifier", return rightKey directly 766 kv1 = new KeyValue(Bytes.toBytes("ilovehbase"), family, qualA, 5, Type.Put); 767 kv2 = new KeyValue(Bytes.toBytes("ilovehbase"), family, qualA, 0, Type.Put); 768 assertTrue(keyComparator.compare(kv1, kv2) < 0); 769 newKey = HFileWriterImpl.getMidpoint(keyComparator, kv1, kv2); 770 assertTrue(keyComparator.compare(kv1, newKey) < 0); 771 assertTrue((keyComparator.compare(kv2, newKey)) == 0); 772 kv1 = new KeyValue(Bytes.toBytes("ilovehbase"), family, qualA, -5, Type.Put); 773 kv2 = new KeyValue(Bytes.toBytes("ilovehbase"), family, qualA, -10, Type.Put); 774 assertTrue(keyComparator.compare(kv1, kv2) < 0); 775 newKey = HFileWriterImpl.getMidpoint(keyComparator, kv1, kv2); 776 assertTrue(keyComparator.compare(kv1, newKey) < 0); 777 assertTrue((keyComparator.compare(kv2, newKey)) == 0); 778 779 // verify: same with row, different with qualifier 780 kv1 = new KeyValue(Bytes.toBytes("ilovehbase"), family, qualA, 5, Type.Put); 781 kv2 = new KeyValue(Bytes.toBytes("ilovehbase"), family, qualB, 5, Type.Put); 782 assertTrue(keyComparator.compare(kv1, kv2) < 0); 783 newKey = HFileWriterImpl.getMidpoint(keyComparator, kv1, kv2); 784 assertTrue(keyComparator.compare(kv1, newKey) < 0); 785 assertTrue((keyComparator.compare(kv2, newKey)) > 0); 786 assertTrue(Arrays.equals(CellUtil.cloneFamily(newKey), family)); 787 assertTrue(Arrays.equals(CellUtil.cloneQualifier(newKey), qualB)); 788 assertTrue(newKey.getTimestamp() == HConstants.LATEST_TIMESTAMP); 789 assertTrue(newKey.getTypeByte() == Type.Maximum.getCode()); 790 791 // verify metaKeyComparator's getShortMidpointKey output 792 final CellComparatorImpl metaKeyComparator = MetaCellComparator.META_COMPARATOR; 793 kv1 = new KeyValue(Bytes.toBytes("ilovehbase123"), family, qualA, 5, Type.Put); 794 kv2 = new KeyValue(Bytes.toBytes("ilovehbase234"), family, qualA, 0, Type.Put); 795 newKey = HFileWriterImpl.getMidpoint(metaKeyComparator, kv1, kv2); 796 assertTrue(metaKeyComparator.compare(kv1, newKey) < 0); 797 assertTrue((metaKeyComparator.compare(kv2, newKey) == 0)); 798 799 // verify common fix scenario 800 kv1 = new KeyValue(Bytes.toBytes("ilovehbase"), family, qualA, ts, Type.Put); 801 kv2 = new KeyValue(Bytes.toBytes("ilovehbaseandhdfs"), family, qualA, ts, Type.Put); 802 assertTrue(keyComparator.compare(kv1, kv2) < 0); 803 newKey = HFileWriterImpl.getMidpoint(keyComparator, kv1, kv2); 804 assertTrue(keyComparator.compare(kv1, newKey) < 0); 805 assertTrue((keyComparator.compare(kv2, newKey)) > 0); 806 expectedArray = Bytes.toBytes("ilovehbasea"); 807 Bytes.equals(newKey.getRowArray(), newKey.getRowOffset(), newKey.getRowLength(), expectedArray, 808 0, expectedArray.length); 809 // verify only 1 offset scenario 810 kv1 = new KeyValue(Bytes.toBytes("100abcdefg"), family, qualA, ts, Type.Put); 811 kv2 = new KeyValue(Bytes.toBytes("101abcdefg"), family, qualA, ts, Type.Put); 812 assertTrue(keyComparator.compare(kv1, kv2) < 0); 813 newKey = HFileWriterImpl.getMidpoint(keyComparator, kv1, kv2); 814 assertTrue(keyComparator.compare(kv1, newKey) < 0); 815 assertTrue((keyComparator.compare(kv2, newKey)) > 0); 816 expectedArray = Bytes.toBytes("101"); 817 Bytes.equals(newKey.getRowArray(), newKey.getRowOffset(), newKey.getRowLength(), expectedArray, 818 0, expectedArray.length); 819 } 820 821 @Test 822 public void testDBEShipped() throws IOException { 823 for (DataBlockEncoding encoding : DataBlockEncoding.values()) { 824 DataBlockEncoder encoder = encoding.getEncoder(); 825 if (encoder == null) { 826 continue; 827 } 828 Path f = new Path(ROOT_DIR, testName.getMethodName() + "_" + encoding); 829 HFileContext context = new HFileContextBuilder() 830 .withIncludesTags(false) 831 .withDataBlockEncoding(encoding).build(); 832 HFileWriterImpl writer = (HFileWriterImpl) HFile.getWriterFactory(conf, cacheConf) 833 .withPath(fs, f).withFileContext(context).create(); 834 835 KeyValue kv = new KeyValue(Bytes.toBytes("testkey1"), Bytes.toBytes("family"), 836 Bytes.toBytes("qual"), Bytes.toBytes("testvalue")); 837 KeyValue kv2 = new KeyValue(Bytes.toBytes("testkey2"), Bytes.toBytes("family"), 838 Bytes.toBytes("qual"), Bytes.toBytes("testvalue")); 839 KeyValue kv3 = new KeyValue(Bytes.toBytes("testkey3"), Bytes.toBytes("family"), 840 Bytes.toBytes("qual"), Bytes.toBytes("testvalue")); 841 842 ByteBuffer buffer = ByteBuffer.wrap(kv.getBuffer()); 843 ByteBuffer buffer2 = ByteBuffer.wrap(kv2.getBuffer()); 844 ByteBuffer buffer3 = ByteBuffer.wrap(kv3.getBuffer()); 845 846 writer.append(new ByteBufferKeyValue(buffer, 0, buffer.remaining())); 847 writer.beforeShipped(); 848 849 // pollute first cell's backing ByteBuffer 850 ByteBufferUtils.copyFromBufferToBuffer(buffer3, buffer); 851 852 // write another cell, if DBE not Shipped, test will fail 853 writer.append(new ByteBufferKeyValue(buffer2, 0, buffer2.remaining())); 854 writer.close(); 855 } 856 } 857}