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.CellBuilder;
053import org.apache.hadoop.hbase.CellBuilderFactory;
054import org.apache.hadoop.hbase.CellBuilderType;
055import org.apache.hadoop.hbase.CellComparatorImpl;
056import org.apache.hadoop.hbase.CellUtil;
057import org.apache.hadoop.hbase.ExtendedCellBuilderFactory;
058import org.apache.hadoop.hbase.HBaseClassTestRule;
059import org.apache.hadoop.hbase.HBaseCommonTestingUtility;
060import org.apache.hadoop.hbase.HBaseConfiguration;
061import org.apache.hadoop.hbase.HBaseTestingUtility;
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 HBaseTestingUtility TEST_UTIL = new HBaseTestingUtility();
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    CellBuilder cellBuilder = CellBuilderFactory.create(CellBuilderType.SHALLOW_COPY);
498    byte[] row = Bytes.toBytes("foo");
499    byte[] qualifier = Bytes.toBytes("qualifier");
500    byte[] cf = Bytes.toBytes(columnFamily);
501    byte[] val = Bytes.toBytes("fooVal");
502    long firstTS = 100L;
503    long secondTS = 101L;
504    Cell firstCell = cellBuilder.setRow(row).setValue(val).setTimestamp(firstTS)
505      .setQualifier(qualifier).setFamily(cf).setType(Cell.Type.Put).build();
506    Cell secondCell = cellBuilder.setRow(row).setValue(val).setTimestamp(secondTS)
507      .setQualifier(qualifier).setFamily(cf).setType(Cell.Type.Put).build();
508    // second Cell will sort "higher" than the first because later timestamps should come first
509    writer.append(firstCell);
510    try {
511      writer.append(secondCell);
512    } catch (IOException ie) {
513      String message = ie.getMessage();
514      Assert.assertTrue(message.contains("not lexically larger"));
515      Assert.assertTrue(message.contains(tableName));
516      Assert.assertTrue(message.contains(columnFamily));
517      return;
518    }
519    Assert.fail("Exception wasn't thrown even though Cells were appended in the wrong order!");
520  }
521
522  public static void truncateFile(FileSystem fs, Path src, Path dst) throws IOException {
523    FileStatus fst = fs.getFileStatus(src);
524    long len = fst.getLen();
525    len = len / 2;
526
527    // create a truncated hfile
528    FSDataOutputStream fdos = fs.create(dst);
529    byte[] buf = new byte[(int) len];
530    FSDataInputStream fdis = fs.open(src);
531    fdis.read(buf);
532    fdos.write(buf);
533    fdis.close();
534    fdos.close();
535  }
536
537  /**
538   * Create a truncated hfile and verify that exception thrown.
539   */
540  @Test
541  public void testCorruptTruncatedHFile() throws IOException {
542    Path f = new Path(ROOT_DIR, testName.getMethodName());
543    HFileContext context = new HFileContextBuilder().build();
544    Writer w = HFile.getWriterFactory(conf, cacheConf).withPath(this.fs, f).withFileContext(context)
545      .create();
546    writeSomeRecords(w, 0, 100, false);
547    w.close();
548
549    Path trunc = new Path(f.getParent(), "trucated");
550    truncateFile(fs, w.getPath(), trunc);
551
552    try {
553      HFile.createReader(fs, trunc, cacheConf, true, conf);
554    } catch (CorruptHFileException | IllegalArgumentException che) {
555      // Expected failure
556      return;
557    }
558    fail("Should have thrown exception");
559  }
560
561  // write some records into the hfile
562  // write them twice
563  private int writeSomeRecords(Writer writer, int start, int n, boolean useTags)
564    throws IOException {
565    String value = "value";
566    KeyValue kv;
567    for (int i = start; i < (start + n); i++) {
568      String key = String.format(localFormatter, Integer.valueOf(i));
569      if (useTags) {
570        Tag t = new ArrayBackedTag((byte) 1, "myTag1");
571        Tag[] tags = new Tag[1];
572        tags[0] = t;
573        kv = new KeyValue(Bytes.toBytes(key), Bytes.toBytes("family"), Bytes.toBytes("qual"),
574          HConstants.LATEST_TIMESTAMP, Bytes.toBytes(value + key), tags);
575        writer.append(kv);
576      } else {
577        kv = new KeyValue(Bytes.toBytes(key), Bytes.toBytes("family"), Bytes.toBytes("qual"),
578          Bytes.toBytes(value + key));
579        writer.append(kv);
580      }
581    }
582    return (start + n);
583  }
584
585  private void readAllRecords(HFileScanner scanner) throws IOException {
586    readAndCheckbytes(scanner, 0, 100);
587  }
588
589  // read the records and check
590  private int readAndCheckbytes(HFileScanner scanner, int start, int n) throws IOException {
591    String value = "value";
592    int i = start;
593    for (; i < (start + n); i++) {
594      ByteBuffer key = ByteBuffer.wrap(((KeyValue) scanner.getKey()).getKey());
595      ByteBuffer val = scanner.getValue();
596      String keyStr = String.format(localFormatter, Integer.valueOf(i));
597      String valStr = value + keyStr;
598      KeyValue kv = new KeyValue(Bytes.toBytes(keyStr), Bytes.toBytes("family"),
599        Bytes.toBytes("qual"), Bytes.toBytes(valStr));
600      byte[] keyBytes =
601        new KeyValue.KeyOnlyKeyValue(Bytes.toBytes(key), 0, Bytes.toBytes(key).length).getKey();
602      assertTrue("bytes for keys do not match " + keyStr + " " + Bytes.toString(Bytes.toBytes(key)),
603        Arrays.equals(kv.getKey(), keyBytes));
604      byte[] valBytes = Bytes.toBytes(val);
605      assertTrue("bytes for vals do not match " + valStr + " " + Bytes.toString(valBytes),
606        Arrays.equals(Bytes.toBytes(valStr), valBytes));
607      if (!scanner.next()) {
608        break;
609      }
610    }
611    assertEquals(i, start + n - 1);
612    return (start + n);
613  }
614
615  private byte[] getSomeKey(int rowId) {
616    KeyValue kv = new KeyValue(String.format(localFormatter, Integer.valueOf(rowId)).getBytes(),
617      Bytes.toBytes("family"), Bytes.toBytes("qual"), HConstants.LATEST_TIMESTAMP, Type.Put);
618    return kv.getKey();
619  }
620
621  private void writeRecords(Writer writer, boolean useTags) throws IOException {
622    writeSomeRecords(writer, 0, 100, useTags);
623    writer.close();
624  }
625
626  private FSDataOutputStream createFSOutput(Path name) throws IOException {
627    // if (fs.exists(name)) fs.delete(name, true);
628    FSDataOutputStream fout = fs.create(name);
629    return fout;
630  }
631
632  /**
633   * test none codecs
634   */
635  void basicWithSomeCodec(String codec, boolean useTags) throws IOException {
636    if (useTags) {
637      conf.setInt("hfile.format.version", 3);
638    }
639    Path ncHFile = new Path(ROOT_DIR, "basic.hfile." + codec.toString() + useTags);
640    FSDataOutputStream fout = createFSOutput(ncHFile);
641    HFileContext meta = new HFileContextBuilder().withBlockSize(minBlockSize)
642      .withCompression(HFileWriterImpl.compressionByName(codec)).build();
643    Writer writer =
644      HFile.getWriterFactory(conf, cacheConf).withOutputStream(fout).withFileContext(meta).create();
645    LOG.info(Objects.toString(writer));
646    writeRecords(writer, useTags);
647    fout.close();
648    FSDataInputStream fin = fs.open(ncHFile);
649    ReaderContext context = new ReaderContextBuilder().withFileSystemAndPath(fs, ncHFile).build();
650    Reader reader = createReaderFromStream(context, cacheConf, conf);
651    System.out.println(cacheConf.toString());
652    // Load up the index.
653    // Get a scanner that caches and that does not use pread.
654    HFileScanner scanner = reader.getScanner(conf, true, false);
655    // Align scanner at start of the file.
656    scanner.seekTo();
657    readAllRecords(scanner);
658    int seekTo = scanner.seekTo(KeyValueUtil.createKeyValueFromKey(getSomeKey(50)));
659    System.out.println(seekTo);
660    assertTrue("location lookup failed",
661      scanner.seekTo(KeyValueUtil.createKeyValueFromKey(getSomeKey(50))) == 0);
662    // read the key and see if it matches
663    ByteBuffer readKey = ByteBuffer.wrap(((KeyValue) scanner.getKey()).getKey());
664    assertTrue("seeked key does not match", Arrays.equals(getSomeKey(50), Bytes.toBytes(readKey)));
665
666    scanner.seekTo(KeyValueUtil.createKeyValueFromKey(getSomeKey(0)));
667    ByteBuffer val1 = scanner.getValue();
668    scanner.seekTo(KeyValueUtil.createKeyValueFromKey(getSomeKey(0)));
669    ByteBuffer val2 = scanner.getValue();
670    assertTrue(Arrays.equals(Bytes.toBytes(val1), Bytes.toBytes(val2)));
671
672    reader.close();
673    fin.close();
674    fs.delete(ncHFile, true);
675  }
676
677  @Test
678  public void testTFileFeatures() throws IOException {
679    testHFilefeaturesInternals(false);
680    testHFilefeaturesInternals(true);
681  }
682
683  protected void testHFilefeaturesInternals(boolean useTags) throws IOException {
684    basicWithSomeCodec("none", useTags);
685    basicWithSomeCodec("gz", useTags);
686  }
687
688  private void writeNumMetablocks(Writer writer, int n) {
689    for (int i = 0; i < n; i++) {
690      writer.appendMetaBlock("HFileMeta" + i, new Writable() {
691        private int val;
692
693        public Writable setVal(int val) {
694          this.val = val;
695          return this;
696        }
697
698        @Override
699        public void write(DataOutput out) throws IOException {
700          out.write(("something to test" + val).getBytes());
701        }
702
703        @Override
704        public void readFields(DataInput in) throws IOException {
705        }
706      }.setVal(i));
707    }
708  }
709
710  private void someTestingWithMetaBlock(Writer writer) {
711    writeNumMetablocks(writer, 10);
712  }
713
714  private void readNumMetablocks(Reader reader, int n) throws IOException {
715    for (int i = 0; i < n; i++) {
716      ByteBuff actual = reader.getMetaBlock("HFileMeta" + i, false).getBufferWithoutHeader();
717      ByteBuffer expected = ByteBuffer.wrap(("something to test" + i).getBytes());
718      assertEquals("failed to match metadata", Bytes.toStringBinary(expected), Bytes.toStringBinary(
719        actual.array(), actual.arrayOffset() + actual.position(), actual.capacity()));
720    }
721  }
722
723  private void someReadingWithMetaBlock(Reader reader) throws IOException {
724    readNumMetablocks(reader, 10);
725  }
726
727  private void metablocks(final String compress) throws Exception {
728    Path mFile = new Path(ROOT_DIR, "meta.hfile");
729    FSDataOutputStream fout = createFSOutput(mFile);
730    HFileContext meta =
731      new HFileContextBuilder().withCompression(HFileWriterImpl.compressionByName(compress))
732        .withBlockSize(minBlockSize).build();
733    Writer writer =
734      HFile.getWriterFactory(conf, cacheConf).withOutputStream(fout).withFileContext(meta).create();
735    someTestingWithMetaBlock(writer);
736    writer.close();
737    fout.close();
738    ReaderContext context = new ReaderContextBuilder().withFileSystemAndPath(fs, mFile).build();
739    Reader reader = createReaderFromStream(context, cacheConf, conf);
740    // No data -- this should return false.
741    assertFalse(reader.getScanner(conf, false, false).seekTo());
742    someReadingWithMetaBlock(reader);
743    fs.delete(mFile, true);
744    reader.close();
745  }
746
747  // test meta blocks for hfiles
748  @Test
749  public void testMetaBlocks() throws Exception {
750    metablocks("none");
751    metablocks("gz");
752  }
753
754  @Test
755  public void testNullMetaBlocks() throws Exception {
756    for (Compression.Algorithm compressAlgo : HBaseCommonTestingUtility.COMPRESSION_ALGORITHMS) {
757      Path mFile = new Path(ROOT_DIR, "nometa_" + compressAlgo + ".hfile");
758      FSDataOutputStream fout = createFSOutput(mFile);
759      HFileContext meta =
760        new HFileContextBuilder().withCompression(compressAlgo).withBlockSize(minBlockSize).build();
761      Writer writer = HFile.getWriterFactory(conf, cacheConf).withOutputStream(fout)
762        .withFileContext(meta).create();
763      KeyValue kv = new KeyValue("foo".getBytes(), "f1".getBytes(), null, "value".getBytes());
764      writer.append(kv);
765      writer.close();
766      fout.close();
767      Reader reader = HFile.createReader(fs, mFile, cacheConf, true, conf);
768      assertNull(reader.getMetaBlock("non-existant", false));
769    }
770  }
771
772  /**
773   * Make sure the ordinals for our compression algorithms do not change on us.
774   */
775  @Test
776  public void testCompressionOrdinance() {
777    assertTrue(Compression.Algorithm.LZO.ordinal() == 0);
778    assertTrue(Compression.Algorithm.GZ.ordinal() == 1);
779    assertTrue(Compression.Algorithm.NONE.ordinal() == 2);
780    assertTrue(Compression.Algorithm.SNAPPY.ordinal() == 3);
781    assertTrue(Compression.Algorithm.LZ4.ordinal() == 4);
782  }
783
784  @Test
785  public void testShortMidpointSameQual() {
786    Cell left = CellUtil.createCell(Bytes.toBytes("a"), Bytes.toBytes("a"), Bytes.toBytes("a"), 11,
787      KeyValue.Type.Maximum.getCode(), HConstants.EMPTY_BYTE_ARRAY);
788    Cell right = CellUtil.createCell(Bytes.toBytes("a"), Bytes.toBytes("a"), Bytes.toBytes("a"), 9,
789      KeyValue.Type.Maximum.getCode(), HConstants.EMPTY_BYTE_ARRAY);
790    Cell mid = HFileWriterImpl.getMidpoint(CellComparatorImpl.COMPARATOR, left, right);
791    assertTrue(
792      PrivateCellUtil.compareKeyIgnoresMvcc(CellComparatorImpl.COMPARATOR, left, mid) <= 0);
793    assertTrue(
794      PrivateCellUtil.compareKeyIgnoresMvcc(CellComparatorImpl.COMPARATOR, mid, right) == 0);
795  }
796
797  private Cell getCell(byte[] row, byte[] family, byte[] qualifier) {
798    return ExtendedCellBuilderFactory.create(CellBuilderType.DEEP_COPY).setRow(row)
799      .setFamily(family).setQualifier(qualifier).setTimestamp(HConstants.LATEST_TIMESTAMP)
800      .setType(KeyValue.Type.Maximum.getCode()).setValue(HConstants.EMPTY_BYTE_ARRAY).build();
801  }
802
803  @Test
804  public void testGetShortMidpoint() {
805    Cell left = CellUtil.createCell(Bytes.toBytes("a"), Bytes.toBytes("a"), Bytes.toBytes("a"));
806    Cell right = CellUtil.createCell(Bytes.toBytes("a"), Bytes.toBytes("a"), Bytes.toBytes("a"));
807    Cell mid = HFileWriterImpl.getMidpoint(CellComparatorImpl.COMPARATOR, left, right);
808    assertTrue(
809      PrivateCellUtil.compareKeyIgnoresMvcc(CellComparatorImpl.COMPARATOR, left, mid) <= 0);
810    assertTrue(
811      PrivateCellUtil.compareKeyIgnoresMvcc(CellComparatorImpl.COMPARATOR, mid, right) <= 0);
812
813    left = CellUtil.createCell(Bytes.toBytes("a"), Bytes.toBytes("a"), Bytes.toBytes("a"));
814    right = CellUtil.createCell(Bytes.toBytes("b"), Bytes.toBytes("a"), Bytes.toBytes("a"));
815    mid = HFileWriterImpl.getMidpoint(CellComparatorImpl.COMPARATOR, left, right);
816    assertTrue(PrivateCellUtil.compareKeyIgnoresMvcc(CellComparatorImpl.COMPARATOR, left, mid) < 0);
817    assertTrue(
818      PrivateCellUtil.compareKeyIgnoresMvcc(CellComparatorImpl.COMPARATOR, mid, right) <= 0);
819
820    left = CellUtil.createCell(Bytes.toBytes("g"), Bytes.toBytes("a"), Bytes.toBytes("a"));
821    right = CellUtil.createCell(Bytes.toBytes("i"), Bytes.toBytes("a"), Bytes.toBytes("a"));
822    mid = HFileWriterImpl.getMidpoint(CellComparatorImpl.COMPARATOR, left, right);
823    assertTrue(PrivateCellUtil.compareKeyIgnoresMvcc(CellComparatorImpl.COMPARATOR, left, mid) < 0);
824    assertTrue(
825      PrivateCellUtil.compareKeyIgnoresMvcc(CellComparatorImpl.COMPARATOR, mid, right) <= 0);
826
827    left = CellUtil.createCell(Bytes.toBytes("a"), Bytes.toBytes("a"), Bytes.toBytes("a"));
828    right = CellUtil.createCell(Bytes.toBytes("bbbbbbb"), Bytes.toBytes("a"), Bytes.toBytes("a"));
829    mid = HFileWriterImpl.getMidpoint(CellComparatorImpl.COMPARATOR, left, right);
830    assertTrue(PrivateCellUtil.compareKeyIgnoresMvcc(CellComparatorImpl.COMPARATOR, left, mid) < 0);
831    assertTrue(
832      PrivateCellUtil.compareKeyIgnoresMvcc(CellComparatorImpl.COMPARATOR, mid, right) < 0);
833    assertEquals(1, mid.getRowLength());
834
835    left = CellUtil.createCell(Bytes.toBytes("a"), Bytes.toBytes("a"), Bytes.toBytes("a"));
836    right = CellUtil.createCell(Bytes.toBytes("a"), Bytes.toBytes("b"), Bytes.toBytes("a"));
837    mid = HFileWriterImpl.getMidpoint(CellComparatorImpl.COMPARATOR, left, right);
838    assertTrue(PrivateCellUtil.compareKeyIgnoresMvcc(CellComparatorImpl.COMPARATOR, left, mid) < 0);
839    assertTrue(
840      PrivateCellUtil.compareKeyIgnoresMvcc(CellComparatorImpl.COMPARATOR, mid, right) <= 0);
841
842    left = CellUtil.createCell(Bytes.toBytes("a"), Bytes.toBytes("a"), Bytes.toBytes("a"));
843    right = CellUtil.createCell(Bytes.toBytes("a"), Bytes.toBytes("aaaaaaaa"), Bytes.toBytes("b"));
844    mid = HFileWriterImpl.getMidpoint(CellComparatorImpl.COMPARATOR, left, right);
845    assertTrue(PrivateCellUtil.compareKeyIgnoresMvcc(CellComparatorImpl.COMPARATOR, left, mid) < 0);
846    assertTrue(
847      PrivateCellUtil.compareKeyIgnoresMvcc(CellComparatorImpl.COMPARATOR, mid, right) < 0);
848    assertEquals(2, mid.getFamilyLength());
849
850    left = CellUtil.createCell(Bytes.toBytes("a"), Bytes.toBytes("a"), Bytes.toBytes("a"));
851    right = CellUtil.createCell(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
858    left = CellUtil.createCell(Bytes.toBytes("a"), Bytes.toBytes("a"), Bytes.toBytes("a"));
859    right = CellUtil.createCell(Bytes.toBytes("a"), Bytes.toBytes("a"), Bytes.toBytes("b"));
860    mid = HFileWriterImpl.getMidpoint(CellComparatorImpl.COMPARATOR, left, right);
861    assertTrue(PrivateCellUtil.compareKeyIgnoresMvcc(CellComparatorImpl.COMPARATOR, left, mid) < 0);
862    assertTrue(
863      PrivateCellUtil.compareKeyIgnoresMvcc(CellComparatorImpl.COMPARATOR, mid, right) <= 0);
864    assertEquals(1, mid.getQualifierLength());
865
866    // Verify boundary conditions
867    left = getCell(Bytes.toBytes("a"), Bytes.toBytes("a"), new byte[] { 0x00, (byte) 0xFE });
868    right = getCell(Bytes.toBytes("a"), Bytes.toBytes("a"), new byte[] { 0x00, (byte) 0xFF });
869    mid = HFileWriterImpl.getMidpoint(CellComparatorImpl.COMPARATOR, left, right);
870    assertTrue(PrivateCellUtil.compareKeyIgnoresMvcc(CellComparatorImpl.COMPARATOR, left, mid) < 0);
871    assertTrue(
872      PrivateCellUtil.compareKeyIgnoresMvcc(CellComparatorImpl.COMPARATOR, mid, right) == 0);
873    left = getCell(Bytes.toBytes("a"), Bytes.toBytes("a"), new byte[] { 0x00, 0x12 });
874    right = getCell(Bytes.toBytes("a"), Bytes.toBytes("a"), new byte[] { 0x00, 0x12, 0x00 });
875    mid = HFileWriterImpl.getMidpoint(CellComparatorImpl.COMPARATOR, left, right);
876    assertTrue(PrivateCellUtil.compareKeyIgnoresMvcc(CellComparatorImpl.COMPARATOR, left, mid) < 0);
877    assertTrue(
878      PrivateCellUtil.compareKeyIgnoresMvcc(CellComparatorImpl.COMPARATOR, mid, right) == 0);
879
880    // Assert that if meta comparator, it returns the right cell -- i.e. no
881    // optimization done.
882    left = CellUtil.createCell(Bytes.toBytes("g"), Bytes.toBytes("a"), Bytes.toBytes("a"));
883    right = CellUtil.createCell(Bytes.toBytes("i"), Bytes.toBytes("a"), Bytes.toBytes("a"));
884    mid = HFileWriterImpl.getMidpoint(MetaCellComparator.META_COMPARATOR, left, right);
885    assertTrue(PrivateCellUtil.compareKeyIgnoresMvcc(CellComparatorImpl.COMPARATOR, left, mid) < 0);
886    assertTrue(
887      PrivateCellUtil.compareKeyIgnoresMvcc(CellComparatorImpl.COMPARATOR, mid, right) == 0);
888
889    /**
890     * See HBASE-7845
891     */
892    byte[] rowA = Bytes.toBytes("rowA");
893    byte[] rowB = Bytes.toBytes("rowB");
894
895    byte[] family = Bytes.toBytes("family");
896    byte[] qualA = Bytes.toBytes("qfA");
897    byte[] qualB = Bytes.toBytes("qfB");
898    final CellComparatorImpl keyComparator = CellComparatorImpl.COMPARATOR;
899    // verify that faked shorter rowkey could be generated
900    long ts = 5;
901    KeyValue kv1 = new KeyValue(Bytes.toBytes("the quick brown fox"), family, qualA, ts, Type.Put);
902    KeyValue kv2 = new KeyValue(Bytes.toBytes("the who test text"), family, qualA, ts, Type.Put);
903    Cell newKey = HFileWriterImpl.getMidpoint(keyComparator, kv1, kv2);
904    assertTrue(keyComparator.compare(kv1, newKey) < 0);
905    assertTrue((keyComparator.compare(kv2, newKey)) > 0);
906    byte[] expectedArray = Bytes.toBytes("the r");
907    Bytes.equals(newKey.getRowArray(), newKey.getRowOffset(), newKey.getRowLength(), expectedArray,
908      0, expectedArray.length);
909
910    // verify: same with "row + family + qualifier", return rightKey directly
911    kv1 = new KeyValue(Bytes.toBytes("ilovehbase"), family, qualA, 5, Type.Put);
912    kv2 = new KeyValue(Bytes.toBytes("ilovehbase"), family, qualA, 0, Type.Put);
913    assertTrue(keyComparator.compare(kv1, kv2) < 0);
914    newKey = HFileWriterImpl.getMidpoint(keyComparator, kv1, kv2);
915    assertTrue(keyComparator.compare(kv1, newKey) < 0);
916    assertTrue((keyComparator.compare(kv2, newKey)) == 0);
917    kv1 = new KeyValue(Bytes.toBytes("ilovehbase"), family, qualA, -5, Type.Put);
918    kv2 = new KeyValue(Bytes.toBytes("ilovehbase"), family, qualA, -10, 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
924    // verify: same with row, different with qualifier
925    kv1 = new KeyValue(Bytes.toBytes("ilovehbase"), family, qualA, 5, Type.Put);
926    kv2 = new KeyValue(Bytes.toBytes("ilovehbase"), family, qualB, 5, Type.Put);
927    assertTrue(keyComparator.compare(kv1, kv2) < 0);
928    newKey = HFileWriterImpl.getMidpoint(keyComparator, kv1, kv2);
929    assertTrue(keyComparator.compare(kv1, newKey) < 0);
930    assertTrue((keyComparator.compare(kv2, newKey)) > 0);
931    assertTrue(Arrays.equals(CellUtil.cloneFamily(newKey), family));
932    assertTrue(Arrays.equals(CellUtil.cloneQualifier(newKey), qualB));
933    assertTrue(newKey.getTimestamp() == HConstants.LATEST_TIMESTAMP);
934    assertTrue(newKey.getTypeByte() == Type.Maximum.getCode());
935
936    // verify metaKeyComparator's getShortMidpointKey output
937    final CellComparatorImpl metaKeyComparator = MetaCellComparator.META_COMPARATOR;
938    kv1 = new KeyValue(Bytes.toBytes("ilovehbase123"), family, qualA, 5, Type.Put);
939    kv2 = new KeyValue(Bytes.toBytes("ilovehbase234"), family, qualA, 0, Type.Put);
940    newKey = HFileWriterImpl.getMidpoint(metaKeyComparator, kv1, kv2);
941    assertTrue(metaKeyComparator.compare(kv1, newKey) < 0);
942    assertTrue((metaKeyComparator.compare(kv2, newKey) == 0));
943
944    // verify common fix scenario
945    kv1 = new KeyValue(Bytes.toBytes("ilovehbase"), family, qualA, ts, Type.Put);
946    kv2 = new KeyValue(Bytes.toBytes("ilovehbaseandhdfs"), family, qualA, ts, Type.Put);
947    assertTrue(keyComparator.compare(kv1, kv2) < 0);
948    newKey = HFileWriterImpl.getMidpoint(keyComparator, kv1, kv2);
949    assertTrue(keyComparator.compare(kv1, newKey) < 0);
950    assertTrue((keyComparator.compare(kv2, newKey)) > 0);
951    expectedArray = Bytes.toBytes("ilovehbasea");
952    Bytes.equals(newKey.getRowArray(), newKey.getRowOffset(), newKey.getRowLength(), expectedArray,
953      0, expectedArray.length);
954    // verify only 1 offset scenario
955    kv1 = new KeyValue(Bytes.toBytes("100abcdefg"), family, qualA, ts, Type.Put);
956    kv2 = new KeyValue(Bytes.toBytes("101abcdefg"), family, qualA, ts, Type.Put);
957    assertTrue(keyComparator.compare(kv1, kv2) < 0);
958    newKey = HFileWriterImpl.getMidpoint(keyComparator, kv1, kv2);
959    assertTrue(keyComparator.compare(kv1, newKey) < 0);
960    assertTrue((keyComparator.compare(kv2, newKey)) > 0);
961    expectedArray = Bytes.toBytes("101");
962    Bytes.equals(newKey.getRowArray(), newKey.getRowOffset(), newKey.getRowLength(), expectedArray,
963      0, expectedArray.length);
964  }
965
966  @Test
967  public void testDBEShipped() throws IOException {
968    for (DataBlockEncoding encoding : DataBlockEncoding.values()) {
969      DataBlockEncoder encoder = encoding.getEncoder();
970      if (encoder == null) {
971        continue;
972      }
973      Path f = new Path(ROOT_DIR, testName.getMethodName() + "_" + encoding);
974      HFileContext context =
975        new HFileContextBuilder().withIncludesTags(false).withDataBlockEncoding(encoding).build();
976      HFileWriterImpl writer = (HFileWriterImpl) HFile.getWriterFactory(conf, cacheConf)
977        .withPath(fs, f).withFileContext(context).create();
978
979      KeyValue kv = new KeyValue(Bytes.toBytes("testkey1"), Bytes.toBytes("family"),
980        Bytes.toBytes("qual"), Bytes.toBytes("testvalue"));
981      KeyValue kv2 = new KeyValue(Bytes.toBytes("testkey2"), Bytes.toBytes("family"),
982        Bytes.toBytes("qual"), Bytes.toBytes("testvalue"));
983      KeyValue kv3 = new KeyValue(Bytes.toBytes("testkey3"), Bytes.toBytes("family"),
984        Bytes.toBytes("qual"), Bytes.toBytes("testvalue"));
985
986      ByteBuffer buffer = ByteBuffer.wrap(kv.getBuffer());
987      ByteBuffer buffer2 = ByteBuffer.wrap(kv2.getBuffer());
988      ByteBuffer buffer3 = ByteBuffer.wrap(kv3.getBuffer());
989
990      writer.append(new ByteBufferKeyValue(buffer, 0, buffer.remaining()));
991      writer.beforeShipped();
992
993      // pollute first cell's backing ByteBuffer
994      ByteBufferUtils.copyFromBufferToBuffer(buffer3, buffer);
995
996      // write another cell, if DBE not Shipped, test will fail
997      writer.append(new ByteBufferKeyValue(buffer2, 0, buffer2.remaining()));
998      writer.close();
999    }
1000  }
1001
1002  /**
1003   * Test case for CombinedBlockCache with TinyLfu as L1 cache
1004   */
1005  @Test
1006  public void testReaderWithTinyLfuCombinedBlockCache() throws Exception {
1007    testReaderCombinedCache("TinyLfu");
1008  }
1009
1010  /**
1011   * Test case for CombinedBlockCache with AdaptiveLRU as L1 cache
1012   */
1013  @Test
1014  public void testReaderWithAdaptiveLruCombinedBlockCache() throws Exception {
1015    testReaderCombinedCache("AdaptiveLRU");
1016  }
1017
1018  /**
1019   * Test case for CombinedBlockCache with AdaptiveLRU as L1 cache
1020   */
1021  @Test
1022  public void testReaderWithLruCombinedBlockCache() throws Exception {
1023    testReaderCombinedCache("LRU");
1024  }
1025
1026  private void testReaderCombinedCache(final String l1CachePolicy) throws Exception {
1027    int bufCount = 1024;
1028    int blockSize = 64 * 1024;
1029    ByteBuffAllocator alloc = initAllocator(true, bufCount, blockSize, 0);
1030    fillByteBuffAllocator(alloc, bufCount);
1031    Path storeFilePath = writeStoreFile();
1032    // Open the file reader with CombinedBlockCache
1033    BlockCache combined = initCombinedBlockCache(l1CachePolicy);
1034    conf.setBoolean(EVICT_BLOCKS_ON_CLOSE_KEY, true);
1035    CacheConfig cacheConfig = new CacheConfig(conf, null, combined, alloc);
1036    HFile.Reader reader = HFile.createReader(fs, storeFilePath, cacheConfig, true, conf);
1037    long offset = 0;
1038    Cacheable cachedBlock = null;
1039    while (offset < reader.getTrailer().getLoadOnOpenDataOffset()) {
1040      BlockCacheKey key = new BlockCacheKey(storeFilePath.getName(), offset);
1041      HFileBlock block = reader.readBlock(offset, -1, true, true, false, true, null, null);
1042      offset += block.getOnDiskSizeWithHeader();
1043      // Read the cached block.
1044      cachedBlock = combined.getBlock(key, false, false, true);
1045      try {
1046        Assert.assertNotNull(cachedBlock);
1047        Assert.assertTrue(cachedBlock instanceof HFileBlock);
1048        HFileBlock hfb = (HFileBlock) cachedBlock;
1049        // Data block will be cached in BucketCache, so it should be an off-heap block.
1050        if (hfb.getBlockType().isData()) {
1051          Assert.assertTrue(hfb.isSharedMem());
1052        } else if (!l1CachePolicy.equals("TinyLfu")) {
1053          Assert.assertFalse(hfb.isSharedMem());
1054        }
1055      } finally {
1056        cachedBlock.release();
1057      }
1058      block.release(); // return back the ByteBuffer back to allocator.
1059    }
1060    reader.close();
1061    combined.shutdown();
1062    if (cachedBlock != null) {
1063      Assert.assertEquals(0, cachedBlock.refCnt());
1064    }
1065    Assert.assertEquals(bufCount, alloc.getFreeBufferCount());
1066    alloc.clean();
1067  }
1068
1069  @Test
1070  public void testHFileContextBuilderWithIndexEncoding() throws IOException {
1071    HFileContext context =
1072      new HFileContextBuilder().withIndexBlockEncoding(IndexBlockEncoding.PREFIX_TREE).build();
1073    HFileContext newContext = new HFileContextBuilder(context).build();
1074    assertTrue(newContext.getIndexBlockEncoding() == IndexBlockEncoding.PREFIX_TREE);
1075  }
1076}