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.assertEquals;
021import static org.junit.Assert.assertTrue;
022import java.io.IOException;
023import java.util.ArrayList;
024import java.util.List;
025import org.apache.hadoop.hbase.client.Put;
026import org.apache.hadoop.hbase.client.Result;
027import org.apache.hadoop.hbase.client.ResultScanner;
028import org.apache.hadoop.hbase.client.Scan;
029import org.apache.hadoop.hbase.client.Table;
030import org.apache.hadoop.hbase.client.metrics.ScanMetrics;
031import org.apache.hadoop.hbase.client.metrics.ServerSideScanMetrics;
032import org.apache.hadoop.hbase.filter.BinaryComparator;
033import org.apache.hadoop.hbase.filter.ColumnPrefixFilter;
034import org.apache.hadoop.hbase.filter.Filter;
035import org.apache.hadoop.hbase.filter.FilterList;
036import org.apache.hadoop.hbase.filter.FilterList.Operator;
037import org.apache.hadoop.hbase.filter.FirstKeyOnlyFilter;
038import org.apache.hadoop.hbase.filter.RowFilter;
039import org.apache.hadoop.hbase.filter.SingleColumnValueExcludeFilter;
040import org.apache.hadoop.hbase.filter.SingleColumnValueFilter;
041import org.apache.hadoop.hbase.testclassification.LargeTests;
042import org.apache.hadoop.hbase.util.Bytes;
043import org.junit.AfterClass;
044import org.junit.BeforeClass;
045import org.junit.ClassRule;
046import org.junit.Test;
047import org.junit.experimental.categories.Category;
048import org.slf4j.Logger;
049import org.slf4j.LoggerFactory;
050
051@Category(LargeTests.class)
052public class TestServerSideScanMetricsFromClientSide {
053  private static final Logger LOG =
054    LoggerFactory.getLogger(TestServerSideScanMetricsFromClientSide.class);
055
056  @ClassRule
057  public static final HBaseClassTestRule CLASS_RULE =
058      HBaseClassTestRule.forClass(TestServerSideScanMetricsFromClientSide.class);
059
060  private final static HBaseTestingUtility TEST_UTIL = new HBaseTestingUtility();
061
062  private static Table TABLE = null;
063
064  /**
065   * Table configuration
066   */
067  private static TableName TABLE_NAME = TableName.valueOf("testTable");
068
069  private static int NUM_ROWS = 10;
070  private static byte[] ROW = Bytes.toBytes("testRow");
071  private static byte[][] ROWS = HTestConst.makeNAscii(ROW, NUM_ROWS);
072
073  // Should keep this value below 10 to keep generation of expected kv's simple. If above 10 then
074  // table/row/cf1/... will be followed by table/row/cf10/... instead of table/row/cf2/... which
075  // breaks the simple generation of expected kv's
076  private static int NUM_FAMILIES = 1;
077  private static byte[] FAMILY = Bytes.toBytes("testFamily");
078  private static byte[][] FAMILIES = HTestConst.makeNAscii(FAMILY, NUM_FAMILIES);
079
080  private static int NUM_QUALIFIERS = 1;
081  private static byte[] QUALIFIER = Bytes.toBytes("testQualifier");
082  private static byte[][] QUALIFIERS = HTestConst.makeNAscii(QUALIFIER, NUM_QUALIFIERS);
083
084  private static int VALUE_SIZE = 10;
085  private static byte[] VALUE = Bytes.createMaxByteArray(VALUE_SIZE);
086
087  private static int NUM_COLS = NUM_FAMILIES * NUM_QUALIFIERS;
088
089  // Approximation of how large the heap size of cells in our table. Should be accessed through
090  // getCellHeapSize().
091  private static long CELL_HEAP_SIZE = -1;
092
093  @BeforeClass
094  public static void setUpBeforeClass() throws Exception {
095    TEST_UTIL.startMiniCluster(3);
096    TABLE = createTestTable(TABLE_NAME, ROWS, FAMILIES, QUALIFIERS, VALUE);
097  }
098
099  static Table createTestTable(TableName name, byte[][] rows, byte[][] families,
100      byte[][] qualifiers, byte[] cellValue) throws IOException {
101    Table ht = TEST_UTIL.createTable(name, families);
102    List<Put> puts = createPuts(rows, families, qualifiers, cellValue);
103    ht.put(puts);
104
105    return ht;
106  }
107
108  @AfterClass
109  public static void tearDownAfterClass() throws Exception {
110    TEST_UTIL.shutdownMiniCluster();
111  }
112
113  /**
114   * Make puts to put the input value into each combination of row, family, and qualifier
115   * @param rows the rows to use
116   * @param families the column families to use
117   * @param qualifiers the column qualifiers to use
118   * @param value the value to put
119   * @return the putted input values added in puts
120   * @throws IOException If an IO problem is encountered
121   */
122  static ArrayList<Put> createPuts(byte[][] rows, byte[][] families, byte[][] qualifiers,
123      byte[] value) throws IOException {
124    Put put;
125    ArrayList<Put> puts = new ArrayList<>();
126
127    for (int row = 0; row < rows.length; row++) {
128      put = new Put(rows[row]);
129      for (int fam = 0; fam < families.length; fam++) {
130        for (int qual = 0; qual < qualifiers.length; qual++) {
131          KeyValue kv = new KeyValue(rows[row], families[fam], qualifiers[qual], qual, value);
132          put.add(kv);
133        }
134      }
135      puts.add(put);
136    }
137
138    return puts;
139  }
140
141  /**
142   * @return The approximate heap size of a cell in the test table. All cells should have
143   *         approximately the same heap size, so the value is cached to avoid repeating the
144   *         calculation
145   * @throws Exception on unexpected failure
146   */
147  private long getCellHeapSize() throws Exception {
148    if (CELL_HEAP_SIZE == -1) {
149      // Do a partial scan that will return a single result with a single cell
150      Scan scan = new Scan();
151      scan.setMaxResultSize(1);
152      scan.setAllowPartialResults(true);
153      ResultScanner scanner = TABLE.getScanner(scan);
154
155      Result result = scanner.next();
156
157      assertTrue(result != null);
158      assertTrue(result.rawCells() != null);
159      assertTrue(result.rawCells().length == 1);
160
161      CELL_HEAP_SIZE = result.rawCells()[0].heapSize();
162      scanner.close();
163    }
164
165    return CELL_HEAP_SIZE;
166  }
167
168  @Test
169  public void testRowsSeenMetricWithSync() throws Exception {
170    testRowsSeenMetric(false);
171  }
172
173  @Test
174  public void testRowsSeenMetricWithAsync() throws Exception {
175    testRowsSeenMetric(true);
176  }
177
178  private void testRowsSeenMetric(boolean async) throws Exception {
179    // Base scan configuration
180    Scan baseScan;
181    baseScan = new Scan();
182    baseScan.setScanMetricsEnabled(true);
183    baseScan.setAsyncPrefetch(async);
184    try {
185      testRowsSeenMetric(baseScan);
186
187      // Test case that only a single result will be returned per RPC to the serer
188      baseScan.setCaching(1);
189      testRowsSeenMetric(baseScan);
190
191      // Test case that partial results are returned from the server. At most one cell will be
192      // contained in each response
193      baseScan.setMaxResultSize(1);
194      testRowsSeenMetric(baseScan);
195
196      // Test case that size limit is set such that a few cells are returned per partial result from
197      // the server
198      baseScan.setCaching(NUM_ROWS);
199      baseScan.setMaxResultSize(getCellHeapSize() * (NUM_COLS - 1));
200      testRowsSeenMetric(baseScan);
201    } catch (Throwable t) {
202      LOG.error("FAIL", t);
203      throw t;
204    }
205  }
206
207  public void testRowsSeenMetric(Scan baseScan) throws Exception {
208    Scan scan;
209    scan = new Scan(baseScan);
210    testMetric(scan, ServerSideScanMetrics.COUNT_OF_ROWS_SCANNED_KEY_METRIC_NAME, NUM_ROWS);
211
212    for (int i = 0; i < ROWS.length - 1; i++) {
213      scan = new Scan(baseScan);
214      scan.withStartRow(ROWS[0]);
215      scan.withStopRow(ROWS[i + 1]);
216      testMetric(scan, ServerSideScanMetrics.COUNT_OF_ROWS_SCANNED_KEY_METRIC_NAME, i + 1);
217    }
218
219    for (int i = ROWS.length - 1; i > 0; i--) {
220      scan = new Scan(baseScan);
221      scan.withStartRow(ROWS[i - 1]);
222      scan.withStopRow(ROWS[ROWS.length - 1]);
223      testMetric(scan, ServerSideScanMetrics.COUNT_OF_ROWS_SCANNED_KEY_METRIC_NAME,
224        ROWS.length - i);
225    }
226
227    // The filter should filter out all rows, but we still expect to see every row.
228    Filter filter =
229        new RowFilter(CompareOperator.EQUAL, new BinaryComparator(Bytes.toBytes("xyz")));
230    scan = new Scan(baseScan);
231    scan.setFilter(filter);
232    testMetric(scan, ServerSideScanMetrics.COUNT_OF_ROWS_SCANNED_KEY_METRIC_NAME, ROWS.length);
233
234    // Filter should pass on all rows
235    SingleColumnValueFilter singleColumnValueFilter =
236        new SingleColumnValueFilter(FAMILIES[0], QUALIFIERS[0], CompareOperator.EQUAL, VALUE);
237    scan = new Scan(baseScan);
238    scan.setFilter(singleColumnValueFilter);
239    testMetric(scan, ServerSideScanMetrics.COUNT_OF_ROWS_SCANNED_KEY_METRIC_NAME, ROWS.length);
240
241    // Filter should filter out all rows
242    singleColumnValueFilter =
243        new SingleColumnValueFilter(FAMILIES[0], QUALIFIERS[0], CompareOperator.NOT_EQUAL, VALUE);
244    scan = new Scan(baseScan);
245    scan.setFilter(singleColumnValueFilter);
246    testMetric(scan, ServerSideScanMetrics.COUNT_OF_ROWS_SCANNED_KEY_METRIC_NAME, ROWS.length);
247  }
248
249  @Test
250  public void testRowsFilteredMetric() throws Exception {
251    // Base scan configuration
252    Scan baseScan;
253    baseScan = new Scan();
254    baseScan.setScanMetricsEnabled(true);
255
256    // Test case where scan uses default values
257    testRowsFilteredMetric(baseScan);
258
259    // Test case where at most one Result is retrieved per RPC
260    baseScan.setCaching(1);
261    testRowsFilteredMetric(baseScan);
262
263    // Test case where size limit is very restrictive and partial results will be returned from
264    // server
265    baseScan.setMaxResultSize(1);
266    testRowsFilteredMetric(baseScan);
267
268    // Test a case where max result size limits response from server to only a few cells (not all
269    // cells from the row)
270    baseScan.setCaching(NUM_ROWS);
271    baseScan.setMaxResultSize(getCellHeapSize() * (NUM_COLS - 1));
272    testRowsSeenMetric(baseScan);
273  }
274
275  public void testRowsFilteredMetric(Scan baseScan) throws Exception {
276    testRowsFilteredMetric(baseScan, null, 0);
277
278    // Row filter doesn't match any row key. All rows should be filtered
279    Filter filter =
280        new RowFilter(CompareOperator.EQUAL, new BinaryComparator(Bytes.toBytes("xyz")));
281    testRowsFilteredMetric(baseScan, filter, ROWS.length);
282
283    // Filter will return results containing only the first key. Number of entire rows filtered
284    // should be 0.
285    filter = new FirstKeyOnlyFilter();
286    testRowsFilteredMetric(baseScan, filter, 0);
287
288    // Column prefix will find some matching qualifier on each row. Number of entire rows filtered
289    // should be 0
290    filter = new ColumnPrefixFilter(QUALIFIERS[0]);
291    testRowsFilteredMetric(baseScan, filter, 0);
292
293    // Column prefix will NOT find any matching qualifier on any row. All rows should be filtered
294    filter = new ColumnPrefixFilter(Bytes.toBytes("xyz"));
295    testRowsFilteredMetric(baseScan, filter, ROWS.length);
296
297    // Matching column value should exist in each row. No rows should be filtered.
298    filter = new SingleColumnValueFilter(FAMILIES[0], QUALIFIERS[0], CompareOperator.EQUAL, VALUE);
299    testRowsFilteredMetric(baseScan, filter, 0);
300
301    // No matching column value should exist in any row. Filter all rows
302    filter = new SingleColumnValueFilter(FAMILIES[0], QUALIFIERS[0],
303      CompareOperator.NOT_EQUAL, VALUE);
304    testRowsFilteredMetric(baseScan, filter, ROWS.length);
305
306    List<Filter> filters = new ArrayList<>();
307    filters.add(new RowFilter(CompareOperator.EQUAL, new BinaryComparator(ROWS[0])));
308    filters.add(new RowFilter(CompareOperator.EQUAL, new BinaryComparator(ROWS[3])));
309    int numberOfMatchingRowFilters = filters.size();
310    filter = new FilterList(Operator.MUST_PASS_ONE, filters);
311    testRowsFilteredMetric(baseScan, filter, ROWS.length - numberOfMatchingRowFilters);
312    filters.clear();
313
314    // Add a single column value exclude filter for each column... The net effect is that all
315    // columns will be excluded when scanning on the server side. This will result in an empty cell
316    // array in RegionScanner#nextInternal which should be interpreted as a row being filtered.
317    for (int family = 0; family < FAMILIES.length; family++) {
318      for (int qualifier = 0; qualifier < QUALIFIERS.length; qualifier++) {
319        filters.add(new SingleColumnValueExcludeFilter(FAMILIES[family], QUALIFIERS[qualifier],
320          CompareOperator.EQUAL, VALUE));
321      }
322    }
323    filter = new FilterList(Operator.MUST_PASS_ONE, filters);
324    testRowsFilteredMetric(baseScan, filter, ROWS.length);
325  }
326
327  public void testRowsFilteredMetric(Scan baseScan, Filter filter, int expectedNumFiltered)
328      throws Exception {
329    Scan scan = new Scan(baseScan);
330    if (filter != null) {
331      scan.setFilter(filter);
332    }
333    testMetric(scan, ServerSideScanMetrics.COUNT_OF_ROWS_FILTERED_KEY_METRIC_NAME,
334      expectedNumFiltered);
335  }
336
337  /**
338   * Run the scan to completion and check the metric against the specified value
339   * @param scan The scan instance to use to record metrics
340   * @param metricKey The metric key name
341   * @param expectedValue The expected value of metric
342   * @throws Exception on unexpected failure
343   */
344  public void testMetric(Scan scan, String metricKey, long expectedValue) throws Exception {
345    assertTrue("Scan should be configured to record metrics", scan.isScanMetricsEnabled());
346    ResultScanner scanner = TABLE.getScanner(scan);
347    // Iterate through all the results
348    while (scanner.next() != null) {
349      continue;
350    }
351    scanner.close();
352    ScanMetrics metrics = scanner.getScanMetrics();
353    assertTrue("Metrics are null", metrics != null);
354    assertTrue("Metric : " + metricKey + " does not exist", metrics.hasCounter(metricKey));
355    final long actualMetricValue = metrics.getCounter(metricKey).get();
356    assertEquals("Metric: " + metricKey + " Expected: " + expectedValue + " Actual: "
357        + actualMetricValue, expectedValue, actualMetricValue);
358
359  }
360}