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