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.regionserver;
019
020import java.io.IOException;
021import java.util.List;
022import java.util.Random;
023import org.apache.hadoop.conf.Configuration;
024import org.apache.hadoop.fs.FileStatus;
025import org.apache.hadoop.fs.FileSystem;
026import org.apache.hadoop.fs.Path;
027import org.apache.hadoop.hbase.Cell;
028import org.apache.hadoop.hbase.CellUtil;
029import org.apache.hadoop.hbase.HBaseClassTestRule;
030import org.apache.hadoop.hbase.HBaseTestingUtility;
031import org.apache.hadoop.hbase.HColumnDescriptor;
032import org.apache.hadoop.hbase.HTableDescriptor;
033import org.apache.hadoop.hbase.TableName;
034import org.apache.hadoop.hbase.client.Admin;
035import org.apache.hadoop.hbase.client.ConnectionConfiguration;
036import org.apache.hadoop.hbase.client.ConnectionFactory;
037import org.apache.hadoop.hbase.client.Get;
038import org.apache.hadoop.hbase.client.Put;
039import org.apache.hadoop.hbase.client.RegionInfo;
040import org.apache.hadoop.hbase.client.Result;
041import org.apache.hadoop.hbase.client.ResultScanner;
042import org.apache.hadoop.hbase.client.Scan;
043import org.apache.hadoop.hbase.client.Table;
044import org.apache.hadoop.hbase.io.hfile.CorruptHFileException;
045import org.apache.hadoop.hbase.io.hfile.TestHFile;
046import org.apache.hadoop.hbase.mob.MobConstants;
047import org.apache.hadoop.hbase.mob.MobTestUtil;
048import org.apache.hadoop.hbase.mob.MobUtils;
049import org.apache.hadoop.hbase.testclassification.MediumTests;
050import org.apache.hadoop.hbase.util.Bytes;
051import org.apache.hadoop.hbase.util.FSUtils;
052import org.apache.hadoop.hbase.util.HFileArchiveUtil;
053import org.junit.AfterClass;
054import org.junit.Assert;
055import org.junit.BeforeClass;
056import org.junit.ClassRule;
057import org.junit.Rule;
058import org.junit.Test;
059import org.junit.experimental.categories.Category;
060import org.junit.rules.TestName;
061
062@Category(MediumTests.class)
063public class TestMobStoreScanner {
064
065  @ClassRule
066  public static final HBaseClassTestRule CLASS_RULE =
067      HBaseClassTestRule.forClass(TestMobStoreScanner.class);
068
069  private final static HBaseTestingUtility TEST_UTIL = new HBaseTestingUtility();
070  private final static byte [] row1 = Bytes.toBytes("row1");
071  private final static byte [] row2 = Bytes.toBytes("row2");
072  private final static byte [] family = Bytes.toBytes("family");
073  private final static byte [] qf1 = Bytes.toBytes("qualifier1");
074  private final static byte [] qf2 = Bytes.toBytes("qualifier2");
075  protected final byte[] qf3 = Bytes.toBytes("qualifier3");
076  private static Table table;
077  private static Admin admin;
078  private static HColumnDescriptor hcd;
079  private static HTableDescriptor desc;
080  private static Random random = new Random();
081  private static long defaultThreshold = 10;
082  private FileSystem fs;
083  private Configuration conf;
084
085  @Rule
086  public TestName name = new TestName();
087
088  @BeforeClass
089  public static void setUpBeforeClass() throws Exception {
090    TEST_UTIL.getConfiguration().setInt(ConnectionConfiguration.MAX_KEYVALUE_SIZE_KEY,
091        100 * 1024 * 1024);
092    TEST_UTIL.getConfiguration().setInt(HRegion.HBASE_MAX_CELL_SIZE_KEY, 100 * 1024 * 1024);
093    TEST_UTIL.startMiniCluster(1);
094  }
095
096  @AfterClass
097  public static void tearDownAfterClass() throws Exception {
098    TEST_UTIL.shutdownMiniCluster();
099  }
100
101  public void setUp(long threshold, TableName tn) throws Exception {
102    conf = TEST_UTIL.getConfiguration();
103    fs = FileSystem.get(conf);
104    desc = new HTableDescriptor(tn);
105    hcd = new HColumnDescriptor(family);
106    hcd.setMobEnabled(true);
107    hcd.setMobThreshold(threshold);
108    hcd.setMaxVersions(4);
109    desc.addFamily(hcd);
110    admin = TEST_UTIL.getAdmin();
111    admin.createTable(desc);
112    table = ConnectionFactory.createConnection(TEST_UTIL.getConfiguration())
113            .getTable(tn);
114  }
115
116  /**
117   * Generate the mob value.
118   *
119   * @param size the size of the value
120   * @return the mob value generated
121   */
122  private static byte[] generateMobValue(int size) {
123    byte[] mobVal = new byte[size];
124    random.nextBytes(mobVal);
125    return mobVal;
126  }
127
128  /**
129   * Set the scan attribute
130   *
131   * @param reversed if true, scan will be backward order
132   * @param mobScanRaw if true, scan will get the mob reference
133   */
134  public void setScan(Scan scan, boolean reversed, boolean mobScanRaw) {
135    scan.setReversed(reversed);
136    scan.setMaxVersions(4);
137    if(mobScanRaw) {
138      scan.setAttribute(MobConstants.MOB_SCAN_RAW, Bytes.toBytes(Boolean.TRUE));
139    }
140  }
141
142  @Test
143  public void testMobStoreScanner() throws Exception {
144    testGetFromFiles(false);
145    testGetFromMemStore(false);
146    testGetReferences(false);
147    testMobThreshold(false);
148    testGetFromArchive(false);
149  }
150
151  @Test
152  public void testReversedMobStoreScanner() throws Exception {
153    testGetFromFiles(true);
154    testGetFromMemStore(true);
155    testGetReferences(true);
156    testMobThreshold(true);
157    testGetFromArchive(true);
158  }
159
160  @Test
161  public void testGetMassive() throws Exception {
162    setUp(defaultThreshold, TableName.valueOf(name.getMethodName()));
163
164    // Put some data 5 10, 15, 20  mb ok  (this would be right below protobuf
165    // default max size of 64MB.
166    // 25, 30, 40 fail.  these is above protobuf max size of 64MB
167    byte[] bigValue = new byte[25*1024*1024];
168
169    Put put = new Put(row1);
170    put.addColumn(family, qf1, bigValue);
171    put.addColumn(family, qf2, bigValue);
172    put.addColumn(family, qf3, bigValue);
173    table.put(put);
174
175    Get g = new Get(row1);
176    table.get(g);
177    // should not have blown up.
178  }
179
180  @Test
181  public void testReadPt() throws Exception {
182    final TableName tableName = TableName.valueOf(name.getMethodName());
183    setUp(0L, tableName);
184    long ts = System.currentTimeMillis();
185    byte[] value1 = Bytes.toBytes("value1");
186    Put put1 = new Put(row1);
187    put1.addColumn(family, qf1, ts, value1);
188    table.put(put1);
189    Put put2 = new Put(row2);
190    byte[] value2 = Bytes.toBytes("value2");
191    put2.addColumn(family, qf1, ts, value2);
192    table.put(put2);
193
194    Scan scan = new Scan();
195    scan.setCaching(1);
196    ResultScanner rs = table.getScanner(scan);
197    Result result = rs.next();
198    Put put3 = new Put(row1);
199    byte[] value3 = Bytes.toBytes("value3");
200    put3.addColumn(family, qf1, ts, value3);
201    table.put(put3);
202    Put put4 = new Put(row2);
203    byte[] value4 = Bytes.toBytes("value4");
204    put4.addColumn(family, qf1, ts, value4);
205    table.put(put4);
206
207    Cell cell = result.getColumnLatestCell(family, qf1);
208    Assert.assertArrayEquals(value1, CellUtil.cloneValue(cell));
209
210    admin.flush(tableName);
211    result = rs.next();
212    cell = result.getColumnLatestCell(family, qf1);
213    Assert.assertArrayEquals(value2, CellUtil.cloneValue(cell));
214  }
215
216  @Test
217  public void testReadFromCorruptMobFilesWithReadEmptyValueOnMobCellMiss() throws Exception {
218    final TableName tableName = TableName.valueOf(name.getMethodName());
219    setUp(0, tableName);
220    createRecordAndCorruptMobFile(tableName, row1, family, qf1, Bytes.toBytes("value1"));
221    Get get = new Get(row1);
222    get.setAttribute(MobConstants.EMPTY_VALUE_ON_MOBCELL_MISS, Bytes.toBytes(true));
223    Result result = table.get(get);
224    Cell cell = result.getColumnLatestCell(family, qf1);
225    Assert.assertEquals(0, cell.getValueLength());
226  }
227
228  @Test
229  public void testReadFromCorruptMobFiles() throws Exception {
230    final TableName tableName = TableName.valueOf(name.getMethodName());
231    setUp(0, tableName);
232    createRecordAndCorruptMobFile(tableName, row1, family, qf1, Bytes.toBytes("value1"));
233    Get get = new Get(row1);
234    IOException ioe = null;
235    try {
236      table.get(get);
237    } catch (IOException e) {
238      ioe = e;
239    }
240    Assert.assertNotNull(ioe);
241    Assert.assertEquals(CorruptHFileException.class.getName(), ioe.getClass().getName());
242  }
243
244  private void createRecordAndCorruptMobFile(TableName tn, byte[] row, byte[] family, byte[] qf,
245    byte[] value) throws IOException {
246    Put put1 = new Put(row);
247    put1.addColumn(family, qf, value);
248    table.put(put1);
249    admin.flush(tn);
250    Path mobFile = getFlushedMobFile(conf, fs, tn, Bytes.toString(family));
251    Assert.assertNotNull(mobFile);
252    // create new corrupt mob file.
253    Path corruptFile = new Path(mobFile.getParent(), "dummy");
254    TestHFile.truncateFile(fs, mobFile, corruptFile);
255    fs.delete(mobFile, true);
256    fs.rename(corruptFile, mobFile);
257  }
258
259  private Path getFlushedMobFile(Configuration conf, FileSystem fs, TableName table, String family)
260    throws IOException {
261    Path famDir = MobUtils.getMobFamilyPath(conf, table, family);
262    FileStatus[] hfFss = fs.listStatus(famDir);
263    for (FileStatus hfs : hfFss) {
264      if (!hfs.isDirectory()) {
265        return hfs.getPath();
266      }
267    }
268    return null;
269  }
270
271  private void testGetFromFiles(boolean reversed) throws Exception {
272    TableName tn = TableName.valueOf("testGetFromFiles" + reversed);
273    testGet(tn, reversed, true);
274  }
275
276  private void testGetFromMemStore(boolean reversed) throws Exception {
277    TableName tn = TableName.valueOf("testGetFromMemStore" + reversed);
278    testGet(tn, reversed, false);
279  }
280
281  private void testGet(TableName tableName, boolean reversed, boolean doFlush)
282      throws Exception {
283    setUp(defaultThreshold, tableName);
284    long ts1 = System.currentTimeMillis();
285    long ts2 = ts1 + 1;
286    long ts3 = ts1 + 2;
287    byte [] value = generateMobValue((int)defaultThreshold+1);
288
289    Put put1 = new Put(row1);
290    put1.addColumn(family, qf1, ts3, value);
291    put1.addColumn(family, qf2, ts2, value);
292    put1.addColumn(family, qf3, ts1, value);
293    table.put(put1);
294
295    if (doFlush) {
296      admin.flush(tableName);
297    }
298
299    Scan scan = new Scan();
300    setScan(scan, reversed, false);
301    MobTestUtil.assertCellsValue(table, scan, value, 3);
302  }
303
304  private void testGetReferences(boolean reversed) throws Exception {
305    TableName tn = TableName.valueOf("testGetReferences" + reversed);
306    setUp(defaultThreshold, tn);
307    long ts1 = System.currentTimeMillis();
308    long ts2 = ts1 + 1;
309    long ts3 = ts1 + 2;
310    byte [] value = generateMobValue((int)defaultThreshold+1);;
311
312    Put put1 = new Put(row1);
313    put1.addColumn(family, qf1, ts3, value);
314    put1.addColumn(family, qf2, ts2, value);
315    put1.addColumn(family, qf3, ts1, value);
316    table.put(put1);
317
318    admin.flush(tn);
319
320    Scan scan = new Scan();
321    setScan(scan, reversed, true);
322
323    ResultScanner results = table.getScanner(scan);
324    int count = 0;
325    for (Result res : results) {
326      List<Cell> cells = res.listCells();
327      for(Cell cell : cells) {
328        // Verify the value
329        assertIsMobReference(cell, row1, family, value, tn);
330        count++;
331      }
332    }
333    results.close();
334    Assert.assertEquals(3, count);
335  }
336
337  private void testMobThreshold(boolean reversed) throws Exception {
338    TableName tn = TableName.valueOf("testMobThreshold" + reversed);
339    setUp(defaultThreshold, tn);
340    byte [] valueLess = generateMobValue((int)defaultThreshold-1);
341    byte [] valueEqual = generateMobValue((int)defaultThreshold);
342    byte [] valueGreater = generateMobValue((int)defaultThreshold+1);
343    long ts1 = System.currentTimeMillis();
344    long ts2 = ts1 + 1;
345    long ts3 = ts1 + 2;
346
347    Put put1 = new Put(row1);
348    put1.addColumn(family, qf1, ts3, valueLess);
349    put1.addColumn(family, qf2, ts2, valueEqual);
350    put1.addColumn(family, qf3, ts1, valueGreater);
351    table.put(put1);
352
353    admin.flush(tn);
354
355    Scan scan = new Scan();
356    setScan(scan, reversed, true);
357
358    Cell cellLess= null;
359    Cell cellEqual = null;
360    Cell cellGreater = null;
361    ResultScanner results = table.getScanner(scan);
362    int count = 0;
363    for (Result res : results) {
364      List<Cell> cells = res.listCells();
365      for(Cell cell : cells) {
366        // Verify the value
367        String qf = Bytes.toString(CellUtil.cloneQualifier(cell));
368        if(qf.equals(Bytes.toString(qf1))) {
369          cellLess = cell;
370        }
371        if(qf.equals(Bytes.toString(qf2))) {
372          cellEqual = cell;
373        }
374        if(qf.equals(Bytes.toString(qf3))) {
375          cellGreater = cell;
376        }
377        count++;
378      }
379    }
380    Assert.assertEquals(3, count);
381    assertNotMobReference(cellLess, row1, family, valueLess);
382    assertNotMobReference(cellEqual, row1, family, valueEqual);
383    assertIsMobReference(cellGreater, row1, family, valueGreater, tn);
384    results.close();
385  }
386
387  private void testGetFromArchive(boolean reversed) throws Exception {
388    TableName tn = TableName.valueOf("testGetFromArchive" + reversed);
389    setUp(defaultThreshold, tn);
390    long ts1 = System.currentTimeMillis();
391    long ts2 = ts1 + 1;
392    long ts3 = ts1 + 2;
393    byte [] value = generateMobValue((int)defaultThreshold+1);;
394    // Put some data
395    Put put1 = new Put(row1);
396    put1.addColumn(family, qf1, ts3, value);
397    put1.addColumn(family, qf2, ts2, value);
398    put1.addColumn(family, qf3, ts1, value);
399    table.put(put1);
400
401    admin.flush(tn);
402
403    // Get the files in the mob path
404    Path mobFamilyPath;
405    mobFamilyPath = MobUtils.getMobFamilyPath(
406      TEST_UTIL.getConfiguration(), tn, hcd.getNameAsString());
407    FileSystem fs = FileSystem.get(TEST_UTIL.getConfiguration());
408    FileStatus[] files = fs.listStatus(mobFamilyPath);
409
410    // Get the archive path
411    Path rootDir = FSUtils.getRootDir(TEST_UTIL.getConfiguration());
412    Path tableDir = FSUtils.getTableDir(rootDir, tn);
413    RegionInfo regionInfo = MobUtils.getMobRegionInfo(tn);
414    Path storeArchiveDir = HFileArchiveUtil.getStoreArchivePath(TEST_UTIL.getConfiguration(),
415        regionInfo, tableDir, family);
416
417    // Move the files from mob path to archive path
418    fs.mkdirs(storeArchiveDir);
419    int fileCount = 0;
420    for(FileStatus file : files) {
421      fileCount++;
422      Path filePath = file.getPath();
423      Path src = new Path(mobFamilyPath, filePath.getName());
424      Path dst = new Path(storeArchiveDir, filePath.getName());
425      fs.rename(src, dst);
426    }
427
428    // Verify the moving success
429    FileStatus[] files1 = fs.listStatus(mobFamilyPath);
430    Assert.assertEquals(0, files1.length);
431    FileStatus[] files2 = fs.listStatus(storeArchiveDir);
432    Assert.assertEquals(fileCount, files2.length);
433
434    // Scan from archive
435    Scan scan = new Scan();
436    setScan(scan, reversed, false);
437    MobTestUtil.assertCellsValue(table, scan, value, 3);
438  }
439
440  /**
441   * Assert the value is not store in mob.
442   */
443  private static void assertNotMobReference(Cell cell, byte[] row, byte[] family,
444      byte[] value) throws IOException {
445    Assert.assertArrayEquals(row, CellUtil.cloneRow(cell));
446    Assert.assertArrayEquals(family, CellUtil.cloneFamily(cell));
447    Assert.assertArrayEquals(value, CellUtil.cloneValue(cell));
448  }
449
450  /**
451   * Assert the value is store in mob.
452   */
453  private static void assertIsMobReference(Cell cell, byte[] row, byte[] family,
454      byte[] value, TableName tn) throws IOException {
455    Assert.assertArrayEquals(row, CellUtil.cloneRow(cell));
456    Assert.assertArrayEquals(family, CellUtil.cloneFamily(cell));
457    Assert.assertFalse(Bytes.equals(value, CellUtil.cloneValue(cell)));
458    byte[] referenceValue = CellUtil.cloneValue(cell);
459    String fileName = MobUtils.getMobFileName(cell);
460    int valLen = Bytes.toInt(referenceValue, 0, Bytes.SIZEOF_INT);
461    Assert.assertEquals(value.length, valLen);
462    Path mobFamilyPath = MobUtils.getMobFamilyPath(
463      TEST_UTIL.getConfiguration(), tn, hcd.getNameAsString());
464    Path targetPath = new Path(mobFamilyPath, fileName);
465    FileSystem fs = FileSystem.get(TEST_UTIL.getConfiguration());
466    Assert.assertTrue(fs.exists(targetPath));
467  }
468}