You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@hbase.apache.org by zh...@apache.org on 2023/05/04 14:23:55 UTC

[hbase] branch branch-2.5 updated: HBASE-27821 Split TestFuzzyRowFilterEndToEnd (#5209)

This is an automated email from the ASF dual-hosted git repository.

zhangduo pushed a commit to branch branch-2.5
in repository https://gitbox.apache.org/repos/asf/hbase.git


The following commit(s) were added to refs/heads/branch-2.5 by this push:
     new 3088b2d86e5 HBASE-27821 Split TestFuzzyRowFilterEndToEnd (#5209)
3088b2d86e5 is described below

commit 3088b2d86e5a4f5902a2727f6269bbdbf0bf31a9
Author: Duo Zhang <zh...@apache.org>
AuthorDate: Thu May 4 20:58:20 2023 +0800

    HBASE-27821 Split TestFuzzyRowFilterEndToEnd (#5209)
    
    Signed-off-by: Liangjun He <he...@apache.org>
    (cherry picked from commit 4e69921a001decc73a3fb0bc0d7e94789b440afb)
---
 .../hbase/filter/TestFuzzyRowFilterEndToEnd.java   | 182 +----------------
 .../filter/TestFuzzyRowFilterEndToEndLarge.java    | 223 +++++++++++++++++++++
 2 files changed, 228 insertions(+), 177 deletions(-)

diff --git a/hbase-server/src/test/java/org/apache/hadoop/hbase/filter/TestFuzzyRowFilterEndToEnd.java b/hbase-server/src/test/java/org/apache/hadoop/hbase/filter/TestFuzzyRowFilterEndToEnd.java
index d3c9cfe3eca..7cf76433a0d 100644
--- a/hbase-server/src/test/java/org/apache/hadoop/hbase/filter/TestFuzzyRowFilterEndToEnd.java
+++ b/hbase-server/src/test/java/org/apache/hadoop/hbase/filter/TestFuzzyRowFilterEndToEnd.java
@@ -41,16 +41,12 @@ import org.apache.hadoop.hbase.client.Scan;
 import org.apache.hadoop.hbase.client.Table;
 import org.apache.hadoop.hbase.filter.FilterList.Operator;
 import org.apache.hadoop.hbase.regionserver.ConstantSizeRegionSplitPolicy;
-import org.apache.hadoop.hbase.regionserver.HRegion;
-import org.apache.hadoop.hbase.regionserver.RegionScanner;
 import org.apache.hadoop.hbase.testclassification.FilterTests;
-import org.apache.hadoop.hbase.testclassification.LargeTests;
+import org.apache.hadoop.hbase.testclassification.MediumTests;
 import org.apache.hadoop.hbase.util.Bytes;
 import org.apache.hadoop.hbase.util.EnvironmentEdgeManager;
 import org.apache.hadoop.hbase.util.Pair;
-import org.junit.After;
 import org.junit.AfterClass;
-import org.junit.Before;
 import org.junit.BeforeClass;
 import org.junit.ClassRule;
 import org.junit.Rule;
@@ -62,31 +58,22 @@ import org.slf4j.LoggerFactory;
 
 import org.apache.hbase.thirdparty.com.google.common.collect.Lists;
 
-@Category({ FilterTests.class, LargeTests.class })
+@Category({ FilterTests.class, MediumTests.class })
 public class TestFuzzyRowFilterEndToEnd {
 
   @ClassRule
   public static final HBaseClassTestRule CLASS_RULE =
     HBaseClassTestRule.forClass(TestFuzzyRowFilterEndToEnd.class);
 
-  private final static HBaseTestingUtility TEST_UTIL = new HBaseTestingUtility();
-  private final static byte fuzzyValue = (byte) 63;
   private static final Logger LOG = LoggerFactory.getLogger(TestFuzzyRowFilterEndToEnd.class);
 
-  private static int firstPartCardinality = 50;
-  private static int secondPartCardinality = 50;
-  private static int thirdPartCardinality = 50;
-  private static int colQualifiersTotal = 5;
-  private static int totalFuzzyKeys = thirdPartCardinality / 2;
+  private static final HBaseTestingUtility TEST_UTIL = new HBaseTestingUtility();
 
-  private static String table = "TestFuzzyRowFilterEndToEnd";
+  private static final byte fuzzyValue = (byte) 63;
 
   @Rule
   public TestName name = new TestName();
 
-  /**
-   * @throws java.lang.Exception
-   */
   @BeforeClass
   public static void setUpBeforeClass() throws Exception {
     Configuration conf = TEST_UTIL.getConfiguration();
@@ -99,30 +86,11 @@ public class TestFuzzyRowFilterEndToEnd {
     TEST_UTIL.startMiniCluster();
   }
 
-  /**
-   * @throws java.lang.Exception
-   */
   @AfterClass
   public static void tearDownAfterClass() throws Exception {
     TEST_UTIL.shutdownMiniCluster();
   }
 
-  /**
-   * @throws java.lang.Exception
-   */
-  @Before
-  public void setUp() throws Exception {
-    // Nothing to do.
-  }
-
-  /**
-   * @throws java.lang.Exception
-   */
-  @After
-  public void tearDown() throws Exception {
-    // Nothing to do.
-  }
-
   // HBASE-15676 Test that fuzzy info of all fixed bits (0s) finds matching row.
   @Test
   public void testAllFixedBits() throws IOException {
@@ -149,7 +117,7 @@ public class TestFuzzyRowFilterEndToEnd {
     testAllFixedBitsRunScanWithMask(ht, rows.length, FuzzyRowFilter.V1_PROCESSED_WILDCARD_MASK);
     testAllFixedBitsRunScanWithMask(ht, 2, FuzzyRowFilter.V2_PROCESSED_WILDCARD_MASK);
 
-    TEST_UTIL.deleteTable(TableName.valueOf(table));
+    TEST_UTIL.deleteTable(TableName.valueOf(name.getMethodName()));
   }
 
   private void testAllFixedBitsRunScanWithMask(Table ht, int expectedRows, byte processedRowMask)
@@ -232,146 +200,6 @@ public class TestFuzzyRowFilterEndToEnd {
     assertEquals(expectedRows, total);
   }
 
-  @Test
-  public void testEndToEnd() throws Exception {
-    String cf = "f";
-
-    Table ht =
-      TEST_UTIL.createTable(TableName.valueOf(table), Bytes.toBytes(cf), Integer.MAX_VALUE);
-
-    // 10 byte row key - (2 bytes 4 bytes 4 bytes)
-    // 4 byte qualifier
-    // 4 byte value
-
-    for (int i0 = 0; i0 < firstPartCardinality; i0++) {
-
-      for (int i1 = 0; i1 < secondPartCardinality; i1++) {
-
-        for (int i2 = 0; i2 < thirdPartCardinality; i2++) {
-          byte[] rk = new byte[10];
-
-          ByteBuffer buf = ByteBuffer.wrap(rk);
-          buf.clear();
-          buf.putShort((short) i0);
-          buf.putInt(i1);
-          buf.putInt(i2);
-          for (int c = 0; c < colQualifiersTotal; c++) {
-            byte[] cq = new byte[4];
-            Bytes.putBytes(cq, 0, Bytes.toBytes(c), 0, 4);
-
-            Put p = new Put(rk);
-            p.setDurability(Durability.SKIP_WAL);
-            p.addColumn(cf.getBytes(), cq, Bytes.toBytes(c));
-            ht.put(p);
-          }
-        }
-      }
-    }
-
-    TEST_UTIL.flush();
-
-    // test passes
-    runTest1(ht, FuzzyRowFilter.V1_PROCESSED_WILDCARD_MASK);
-    runTest1(ht, FuzzyRowFilter.V2_PROCESSED_WILDCARD_MASK);
-    runTest2(ht, FuzzyRowFilter.V1_PROCESSED_WILDCARD_MASK);
-    runTest2(ht, FuzzyRowFilter.V2_PROCESSED_WILDCARD_MASK);
-
-  }
-
-  private void runTest1(Table hTable, byte processedWildcardMask) throws IOException {
-    // [0, 2, ?, ?, ?, ?, 0, 0, 0, 1]
-
-    byte[] mask = new byte[] { 0, 0, 1, 1, 1, 1, 0, 0, 0, 0 };
-
-    List<Pair<byte[], byte[]>> list = new ArrayList<>();
-    for (int i = 0; i < totalFuzzyKeys; i++) {
-      byte[] fuzzyKey = new byte[10];
-      ByteBuffer buf = ByteBuffer.wrap(fuzzyKey);
-      buf.clear();
-      buf.putShort((short) 2);
-      for (int j = 0; j < 4; j++) {
-        buf.put(fuzzyValue);
-      }
-      buf.putInt(i);
-
-      Pair<byte[], byte[]> pair = new Pair<>(fuzzyKey, mask);
-      list.add(pair);
-    }
-
-    int expectedSize = secondPartCardinality * totalFuzzyKeys * colQualifiersTotal;
-    FuzzyRowFilter fuzzyRowFilter0 = new FuzzyRowFilter(list, processedWildcardMask);
-    // Filters are not stateless - we can't reuse them
-    FuzzyRowFilter fuzzyRowFilter1 = new FuzzyRowFilter(list, processedWildcardMask);
-
-    // regular test
-    runScanner(hTable, expectedSize, fuzzyRowFilter0);
-    // optimized from block cache
-    runScanner(hTable, expectedSize, fuzzyRowFilter1);
-
-  }
-
-  private void runTest2(Table hTable, byte processedWildcardMask) throws IOException {
-    // [0, 0, ?, ?, ?, ?, 0, 0, 0, 0] , [0, 1, ?, ?, ?, ?, 0, 0, 0, 1]...
-
-    byte[] mask = new byte[] { 0, 0, 1, 1, 1, 1, 0, 0, 0, 0 };
-
-    List<Pair<byte[], byte[]>> list = new ArrayList<>();
-
-    for (int i = 0; i < totalFuzzyKeys; i++) {
-      byte[] fuzzyKey = new byte[10];
-      ByteBuffer buf = ByteBuffer.wrap(fuzzyKey);
-      buf.clear();
-      buf.putShort((short) (i * 2));
-      for (int j = 0; j < 4; j++) {
-        buf.put(fuzzyValue);
-      }
-      buf.putInt(i * 2);
-
-      Pair<byte[], byte[]> pair = new Pair<>(fuzzyKey, mask);
-      list.add(pair);
-    }
-
-    int expectedSize = totalFuzzyKeys * secondPartCardinality * colQualifiersTotal;
-
-    FuzzyRowFilter fuzzyRowFilter0 = new FuzzyRowFilter(list, processedWildcardMask);
-    // Filters are not stateless - we can't reuse them
-    FuzzyRowFilter fuzzyRowFilter1 = new FuzzyRowFilter(list, processedWildcardMask);
-
-    // regular test
-    runScanner(hTable, expectedSize, fuzzyRowFilter0);
-    // optimized from block cache
-    runScanner(hTable, expectedSize, fuzzyRowFilter1);
-
-  }
-
-  private void runScanner(Table hTable, int expectedSize, Filter filter) throws IOException {
-
-    String cf = "f";
-    Scan scan = new Scan();
-    scan.addFamily(cf.getBytes());
-    scan.setFilter(filter);
-    List<HRegion> regions = TEST_UTIL.getHBaseCluster().getRegions(table.getBytes());
-    HRegion first = regions.get(0);
-    first.getScanner(scan);
-    RegionScanner scanner = first.getScanner(scan);
-    List<Cell> results = new ArrayList<>();
-    // Result result;
-    long timeBeforeScan = EnvironmentEdgeManager.currentTime();
-    int found = 0;
-    while (scanner.next(results)) {
-      found += results.size();
-      results.clear();
-    }
-    found += results.size();
-    long scanTime = EnvironmentEdgeManager.currentTime() - timeBeforeScan;
-    scanner.close();
-
-    LOG.info("\nscan time = " + scanTime + "ms");
-    LOG.info("found " + found + " results\n");
-
-    assertEquals(expectedSize, found);
-  }
-
   @Test
   public void testFilterList() throws Exception {
     String cf = "f";
diff --git a/hbase-server/src/test/java/org/apache/hadoop/hbase/filter/TestFuzzyRowFilterEndToEndLarge.java b/hbase-server/src/test/java/org/apache/hadoop/hbase/filter/TestFuzzyRowFilterEndToEndLarge.java
new file mode 100644
index 00000000000..a8c7b7c9151
--- /dev/null
+++ b/hbase-server/src/test/java/org/apache/hadoop/hbase/filter/TestFuzzyRowFilterEndToEndLarge.java
@@ -0,0 +1,223 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.apache.hadoop.hbase.filter;
+
+import static org.junit.Assert.assertEquals;
+
+import java.io.IOException;
+import java.nio.ByteBuffer;
+import java.util.ArrayList;
+import java.util.List;
+import org.apache.hadoop.conf.Configuration;
+import org.apache.hadoop.hbase.Cell;
+import org.apache.hadoop.hbase.HBaseClassTestRule;
+import org.apache.hadoop.hbase.HBaseTestingUtility;
+import org.apache.hadoop.hbase.HConstants;
+import org.apache.hadoop.hbase.TableName;
+import org.apache.hadoop.hbase.client.Durability;
+import org.apache.hadoop.hbase.client.Put;
+import org.apache.hadoop.hbase.client.Scan;
+import org.apache.hadoop.hbase.client.Table;
+import org.apache.hadoop.hbase.regionserver.ConstantSizeRegionSplitPolicy;
+import org.apache.hadoop.hbase.regionserver.HRegion;
+import org.apache.hadoop.hbase.regionserver.RegionScanner;
+import org.apache.hadoop.hbase.testclassification.FilterTests;
+import org.apache.hadoop.hbase.testclassification.LargeTests;
+import org.apache.hadoop.hbase.util.Bytes;
+import org.apache.hadoop.hbase.util.EnvironmentEdgeManager;
+import org.apache.hadoop.hbase.util.Pair;
+import org.junit.AfterClass;
+import org.junit.BeforeClass;
+import org.junit.ClassRule;
+import org.junit.Test;
+import org.junit.experimental.categories.Category;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
+@Category({ FilterTests.class, LargeTests.class })
+public class TestFuzzyRowFilterEndToEndLarge {
+
+  @ClassRule
+  public static final HBaseClassTestRule CLASS_RULE =
+    HBaseClassTestRule.forClass(TestFuzzyRowFilterEndToEndLarge.class);
+
+  private static final Logger LOG = LoggerFactory.getLogger(TestFuzzyRowFilterEndToEndLarge.class);
+
+  private final static HBaseTestingUtility TEST_UTIL = new HBaseTestingUtility();
+
+  private final static byte fuzzyValue = (byte) 63;
+
+  private static int firstPartCardinality = 50;
+  private static int secondPartCardinality = 50;
+  private static int thirdPartCardinality = 50;
+  private static int colQualifiersTotal = 5;
+  private static int totalFuzzyKeys = thirdPartCardinality / 2;
+
+  private static String table = "TestFuzzyRowFilterEndToEndLarge";
+
+  @BeforeClass
+  public static void setUpBeforeClass() throws Exception {
+    Configuration conf = TEST_UTIL.getConfiguration();
+    conf.setInt("hbase.client.scanner.caching", 1000);
+    conf.set(HConstants.HBASE_REGION_SPLIT_POLICY_KEY,
+      ConstantSizeRegionSplitPolicy.class.getName());
+    // set no splits
+    conf.setLong(HConstants.HREGION_MAX_FILESIZE, (1024L) * 1024 * 1024 * 10);
+
+    TEST_UTIL.startMiniCluster();
+  }
+
+  @AfterClass
+  public static void tearDownAfterClass() throws Exception {
+    TEST_UTIL.shutdownMiniCluster();
+  }
+
+  @Test
+  public void testEndToEnd() throws Exception {
+    String cf = "f";
+
+    Table ht =
+      TEST_UTIL.createTable(TableName.valueOf(table), Bytes.toBytes(cf), Integer.MAX_VALUE);
+
+    // 10 byte row key - (2 bytes 4 bytes 4 bytes)
+    // 4 byte qualifier
+    // 4 byte value
+
+    for (int i0 = 0; i0 < firstPartCardinality; i0++) {
+      for (int i1 = 0; i1 < secondPartCardinality; i1++) {
+        for (int i2 = 0; i2 < thirdPartCardinality; i2++) {
+          byte[] rk = new byte[10];
+
+          ByteBuffer buf = ByteBuffer.wrap(rk);
+          buf.clear();
+          buf.putShort((short) i0);
+          buf.putInt(i1);
+          buf.putInt(i2);
+          for (int c = 0; c < colQualifiersTotal; c++) {
+            byte[] cq = new byte[4];
+            Bytes.putBytes(cq, 0, Bytes.toBytes(c), 0, 4);
+
+            Put p = new Put(rk);
+            p.setDurability(Durability.SKIP_WAL);
+            p.addColumn(cf.getBytes(), cq, Bytes.toBytes(c));
+            ht.put(p);
+          }
+        }
+      }
+    }
+
+    TEST_UTIL.flush();
+
+    // test passes
+    runTest1(ht, FuzzyRowFilter.V1_PROCESSED_WILDCARD_MASK);
+    runTest1(ht, FuzzyRowFilter.V2_PROCESSED_WILDCARD_MASK);
+    runTest2(ht, FuzzyRowFilter.V1_PROCESSED_WILDCARD_MASK);
+    runTest2(ht, FuzzyRowFilter.V2_PROCESSED_WILDCARD_MASK);
+  }
+
+  private void runTest1(Table hTable, byte processedWildcardMask) throws IOException {
+    // [0, 2, ?, ?, ?, ?, 0, 0, 0, 1]
+    byte[] mask = new byte[] { 0, 0, 1, 1, 1, 1, 0, 0, 0, 0 };
+
+    List<Pair<byte[], byte[]>> list = new ArrayList<>();
+    for (int i = 0; i < totalFuzzyKeys; i++) {
+      byte[] fuzzyKey = new byte[10];
+      ByteBuffer buf = ByteBuffer.wrap(fuzzyKey);
+      buf.clear();
+      buf.putShort((short) 2);
+      for (int j = 0; j < 4; j++) {
+        buf.put(fuzzyValue);
+      }
+      buf.putInt(i);
+
+      Pair<byte[], byte[]> pair = new Pair<>(fuzzyKey, mask);
+      list.add(pair);
+    }
+
+    int expectedSize = secondPartCardinality * totalFuzzyKeys * colQualifiersTotal;
+    FuzzyRowFilter fuzzyRowFilter0 = new FuzzyRowFilter(list, processedWildcardMask);
+    // Filters are not stateless - we can't reuse them
+    FuzzyRowFilter fuzzyRowFilter1 = new FuzzyRowFilter(list, processedWildcardMask);
+
+    // regular test
+    runScanner(hTable, expectedSize, fuzzyRowFilter0);
+    // optimized from block cache
+    runScanner(hTable, expectedSize, fuzzyRowFilter1);
+
+  }
+
+  private void runTest2(Table hTable, byte processedWildcardMask) throws IOException {
+    // [0, 0, ?, ?, ?, ?, 0, 0, 0, 0] , [0, 1, ?, ?, ?, ?, 0, 0, 0, 1]...
+    byte[] mask = new byte[] { 0, 0, 1, 1, 1, 1, 0, 0, 0, 0 };
+
+    List<Pair<byte[], byte[]>> list = new ArrayList<>();
+
+    for (int i = 0; i < totalFuzzyKeys; i++) {
+      byte[] fuzzyKey = new byte[10];
+      ByteBuffer buf = ByteBuffer.wrap(fuzzyKey);
+      buf.clear();
+      buf.putShort((short) (i * 2));
+      for (int j = 0; j < 4; j++) {
+        buf.put(fuzzyValue);
+      }
+      buf.putInt(i * 2);
+
+      Pair<byte[], byte[]> pair = new Pair<>(fuzzyKey, mask);
+      list.add(pair);
+    }
+
+    int expectedSize = totalFuzzyKeys * secondPartCardinality * colQualifiersTotal;
+
+    FuzzyRowFilter fuzzyRowFilter0 = new FuzzyRowFilter(list, processedWildcardMask);
+    // Filters are not stateless - we can't reuse them
+    FuzzyRowFilter fuzzyRowFilter1 = new FuzzyRowFilter(list, processedWildcardMask);
+
+    // regular test
+    runScanner(hTable, expectedSize, fuzzyRowFilter0);
+    // optimized from block cache
+    runScanner(hTable, expectedSize, fuzzyRowFilter1);
+
+  }
+
+  private void runScanner(Table hTable, int expectedSize, Filter filter) throws IOException {
+    String cf = "f";
+    Scan scan = new Scan();
+    scan.addFamily(cf.getBytes());
+    scan.setFilter(filter);
+    List<HRegion> regions = TEST_UTIL.getHBaseCluster().getRegions(table.getBytes());
+    HRegion first = regions.get(0);
+    first.getScanner(scan);
+    RegionScanner scanner = first.getScanner(scan);
+    List<Cell> results = new ArrayList<>();
+    // Result result;
+    long timeBeforeScan = EnvironmentEdgeManager.currentTime();
+    int found = 0;
+    while (scanner.next(results)) {
+      found += results.size();
+      results.clear();
+    }
+    found += results.size();
+    long scanTime = EnvironmentEdgeManager.currentTime() - timeBeforeScan;
+    scanner.close();
+
+    LOG.info("\nscan time = " + scanTime + "ms");
+    LOG.info("found " + found + " results\n");
+
+    assertEquals(expectedSize, found);
+  }
+}