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.client;
019
020import static org.hamcrest.MatcherAssert.assertThat;
021import static org.hamcrest.Matchers.greaterThan;
022import static org.hamcrest.Matchers.lessThan;
023import static org.junit.Assert.assertArrayEquals;
024import static org.junit.Assert.assertEquals;
025import static org.junit.Assert.assertFalse;
026import static org.junit.Assert.assertNotSame;
027import static org.junit.Assert.assertNull;
028import static org.junit.Assert.assertSame;
029import static org.junit.Assert.assertTrue;
030import static org.junit.Assert.fail;
031
032import java.io.IOException;
033import java.nio.ByteBuffer;
034import java.util.ArrayList;
035import java.util.Arrays;
036import java.util.List;
037import org.apache.hadoop.hbase.ArrayBackedTag;
038import org.apache.hadoop.hbase.ByteBufferKeyValue;
039import org.apache.hadoop.hbase.Cell;
040import org.apache.hadoop.hbase.CellComparator;
041import org.apache.hadoop.hbase.CellScanner;
042import org.apache.hadoop.hbase.CellUtil;
043import org.apache.hadoop.hbase.HBaseClassTestRule;
044import org.apache.hadoop.hbase.KeyValue;
045import org.apache.hadoop.hbase.Tag;
046import org.apache.hadoop.hbase.testclassification.ClientTests;
047import org.apache.hadoop.hbase.testclassification.SmallTests;
048import org.apache.hadoop.hbase.util.ByteBufferUtils;
049import org.apache.hadoop.hbase.util.Bytes;
050import org.apache.hadoop.hbase.util.EnvironmentEdgeManager;
051import org.junit.ClassRule;
052import org.junit.Test;
053import org.junit.experimental.categories.Category;
054import org.slf4j.Logger;
055import org.slf4j.LoggerFactory;
056
057@Category({ SmallTests.class, ClientTests.class })
058public class TestResult {
059
060  @ClassRule
061  public static final HBaseClassTestRule CLASS_RULE = HBaseClassTestRule.forClass(TestResult.class);
062
063  private static final Logger LOG = LoggerFactory.getLogger(TestResult.class.getName());
064
065  static KeyValue[] genKVs(final byte[] row, final byte[] family, final byte[] value,
066    final long timestamp, final int cols) {
067    KeyValue[] kvs = new KeyValue[cols];
068
069    for (int i = 0; i < cols; i++) {
070      kvs[i] =
071        new KeyValue(row, family, Bytes.toBytes(i), timestamp, Bytes.add(value, Bytes.toBytes(i)));
072    }
073    return kvs;
074  }
075
076  static final byte[] row = Bytes.toBytes("row");
077  static final byte[] family = Bytes.toBytes("family");
078  static final byte[] value = Bytes.toBytes("value");
079  static final byte[] qual = Bytes.toBytes("qual");
080
081  /**
082   * Run some tests to ensure Result acts like a proper CellScanner.
083   */
084  @Test
085  public void testResultAsCellScanner() throws IOException {
086    Cell[] cells = genKVs(row, family, value, 1, 10);
087    Arrays.sort(cells, CellComparator.getInstance());
088    Result r = Result.create(cells);
089    assertCellsSame(r, cells);
090    // Assert I run over same result multiple times.
091    assertCellsSame(r.cellScanner(), cells);
092    assertCellsSame(r.cellScanner(), cells);
093    // Assert we are not creating new object when doing cellscanner
094    assertTrue(r == r.cellScanner());
095  }
096
097  private void assertCellsSame(final CellScanner cellScanner, final Cell[] cells)
098    throws IOException {
099    int count = 0;
100    while (cellScanner.advance()) {
101      assertTrue(cells[count].equals(cellScanner.current()));
102      count++;
103    }
104    assertEquals(cells.length, count);
105  }
106
107  @Test
108  public void testBasicGetColumn() throws Exception {
109    KeyValue[] kvs = genKVs(row, family, value, 1, 100);
110
111    Arrays.sort(kvs, CellComparator.getInstance());
112
113    Result r = Result.create(kvs);
114
115    for (int i = 0; i < 100; ++i) {
116      final byte[] qf = Bytes.toBytes(i);
117
118      List<Cell> ks = r.getColumnCells(family, qf);
119      assertEquals(1, ks.size());
120      assertTrue(CellUtil.matchingQualifier(ks.get(0), qf));
121      assertEquals(ks.get(0), r.getColumnLatestCell(family, qf));
122    }
123  }
124
125  @Test
126  public void testCurrentOnEmptyCell() throws IOException {
127    Result r = Result.create(new Cell[0]);
128    assertFalse(r.advance());
129    assertNull(r.current());
130  }
131
132  @Test
133  public void testAdvanceMultipleOnEmptyCell() throws IOException {
134    Result r = Result.create(new Cell[0]);
135    // After HBASE-26688, advance of result with empty cell list will always return false.
136    // Here 10 is an arbitrary number to test the logic.
137    for (int i = 0; i < 10; i++) {
138      assertFalse(r.advance());
139    }
140  }
141
142  @Test
143  public void testMultiVersionGetColumn() throws Exception {
144    KeyValue[] kvs1 = genKVs(row, family, value, 1, 100);
145    KeyValue[] kvs2 = genKVs(row, family, value, 200, 100);
146
147    KeyValue[] kvs = new KeyValue[kvs1.length + kvs2.length];
148    System.arraycopy(kvs1, 0, kvs, 0, kvs1.length);
149    System.arraycopy(kvs2, 0, kvs, kvs1.length, kvs2.length);
150
151    Arrays.sort(kvs, CellComparator.getInstance());
152
153    Result r = Result.create(kvs);
154    for (int i = 0; i < 100; ++i) {
155      final byte[] qf = Bytes.toBytes(i);
156
157      List<Cell> ks = r.getColumnCells(family, qf);
158      assertEquals(2, ks.size());
159      assertTrue(CellUtil.matchingQualifier(ks.get(0), qf));
160      assertEquals(200, ks.get(0).getTimestamp());
161      assertEquals(ks.get(0), r.getColumnLatestCell(family, qf));
162    }
163  }
164
165  @Test
166  public void testBasicGetValue() throws Exception {
167    KeyValue[] kvs = genKVs(row, family, value, 1, 100);
168
169    Arrays.sort(kvs, CellComparator.getInstance());
170
171    Result r = Result.create(kvs);
172
173    for (int i = 0; i < 100; ++i) {
174      final byte[] qf = Bytes.toBytes(i);
175
176      assertArrayEquals(Bytes.add(value, Bytes.toBytes(i)), r.getValue(family, qf));
177      assertTrue(r.containsColumn(family, qf));
178    }
179  }
180
181  @Test
182  public void testMultiVersionGetValue() throws Exception {
183    KeyValue[] kvs1 = genKVs(row, family, value, 1, 100);
184    KeyValue[] kvs2 = genKVs(row, family, value, 200, 100);
185
186    KeyValue[] kvs = new KeyValue[kvs1.length + kvs2.length];
187    System.arraycopy(kvs1, 0, kvs, 0, kvs1.length);
188    System.arraycopy(kvs2, 0, kvs, kvs1.length, kvs2.length);
189
190    Arrays.sort(kvs, CellComparator.getInstance());
191
192    Result r = Result.create(kvs);
193    for (int i = 0; i < 100; ++i) {
194      final byte[] qf = Bytes.toBytes(i);
195
196      assertArrayEquals(Bytes.add(value, Bytes.toBytes(i)), r.getValue(family, qf));
197      assertTrue(r.containsColumn(family, qf));
198    }
199  }
200
201  @Test
202  public void testBasicLoadValue() throws Exception {
203    KeyValue[] kvs = genKVs(row, family, value, 1, 100);
204
205    Arrays.sort(kvs, CellComparator.getInstance());
206
207    Result r = Result.create(kvs);
208    ByteBuffer loadValueBuffer = ByteBuffer.allocate(1024);
209
210    for (int i = 0; i < 100; ++i) {
211      final byte[] qf = Bytes.toBytes(i);
212
213      loadValueBuffer.clear();
214      r.loadValue(family, qf, loadValueBuffer);
215      loadValueBuffer.flip();
216      assertEquals(loadValueBuffer, ByteBuffer.wrap(Bytes.add(value, Bytes.toBytes(i))));
217      assertEquals(ByteBuffer.wrap(Bytes.add(value, Bytes.toBytes(i))),
218        r.getValueAsByteBuffer(family, qf));
219    }
220  }
221
222  @Test
223  public void testMultiVersionLoadValue() throws Exception {
224    KeyValue[] kvs1 = genKVs(row, family, value, 1, 100);
225    KeyValue[] kvs2 = genKVs(row, family, value, 200, 100);
226
227    KeyValue[] kvs = new KeyValue[kvs1.length + kvs2.length];
228    System.arraycopy(kvs1, 0, kvs, 0, kvs1.length);
229    System.arraycopy(kvs2, 0, kvs, kvs1.length, kvs2.length);
230
231    Arrays.sort(kvs, CellComparator.getInstance());
232
233    ByteBuffer loadValueBuffer = ByteBuffer.allocate(1024);
234
235    Result r = Result.create(kvs);
236    for (int i = 0; i < 100; ++i) {
237      final byte[] qf = Bytes.toBytes(i);
238
239      loadValueBuffer.clear();
240      r.loadValue(family, qf, loadValueBuffer);
241      loadValueBuffer.flip();
242      assertEquals(loadValueBuffer, ByteBuffer.wrap(Bytes.add(value, Bytes.toBytes(i))));
243      assertEquals(ByteBuffer.wrap(Bytes.add(value, Bytes.toBytes(i))),
244        r.getValueAsByteBuffer(family, qf));
245    }
246  }
247
248  /**
249   * Verify that Result.compareResults(...) behaves correctly.
250   */
251  @Test
252  public void testCompareResults() throws Exception {
253    byte[] value1 = Bytes.toBytes("value1");
254    byte[] qual = Bytes.toBytes("qual");
255
256    KeyValue kv1 = new KeyValue(row, family, qual, value);
257    KeyValue kv2 = new KeyValue(row, family, qual, value1);
258
259    Result r1 = Result.create(new KeyValue[] { kv1 });
260    Result r2 = Result.create(new KeyValue[] { kv2 });
261    // no exception thrown
262    Result.compareResults(r1, r1);
263    try {
264      // these are different (HBASE-4800)
265      Result.compareResults(r1, r2);
266      fail();
267    } catch (Exception x) {
268      assertTrue(x.getMessage().startsWith("This result was different:"));
269    }
270  }
271
272  @Test
273  public void testCompareResultsWithTags() throws Exception {
274    Tag t1 = new ArrayBackedTag((byte) 1, Bytes.toBytes("TAG1"));
275    Tag t2 = new ArrayBackedTag((byte) 2, Bytes.toBytes("TAG2"));
276    // Both BB backed tags KV are null
277    Result result1 = getByteBufferBackedTagResult(null);
278    Result result2 = getByteBufferBackedTagResult(null);
279    Result.compareResults(result1, result2);
280
281    // Test both byte buffer backed tags KeyValue
282    result1 = getByteBufferBackedTagResult(t1);
283    result2 = getByteBufferBackedTagResult(t1);
284    Result.compareResults(result1, result2);
285
286    // Both array backed tags KV are null
287    result1 = getArrayBackedTagResult(null);
288    result2 = getArrayBackedTagResult(null);
289    Result.compareResults(result1, result2);
290
291    // Test both array backed tags KeyValue
292    result1 = getArrayBackedTagResult(t1);
293    result2 = getArrayBackedTagResult(t1);
294    Result.compareResults(result1, result2);
295
296    // left instance of byte buffer and right instance of array backed
297    result1 = getByteBufferBackedTagResult(t1);
298    result2 = getArrayBackedTagResult(t1);
299    Result.compareResults(result1, result2);
300
301    // left instance of array backed and right instance of byte buffer backed.
302    result1 = getArrayBackedTagResult(t1);
303    result2 = getByteBufferBackedTagResult(t1);
304    Result.compareResults(result1, result2);
305
306    // Left BB backed null tag and right BB backed non null tag
307    result1 = getByteBufferBackedTagResult(null);
308    result2 = getByteBufferBackedTagResult(t2);
309    try {
310      Result.compareResults(result1, result2);
311      fail();
312    } catch (Exception e) {
313      // Expected
314    }
315
316    // Left BB backed non null tag and right BB backed null tag
317    result1 = getByteBufferBackedTagResult(t1);
318    result2 = getByteBufferBackedTagResult(null);
319    try {
320      Result.compareResults(result1, result2);
321      fail();
322    } catch (Exception e) {
323      // Expected
324    }
325
326    // Both byte buffer backed tags KV are different
327    result1 = getByteBufferBackedTagResult(t1);
328    result2 = getByteBufferBackedTagResult(t2);
329    try {
330      Result.compareResults(result1, result2);
331      fail();
332    } catch (Exception e) {
333      // Expected
334    }
335
336    // Left array backed non null tag and right array backed null tag
337    result1 = getArrayBackedTagResult(t1);
338    result2 = getArrayBackedTagResult(null);
339    try {
340      Result.compareResults(result1, result2);
341      fail();
342    } catch (Exception e) {
343      // Expected
344    }
345
346    // Left array backed null tag and right array backed non null tag
347    result1 = getByteBufferBackedTagResult(null);
348    result2 = getByteBufferBackedTagResult(t2);
349    try {
350      Result.compareResults(result1, result2);
351      fail();
352    } catch (Exception e) {
353      // Expected
354    }
355
356    // Both array backed tags KV are different
357    result1 = getArrayBackedTagResult(t1);
358    result2 = getArrayBackedTagResult(t2);
359    try {
360      Result.compareResults(result1, result2);
361      fail();
362    } catch (Exception e) {
363      // Expected
364    }
365
366    // left instance of byte buffer and right instance of array backed are different
367    result1 = getByteBufferBackedTagResult(t1);
368    result2 = getArrayBackedTagResult(t2);
369    try {
370      Result.compareResults(result1, result2);
371      fail();
372    } catch (Exception e) {
373      // Expected
374    }
375
376    // left instance of array backed and right instance of byte buffer backed are different
377    result1 = getArrayBackedTagResult(t1);
378    result2 = getByteBufferBackedTagResult(t2);
379    try {
380      Result.compareResults(result1, result2);
381      fail();
382    } catch (Exception e) {
383      // Expected
384    }
385  }
386
387  @Test
388  public void testCompareResultMemoryUsage() {
389    List<Cell> cells1 = new ArrayList<>();
390    for (long i = 0; i < 100; i++) {
391      cells1.add(new KeyValue(row, family, Bytes.toBytes(i), value));
392    }
393
394    List<Cell> cells2 = new ArrayList<>();
395    for (long i = 0; i < 100; i++) {
396      cells2.add(new KeyValue(row, family, Bytes.toBytes(i), Bytes.toBytes(i)));
397    }
398
399    Result r1 = Result.create(cells1);
400    Result r2 = Result.create(cells2);
401    try {
402      Result.compareResults(r1, r2);
403      fail();
404    } catch (Exception x) {
405      assertTrue(x.getMessage().startsWith("This result was different:"));
406      assertThat(x.getMessage().length(), greaterThan(100));
407    }
408
409    try {
410      Result.compareResults(r1, r2, false);
411      fail();
412    } catch (Exception x) {
413      assertEquals("This result was different: row=row", x.getMessage());
414      assertThat(x.getMessage().length(), lessThan(100));
415    }
416  }
417
418  private Result getArrayBackedTagResult(Tag tag) {
419    List<Tag> tags = null;
420    if (tag != null) {
421      tags = Arrays.asList(tag);
422    }
423    KeyValue kvCell = new KeyValue(row, family, qual, 0L, KeyValue.Type.Put, value, tags);
424    return Result.create(new Cell[] { kvCell });
425  }
426
427  private Result getByteBufferBackedTagResult(Tag tag) {
428    List<Tag> tags = null;
429    if (tag != null) {
430      tags = Arrays.asList(tag);
431    }
432    KeyValue kvCell = new KeyValue(row, family, qual, 0L, KeyValue.Type.Put, value, tags);
433    ByteBuffer buf = ByteBuffer.allocateDirect(kvCell.getBuffer().length);
434    ByteBufferUtils.copyFromArrayToBuffer(buf, kvCell.getBuffer(), 0, kvCell.getBuffer().length);
435    ByteBufferKeyValue bbKV = new ByteBufferKeyValue(buf, 0, buf.capacity(), 0L);
436    return Result.create(new Cell[] { bbKV });
437  }
438
439  /**
440   * Verifies that one can't modify instance of EMPTY_RESULT.
441   */
442  @Test
443  public void testEmptyResultIsReadonly() {
444    Result emptyResult = Result.EMPTY_RESULT;
445    Result otherResult = new Result();
446
447    try {
448      emptyResult.copyFrom(otherResult);
449      fail("UnsupportedOperationException should have been thrown!");
450    } catch (UnsupportedOperationException ex) {
451      LOG.debug("As expected: " + ex.getMessage());
452    }
453    try {
454      emptyResult.setExists(true);
455      fail("UnsupportedOperationException should have been thrown!");
456    } catch (UnsupportedOperationException ex) {
457      LOG.debug("As expected: " + ex.getMessage());
458    }
459  }
460
461  /**
462   * Microbenchmark that compares {@link Result#getValue} and {@link Result#loadValue} performance.
463   */
464  public void doReadBenchmark() throws Exception {
465    final int n = 5;
466    final int m = 100000000;
467
468    StringBuilder valueSB = new StringBuilder();
469    for (int i = 0; i < 100; i++) {
470      valueSB.append((byte) (Math.random() * 10));
471    }
472
473    StringBuilder rowSB = new StringBuilder();
474    for (int i = 0; i < 50; i++) {
475      rowSB.append((byte) (Math.random() * 10));
476    }
477
478    KeyValue[] kvs =
479      genKVs(Bytes.toBytes(rowSB.toString()), family, Bytes.toBytes(valueSB.toString()), 1, n);
480    Arrays.sort(kvs, CellComparator.getInstance());
481    ByteBuffer loadValueBuffer = ByteBuffer.allocate(1024);
482    Result r = Result.create(kvs);
483
484    byte[][] qfs = new byte[n][Bytes.SIZEOF_INT];
485    for (int i = 0; i < n; ++i) {
486      System.arraycopy(qfs[i], 0, Bytes.toBytes(i), 0, Bytes.SIZEOF_INT);
487    }
488
489    // warm up
490    for (int k = 0; k < 100000; k++) {
491      for (int i = 0; i < n; ++i) {
492        r.getValue(family, qfs[i]);
493        loadValueBuffer.clear();
494        r.loadValue(family, qfs[i], loadValueBuffer);
495        loadValueBuffer.flip();
496      }
497    }
498
499    System.gc();
500    long start = System.nanoTime();
501    for (int k = 0; k < m; k++) {
502      for (int i = 0; i < n; ++i) {
503        loadValueBuffer.clear();
504        r.loadValue(family, qfs[i], loadValueBuffer);
505        loadValueBuffer.flip();
506      }
507    }
508    long stop = System.nanoTime();
509    System.out.println("loadValue(): " + (stop - start));
510
511    System.gc();
512    start = System.nanoTime();
513    for (int k = 0; k < m; k++) {
514      for (int i = 0; i < n; i++) {
515        r.getValue(family, qfs[i]);
516      }
517    }
518    stop = System.nanoTime();
519    System.out.println("getValue():  " + (stop - start));
520  }
521
522  @Test
523  public void testCreateResultWithCellArray() {
524    Cell[] cells = genKVs(row, family, value, EnvironmentEdgeManager.currentTime(), 5);
525    Result r = Result.create(cells);
526    // the cells is actually a KeyValue[], which can be cast to ExtendedCell[] directly, so we
527    // should get the same one without copying
528    assertSame(cells, r.rawCells());
529
530    Cell[] emptyCells = new Cell[0];
531    Result emptyResult = Result.create(emptyCells);
532    // emptyCells is a Cell[] instead of ExtendedCell[], so we need to copy it to a new array
533    assertNotSame(emptyCells, emptyResult.rawCells());
534  }
535
536  /**
537   * Calls non-functional test methods.
538   */
539  public static void main(String[] args) {
540    TestResult testResult = new TestResult();
541    try {
542      testResult.doReadBenchmark();
543    } catch (Exception e) {
544      LOG.error("Unexpected exception", e);
545    }
546  }
547}