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;
019
020import static org.junit.Assert.assertArrayEquals;
021import static org.junit.Assert.assertEquals;
022import static org.junit.Assert.assertFalse;
023import static org.junit.Assert.assertNull;
024import static org.junit.Assert.assertTrue;
025import static org.junit.Assert.fail;
026
027import java.io.IOException;
028import java.util.ArrayList;
029import java.util.List;
030import org.apache.hadoop.hbase.client.Delete;
031import org.apache.hadoop.hbase.client.Put;
032import org.apache.hadoop.hbase.client.RegionInfo;
033import org.apache.hadoop.hbase.client.Result;
034import org.apache.hadoop.hbase.client.ResultScanner;
035import org.apache.hadoop.hbase.client.Scan;
036import org.apache.hadoop.hbase.client.Table;
037import org.apache.hadoop.hbase.filter.ColumnPrefixFilter;
038import org.apache.hadoop.hbase.filter.ColumnRangeFilter;
039import org.apache.hadoop.hbase.filter.Filter;
040import org.apache.hadoop.hbase.filter.FirstKeyOnlyFilter;
041import org.apache.hadoop.hbase.filter.RandomRowFilter;
042import org.apache.hadoop.hbase.testclassification.LargeTests;
043import org.apache.hadoop.hbase.util.Bytes;
044import org.apache.hadoop.hbase.util.ClassSize;
045import org.apache.hadoop.hbase.util.Pair;
046import org.junit.AfterClass;
047import org.junit.BeforeClass;
048import org.junit.ClassRule;
049import org.junit.Rule;
050import org.junit.Test;
051import org.junit.experimental.categories.Category;
052import org.junit.rules.TestName;
053import org.slf4j.Logger;
054import org.slf4j.LoggerFactory;
055
056/**
057 * These tests are focused on testing how partial results appear to a client. Partial results are
058 * {@link Result}s that contain only a portion of a row's complete list of cells. Partial results
059 * are formed when the server breaches its maximum result size when trying to service a client's RPC
060 * request. It is the responsibility of the scanner on the client side to recognize when partial
061 * results have been returned and to take action to form the complete results.
062 * <p>
063 * Unless the flag {@link Scan#setAllowPartialResults(boolean)} has been set to true, the caller of
064 * {@link ResultScanner#next()} should never see partial results.
065 */
066@Category(LargeTests.class)
067public class TestPartialResultsFromClientSide {
068
069  @ClassRule
070  public static final HBaseClassTestRule CLASS_RULE =
071    HBaseClassTestRule.forClass(TestPartialResultsFromClientSide.class);
072
073  private static final Logger LOG = LoggerFactory.getLogger(TestPartialResultsFromClientSide.class);
074
075  private final static HBaseTestingUtil TEST_UTIL = new HBaseTestingUtil();
076  private final static int MINICLUSTER_SIZE = 5;
077  private static Table TABLE = null;
078
079  /**
080   * Table configuration
081   */
082  private static TableName TABLE_NAME = TableName.valueOf("testTable");
083
084  private static int NUM_ROWS = 5;
085  private static byte[] ROW = Bytes.toBytes("testRow");
086  private static byte[][] ROWS = HTestConst.makeNAscii(ROW, NUM_ROWS);
087
088  // Should keep this value below 10 to keep generation of expected kv's simple. If above 10 then
089  // table/row/cf1/... will be followed by table/row/cf10/... instead of table/row/cf2/... which
090  // breaks the simple generation of expected kv's
091  private static int NUM_FAMILIES = 10;
092  private static byte[] FAMILY = Bytes.toBytes("testFamily");
093  private static byte[][] FAMILIES = HTestConst.makeNAscii(FAMILY, NUM_FAMILIES);
094
095  private static int NUM_QUALIFIERS = 10;
096  private static byte[] QUALIFIER = Bytes.toBytes("testQualifier");
097  private static byte[][] QUALIFIERS = HTestConst.makeNAscii(QUALIFIER, NUM_QUALIFIERS);
098
099  private static int VALUE_SIZE = 1024;
100  private static byte[] VALUE = Bytes.createMaxByteArray(VALUE_SIZE);
101
102  private static int NUM_COLS = NUM_FAMILIES * NUM_QUALIFIERS;
103
104  // Approximation of how large the heap size of cells in our table. Should be accessed through
105  // getCellHeapSize().
106  private static long CELL_HEAP_SIZE = -1;
107
108  private static long timeout = 10000;
109
110  @Rule
111  public TestName name = new TestName();
112
113  @BeforeClass
114  public static void setUpBeforeClass() throws Exception {
115    TEST_UTIL.getConfiguration().setLong(HConstants.HBASE_CLIENT_SCANNER_TIMEOUT_PERIOD, timeout);
116    TEST_UTIL.startMiniCluster(MINICLUSTER_SIZE);
117    TEST_UTIL.getAdmin().balancerSwitch(false, true);
118    TABLE = createTestTable(TABLE_NAME, ROWS, FAMILIES, QUALIFIERS, VALUE);
119  }
120
121  static Table createTestTable(TableName name, byte[][] rows, byte[][] families,
122    byte[][] qualifiers, byte[] cellValue) throws IOException {
123    Table ht = TEST_UTIL.createTable(name, families);
124    List<Put> puts = createPuts(rows, families, qualifiers, cellValue);
125    ht.put(puts);
126
127    return ht;
128  }
129
130  @AfterClass
131  public static void tearDownAfterClass() throws Exception {
132    TEST_UTIL.shutdownMiniCluster();
133  }
134
135  /**
136   * Ensure that the expected key values appear in a result returned from a scanner that is
137   * combining partial results into complete results
138   */
139  @Test
140  public void testExpectedValuesOfPartialResults() throws Exception {
141    testExpectedValuesOfPartialResults(false);
142    testExpectedValuesOfPartialResults(true);
143  }
144
145  public void testExpectedValuesOfPartialResults(boolean reversed) throws Exception {
146    Scan partialScan = new Scan();
147    partialScan.readAllVersions();
148    // Max result size of 1 ensures that each RPC request will return a single cell. The scanner
149    // will need to reconstruct the results into a complete result before returning to the caller
150    partialScan.setMaxResultSize(1);
151    partialScan.setReversed(reversed);
152    ResultScanner partialScanner = TABLE.getScanner(partialScan);
153
154    final int startRow = reversed ? ROWS.length - 1 : 0;
155    final int endRow = reversed ? -1 : ROWS.length;
156    final int loopDelta = reversed ? -1 : 1;
157    String message;
158
159    for (int row = startRow; row != endRow; row = row + loopDelta) {
160      message = "Ensuring the expected keyValues are present for row " + row;
161      List<Cell> expectedKeyValues = createKeyValuesForRow(ROWS[row], FAMILIES, QUALIFIERS, VALUE);
162      Result result = partialScanner.next();
163      assertFalse(result.mayHaveMoreCellsInRow());
164      verifyResult(result, expectedKeyValues, message);
165    }
166
167    partialScanner.close();
168  }
169
170  /**
171   * Ensure that we only see Results marked as partial when the allowPartial flag is set
172   */
173  @Test
174  public void testAllowPartialResults() throws Exception {
175    Scan scan = new Scan();
176    scan.setAllowPartialResults(true);
177    scan.setMaxResultSize(1);
178    ResultScanner scanner = TABLE.getScanner(scan);
179    Result result = scanner.next();
180
181    assertTrue(result != null);
182    assertTrue(result.mayHaveMoreCellsInRow());
183    assertTrue(result.rawCells() != null);
184    assertTrue(result.rawCells().length == 1);
185
186    scanner.close();
187
188    scan.setAllowPartialResults(false);
189    scanner = TABLE.getScanner(scan);
190    result = scanner.next();
191
192    assertTrue(result != null);
193    assertTrue(!result.mayHaveMoreCellsInRow());
194    assertTrue(result.rawCells() != null);
195    assertTrue(result.rawCells().length == NUM_COLS);
196
197    scanner.close();
198  }
199
200  /**
201   * Ensure that the results returned from a scanner that retrieves all results in a single RPC call
202   * matches the results that are returned from a scanner that must incrementally combine partial
203   * results into complete results. A variety of scan configurations can be tested
204   */
205  @Test
206  public void testEquivalenceOfScanResults() throws Exception {
207    Scan oneShotScan = new Scan();
208    oneShotScan.setMaxResultSize(Long.MAX_VALUE);
209
210    Scan partialScan = new Scan(oneShotScan);
211    partialScan.setMaxResultSize(1);
212
213    testEquivalenceOfScanResults(TABLE, oneShotScan, partialScan);
214  }
215
216  public void testEquivalenceOfScanResults(Table table, Scan scan1, Scan scan2) throws Exception {
217    ResultScanner scanner1 = table.getScanner(scan1);
218    ResultScanner scanner2 = table.getScanner(scan2);
219
220    Result r1 = null;
221    Result r2 = null;
222    int count = 0;
223
224    while ((r1 = scanner1.next()) != null) {
225      r2 = scanner2.next();
226
227      assertTrue(r2 != null);
228      compareResults(r1, r2, "Comparing result #" + count);
229      count++;
230    }
231
232    r2 = scanner2.next();
233    assertTrue("r2: " + r2 + " Should be null", r2 == null);
234
235    scanner1.close();
236    scanner2.close();
237  }
238
239  /**
240   * Order of cells in partial results matches the ordering of cells from complete results
241   */
242  @Test
243  public void testOrderingOfCellsInPartialResults() throws Exception {
244    Scan scan = new Scan();
245
246    for (int col = 1; col <= NUM_COLS; col++) {
247      scan.setMaxResultSize(getResultSizeForNumberOfCells(col));
248      testOrderingOfCellsInPartialResults(scan);
249
250      // Test again with a reversed scanner
251      scan.setReversed(true);
252      testOrderingOfCellsInPartialResults(scan);
253    }
254  }
255
256  public void testOrderingOfCellsInPartialResults(final Scan basePartialScan) throws Exception {
257    // Scan that retrieves results in pieces (partials). By setting allowPartialResults to be true
258    // the results will NOT be reconstructed and instead the caller will see the partial results
259    // returned by the server
260    Scan partialScan = new Scan(basePartialScan);
261    partialScan.setAllowPartialResults(true);
262    ResultScanner partialScanner = TABLE.getScanner(partialScan);
263
264    // Scan that retrieves all table results in single RPC request
265    Scan oneShotScan = new Scan(basePartialScan);
266    oneShotScan.setMaxResultSize(Long.MAX_VALUE);
267    oneShotScan.setCaching(ROWS.length);
268    ResultScanner oneShotScanner = TABLE.getScanner(oneShotScan);
269
270    Result oneShotResult = oneShotScanner.next();
271    Result partialResult = null;
272    int iterationCount = 0;
273
274    while (oneShotResult != null && oneShotResult.rawCells() != null) {
275      List<Cell> aggregatePartialCells = new ArrayList<>();
276      do {
277        partialResult = partialScanner.next();
278        assertTrue("Partial Result is null. iteration: " + iterationCount, partialResult != null);
279        assertTrue("Partial cells are null. iteration: " + iterationCount,
280          partialResult.rawCells() != null);
281
282        for (Cell c : partialResult.rawCells()) {
283          aggregatePartialCells.add(c);
284        }
285      } while (partialResult.mayHaveMoreCellsInRow());
286
287      assertTrue("Number of cells differs. iteration: " + iterationCount,
288        oneShotResult.rawCells().length == aggregatePartialCells.size());
289      final Cell[] oneShotCells = oneShotResult.rawCells();
290      for (int cell = 0; cell < oneShotCells.length; cell++) {
291        Cell oneShotCell = oneShotCells[cell];
292        Cell partialCell = aggregatePartialCells.get(cell);
293
294        assertTrue("One shot cell was null", oneShotCell != null);
295        assertTrue("Partial cell was null", partialCell != null);
296        assertTrue("Cell differs. oneShotCell:" + oneShotCell + " partialCell:" + partialCell,
297          oneShotCell.equals(partialCell));
298      }
299
300      oneShotResult = oneShotScanner.next();
301      iterationCount++;
302    }
303
304    assertTrue(partialScanner.next() == null);
305
306    partialScanner.close();
307    oneShotScanner.close();
308  }
309
310  /**
311   * Setting the max result size allows us to control how many cells we expect to see on each call
312   * to next on the scanner. Test a variety of different sizes for correctness
313   */
314  @Test
315  public void testExpectedNumberOfCellsPerPartialResult() throws Exception {
316    Scan scan = new Scan();
317    testExpectedNumberOfCellsPerPartialResult(scan);
318
319    scan.setReversed(true);
320    testExpectedNumberOfCellsPerPartialResult(scan);
321  }
322
323  public void testExpectedNumberOfCellsPerPartialResult(Scan baseScan) throws Exception {
324    for (int expectedCells = 1; expectedCells <= NUM_COLS; expectedCells++) {
325      testExpectedNumberOfCellsPerPartialResult(baseScan, expectedCells);
326    }
327  }
328
329  public void testExpectedNumberOfCellsPerPartialResult(Scan baseScan, int expectedNumberOfCells)
330    throws Exception {
331
332    if (LOG.isInfoEnabled()) LOG.info("groupSize:" + expectedNumberOfCells);
333
334    // Use the cellHeapSize to set maxResultSize such that we know how many cells to expect back
335    // from the call. The returned results should NOT exceed expectedNumberOfCells but may be less
336    // than it in cases where expectedNumberOfCells is not an exact multiple of the number of
337    // columns in the table.
338    Scan scan = new Scan(baseScan);
339    scan.setAllowPartialResults(true);
340    scan.setMaxResultSize(getResultSizeForNumberOfCells(expectedNumberOfCells));
341
342    ResultScanner scanner = TABLE.getScanner(scan);
343    Result result = null;
344    byte[] prevRow = null;
345    while ((result = scanner.next()) != null) {
346      assertTrue(result.rawCells() != null);
347
348      // Cases when cell count won't equal expectedNumberOfCells:
349      // 1. Returned result is the final result needed to form the complete result for that row
350      // 2. It is the first result we have seen for that row and thus may have been fetched as
351      // the last group of cells that fit inside the maxResultSize
352      assertTrue("Result's cell count differed from expected number. result: " + result,
353        result.rawCells().length == expectedNumberOfCells || !result.mayHaveMoreCellsInRow()
354          || !Bytes.equals(prevRow, result.getRow()));
355      prevRow = result.getRow();
356    }
357
358    scanner.close();
359  }
360
361  /**
362   * @return The approximate heap size of a cell in the test table. All cells should have
363   *         approximately the same heap size, so the value is cached to avoid repeating the
364   *         calculation
365   */
366  private long getCellHeapSize() throws Exception {
367    if (CELL_HEAP_SIZE == -1) {
368      // Do a partial scan that will return a single result with a single cell
369      Scan scan = new Scan();
370      scan.setMaxResultSize(2);
371      scan.setAllowPartialResults(true);
372      ResultScanner scanner = TABLE.getScanner(scan);
373
374      Result result = scanner.next();
375
376      assertTrue(result != null);
377      assertTrue(result.rawCells() != null);
378      assertTrue(result.rawCells().length == 1);
379
380      // Estimate the cell heap size. One difference is that on server side, the KV Heap size is
381      // estimated differently in case the cell is backed up by MSLAB byte[] (no overhead for
382      // backing array). Thus below calculation is a bit brittle.
383      CELL_HEAP_SIZE = result.rawCells()[0].heapSize() - (ClassSize.ARRAY + 3);
384      if (LOG.isInfoEnabled()) LOG.info("Cell heap size: " + CELL_HEAP_SIZE);
385      scanner.close();
386    }
387
388    return CELL_HEAP_SIZE;
389  }
390
391  /**
392   * @return the result size that should be used in {@link Scan#setMaxResultSize(long)} if you want
393   *         the server to return exactly numberOfCells cells
394   */
395  private long getResultSizeForNumberOfCells(int numberOfCells) throws Exception {
396    return getCellHeapSize() * numberOfCells;
397  }
398
399  /**
400   * Test various combinations of batching and partial results for correctness
401   */
402  @Test
403  public void testPartialResultsAndBatch() throws Exception {
404    for (int batch = 1; batch <= NUM_COLS / 4; batch++) {
405      for (int cellsPerPartial = 1; cellsPerPartial <= NUM_COLS / 4; cellsPerPartial++) {
406        testPartialResultsAndBatch(batch, cellsPerPartial);
407      }
408    }
409  }
410
411  public void testPartialResultsAndBatch(final int batch, final int cellsPerPartialResult)
412    throws Exception {
413    if (LOG.isInfoEnabled()) {
414      LOG.info("batch: " + batch + " cellsPerPartialResult: " + cellsPerPartialResult);
415    }
416
417    Scan scan = new Scan();
418    scan.setMaxResultSize(getResultSizeForNumberOfCells(cellsPerPartialResult));
419    scan.setBatch(batch);
420    ResultScanner scanner = TABLE.getScanner(scan);
421    Result result = scanner.next();
422    int repCount = 0;
423
424    while ((result = scanner.next()) != null) {
425      assertTrue(result.rawCells() != null);
426
427      if (result.mayHaveMoreCellsInRow()) {
428        final String error = "Cells:" + result.rawCells().length + " Batch size:" + batch
429          + " cellsPerPartialResult:" + cellsPerPartialResult + " rep:" + repCount;
430        assertTrue(error, result.rawCells().length == batch);
431      } else {
432        assertTrue(result.rawCells().length <= batch);
433      }
434      repCount++;
435    }
436
437    scanner.close();
438  }
439
440  /**
441   * Test the method {@link Result#createCompleteResult(Iterable)}
442   */
443  @Test
444  public void testPartialResultsReassembly() throws Exception {
445    Scan scan = new Scan();
446    testPartialResultsReassembly(scan);
447    scan.setReversed(true);
448    testPartialResultsReassembly(scan);
449  }
450
451  public void testPartialResultsReassembly(Scan scanBase) throws Exception {
452    Scan partialScan = new Scan(scanBase);
453    partialScan.setMaxResultSize(1);
454    partialScan.setAllowPartialResults(true);
455    ResultScanner partialScanner = TABLE.getScanner(partialScan);
456
457    Scan oneShotScan = new Scan(scanBase);
458    oneShotScan.setMaxResultSize(Long.MAX_VALUE);
459    ResultScanner oneShotScanner = TABLE.getScanner(oneShotScan);
460
461    ArrayList<Result> partials = new ArrayList<>();
462    for (int i = 0; i < NUM_ROWS; i++) {
463      Result partialResult = null;
464      Result completeResult = null;
465      Result oneShotResult = null;
466      partials.clear();
467
468      do {
469        partialResult = partialScanner.next();
470        partials.add(partialResult);
471      } while (partialResult != null && partialResult.mayHaveMoreCellsInRow());
472
473      completeResult = Result.createCompleteResult(partials);
474      oneShotResult = oneShotScanner.next();
475
476      compareResults(completeResult, oneShotResult, null);
477    }
478
479    assertTrue(oneShotScanner.next() == null);
480    assertTrue(partialScanner.next() == null);
481
482    oneShotScanner.close();
483    partialScanner.close();
484  }
485
486  /**
487   * When reconstructing the complete result from its partials we ensure that the row of each
488   * partial result is the same. If one of the rows differs, an exception is thrown.
489   */
490  @Test
491  public void testExceptionThrownOnMismatchedPartialResults() throws IOException {
492    assertTrue(NUM_ROWS >= 2);
493
494    ArrayList<Result> partials = new ArrayList<>();
495    Scan scan = new Scan();
496    scan.setMaxResultSize(Long.MAX_VALUE);
497    ResultScanner scanner = TABLE.getScanner(scan);
498    Result r1 = scanner.next();
499    partials.add(r1);
500    Result r2 = scanner.next();
501    partials.add(r2);
502
503    assertFalse(Bytes.equals(r1.getRow(), r2.getRow()));
504
505    try {
506      Result.createCompleteResult(partials);
507      fail("r1 and r2 are from different rows. It should not be possible to combine them into"
508        + " a single result");
509    } catch (IOException e) {
510    }
511
512    scanner.close();
513  }
514
515  /**
516   * When a scan has a filter where {@link org.apache.hadoop.hbase.filter.Filter#hasFilterRow()} is
517   * true, the scanner should not return partial results. The scanner cannot return partial results
518   * because the entire row needs to be read for the include/exclude decision to be made
519   */
520  @Test
521  public void testNoPartialResultsWhenRowFilterPresent() throws Exception {
522    Scan scan = new Scan();
523    scan.setMaxResultSize(1);
524    scan.setAllowPartialResults(true);
525    // If a filter hasFilter() is true then partial results should not be returned else filter
526    // application server side would break.
527    scan.setFilter(new RandomRowFilter(1.0f));
528    ResultScanner scanner = TABLE.getScanner(scan);
529
530    Result r = null;
531    while ((r = scanner.next()) != null) {
532      assertFalse(r.mayHaveMoreCellsInRow());
533    }
534
535    scanner.close();
536  }
537
538  /**
539   * Examine the interaction between the maxResultSize and caching. If the caching limit is reached
540   * before the maxResultSize limit, we should not see partial results. On the other hand, if the
541   * maxResultSize limit is reached before the caching limit, it is likely that partial results will
542   * be seen.
543   */
544  @Test
545  public void testPartialResultsAndCaching() throws Exception {
546    for (int caching = 1; caching <= NUM_ROWS; caching++) {
547      for (int maxResultRows = 0; maxResultRows <= NUM_ROWS; maxResultRows++) {
548        testPartialResultsAndCaching(maxResultRows, caching);
549      }
550    }
551  }
552
553  /**
554   * @param resultSizeRowLimit The row limit that will be enforced through maxResultSize
555   * @param cachingRowLimit    The row limit that will be enforced through caching
556   */
557  public void testPartialResultsAndCaching(int resultSizeRowLimit, int cachingRowLimit)
558    throws Exception {
559    Scan scan = new Scan();
560    scan.setAllowPartialResults(true);
561
562    // The number of cells specified in the call to getResultSizeForNumberOfCells is offset to
563    // ensure that the result size we specify is not an exact multiple of the number of cells
564    // in a row. This ensures that partial results will be returned when the result size limit
565    // is reached before the caching limit.
566    int cellOffset = NUM_COLS / 3;
567    long maxResultSize = getResultSizeForNumberOfCells(resultSizeRowLimit * NUM_COLS + cellOffset);
568    scan.setMaxResultSize(maxResultSize);
569    scan.setCaching(cachingRowLimit);
570
571    try (ResultScanner scanner = TABLE.getScanner(scan)) {
572      Result r = null;
573      // Approximate the number of rows we expect will fit into the specified max rsult size. If
574      // this approximation is less than caching, then we expect that the max result size limit will
575      // be hit before the caching limit and thus partial results may be seen
576      boolean expectToSeePartialResults = resultSizeRowLimit < cachingRowLimit;
577      while ((r = scanner.next()) != null) {
578        assertTrue(!r.mayHaveMoreCellsInRow() || expectToSeePartialResults);
579      }
580    }
581  }
582
583  /**
584   * Make puts to put the input value into each combination of row, family, and qualifier
585   * @param rows       the rows to use
586   * @param families   the families to use
587   * @param qualifiers the qualifiers to use
588   * @param value      the values to use
589   * @return the dot product of the given rows, families, qualifiers, and values
590   * @throws IOException if there is a problem creating one of the Put objects
591   */
592  static ArrayList<Put> createPuts(byte[][] rows, byte[][] families, byte[][] qualifiers,
593    byte[] value) throws IOException {
594    Put put;
595    ArrayList<Put> puts = new ArrayList<>();
596
597    for (int row = 0; row < rows.length; row++) {
598      put = new Put(rows[row]);
599      for (int fam = 0; fam < families.length; fam++) {
600        for (int qual = 0; qual < qualifiers.length; qual++) {
601          KeyValue kv = new KeyValue(rows[row], families[fam], qualifiers[qual], qual, value);
602          put.add(kv);
603        }
604      }
605      puts.add(put);
606    }
607
608    return puts;
609  }
610
611  /**
612   * Make key values to represent each possible combination of family and qualifier in the specified
613   * row.
614   * @param row        the row to use
615   * @param families   the families to use
616   * @param qualifiers the qualifiers to use
617   * @param value      the values to use
618   * @return the dot product of the given families, qualifiers, and values for a given row
619   */
620  static ArrayList<Cell> createKeyValuesForRow(byte[] row, byte[][] families, byte[][] qualifiers,
621    byte[] value) {
622    ArrayList<Cell> outList = new ArrayList<>();
623    for (int fam = 0; fam < families.length; fam++) {
624      for (int qual = 0; qual < qualifiers.length; qual++) {
625        outList.add(new KeyValue(row, families[fam], qualifiers[qual], qual, value));
626      }
627    }
628    return outList;
629  }
630
631  /**
632   * Verifies that result contains all the key values within expKvList. Fails the test otherwise
633   */
634  static void verifyResult(Result result, List<Cell> expKvList, String msg) {
635    if (LOG.isInfoEnabled()) {
636      LOG.info(msg);
637      LOG.info("Expected count: " + expKvList.size());
638      LOG.info("Actual count: " + result.size());
639    }
640
641    if (expKvList.isEmpty()) return;
642
643    int i = 0;
644    for (Cell kv : result.rawCells()) {
645      if (i >= expKvList.size()) {
646        break; // we will check the size later
647      }
648
649      Cell kvExp = expKvList.get(i++);
650      assertTrue("Not equal. get kv: " + kv.toString() + " exp kv: " + kvExp.toString(),
651        kvExp.equals(kv));
652    }
653
654    assertEquals(expKvList.size(), result.size());
655  }
656
657  /**
658   * Compares two results and fails the test if the results are different
659   */
660  static void compareResults(Result r1, Result r2, final String message) {
661    if (LOG.isInfoEnabled()) {
662      if (message != null) LOG.info(message);
663      LOG.info("r1: " + r1);
664      LOG.info("r2: " + r2);
665    }
666
667    final String failureMessage = "Results r1:" + r1 + " \nr2:" + r2 + " are not equivalent";
668    if (r1 == null && r2 == null) fail(failureMessage);
669    else if (r1 == null || r2 == null) fail(failureMessage);
670
671    try {
672      Result.compareResults(r1, r2);
673    } catch (Exception e) {
674      fail(failureMessage);
675    }
676  }
677
678  @Test
679  public void testReadPointAndPartialResults() throws Exception {
680    final TableName tableName = TableName.valueOf(name.getMethodName());
681    int numRows = 5;
682    int numFamilies = 5;
683    int numQualifiers = 5;
684    byte[][] rows = HTestConst.makeNAscii(Bytes.toBytes("testRow"), numRows);
685    byte[][] families = HTestConst.makeNAscii(Bytes.toBytes("testFamily"), numFamilies);
686    byte[][] qualifiers = HTestConst.makeNAscii(Bytes.toBytes("testQualifier"), numQualifiers);
687    byte[] value = Bytes.createMaxByteArray(100);
688
689    Table tmpTable = createTestTable(tableName, rows, families, qualifiers, value);
690    // Open scanner before deletes
691    ResultScanner scanner =
692      tmpTable.getScanner(new Scan().setMaxResultSize(1).setAllowPartialResults(true));
693    // now the openScanner will also fetch data and will be executed lazily, i.e, only openScanner
694    // when you call next, so here we need to make a next call to open scanner. The maxResultSize
695    // limit can make sure that we will not fetch all the data at once, so the test sill works.
696    int scannerCount = scanner.next().rawCells().length;
697    Delete delete1 = new Delete(rows[0]);
698    delete1.addColumn(families[0], qualifiers[0], 0);
699    tmpTable.delete(delete1);
700
701    Delete delete2 = new Delete(rows[1]);
702    delete2.addColumn(families[1], qualifiers[1], 1);
703    tmpTable.delete(delete2);
704
705    // Should see all cells because scanner was opened prior to deletes
706    scannerCount += countCellsFromScanner(scanner);
707    int expectedCount = numRows * numFamilies * numQualifiers;
708    assertTrue("scannerCount: " + scannerCount + " expectedCount: " + expectedCount,
709      scannerCount == expectedCount);
710
711    // Minus 2 for the two cells that were deleted
712    scanner = tmpTable.getScanner(new Scan().setMaxResultSize(1).setAllowPartialResults(true));
713    scannerCount = countCellsFromScanner(scanner);
714    expectedCount = numRows * numFamilies * numQualifiers - 2;
715    assertTrue("scannerCount: " + scannerCount + " expectedCount: " + expectedCount,
716      scannerCount == expectedCount);
717
718    scanner = tmpTable.getScanner(new Scan().setMaxResultSize(1).setAllowPartialResults(true));
719    scannerCount = scanner.next().rawCells().length;
720    // Put in 2 new rows. The timestamps differ from the deleted rows
721    Put put1 = new Put(rows[0]);
722    put1.add(new KeyValue(rows[0], families[0], qualifiers[0], 1, value));
723    tmpTable.put(put1);
724
725    Put put2 = new Put(rows[1]);
726    put2.add(new KeyValue(rows[1], families[1], qualifiers[1], 2, value));
727    tmpTable.put(put2);
728
729    // Scanner opened prior to puts. Cell count shouldn't have changed
730    scannerCount += countCellsFromScanner(scanner);
731    expectedCount = numRows * numFamilies * numQualifiers - 2;
732    assertTrue("scannerCount: " + scannerCount + " expectedCount: " + expectedCount,
733      scannerCount == expectedCount);
734
735    // Now the scanner should see the cells that were added by puts
736    scanner = tmpTable.getScanner(new Scan().setMaxResultSize(1).setAllowPartialResults(true));
737    scannerCount = countCellsFromScanner(scanner);
738    expectedCount = numRows * numFamilies * numQualifiers;
739    assertTrue("scannerCount: " + scannerCount + " expectedCount: " + expectedCount,
740      scannerCount == expectedCount);
741
742    TEST_UTIL.deleteTable(tableName);
743  }
744
745  /**
746   * Exhausts the scanner by calling next repetitively. Once completely exhausted, close scanner and
747   * return total cell count
748   * @param scanner the scanner to exhaust
749   * @return the number of cells counted
750   * @throws Exception if there is a problem retrieving cells from the scanner
751   */
752  private int countCellsFromScanner(ResultScanner scanner) throws Exception {
753    Result result = null;
754    int numCells = 0;
755    while ((result = scanner.next()) != null) {
756      numCells += result.rawCells().length;
757    }
758
759    scanner.close();
760    return numCells;
761  }
762
763  /**
764   * Test partial Result re-assembly in the presence of different filters. The Results from the
765   * partial scanner should match the Results returned from a scanner that receives all of the
766   * results in one RPC to the server. The partial scanner is tested with a variety of different
767   * result sizes (all of which are less than the size necessary to fetch an entire row)
768   */
769  @Test
770  public void testPartialResultsWithColumnFilter() throws Exception {
771    testPartialResultsWithColumnFilter(new FirstKeyOnlyFilter());
772    testPartialResultsWithColumnFilter(new ColumnPrefixFilter(Bytes.toBytes("testQualifier5")));
773    testPartialResultsWithColumnFilter(new ColumnRangeFilter(Bytes.toBytes("testQualifer1"), true,
774      Bytes.toBytes("testQualifier7"), true));
775  }
776
777  public void testPartialResultsWithColumnFilter(Filter filter) throws Exception {
778    assertTrue(!filter.hasFilterRow());
779
780    Scan partialScan = new Scan();
781    partialScan.setFilter(filter);
782
783    Scan oneshotScan = new Scan();
784    oneshotScan.setFilter(filter);
785    oneshotScan.setMaxResultSize(Long.MAX_VALUE);
786
787    for (int i = 1; i <= NUM_COLS; i++) {
788      partialScan.setMaxResultSize(getResultSizeForNumberOfCells(i));
789      testEquivalenceOfScanResults(TABLE, partialScan, oneshotScan);
790    }
791  }
792
793  private void moveRegion(Table table, int index) throws IOException {
794    List<Pair<RegionInfo, ServerName>> regions =
795      MetaTableAccessor.getTableRegionsAndLocations(TEST_UTIL.getConnection(), table.getName());
796    assertEquals(1, regions.size());
797    RegionInfo regionInfo = regions.get(0).getFirst();
798    ServerName name = TEST_UTIL.getHBaseCluster().getRegionServer(index).getServerName();
799    TEST_UTIL.getAdmin().move(regionInfo.getEncodedNameAsBytes(), name);
800  }
801
802  private void assertCell(Cell cell, byte[] row, byte[] cf, byte[] cq) {
803    assertArrayEquals(row,
804      Bytes.copy(cell.getRowArray(), cell.getRowOffset(), cell.getRowLength()));
805    assertArrayEquals(cf,
806      Bytes.copy(cell.getFamilyArray(), cell.getFamilyOffset(), cell.getFamilyLength()));
807    assertArrayEquals(cq,
808      Bytes.copy(cell.getQualifierArray(), cell.getQualifierOffset(), cell.getQualifierLength()));
809  }
810
811  @Test
812  public void testPartialResultWhenRegionMove() throws IOException {
813    Table table =
814      createTestTable(TableName.valueOf(name.getMethodName()), ROWS, FAMILIES, QUALIFIERS, VALUE);
815
816    moveRegion(table, 1);
817
818    Scan scan = new Scan();
819    scan.setMaxResultSize(1);
820    scan.setAllowPartialResults(true);
821    ResultScanner scanner = table.getScanner(scan);
822    for (int i = 0; i < NUM_FAMILIES * NUM_QUALIFIERS - 1; i++) {
823      scanner.next();
824    }
825    Result result1 = scanner.next();
826    assertEquals(1, result1.rawCells().length);
827    Cell c1 = result1.rawCells()[0];
828    assertCell(c1, ROWS[0], FAMILIES[NUM_FAMILIES - 1], QUALIFIERS[NUM_QUALIFIERS - 1]);
829    assertFalse(result1.mayHaveMoreCellsInRow());
830
831    moveRegion(table, 2);
832
833    Result result2 = scanner.next();
834    assertEquals(1, result2.rawCells().length);
835    Cell c2 = result2.rawCells()[0];
836    assertCell(c2, ROWS[1], FAMILIES[0], QUALIFIERS[0]);
837    assertTrue(result2.mayHaveMoreCellsInRow());
838
839    moveRegion(table, 3);
840
841    Result result3 = scanner.next();
842    assertEquals(1, result3.rawCells().length);
843    Cell c3 = result3.rawCells()[0];
844    assertCell(c3, ROWS[1], FAMILIES[0], QUALIFIERS[1]);
845    assertTrue(result3.mayHaveMoreCellsInRow());
846
847  }
848
849  @Test
850  public void testReversedPartialResultWhenRegionMove() throws IOException {
851    Table table =
852      createTestTable(TableName.valueOf(name.getMethodName()), ROWS, FAMILIES, QUALIFIERS, VALUE);
853
854    moveRegion(table, 1);
855
856    Scan scan = new Scan();
857    scan.setMaxResultSize(1);
858    scan.setAllowPartialResults(true);
859    scan.setReversed(true);
860    ResultScanner scanner = table.getScanner(scan);
861    for (int i = 0; i < NUM_FAMILIES * NUM_QUALIFIERS - 1; i++) {
862      scanner.next();
863    }
864    Result result1 = scanner.next();
865    assertEquals(1, result1.rawCells().length);
866    Cell c1 = result1.rawCells()[0];
867    assertCell(c1, ROWS[NUM_ROWS - 1], FAMILIES[NUM_FAMILIES - 1], QUALIFIERS[NUM_QUALIFIERS - 1]);
868    assertFalse(result1.mayHaveMoreCellsInRow());
869
870    moveRegion(table, 2);
871
872    Result result2 = scanner.next();
873    assertEquals(1, result2.rawCells().length);
874    Cell c2 = result2.rawCells()[0];
875    assertCell(c2, ROWS[NUM_ROWS - 2], FAMILIES[0], QUALIFIERS[0]);
876    assertTrue(result2.mayHaveMoreCellsInRow());
877
878    moveRegion(table, 3);
879
880    Result result3 = scanner.next();
881    assertEquals(1, result3.rawCells().length);
882    Cell c3 = result3.rawCells()[0];
883    assertCell(c3, ROWS[NUM_ROWS - 2], FAMILIES[0], QUALIFIERS[1]);
884    assertTrue(result3.mayHaveMoreCellsInRow());
885
886  }
887
888  @Test
889  public void testCompleteResultWhenRegionMove() throws IOException {
890    Table table =
891      createTestTable(TableName.valueOf(name.getMethodName()), ROWS, FAMILIES, QUALIFIERS, VALUE);
892
893    moveRegion(table, 1);
894
895    Scan scan = new Scan();
896    scan.setMaxResultSize(1);
897    scan.setCaching(1);
898    ResultScanner scanner = table.getScanner(scan);
899
900    Result result1 = scanner.next();
901    assertEquals(NUM_FAMILIES * NUM_QUALIFIERS, result1.rawCells().length);
902    Cell c1 = result1.rawCells()[0];
903    assertCell(c1, ROWS[0], FAMILIES[0], QUALIFIERS[0]);
904    assertFalse(result1.mayHaveMoreCellsInRow());
905
906    moveRegion(table, 2);
907
908    Result result2 = scanner.next();
909    assertEquals(NUM_FAMILIES * NUM_QUALIFIERS, result2.rawCells().length);
910    Cell c2 = result2.rawCells()[0];
911    assertCell(c2, ROWS[1], FAMILIES[0], QUALIFIERS[0]);
912    assertFalse(result2.mayHaveMoreCellsInRow());
913
914    moveRegion(table, 3);
915
916    Result result3 = scanner.next();
917    assertEquals(NUM_FAMILIES * NUM_QUALIFIERS, result3.rawCells().length);
918    Cell c3 = result3.rawCells()[0];
919    assertCell(c3, ROWS[2], FAMILIES[0], QUALIFIERS[0]);
920    assertFalse(result3.mayHaveMoreCellsInRow());
921
922  }
923
924  @Test
925  public void testReversedCompleteResultWhenRegionMove() throws IOException {
926    Table table =
927      createTestTable(TableName.valueOf(name.getMethodName()), ROWS, FAMILIES, QUALIFIERS, VALUE);
928
929    moveRegion(table, 1);
930
931    Scan scan = new Scan();
932    scan.setMaxResultSize(1);
933    scan.setCaching(1);
934    scan.setReversed(true);
935    ResultScanner scanner = table.getScanner(scan);
936
937    Result result1 = scanner.next();
938    assertEquals(NUM_FAMILIES * NUM_QUALIFIERS, result1.rawCells().length);
939    Cell c1 = result1.rawCells()[0];
940    assertCell(c1, ROWS[NUM_ROWS - 1], FAMILIES[0], QUALIFIERS[0]);
941    assertFalse(result1.mayHaveMoreCellsInRow());
942
943    moveRegion(table, 2);
944
945    Result result2 = scanner.next();
946    assertEquals(NUM_FAMILIES * NUM_QUALIFIERS, result2.rawCells().length);
947    Cell c2 = result2.rawCells()[0];
948    assertCell(c2, ROWS[NUM_ROWS - 2], FAMILIES[0], QUALIFIERS[0]);
949    assertFalse(result2.mayHaveMoreCellsInRow());
950
951    moveRegion(table, 3);
952
953    Result result3 = scanner.next();
954    assertEquals(NUM_FAMILIES * NUM_QUALIFIERS, result3.rawCells().length);
955    Cell c3 = result3.rawCells()[0];
956    assertCell(c3, ROWS[NUM_ROWS - 3], FAMILIES[0], QUALIFIERS[0]);
957    assertFalse(result3.mayHaveMoreCellsInRow());
958
959  }
960
961  @Test
962  public void testBatchingResultWhenRegionMove() throws IOException {
963    // If user setBatch(5) and rpc returns 3+5+5+5+3 cells,
964    // we should return 5+5+5+5+1 to user.
965    // setBatch doesn't mean setAllowPartialResult(true)
966    Table table =
967      createTestTable(TableName.valueOf(name.getMethodName()), ROWS, FAMILIES, QUALIFIERS, VALUE);
968
969    Put put = new Put(ROWS[1]);
970    put.addColumn(FAMILIES[0], QUALIFIERS[1], new byte[VALUE_SIZE * 10]);
971    table.put(put);
972    Delete delete = new Delete(ROWS[1]);
973    delete.addColumn(FAMILIES[NUM_FAMILIES - 1], QUALIFIERS[NUM_QUALIFIERS - 1]);
974    table.delete(delete);
975
976    moveRegion(table, 1);
977
978    Scan scan = new Scan();
979    scan.setCaching(1);
980    scan.setBatch(5);
981    scan.setMaxResultSize(VALUE_SIZE * 6);
982
983    ResultScanner scanner = table.getScanner(scan);
984    for (int i = 0; i < NUM_FAMILIES * NUM_QUALIFIERS / 5 - 1; i++) {
985      assertTrue(scanner.next().mayHaveMoreCellsInRow());
986    }
987    Result result1 = scanner.next();
988    assertEquals(5, result1.rawCells().length);
989    assertCell(result1.rawCells()[0], ROWS[0], FAMILIES[NUM_FAMILIES - 1],
990      QUALIFIERS[NUM_QUALIFIERS - 5]);
991    assertCell(result1.rawCells()[4], ROWS[0], FAMILIES[NUM_FAMILIES - 1],
992      QUALIFIERS[NUM_QUALIFIERS - 1]);
993    assertFalse(result1.mayHaveMoreCellsInRow());
994
995    moveRegion(table, 2);
996
997    Result result2 = scanner.next();
998    assertEquals(5, result2.rawCells().length);
999    assertCell(result2.rawCells()[0], ROWS[1], FAMILIES[0], QUALIFIERS[0]);
1000    assertCell(result2.rawCells()[4], ROWS[1], FAMILIES[0], QUALIFIERS[4]);
1001    assertTrue(result2.mayHaveMoreCellsInRow());
1002
1003    moveRegion(table, 3);
1004
1005    Result result3 = scanner.next();
1006    assertEquals(5, result3.rawCells().length);
1007    assertCell(result3.rawCells()[0], ROWS[1], FAMILIES[0], QUALIFIERS[5]);
1008    assertCell(result3.rawCells()[4], ROWS[1], FAMILIES[0], QUALIFIERS[9]);
1009    assertTrue(result3.mayHaveMoreCellsInRow());
1010
1011    for (int i = 0; i < NUM_FAMILIES * NUM_QUALIFIERS / 5 - 3; i++) {
1012      Result result = scanner.next();
1013      assertEquals(5, result.rawCells().length);
1014      assertTrue(result.mayHaveMoreCellsInRow());
1015    }
1016    Result result = scanner.next();
1017    assertEquals(4, result.rawCells().length);
1018    assertFalse(result.mayHaveMoreCellsInRow());
1019
1020    for (int i = 2; i < NUM_ROWS; i++) {
1021      for (int j = 0; j < NUM_FAMILIES; j++) {
1022        for (int k = 0; k < NUM_QUALIFIERS; k += 5) {
1023          result = scanner.next();
1024          assertCell(result.rawCells()[0], ROWS[i], FAMILIES[j], QUALIFIERS[k]);
1025          assertEquals(5, result.rawCells().length);
1026          if (j == NUM_FAMILIES - 1 && k == NUM_QUALIFIERS - 5) {
1027            assertFalse(result.mayHaveMoreCellsInRow());
1028          } else {
1029            assertTrue(result.mayHaveMoreCellsInRow());
1030          }
1031        }
1032      }
1033    }
1034    assertNull(scanner.next());
1035  }
1036
1037  @Test
1038  public void testDontThrowUnknowScannerExceptionToClient() throws Exception {
1039    Table table =
1040      createTestTable(TableName.valueOf(name.getMethodName()), ROWS, FAMILIES, QUALIFIERS, VALUE);
1041    Scan scan = new Scan();
1042    scan.setCaching(1);
1043    ResultScanner scanner = table.getScanner(scan);
1044    scanner.next();
1045    Thread.sleep(timeout * 2);
1046    int count = 1;
1047    while (scanner.next() != null) {
1048      count++;
1049    }
1050    assertEquals(NUM_ROWS, count);
1051    scanner.close();
1052  }
1053
1054  @Test
1055  public void testMayHaveMoreCellsInRowReturnsTrueAndSetBatch() throws IOException {
1056    Table table =
1057      createTestTable(TableName.valueOf(name.getMethodName()), ROWS, FAMILIES, QUALIFIERS, VALUE);
1058    Scan scan = new Scan();
1059    scan.setBatch(1);
1060    scan.setFilter(new FirstKeyOnlyFilter());
1061    ResultScanner scanner = table.getScanner(scan);
1062    Result result;
1063    while ((result = scanner.next()) != null) {
1064      assertTrue(result.rawCells() != null);
1065      assertEquals(1, result.rawCells().length);
1066    }
1067  }
1068
1069}