HBASE-26567 Remove IndexType from ChunkCreator (#3947)
[hbase.git] / hbase-server / src / test / java / org / apache / hadoop / hbase / TestServerSideScanMetricsFromClientSide.java
blob3eaa2ffd7bd4f9a55b807726dae83e40d559afa1
1 /**
2 * Licensed to the Apache Software Foundation (ASF) under one
3 * or more contributor license agreements. See the NOTICE file
4 * distributed with this work for additional information
5 * regarding copyright ownership. The ASF licenses this file
6 * to you under the Apache License, Version 2.0 (the
7 * "License"); you may not use this file except in compliance
8 * with the License. You may obtain a copy of the License at
10 * http://www.apache.org/licenses/LICENSE-2.0
12 * Unless required by applicable law or agreed to in writing, software
13 * distributed under the License is distributed on an "AS IS" BASIS,
14 * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
15 * See the License for the specific language governing permissions and
16 * limitations under the License.
18 package org.apache.hadoop.hbase;
20 import static org.junit.Assert.assertEquals;
21 import static org.junit.Assert.assertTrue;
22 import java.io.IOException;
23 import java.util.ArrayList;
24 import java.util.List;
25 import org.apache.hadoop.hbase.client.Put;
26 import org.apache.hadoop.hbase.client.Result;
27 import org.apache.hadoop.hbase.client.ResultScanner;
28 import org.apache.hadoop.hbase.client.Scan;
29 import org.apache.hadoop.hbase.client.Table;
30 import org.apache.hadoop.hbase.client.metrics.ScanMetrics;
31 import org.apache.hadoop.hbase.client.metrics.ServerSideScanMetrics;
32 import org.apache.hadoop.hbase.filter.BinaryComparator;
33 import org.apache.hadoop.hbase.filter.ColumnPrefixFilter;
34 import org.apache.hadoop.hbase.filter.Filter;
35 import org.apache.hadoop.hbase.filter.FilterList;
36 import org.apache.hadoop.hbase.filter.FilterList.Operator;
37 import org.apache.hadoop.hbase.filter.FirstKeyOnlyFilter;
38 import org.apache.hadoop.hbase.filter.RowFilter;
39 import org.apache.hadoop.hbase.filter.SingleColumnValueExcludeFilter;
40 import org.apache.hadoop.hbase.filter.SingleColumnValueFilter;
41 import org.apache.hadoop.hbase.testclassification.LargeTests;
42 import org.apache.hadoop.hbase.util.Bytes;
43 import org.junit.AfterClass;
44 import org.junit.BeforeClass;
45 import org.junit.ClassRule;
46 import org.junit.Test;
47 import org.junit.experimental.categories.Category;
48 import org.slf4j.Logger;
49 import org.slf4j.LoggerFactory;
51 @Category(LargeTests.class)
52 public class TestServerSideScanMetricsFromClientSide {
53 private static final Logger LOG =
54 LoggerFactory.getLogger(TestServerSideScanMetricsFromClientSide.class);
56 @ClassRule
57 public static final HBaseClassTestRule CLASS_RULE =
58 HBaseClassTestRule.forClass(TestServerSideScanMetricsFromClientSide.class);
60 private final static HBaseTestingUtil TEST_UTIL = new HBaseTestingUtil();
62 private static Table TABLE = null;
64 /**
65 * Table configuration
67 private static TableName TABLE_NAME = TableName.valueOf("testTable");
69 private static int NUM_ROWS = 10;
70 private static byte[] ROW = Bytes.toBytes("testRow");
71 private static byte[][] ROWS = HTestConst.makeNAscii(ROW, NUM_ROWS);
73 // Should keep this value below 10 to keep generation of expected kv's simple. If above 10 then
74 // table/row/cf1/... will be followed by table/row/cf10/... instead of table/row/cf2/... which
75 // breaks the simple generation of expected kv's
76 private static int NUM_FAMILIES = 1;
77 private static byte[] FAMILY = Bytes.toBytes("testFamily");
78 private static byte[][] FAMILIES = HTestConst.makeNAscii(FAMILY, NUM_FAMILIES);
80 private static int NUM_QUALIFIERS = 1;
81 private static byte[] QUALIFIER = Bytes.toBytes("testQualifier");
82 private static byte[][] QUALIFIERS = HTestConst.makeNAscii(QUALIFIER, NUM_QUALIFIERS);
84 private static int VALUE_SIZE = 10;
85 private static byte[] VALUE = Bytes.createMaxByteArray(VALUE_SIZE);
87 private static int NUM_COLS = NUM_FAMILIES * NUM_QUALIFIERS;
89 // Approximation of how large the heap size of cells in our table. Should be accessed through
90 // getCellHeapSize().
91 private static long CELL_HEAP_SIZE = -1;
93 @BeforeClass
94 public static void setUpBeforeClass() throws Exception {
95 TEST_UTIL.startMiniCluster(3);
96 TABLE = createTestTable(TABLE_NAME, ROWS, FAMILIES, QUALIFIERS, VALUE);
99 private 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);
105 return ht;
108 @AfterClass
109 public static void tearDownAfterClass() throws Exception {
110 TEST_UTIL.shutdownMiniCluster();
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
122 private static ArrayList<Put> createPuts(byte[][] rows, byte[][] families, byte[][] qualifiers,
123 byte[] value) throws IOException {
124 Put put;
125 ArrayList<Put> puts = new ArrayList<>();
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);
135 puts.add(put);
138 return puts;
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
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);
155 Result result = scanner.next();
157 assertTrue(result != null);
158 assertTrue(result.rawCells() != null);
159 assertTrue(result.rawCells().length == 1);
161 CELL_HEAP_SIZE = result.rawCells()[0].heapSize();
162 scanner.close();
165 return CELL_HEAP_SIZE;
168 @Test
169 public void testRowsSeenMetric() throws Exception {
170 // Base scan configuration
171 Scan baseScan;
172 baseScan = new Scan();
173 baseScan.setScanMetricsEnabled(true);
174 try {
175 testRowsSeenMetric(baseScan);
177 // Test case that only a single result will be returned per RPC to the serer
178 baseScan.setCaching(1);
179 testRowsSeenMetric(baseScan);
181 // Test case that partial results are returned from the server. At most one cell will be
182 // contained in each response
183 baseScan.setMaxResultSize(1);
184 testRowsSeenMetric(baseScan);
186 // Test case that size limit is set such that a few cells are returned per partial result from
187 // the server
188 baseScan.setCaching(NUM_ROWS);
189 baseScan.setMaxResultSize(getCellHeapSize() * (NUM_COLS - 1));
190 testRowsSeenMetric(baseScan);
191 } catch (Throwable t) {
192 LOG.error("FAIL", t);
193 throw t;
197 private void testRowsSeenMetric(Scan baseScan) throws Exception {
198 Scan scan;
199 scan = new Scan(baseScan);
200 testMetric(scan, ServerSideScanMetrics.COUNT_OF_ROWS_SCANNED_KEY_METRIC_NAME, NUM_ROWS);
202 for (int i = 0; i < ROWS.length - 1; i++) {
203 scan = new Scan(baseScan);
204 scan.withStartRow(ROWS[0]);
205 scan.withStopRow(ROWS[i + 1]);
206 testMetric(scan, ServerSideScanMetrics.COUNT_OF_ROWS_SCANNED_KEY_METRIC_NAME, i + 1);
209 for (int i = ROWS.length - 1; i > 0; i--) {
210 scan = new Scan(baseScan);
211 scan.withStartRow(ROWS[i - 1]);
212 scan.withStopRow(ROWS[ROWS.length - 1]);
213 testMetric(scan, ServerSideScanMetrics.COUNT_OF_ROWS_SCANNED_KEY_METRIC_NAME,
214 ROWS.length - i);
217 // The filter should filter out all rows, but we still expect to see every row.
218 Filter filter =
219 new RowFilter(CompareOperator.EQUAL, new BinaryComparator(Bytes.toBytes("xyz")));
220 scan = new Scan(baseScan);
221 scan.setFilter(filter);
222 testMetric(scan, ServerSideScanMetrics.COUNT_OF_ROWS_SCANNED_KEY_METRIC_NAME, ROWS.length);
224 // Filter should pass on all rows
225 SingleColumnValueFilter singleColumnValueFilter =
226 new SingleColumnValueFilter(FAMILIES[0], QUALIFIERS[0], CompareOperator.EQUAL, VALUE);
227 scan = new Scan(baseScan);
228 scan.setFilter(singleColumnValueFilter);
229 testMetric(scan, ServerSideScanMetrics.COUNT_OF_ROWS_SCANNED_KEY_METRIC_NAME, ROWS.length);
231 // Filter should filter out all rows
232 singleColumnValueFilter =
233 new SingleColumnValueFilter(FAMILIES[0], QUALIFIERS[0], CompareOperator.NOT_EQUAL, VALUE);
234 scan = new Scan(baseScan);
235 scan.setFilter(singleColumnValueFilter);
236 testMetric(scan, ServerSideScanMetrics.COUNT_OF_ROWS_SCANNED_KEY_METRIC_NAME, ROWS.length);
239 @Test
240 public void testRowsFilteredMetric() throws Exception {
241 // Base scan configuration
242 Scan baseScan;
243 baseScan = new Scan();
244 baseScan.setScanMetricsEnabled(true);
246 // Test case where scan uses default values
247 testRowsFilteredMetric(baseScan);
249 // Test case where at most one Result is retrieved per RPC
250 baseScan.setCaching(1);
251 testRowsFilteredMetric(baseScan);
253 // Test case where size limit is very restrictive and partial results will be returned from
254 // server
255 baseScan.setMaxResultSize(1);
256 testRowsFilteredMetric(baseScan);
258 // Test a case where max result size limits response from server to only a few cells (not all
259 // cells from the row)
260 baseScan.setCaching(NUM_ROWS);
261 baseScan.setMaxResultSize(getCellHeapSize() * (NUM_COLS - 1));
262 testRowsSeenMetric(baseScan);
265 private void testRowsFilteredMetric(Scan baseScan) throws Exception {
266 testRowsFilteredMetric(baseScan, null, 0);
268 // Row filter doesn't match any row key. All rows should be filtered
269 Filter filter =
270 new RowFilter(CompareOperator.EQUAL, new BinaryComparator(Bytes.toBytes("xyz")));
271 testRowsFilteredMetric(baseScan, filter, ROWS.length);
273 // Filter will return results containing only the first key. Number of entire rows filtered
274 // should be 0.
275 filter = new FirstKeyOnlyFilter();
276 testRowsFilteredMetric(baseScan, filter, 0);
278 // Column prefix will find some matching qualifier on each row. Number of entire rows filtered
279 // should be 0
280 filter = new ColumnPrefixFilter(QUALIFIERS[0]);
281 testRowsFilteredMetric(baseScan, filter, 0);
283 // Column prefix will NOT find any matching qualifier on any row. All rows should be filtered
284 filter = new ColumnPrefixFilter(Bytes.toBytes("xyz"));
285 testRowsFilteredMetric(baseScan, filter, ROWS.length);
287 // Matching column value should exist in each row. No rows should be filtered.
288 filter = new SingleColumnValueFilter(FAMILIES[0], QUALIFIERS[0], CompareOperator.EQUAL, VALUE);
289 testRowsFilteredMetric(baseScan, filter, 0);
291 // No matching column value should exist in any row. Filter all rows
292 filter = new SingleColumnValueFilter(FAMILIES[0], QUALIFIERS[0],
293 CompareOperator.NOT_EQUAL, VALUE);
294 testRowsFilteredMetric(baseScan, filter, ROWS.length);
296 List<Filter> filters = new ArrayList<>();
297 filters.add(new RowFilter(CompareOperator.EQUAL, new BinaryComparator(ROWS[0])));
298 filters.add(new RowFilter(CompareOperator.EQUAL, new BinaryComparator(ROWS[3])));
299 int numberOfMatchingRowFilters = filters.size();
300 filter = new FilterList(Operator.MUST_PASS_ONE, filters);
301 testRowsFilteredMetric(baseScan, filter, ROWS.length - numberOfMatchingRowFilters);
302 filters.clear();
304 // Add a single column value exclude filter for each column... The net effect is that all
305 // columns will be excluded when scanning on the server side. This will result in an empty cell
306 // array in RegionScanner#nextInternal which should be interpreted as a row being filtered.
307 for (int family = 0; family < FAMILIES.length; family++) {
308 for (int qualifier = 0; qualifier < QUALIFIERS.length; qualifier++) {
309 filters.add(new SingleColumnValueExcludeFilter(FAMILIES[family], QUALIFIERS[qualifier],
310 CompareOperator.EQUAL, VALUE));
313 filter = new FilterList(Operator.MUST_PASS_ONE, filters);
314 testRowsFilteredMetric(baseScan, filter, ROWS.length);
317 private void testRowsFilteredMetric(Scan baseScan, Filter filter, int expectedNumFiltered)
318 throws Exception {
319 Scan scan = new Scan(baseScan);
320 if (filter != null) {
321 scan.setFilter(filter);
323 testMetric(scan, ServerSideScanMetrics.COUNT_OF_ROWS_FILTERED_KEY_METRIC_NAME,
324 expectedNumFiltered);
328 * Run the scan to completetion and check the metric against the specified value
329 * @param scan The scan instance to use to record metrics
330 * @param metricKey The metric key name
331 * @param expectedValue The expected value of metric
332 * @throws Exception on unexpected failure
334 private void testMetric(Scan scan, String metricKey, long expectedValue) throws Exception {
335 assertTrue("Scan should be configured to record metrics", scan.isScanMetricsEnabled());
336 ResultScanner scanner = TABLE.getScanner(scan);
337 // Iterate through all the results
338 while (scanner.next() != null) {
339 continue;
341 scanner.close();
342 ScanMetrics metrics = scanner.getScanMetrics();
343 assertTrue("Metrics are null", metrics != null);
344 assertTrue("Metric : " + metricKey + " does not exist", metrics.hasCounter(metricKey));
345 final long actualMetricValue = metrics.getCounter(metricKey).get();
346 assertEquals(
347 "Metric: " + metricKey + " Expected: " + expectedValue + " Actual: " + actualMetricValue,
348 expectedValue, actualMetricValue);