You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@pinot.apache.org by ja...@apache.org on 2022/03/21 22:07:28 UTC

[pinot] branch master updated: Group by order by interning (#8376)

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

jackie pushed a commit to branch master
in repository https://gitbox.apache.org/repos/asf/pinot.git


The following commit(s) were added to refs/heads/master by this push:
     new f97a23f  Group by order by interning (#8376)
f97a23f is described below

commit f97a23fb76ec529c3ade23dc80a914b58ba1fe34
Author: Richard Startin <ri...@startree.ai>
AuthorDate: Mon Mar 21 22:07:04 2022 +0000

    Group by order by interning (#8376)
    
    This caches raw values extracted from dictionaries in ND group-by/order-by queries where N > 1 for low cardinality columns. This is an effective optimization because if the cardinality of dimension 1 is x and the cardinality of dimension 2 is y then the dictionary ids of dimension 1 will be decoded up to y times, and those of dimension 2 up to x times. The size of the cache is limited to 10k elements per dimension per group by query, which is similar to what gets allocated elsewhere i [...]
---
 .../groupby/DictionaryBasedGroupKeyGenerator.java  | 30 +++++++++++++++++++---
 .../org/apache/pinot/perf/BenchmarkQueries.java    | 16 +++++++++++-
 2 files changed, 42 insertions(+), 4 deletions(-)

diff --git a/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/groupby/DictionaryBasedGroupKeyGenerator.java b/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/groupby/DictionaryBasedGroupKeyGenerator.java
index f3e868b..e0cb09e 100644
--- a/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/groupby/DictionaryBasedGroupKeyGenerator.java
+++ b/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/groupby/DictionaryBasedGroupKeyGenerator.java
@@ -63,6 +63,7 @@ public class DictionaryBasedGroupKeyGenerator implements GroupKeyGenerator {
   // NOTE: map size = map capacity (power of 2) * load factor
   private static final int INITIAL_MAP_SIZE = (int) ((1 << 9) * 0.75f);
   private static final int MAX_CACHING_MAP_SIZE = (int) ((1 << 20) * 0.75f);
+  private static final int MAX_DICTIONARY_INTERN_TABLE_SIZE = 10000;
 
   @VisibleForTesting
   static final ThreadLocal<IntGroupIdMap> THREAD_LOCAL_INT_MAP = ThreadLocal.withInitial(IntGroupIdMap::new);
@@ -91,6 +92,8 @@ public class DictionaryBasedGroupKeyGenerator implements GroupKeyGenerator {
   // Reusable buffer for multi-value column dictionary ids
   private final int[][][] _multiValueDictIds;
 
+  private final Object[][] _internedDictionaryValues;
+
   private final int _globalGroupIdUpperBound;
   private final RawKeyHolder _rawKeyHolder;
 
@@ -106,6 +109,9 @@ public class DictionaryBasedGroupKeyGenerator implements GroupKeyGenerator {
     _dictionaries = new Dictionary[_numGroupByExpressions];
     _singleValueDictIds = new int[_numGroupByExpressions][];
     _multiValueDictIds = new int[_numGroupByExpressions][][];
+    // no need to intern dictionary values when there is only one group by expression because
+    // only one call will be made to the dictionary to extract each raw value.
+    _internedDictionaryValues = _numGroupByExpressions > 1 ? new Object[_numGroupByExpressions][] : null;
 
     long cardinalityProduct = 1L;
     boolean longOverflow = false;
@@ -114,6 +120,9 @@ public class DictionaryBasedGroupKeyGenerator implements GroupKeyGenerator {
       _dictionaries[i] = transformOperator.getDictionary(groupByExpression);
       int cardinality = _dictionaries[i].length();
       _cardinalities[i] = cardinality;
+      if (_internedDictionaryValues != null && cardinality < MAX_DICTIONARY_INTERN_TABLE_SIZE) {
+        _internedDictionaryValues[i] = new Object[cardinality];
+      }
       if (!longOverflow) {
         if (cardinalityProduct > Long.MAX_VALUE / cardinality) {
           longOverflow = true;
@@ -613,13 +622,28 @@ public class DictionaryBasedGroupKeyGenerator implements GroupKeyGenerator {
       Object[] groupKeys = new Object[_numGroupByExpressions];
       for (int i = 0; i < _numGroupByExpressions; i++) {
         int cardinality = _cardinalities[i];
-        groupKeys[i] = _dictionaries[i].getInternal(rawKey % cardinality);
+        groupKeys[i] = getRawValue(i, rawKey % cardinality);
         rawKey /= cardinality;
       }
       return groupKeys;
     }
   }
 
+  private Object getRawValue(int dictionaryIndex, int dictId) {
+    Dictionary dictionary = _dictionaries[dictionaryIndex];
+    Object[] table = _internedDictionaryValues[dictionaryIndex];
+    if (table == null) {
+      // high cardinality dictionary values aren't interned
+      return dictionary.getInternal(dictId);
+    }
+    Object rawValue = table[dictId];
+    if (rawValue == null) {
+      rawValue = dictionary.getInternal(dictId);
+      table[dictId] = rawValue;
+    }
+    return rawValue;
+  }
+
   /**
    * Helper method to get the string key from the raw key.
    */
@@ -825,7 +849,7 @@ public class DictionaryBasedGroupKeyGenerator implements GroupKeyGenerator {
     Object[] groupKeys = new Object[_numGroupByExpressions];
     for (int i = 0; i < _numGroupByExpressions; i++) {
       int cardinality = _cardinalities[i];
-      groupKeys[i] = _dictionaries[i].getInternal((int) (rawKey % cardinality));
+      groupKeys[i] = getRawValue(i, (int) (rawKey % cardinality));
       rawKey /= cardinality;
     }
     return groupKeys;
@@ -1035,7 +1059,7 @@ public class DictionaryBasedGroupKeyGenerator implements GroupKeyGenerator {
   private Object[] getKeys(IntArray rawKey) {
     Object[] groupKeys = new Object[_numGroupByExpressions];
     for (int i = 0; i < _numGroupByExpressions; i++) {
-      groupKeys[i] = _dictionaries[i].getInternal(rawKey._elements[i]);
+      groupKeys[i] = getRawValue(i, rawKey._elements[i]);
     }
     return groupKeys;
   }
diff --git a/pinot-perf/src/main/java/org/apache/pinot/perf/BenchmarkQueries.java b/pinot-perf/src/main/java/org/apache/pinot/perf/BenchmarkQueries.java
index 7d23c42..597a18b 100644
--- a/pinot-perf/src/main/java/org/apache/pinot/perf/BenchmarkQueries.java
+++ b/pinot-perf/src/main/java/org/apache/pinot/perf/BenchmarkQueries.java
@@ -30,6 +30,7 @@ import java.util.Set;
 import java.util.UUID;
 import java.util.concurrent.TimeUnit;
 import java.util.function.LongSupplier;
+import java.util.stream.IntStream;
 import org.apache.commons.io.FileUtils;
 import org.apache.pinot.common.response.broker.BrokerResponseNative;
 import org.apache.pinot.queries.BaseQueriesTest;
@@ -88,6 +89,7 @@ public class BenchmarkQueries extends BaseQueriesTest {
   private static final String RAW_STRING_COL_NAME = "RAW_STRING_COL";
   private static final String NO_INDEX_INT_COL_NAME = "NO_INDEX_INT_COL";
   private static final String NO_INDEX_STRING_COL = "NO_INDEX_STRING_COL";
+  private static final String LOW_CARDINALITY_STRING_COL = "LOW_CARDINALITY_STRING_COL";
 
   public static final String FILTERED_QUERY = "SELECT SUM(INT_COL) FILTER(WHERE INT_COL > 123 AND INT_COL < 599999),"
       + "MAX(INT_COL) FILTER(WHERE INT_COL > 123 AND INT_COL < 599999) "
@@ -116,13 +118,21 @@ public class BenchmarkQueries extends BaseQueriesTest {
   public static final String NO_INDEX_LIKE_QUERY = "SELECT RAW_INT_COL FROM MyTable "
       + "WHERE NO_INDEX_STRING_COL LIKE '%foo%'";
 
+  public static final String MULTI_GROUP_BY_ORDER_BY = "SELECT NO_INDEX_STRING_COL,INT_COL,COUNT(*) "
+      + "FROM MyTable GROUP BY NO_INDEX_STRING_COL,INT_COL ORDER BY NO_INDEX_STRING_COL, INT_COL ASC";
+
+  public static final String MULTI_GROUP_BY_ORDER_BY_LOW_HIGH = "SELECT "
+      + "NO_INDEX_STRING_COL,LOW_CARDINALITY_STRING_COL,COUNT(*) "
+      + "FROM MyTable GROUP BY LOW_CARDINALITY_STRING_COL,NO_INDEX_STRING_COL "
+      + "ORDER BY LOW_CARDINALITY_STRING_COL, NO_INDEX_STRING_COL ASC";
+
   @Param("1500000")
   private int _numRows;
   @Param({"EXP(0.001)", "EXP(0.5)", "EXP(0.999)"})
   String _scenario;
   @Param({
       MULTI_GROUP_BY_WITH_RAW_QUERY, MULTI_GROUP_BY_WITH_RAW_QUERY_2, FILTERED_QUERY, NON_FILTERED_QUERY,
-      SUM_QUERY, NO_INDEX_LIKE_QUERY
+      SUM_QUERY, NO_INDEX_LIKE_QUERY, MULTI_GROUP_BY_ORDER_BY, MULTI_GROUP_BY_ORDER_BY_LOW_HIGH
   })
   String _query;
   private IndexSegment _indexSegment;
@@ -166,6 +176,8 @@ public class BenchmarkQueries extends BaseQueriesTest {
   private List<GenericRow> createTestData(int numRows) {
     Map<Integer, String> strings = new HashMap<>();
     List<GenericRow> rows = new ArrayList<>();
+    String[] lowCardinalityValues = IntStream.range(0, 10).mapToObj(i -> UUID.randomUUID().toString())
+        .toArray(String[]::new);
     for (int i = 0; i < numRows; i++) {
       GenericRow row = new GenericRow();
       row.putValue(INT_COL_NAME, (int) _supplier.getAsLong());
@@ -174,6 +186,7 @@ public class BenchmarkQueries extends BaseQueriesTest {
       row.putValue(RAW_STRING_COL_NAME,
           strings.computeIfAbsent((int) _supplier.getAsLong(), k -> UUID.randomUUID().toString()));
       row.putValue(NO_INDEX_STRING_COL, row.getValue(RAW_STRING_COL_NAME));
+      row.putValue(LOW_CARDINALITY_STRING_COL, lowCardinalityValues[i % lowCardinalityValues.length]);
       rows.add(row);
     }
     return rows;
@@ -195,6 +208,7 @@ public class BenchmarkQueries extends BaseQueriesTest {
         .addSingleValueDimension(INT_COL_NAME, FieldSpec.DataType.INT)
         .addSingleValueDimension(RAW_STRING_COL_NAME, FieldSpec.DataType.STRING)
         .addSingleValueDimension(NO_INDEX_STRING_COL, FieldSpec.DataType.STRING)
+        .addSingleValueDimension(LOW_CARDINALITY_STRING_COL, FieldSpec.DataType.STRING)
         .build();
     SegmentGeneratorConfig config = new SegmentGeneratorConfig(tableConfig, schema);
     config.setOutDir(INDEX_DIR.getPath());

---------------------------------------------------------------------
To unsubscribe, e-mail: commits-unsubscribe@pinot.apache.org
For additional commands, e-mail: commits-help@pinot.apache.org