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 2d809ff Fix the missing NOT handling (#8366) 2d809ff is described below commit 2d809ffe33e75ae5d6f8ed2c16ec1442fa1d6adf Author: Xiaotian (Jackie) Jiang <17555551+jackie-ji...@users.noreply.github.com> AuthorDate: Mon Mar 21 16:21:34 2022 -0700 Fix the missing NOT handling (#8366) --- .../common/request/context/FilterContext.java | 15 ++-- .../request/context/RequestContextUtils.java | 3 + .../rewriter/PredicateComparisonRewriter.java | 1 + .../recommender/rules/impl/BloomFilterRule.java | 16 ++-- .../NoDictionaryOnHeapDictionaryJointRule.java | 26 +++---- .../rules/impl/PinotTablePartitionRule.java | 19 +++-- .../recommender/rules/impl/RangeIndexRule.java | 11 ++- .../utils/QueryInvertedSortedIndexRecommender.java | 12 ++- ...istinctCountThetaSketchAggregationFunction.java | 50 ++++++++---- .../optimizer/filter/MergeEqInFilterOptimizer.java | 2 +- .../filter/MergeRangeFilterOptimizer.java | 4 +- .../optimizer/filter/NumericalFilterOptimizer.java | 27 ++++--- .../filter/TimePredicateFilterOptimizer.java | 6 +- .../statement/JsonStatementOptimizer.java | 4 +- .../statement/StringPredicateFilterOptimizer.java | 18 ++--- .../query/pruner/ColumnValueSegmentPruner.java | 6 ++ .../core/query/reduce/filter/AndRowMatcher.java | 4 +- .../query/reduce/filter/LiteralValueExtractor.java | 2 +- .../{ValueExtractor.java => NotRowMatcher.java} | 26 +++---- .../core/query/reduce/filter/OrRowMatcher.java | 2 +- .../query/reduce/filter/PredicateRowMatcher.java | 6 +- .../pinot/core/query/reduce/filter/RowMatcher.java | 1 + .../query/reduce/filter/RowMatcherFactory.java | 3 + .../core/query/reduce/filter/ValueExtractor.java | 1 + .../query/reduce/filter/ValueExtractorFactory.java | 1 + .../apache/pinot/core/startree/StarTreeUtils.java | 12 ++- .../tests/ClusterIntegrationTestUtils.java | 90 +++++++++------------- .../tests/BaseClusterIntegrationTestSet.java | 18 +++++ .../realtime/impl/json/MutableJsonIndexImpl.java | 4 +- .../readers/json/ImmutableJsonIndexReader.java | 4 +- 30 files changed, 221 insertions(+), 173 deletions(-) diff --git a/pinot-common/src/main/java/org/apache/pinot/common/request/context/FilterContext.java b/pinot-common/src/main/java/org/apache/pinot/common/request/context/FilterContext.java index 9ade2a8..aa76809 100644 --- a/pinot-common/src/main/java/org/apache/pinot/common/request/context/FilterContext.java +++ b/pinot-common/src/main/java/org/apache/pinot/common/request/context/FilterContext.java @@ -81,8 +81,8 @@ public class FilterContext { return false; } FilterContext that = (FilterContext) o; - return _type == that._type && Objects.equals(_children, that._children) && Objects - .equals(_predicate, that._predicate); + return _type == that._type && Objects.equals(_children, that._children) && Objects.equals(_predicate, + that._predicate); } @Override @@ -94,19 +94,22 @@ public class FilterContext { public String toString() { switch (_type) { case AND: - StringBuilder stringBuilder = new StringBuilder().append('(').append(_children.get(0).toString()); + StringBuilder stringBuilder = new StringBuilder().append('(').append(_children.get(0)); int numChildren = _children.size(); for (int i = 1; i < numChildren; i++) { - stringBuilder.append(" AND ").append(_children.get(i).toString()); + stringBuilder.append(" AND ").append(_children.get(i)); } return stringBuilder.append(')').toString(); case OR: - stringBuilder = new StringBuilder().append('(').append(_children.get(0).toString()); + stringBuilder = new StringBuilder().append('(').append(_children.get(0)); numChildren = _children.size(); for (int i = 1; i < numChildren; i++) { - stringBuilder.append(" OR ").append(_children.get(i).toString()); + stringBuilder.append(" OR ").append(_children.get(i)); } return stringBuilder.append(')').toString(); + case NOT: + assert _children.size() == 1; + return "(NOT " + _children.get(0) + ')'; case PREDICATE: return _predicate.toString(); default: diff --git a/pinot-common/src/main/java/org/apache/pinot/common/request/context/RequestContextUtils.java b/pinot-common/src/main/java/org/apache/pinot/common/request/context/RequestContextUtils.java index ccc4589..6e022c4 100644 --- a/pinot-common/src/main/java/org/apache/pinot/common/request/context/RequestContextUtils.java +++ b/pinot-common/src/main/java/org/apache/pinot/common/request/context/RequestContextUtils.java @@ -284,6 +284,9 @@ public class RequestContextUtils { children.add(getFilter(operand)); } return new FilterContext(FilterContext.Type.OR, children, null); + case NOT: + assert numOperands == 1; + return new FilterContext(FilterContext.Type.NOT, Collections.singletonList(getFilter(operands.get(0))), null); case EQUALS: return new FilterContext(FilterContext.Type.PREDICATE, null, new EqPredicate(operands.get(0), getStringValue(operands.get(1)))); diff --git a/pinot-common/src/main/java/org/apache/pinot/sql/parsers/rewriter/PredicateComparisonRewriter.java b/pinot-common/src/main/java/org/apache/pinot/sql/parsers/rewriter/PredicateComparisonRewriter.java index caa6535..ae5c8ae 100644 --- a/pinot-common/src/main/java/org/apache/pinot/sql/parsers/rewriter/PredicateComparisonRewriter.java +++ b/pinot-common/src/main/java/org/apache/pinot/sql/parsers/rewriter/PredicateComparisonRewriter.java @@ -59,6 +59,7 @@ public class PredicateComparisonRewriter implements QueryRewriter { switch (filterKind) { case AND: case OR: + case NOT: operands.replaceAll(PredicateComparisonRewriter::updateComparisonPredicate); break; case EQUALS: diff --git a/pinot-controller/src/main/java/org/apache/pinot/controller/recommender/rules/impl/BloomFilterRule.java b/pinot-controller/src/main/java/org/apache/pinot/controller/recommender/rules/impl/BloomFilterRule.java index c6135d4..a2e9f2d 100644 --- a/pinot-controller/src/main/java/org/apache/pinot/controller/recommender/rules/impl/BloomFilterRule.java +++ b/pinot-controller/src/main/java/org/apache/pinot/controller/recommender/rules/impl/BloomFilterRule.java @@ -19,6 +19,7 @@ package org.apache.pinot.controller.recommender.rules.impl; import com.google.common.util.concurrent.AtomicDouble; +import java.util.List; import org.apache.pinot.common.request.context.ExpressionContext; import org.apache.pinot.common.request.context.FilterContext; import org.apache.pinot.common.request.context.predicate.Predicate; @@ -92,19 +93,16 @@ public class BloomFilterRule extends AbstractRule { * @return dimension used in eq in this query */ private FixedLenBitset parsePredicateList(FilterContext filterContext) { - FilterContext.Type type = filterContext.getType(); FixedLenBitset ret = mutableEmptySet(); - if (type == FilterContext.Type.AND) { - for (int i = 0; i < filterContext.getChildren().size(); i++) { - FixedLenBitset childResult = parsePredicateList(filterContext.getChildren().get(i)); - ret.union(childResult); - } - } else if (type == FilterContext.Type.OR) { - for (int i = 0; i < filterContext.getChildren().size(); i++) { - FixedLenBitset childResult = parsePredicateList(filterContext.getChildren().get(i)); + List<FilterContext> children = filterContext.getChildren(); + if (children != null) { + // AND, OR, NOT + for (FilterContext child : children) { + FixedLenBitset childResult = parsePredicateList(child); ret.union(childResult); } } else { + // PREDICATE ExpressionContext lhs = filterContext.getPredicate().getLhs(); String colName = lhs.toString(); if (lhs.getType() == ExpressionContext.Type.FUNCTION) { diff --git a/pinot-controller/src/main/java/org/apache/pinot/controller/recommender/rules/impl/NoDictionaryOnHeapDictionaryJointRule.java b/pinot-controller/src/main/java/org/apache/pinot/controller/recommender/rules/impl/NoDictionaryOnHeapDictionaryJointRule.java index ab1fcee..6b07f56 100644 --- a/pinot-controller/src/main/java/org/apache/pinot/controller/recommender/rules/impl/NoDictionaryOnHeapDictionaryJointRule.java +++ b/pinot-controller/src/main/java/org/apache/pinot/controller/recommender/rules/impl/NoDictionaryOnHeapDictionaryJointRule.java @@ -21,11 +21,11 @@ package org.apache.pinot.controller.recommender.rules.impl; import com.google.common.util.concurrent.AtomicDouble; import java.util.HashMap; import java.util.HashSet; +import java.util.List; import java.util.Map; import java.util.Set; import org.apache.pinot.common.request.context.ExpressionContext; import org.apache.pinot.common.request.context.FilterContext; -import org.apache.pinot.common.request.context.predicate.Predicate; import org.apache.pinot.controller.recommender.exceptions.InvalidInputException; import org.apache.pinot.controller.recommender.io.ConfigManager; import org.apache.pinot.controller.recommender.io.InputManager; @@ -207,25 +207,17 @@ public class NoDictionaryOnHeapDictionaryJointRule extends AbstractRule { } public FixedLenBitset parsePredicateList(FilterContext filterContext) { - FilterContext.Type type = filterContext.getType(); FixedLenBitset ret = mutableEmptySet(); - if (type == FilterContext.Type.AND) { - for (int i = 0; i < filterContext.getChildren().size(); i++) { - FixedLenBitset childResult = parsePredicateList(filterContext.getChildren().get(i)); - if (childResult != null) { - ret.union(childResult); - } - } - } else if (type == FilterContext.Type.OR) { - for (int i = 0; i < filterContext.getChildren().size(); i++) { - FixedLenBitset childResult = parsePredicateList(filterContext.getChildren().get(i)); - if (childResult != null) { - ret.union(childResult); - } + List<FilterContext> children = filterContext.getChildren(); + if (children != null) { + // AND, OR, NOT + for (FilterContext child : children) { + FixedLenBitset childResult = parsePredicateList(child); + ret.union(childResult); } } else { - Predicate predicate = filterContext.getPredicate(); - ExpressionContext lhs = predicate.getLhs(); + // PREDICATE + ExpressionContext lhs = filterContext.getPredicate().getLhs(); String colName = lhs.toString(); if (lhs.getType() == ExpressionContext.Type.FUNCTION || _input.isTimeOrDateTimeColumn(colName)) { LOGGER.trace("Skipping this column {}", colName); diff --git a/pinot-controller/src/main/java/org/apache/pinot/controller/recommender/rules/impl/PinotTablePartitionRule.java b/pinot-controller/src/main/java/org/apache/pinot/controller/recommender/rules/impl/PinotTablePartitionRule.java index 7bc21cf..ca879d7 100644 --- a/pinot-controller/src/main/java/org/apache/pinot/controller/recommender/rules/impl/PinotTablePartitionRule.java +++ b/pinot-controller/src/main/java/org/apache/pinot/controller/recommender/rules/impl/PinotTablePartitionRule.java @@ -24,6 +24,7 @@ import java.util.Comparator; import java.util.List; import java.util.Optional; import java.util.function.Function; +import javax.annotation.Nullable; import org.apache.commons.lang3.tuple.Pair; import org.apache.pinot.common.request.context.ExpressionContext; import org.apache.pinot.common.request.context.FilterContext; @@ -169,7 +170,7 @@ public class PinotTablePartitionRule extends AbstractRule { static Optional<String> findBestColumnForPartitioning(List<Pair<String, Double>> columnNameToWeightPairs, Function<String, Double> cardinalityExtractor, double topCandidateRatio, int numPartitions) { return columnNameToWeightPairs.stream().filter(colToWeight -> cardinalityExtractor.apply(colToWeight.getLeft()) - > numPartitions * PartitionRule.ACCEPTABLE_CARDINALITY_TO_NUM_PARTITIONS_RATIO) + > numPartitions * PartitionRule.ACCEPTABLE_CARDINALITY_TO_NUM_PARTITIONS_RATIO) .max(Comparator.comparingDouble(Pair::getRight)).map(Pair::getRight).flatMap(maxWeight -> { double topCandidatesThreshold = maxWeight * topCandidateRatio; return columnNameToWeightPairs.stream().filter(colToWeight -> colToWeight.getRight() > topCandidatesThreshold) @@ -191,10 +192,12 @@ public class PinotTablePartitionRule extends AbstractRule { return parsePredicateList(queryContext.getFilter()); } + @Nullable public FixedLenBitset parsePredicateList(FilterContext filterContext) { FilterContext.Type type = filterContext.getType(); FixedLenBitset ret; - if (type == FilterContext.Type.AND) { // a column can appear in only one sub predicate to partition AND predicate + if (type == FilterContext.Type.AND) { + // a column can appear in only one sub predicate to partition AND predicate ret = mutableEmptySet(); for (int i = 0; i < filterContext.getChildren().size(); i++) { FixedLenBitset childResult = parsePredicateList(filterContext.getChildren().get(i)); @@ -202,7 +205,8 @@ public class PinotTablePartitionRule extends AbstractRule { ret.union(childResult); } } - } else if (type == FilterContext.Type.OR) { // a column must appear in each sub predicate to partition OR predicate + } else if (type == FilterContext.Type.OR) { + // a column must appear in each sub predicate to partition OR predicate ret = null; for (int i = 0; i < filterContext.getChildren().size(); i++) { FixedLenBitset childResult = parsePredicateList(filterContext.getChildren().get(i)); @@ -210,7 +214,11 @@ public class PinotTablePartitionRule extends AbstractRule { ret = (ret == null) ? childResult : ret.intersect(childResult); } } - } else { // a for leaf we consider only IN (with literal size < threshold) / EQ + } else if (type == FilterContext.Type.NOT) { + // partition doesn't help for NOT predicate + ret = null; + } else { + // a for leaf we consider only IN (with literal size < threshold) / EQ ret = mutableEmptySet(); Predicate predicate = filterContext.getPredicate(); Predicate.Type predicateType = predicate.getType(); @@ -226,6 +234,7 @@ public class PinotTablePartitionRule extends AbstractRule { return null; } else if (!_input.isSingleValueColumn(colName)) { // only SV column can be used as partitioning column LOGGER.trace("Skipping the MV column {}", colName); + return null; } else if (predicateType == Predicate.Type.IN) { int numValuesSelected; @@ -248,7 +257,7 @@ public class PinotTablePartitionRule extends AbstractRule { ret.add(_input.colNameToInt(colName)); } } - LOGGER.debug("ret {}", ret.toString()); + LOGGER.debug("ret {}", ret); return ret; } diff --git a/pinot-controller/src/main/java/org/apache/pinot/controller/recommender/rules/impl/RangeIndexRule.java b/pinot-controller/src/main/java/org/apache/pinot/controller/recommender/rules/impl/RangeIndexRule.java index 7065f3a..e393f5f 100644 --- a/pinot-controller/src/main/java/org/apache/pinot/controller/recommender/rules/impl/RangeIndexRule.java +++ b/pinot-controller/src/main/java/org/apache/pinot/controller/recommender/rules/impl/RangeIndexRule.java @@ -19,6 +19,7 @@ package org.apache.pinot.controller.recommender.rules.impl; import com.google.common.util.concurrent.AtomicDouble; +import java.util.List; import org.apache.pinot.common.request.context.ExpressionContext; import org.apache.pinot.common.request.context.FilterContext; import org.apache.pinot.common.request.context.predicate.Predicate; @@ -90,14 +91,16 @@ public class RangeIndexRule extends AbstractRule { * @return FixedLenBitset for range predicates in this query */ private FixedLenBitset parsePredicateList(FilterContext filterContext) { - FilterContext.Type type = filterContext.getType(); FixedLenBitset ret = mutableEmptySet(); - if (type == FilterContext.Type.AND || type == FilterContext.Type.OR) { - for (int i = 0; i < filterContext.getChildren().size(); i++) { - FixedLenBitset childResult = parsePredicateList(filterContext.getChildren().get(i)); + List<FilterContext> children = filterContext.getChildren(); + if (children != null) { + // AND, OR, NOT + for (FilterContext child : children) { + FixedLenBitset childResult = parsePredicateList(child); ret.union(childResult); } } else { + // PREDICATE ExpressionContext lhs = filterContext.getPredicate().getLhs(); String colName = lhs.toString(); if (lhs.getType() == ExpressionContext.Type.FUNCTION) { diff --git a/pinot-controller/src/main/java/org/apache/pinot/controller/recommender/rules/utils/QueryInvertedSortedIndexRecommender.java b/pinot-controller/src/main/java/org/apache/pinot/controller/recommender/rules/utils/QueryInvertedSortedIndexRecommender.java index 8aaf771..55561d8 100644 --- a/pinot-controller/src/main/java/org/apache/pinot/controller/recommender/rules/utils/QueryInvertedSortedIndexRecommender.java +++ b/pinot-controller/src/main/java/org/apache/pinot/controller/recommender/rules/utils/QueryInvertedSortedIndexRecommender.java @@ -289,9 +289,8 @@ public class QueryInvertedSortedIndexRecommender { // case: OR connected top level predicates, recursively run parseTopLevel on each on its children and // simply return all the results. Each result will contribute to the global recommendation equally List<List<PredicateParseResult>> childResults = new ArrayList<>(); - for (int i = 0; i < filterContextTopLevel.getChildren().size(); i++) { - List<List<PredicateParseResult>> childResult = - parseTopLevel(filterContextTopLevel.getChildren().get(i), queryWeight); + for (FilterContext child : filterContextTopLevel.getChildren()) { + List<List<PredicateParseResult>> childResult = parseTopLevel(child, queryWeight); if (childResult != null) { childResults.addAll(childResult); } @@ -302,6 +301,9 @@ public class QueryInvertedSortedIndexRecommender { } else { return childResults; } + } else if (type == FilterContext.Type.NOT) { + assert filterContextTopLevel.getChildren().size() == 1; + return parseTopLevel(filterContextTopLevel.getChildren().get(0), queryWeight); } else { // case: Return result directly. PredicateParseResult predicateParseResult = parseLeafPredicate(filterContextTopLevel, NESTED_TOP_LEVEL); @@ -328,6 +330,7 @@ public class QueryInvertedSortedIndexRecommender { * Recommend inverted index for: * Case AND: The dimension which selects the lowest percentage of rows. * Case OR: All the recommended dimensions from evaluating all its child predicates. + * Case NOT: Same as the underlying predicate * Case Leaf: See {@link QueryInvertedSortedIndexRecommender#parseLeafPredicate(FilterContext, int)} * @param predicateList Single or nested predicates. * @param depth The depth of current AST tree. >= Second level in this function. Top level is handled in @@ -501,6 +504,9 @@ public class QueryInvertedSortedIndexRecommender { .setRecommendationPriorityEnum(RecommendationPriorityEnum.NESTED).setnESI(nESI) .setPercentSelected(percentSelected).setnESIWithIdx(nESI).build(); } + } else if (type == FilterContext.Type.NOT) { + assert predicateList.getChildren().size() == 1; + return parsePredicateList(predicateList.getChildren().get(0), depth); } else { // case:Leaf predicate PredicateParseResult predicateParseResult = parseLeafPredicate(predicateList, depth); diff --git a/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/DistinctCountThetaSketchAggregationFunction.java b/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/DistinctCountThetaSketchAggregationFunction.java index 8f8e226..a371bf2 100644 --- a/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/DistinctCountThetaSketchAggregationFunction.java +++ b/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/DistinctCountThetaSketchAggregationFunction.java @@ -1014,23 +1014,30 @@ public class DistinctCountThetaSketchAggregationFunction */ private static FilterEvaluator getFilterEvaluator(FilterContext filter, Map<ExpressionContext, Integer> expressionIndexMap) { - List<FilterContext> children = filter.getChildren(); - if (children != null) { - // AND/OR - List<FilterEvaluator> childEvaluators = new ArrayList<>(children.size()); - for (FilterContext child : children) { - childEvaluators.add(getFilterEvaluator(child, expressionIndexMap)); - } - if (filter.getType() == FilterContext.Type.AND) { + switch (filter.getType()) { + case AND: + List<FilterContext> children = filter.getChildren(); + List<FilterEvaluator> childEvaluators = new ArrayList<>(children.size()); + for (FilterContext child : children) { + childEvaluators.add(getFilterEvaluator(child, expressionIndexMap)); + } return new AndFilterEvaluator(childEvaluators); - } else { + case OR: + children = filter.getChildren(); + childEvaluators = new ArrayList<>(children.size()); + for (FilterContext child : children) { + childEvaluators.add(getFilterEvaluator(child, expressionIndexMap)); + } return new OrFilterEvaluator(childEvaluators); - } - } else { - // Predicate - Predicate predicate = filter.getPredicate(); - int expressionIndex = expressionIndexMap.get(predicate.getLhs()); - return new PredicateFilterEvaluator(predicate, expressionIndex); + case NOT: + assert filter.getChildren().size() == 1; + return new NotFilterEvaluator(getFilterEvaluator(filter.getChildren().get(0), expressionIndexMap)); + case PREDICATE: + Predicate predicate = filter.getPredicate(); + int expressionIndex = expressionIndexMap.get(predicate.getLhs()); + return new PredicateFilterEvaluator(predicate, expressionIndex); + default: + throw new IllegalStateException(); } } @@ -1368,6 +1375,19 @@ public class DistinctCountThetaSketchAggregationFunction } } + private static class NotFilterEvaluator implements FilterEvaluator { + final FilterEvaluator _child; + + private NotFilterEvaluator(FilterEvaluator child) { + _child = child; + } + + @Override + public boolean evaluate(boolean[] singleValues, DataType[] valueTypes, Object[] valueArrays, int index) { + return !_child.evaluate(singleValues, valueTypes, valueArrays, index); + } + } + private static class PredicateFilterEvaluator implements FilterEvaluator { final Predicate _predicate; final int _expressionIndex; diff --git a/pinot-core/src/main/java/org/apache/pinot/core/query/optimizer/filter/MergeEqInFilterOptimizer.java b/pinot-core/src/main/java/org/apache/pinot/core/query/optimizer/filter/MergeEqInFilterOptimizer.java index 8b89d2d..5f26ece 100644 --- a/pinot-core/src/main/java/org/apache/pinot/core/query/optimizer/filter/MergeEqInFilterOptimizer.java +++ b/pinot-core/src/main/java/org/apache/pinot/core/query/optimizer/filter/MergeEqInFilterOptimizer.java @@ -168,7 +168,7 @@ public class MergeEqInFilterOptimizer implements FilterOptimizer { Function childFunction = child.getFunctionCall(); String childOperator = childFunction.getOperator(); assert !childOperator.equals(FilterKind.OR.name()); - if (childOperator.equals(FilterKind.AND.name())) { + if (childOperator.equals(FilterKind.AND.name()) || childOperator.equals(FilterKind.NOT.name())) { childFunction.getOperands().replaceAll(this::optimize); newChildren.add(child); } else if (childOperator.equals(FilterKind.EQUALS.name())) { diff --git a/pinot-core/src/main/java/org/apache/pinot/core/query/optimizer/filter/MergeRangeFilterOptimizer.java b/pinot-core/src/main/java/org/apache/pinot/core/query/optimizer/filter/MergeRangeFilterOptimizer.java index 46f73e9..2705202 100644 --- a/pinot-core/src/main/java/org/apache/pinot/core/query/optimizer/filter/MergeRangeFilterOptimizer.java +++ b/pinot-core/src/main/java/org/apache/pinot/core/query/optimizer/filter/MergeRangeFilterOptimizer.java @@ -136,7 +136,7 @@ public class MergeRangeFilterOptimizer implements FilterOptimizer { Function childFunction = child.getFunctionCall(); FilterKind filterKind = FilterKind.valueOf(childFunction.getOperator()); assert filterKind != FilterKind.AND; - if (filterKind == FilterKind.OR) { + if (filterKind == FilterKind.OR || filterKind == FilterKind.NOT) { childFunction.getOperands().replaceAll(o -> optimize(o, schema)); newChildren.add(child); } else if (filterKind.isRange()) { @@ -186,7 +186,7 @@ public class MergeRangeFilterOptimizer implements FilterOptimizer { } else { return filterExpression; } - } else if (operator.equals(FilterKind.OR.name())) { + } else if (operator.equals(FilterKind.OR.name()) || operator.equals(FilterKind.NOT.name())) { function.getOperands().replaceAll(c -> optimize(c, schema)); return filterExpression; } else { diff --git a/pinot-core/src/main/java/org/apache/pinot/core/query/optimizer/filter/NumericalFilterOptimizer.java b/pinot-core/src/main/java/org/apache/pinot/core/query/optimizer/filter/NumericalFilterOptimizer.java index 16b4e57..4adace7 100644 --- a/pinot-core/src/main/java/org/apache/pinot/core/query/optimizer/filter/NumericalFilterOptimizer.java +++ b/pinot-core/src/main/java/org/apache/pinot/core/query/optimizer/filter/NumericalFilterOptimizer.java @@ -128,35 +128,44 @@ public class NumericalFilterOptimizer implements FilterOptimizer { */ private static Expression optimizeCurrent(Expression expression) { Function function = expression.getFunctionCall(); + String operator = function.getOperator(); List<Expression> operands = function.getOperands(); - if (function.getOperator().equals(FilterKind.AND.name())) { + if (operator.equals(FilterKind.AND.name())) { // If any of the literal operands are FALSE, then replace AND function with FALSE. for (Expression operand : operands) { if (operand.equals(FALSE)) { - return setExpressionToBoolean(expression, false); + return FALSE; } } // Remove all Literal operands that are TRUE. operands.removeIf(x -> x.equals(TRUE)); if (operands.isEmpty()) { - return setExpressionToBoolean(expression, true); + return TRUE; } - } else if (function.getOperator().equals(FilterKind.OR.name())) { + } else if (operator.equals(FilterKind.OR.name())) { // If any of the literal operands are TRUE, then replace OR function with TRUE for (Expression operand : operands) { if (operand.equals(TRUE)) { - return setExpressionToBoolean(expression, true); + return TRUE; } } // Remove all Literal operands that are FALSE. operands.removeIf(x -> x.equals(FALSE)); if (operands.isEmpty()) { - return setExpressionToBoolean(expression, false); + return FALSE; + } + } else if (operator.equals(FilterKind.NOT.name())) { + assert operands.size() == 1; + Expression operand = operands.get(0); + if (operand.equals(TRUE)) { + return FALSE; + } + if (operand.equals(FALSE)) { + return TRUE; } } - return expression; } @@ -482,11 +491,9 @@ public class NumericalFilterOptimizer implements FilterOptimizer { } /** Change the expression value to boolean literal with given value. */ - private static Expression setExpressionToBoolean(Expression expression, boolean value) { + private static void setExpressionToBoolean(Expression expression, boolean value) { expression.unsetFunctionCall(); expression.setType(ExpressionType.LITERAL); expression.setLiteral(Literal.boolValue(value)); - - return expression; } } diff --git a/pinot-core/src/main/java/org/apache/pinot/core/query/optimizer/filter/TimePredicateFilterOptimizer.java b/pinot-core/src/main/java/org/apache/pinot/core/query/optimizer/filter/TimePredicateFilterOptimizer.java index 3d3913a..234b693 100644 --- a/pinot-core/src/main/java/org/apache/pinot/core/query/optimizer/filter/TimePredicateFilterOptimizer.java +++ b/pinot-core/src/main/java/org/apache/pinot/core/query/optimizer/filter/TimePredicateFilterOptimizer.java @@ -76,7 +76,7 @@ public class TimePredicateFilterOptimizer implements FilterOptimizer { Function filterFunction = filterExpression.getFunctionCall(); FilterKind filterKind = FilterKind.valueOf(filterFunction.getOperator()); List<Expression> operands = filterFunction.getOperands(); - if (filterKind == FilterKind.AND || filterKind == FilterKind.OR) { + if (filterKind == FilterKind.AND || filterKind == FilterKind.OR || filterKind == FilterKind.NOT) { // NOTE: We don't need to replace the children because all the changes are applied in-place for (Expression operand : operands) { optimize(operand); @@ -408,8 +408,8 @@ public class TimePredicateFilterOptimizer implements FilterOptimizer { // Step 3: Rewrite the filter function String rangeString = new Range(lowerValue, lowerInclusive, upperValue, upperInclusive).getRangeString(); filterFunction.setOperator(FilterKind.RANGE.name()); - filterFunction - .setOperands(Arrays.asList(dateTimeConvertOperands.get(0), RequestUtils.getLiteralExpression(rangeString))); + filterFunction.setOperands( + Arrays.asList(dateTimeConvertOperands.get(0), RequestUtils.getLiteralExpression(rangeString))); } catch (Exception e) { LOGGER.warn("Caught exception while optimizing DATE_TIME_CONVERT predicate: {}, skipping the optimization", filterFunction, e); diff --git a/pinot-core/src/main/java/org/apache/pinot/core/query/optimizer/statement/JsonStatementOptimizer.java b/pinot-core/src/main/java/org/apache/pinot/core/query/optimizer/statement/JsonStatementOptimizer.java index 0ee5abf..872508b 100644 --- a/pinot-core/src/main/java/org/apache/pinot/core/query/optimizer/statement/JsonStatementOptimizer.java +++ b/pinot-core/src/main/java/org/apache/pinot/core/query/optimizer/statement/JsonStatementOptimizer.java @@ -316,10 +316,10 @@ public class JsonStatementOptimizer implements StatementOptimizer { List<Expression> operands = function.getOperands(); switch (kind) { case AND: - case OR: { + case OR: + case NOT: operands.forEach(operand -> optimizeJsonPredicate(operand, tableConfig, schema)); break; - } case EQUALS: case NOT_EQUALS: case GREATER_THAN: diff --git a/pinot-core/src/main/java/org/apache/pinot/core/query/optimizer/statement/StringPredicateFilterOptimizer.java b/pinot-core/src/main/java/org/apache/pinot/core/query/optimizer/statement/StringPredicateFilterOptimizer.java index eb9e64d..09abfa9 100644 --- a/pinot-core/src/main/java/org/apache/pinot/core/query/optimizer/statement/StringPredicateFilterOptimizer.java +++ b/pinot-core/src/main/java/org/apache/pinot/core/query/optimizer/statement/StringPredicateFilterOptimizer.java @@ -75,19 +75,13 @@ public class StringPredicateFilterOptimizer implements StatementOptimizer { Function function = expression.getFunctionCall(); String operator = function.getOperator(); List<Expression> operands = function.getOperands(); - FilterKind kind = FilterKind.valueOf(operator); - switch (kind) { - case AND: - case OR: { - for (Expression operand : operands) { - optimizeExpression(operand, schema); - } - break; - } - default: { - replaceMinusWithCompareForStrings(operands.get(0), schema); - break; + if (operator.equals(FilterKind.AND.name()) || operator.equals(FilterKind.OR.name()) || operator.equals( + FilterKind.NOT.name())) { + for (Expression operand : operands) { + optimizeExpression(operand, schema); } + } else { + replaceMinusWithCompareForStrings(operands.get(0), schema); } } diff --git a/pinot-core/src/main/java/org/apache/pinot/core/query/pruner/ColumnValueSegmentPruner.java b/pinot-core/src/main/java/org/apache/pinot/core/query/pruner/ColumnValueSegmentPruner.java index 898b157..ec5ba7e 100644 --- a/pinot-core/src/main/java/org/apache/pinot/core/query/pruner/ColumnValueSegmentPruner.java +++ b/pinot-core/src/main/java/org/apache/pinot/core/query/pruner/ColumnValueSegmentPruner.java @@ -176,6 +176,9 @@ public class ColumnValueSegmentPruner implements SegmentPruner { extractPredicateColumns(child, eqInColumns, rangeColumns); } break; + case NOT: + // Do not track the predicates under NOT filter + break; case PREDICATE: Predicate predicate = filter.getPredicate(); @@ -215,6 +218,9 @@ public class ColumnValueSegmentPruner implements SegmentPruner { } } return true; + case NOT: + // Do not prune NOT filter + return false; case PREDICATE: Predicate predicate = filter.getPredicate(); // Only prune columns diff --git a/pinot-core/src/main/java/org/apache/pinot/core/query/reduce/filter/AndRowMatcher.java b/pinot-core/src/main/java/org/apache/pinot/core/query/reduce/filter/AndRowMatcher.java index a222ba8..0991952 100644 --- a/pinot-core/src/main/java/org/apache/pinot/core/query/reduce/filter/AndRowMatcher.java +++ b/pinot-core/src/main/java/org/apache/pinot/core/query/reduce/filter/AndRowMatcher.java @@ -26,9 +26,9 @@ import org.apache.pinot.common.request.context.FilterContext; * AND filter matcher. */ public class AndRowMatcher implements RowMatcher { - RowMatcher[] _childMatchers; + private final RowMatcher[] _childMatchers; - AndRowMatcher(List<FilterContext> childFilters, ValueExtractorFactory valueExtractorFactory) { + public AndRowMatcher(List<FilterContext> childFilters, ValueExtractorFactory valueExtractorFactory) { int numChildren = childFilters.size(); _childMatchers = new RowMatcher[numChildren]; for (int i = 0; i < numChildren; i++) { diff --git a/pinot-core/src/main/java/org/apache/pinot/core/query/reduce/filter/LiteralValueExtractor.java b/pinot-core/src/main/java/org/apache/pinot/core/query/reduce/filter/LiteralValueExtractor.java index 101a83a..82f06bd 100644 --- a/pinot-core/src/main/java/org/apache/pinot/core/query/reduce/filter/LiteralValueExtractor.java +++ b/pinot-core/src/main/java/org/apache/pinot/core/query/reduce/filter/LiteralValueExtractor.java @@ -25,7 +25,7 @@ import org.apache.pinot.common.utils.DataSchema.ColumnDataType; * Value extractor for a literal. */ public class LiteralValueExtractor implements ValueExtractor { - final String _literal; + private final String _literal; public LiteralValueExtractor(String literal) { _literal = literal; diff --git a/pinot-core/src/main/java/org/apache/pinot/core/query/reduce/filter/ValueExtractor.java b/pinot-core/src/main/java/org/apache/pinot/core/query/reduce/filter/NotRowMatcher.java similarity index 65% copy from pinot-core/src/main/java/org/apache/pinot/core/query/reduce/filter/ValueExtractor.java copy to pinot-core/src/main/java/org/apache/pinot/core/query/reduce/filter/NotRowMatcher.java index 74b329e..4b6b51d 100644 --- a/pinot-core/src/main/java/org/apache/pinot/core/query/reduce/filter/ValueExtractor.java +++ b/pinot-core/src/main/java/org/apache/pinot/core/query/reduce/filter/NotRowMatcher.java @@ -18,25 +18,21 @@ */ package org.apache.pinot.core.query.reduce.filter; -import org.apache.pinot.common.utils.DataSchema.ColumnDataType; +import org.apache.pinot.common.request.context.FilterContext; /** - * Value extractor for the post-aggregation function or pre-aggregation gap fill. + * NOT filter matcher. */ -public interface ValueExtractor { - /** - * Returns the column name for the value extracted. - */ - String getColumnName(); +public class NotRowMatcher implements RowMatcher { + private final RowMatcher _childMatcher; - /** - * Returns the ColumnDataType of the value extracted. - */ - ColumnDataType getColumnDataType(); + public NotRowMatcher(FilterContext childFilter, ValueExtractorFactory valueExtractorFactory) { + _childMatcher = RowMatcherFactory.getRowMatcher(childFilter, valueExtractorFactory); + } - /** - * Extracts the value from the given row. - */ - Object extract(Object[] row); + @Override + public boolean isMatch(Object[] row) { + return !_childMatcher.isMatch(row); + } } diff --git a/pinot-core/src/main/java/org/apache/pinot/core/query/reduce/filter/OrRowMatcher.java b/pinot-core/src/main/java/org/apache/pinot/core/query/reduce/filter/OrRowMatcher.java index dc47c6a..543d518 100644 --- a/pinot-core/src/main/java/org/apache/pinot/core/query/reduce/filter/OrRowMatcher.java +++ b/pinot-core/src/main/java/org/apache/pinot/core/query/reduce/filter/OrRowMatcher.java @@ -26,7 +26,7 @@ import org.apache.pinot.common.request.context.FilterContext; * OR filter matcher. */ public class OrRowMatcher implements RowMatcher { - RowMatcher[] _childMatchers; + private final RowMatcher[] _childMatchers; public OrRowMatcher(List<FilterContext> childFilters, ValueExtractorFactory valueExtractorFactory) { int numChildren = childFilters.size(); diff --git a/pinot-core/src/main/java/org/apache/pinot/core/query/reduce/filter/PredicateRowMatcher.java b/pinot-core/src/main/java/org/apache/pinot/core/query/reduce/filter/PredicateRowMatcher.java index 4645ef5..0429262 100644 --- a/pinot-core/src/main/java/org/apache/pinot/core/query/reduce/filter/PredicateRowMatcher.java +++ b/pinot-core/src/main/java/org/apache/pinot/core/query/reduce/filter/PredicateRowMatcher.java @@ -29,9 +29,9 @@ import org.apache.pinot.spi.data.FieldSpec.DataType; * Predicate matcher. */ public class PredicateRowMatcher implements RowMatcher { - ValueExtractor _valueExtractor; - DataType _valueType; - PredicateEvaluator _predicateEvaluator; + private final ValueExtractor _valueExtractor; + private final DataType _valueType; + private final PredicateEvaluator _predicateEvaluator; public PredicateRowMatcher(Predicate predicate, ValueExtractor valueExtractor) { _valueExtractor = valueExtractor; diff --git a/pinot-core/src/main/java/org/apache/pinot/core/query/reduce/filter/RowMatcher.java b/pinot-core/src/main/java/org/apache/pinot/core/query/reduce/filter/RowMatcher.java index 8986366..984c3d7 100644 --- a/pinot-core/src/main/java/org/apache/pinot/core/query/reduce/filter/RowMatcher.java +++ b/pinot-core/src/main/java/org/apache/pinot/core/query/reduce/filter/RowMatcher.java @@ -22,6 +22,7 @@ package org.apache.pinot.core.query.reduce.filter; * Filter matcher for the rows. */ public interface RowMatcher { + /** * Returns {@code true} if the given row matches the filter, {@code false} otherwise. */ diff --git a/pinot-core/src/main/java/org/apache/pinot/core/query/reduce/filter/RowMatcherFactory.java b/pinot-core/src/main/java/org/apache/pinot/core/query/reduce/filter/RowMatcherFactory.java index 206e7ab..d7f0421 100644 --- a/pinot-core/src/main/java/org/apache/pinot/core/query/reduce/filter/RowMatcherFactory.java +++ b/pinot-core/src/main/java/org/apache/pinot/core/query/reduce/filter/RowMatcherFactory.java @@ -37,6 +37,9 @@ public class RowMatcherFactory { return new AndRowMatcher(filter.getChildren(), valueExtractorFactory); case OR: return new OrRowMatcher(filter.getChildren(), valueExtractorFactory); + case NOT: + assert filter.getChildren().size() == 1; + return new NotRowMatcher(filter.getChildren().get(0), valueExtractorFactory); case PREDICATE: return new PredicateRowMatcher(filter.getPredicate(), valueExtractorFactory.getValueExtractor(filter.getPredicate().getLhs())); diff --git a/pinot-core/src/main/java/org/apache/pinot/core/query/reduce/filter/ValueExtractor.java b/pinot-core/src/main/java/org/apache/pinot/core/query/reduce/filter/ValueExtractor.java index 74b329e..6ffaf48 100644 --- a/pinot-core/src/main/java/org/apache/pinot/core/query/reduce/filter/ValueExtractor.java +++ b/pinot-core/src/main/java/org/apache/pinot/core/query/reduce/filter/ValueExtractor.java @@ -25,6 +25,7 @@ import org.apache.pinot.common.utils.DataSchema.ColumnDataType; * Value extractor for the post-aggregation function or pre-aggregation gap fill. */ public interface ValueExtractor { + /** * Returns the column name for the value extracted. */ diff --git a/pinot-core/src/main/java/org/apache/pinot/core/query/reduce/filter/ValueExtractorFactory.java b/pinot-core/src/main/java/org/apache/pinot/core/query/reduce/filter/ValueExtractorFactory.java index 8aae73a..33e26fe 100644 --- a/pinot-core/src/main/java/org/apache/pinot/core/query/reduce/filter/ValueExtractorFactory.java +++ b/pinot-core/src/main/java/org/apache/pinot/core/query/reduce/filter/ValueExtractorFactory.java @@ -25,6 +25,7 @@ import org.apache.pinot.common.request.context.ExpressionContext; * Factory for {@link ValueExtractor}. */ public interface ValueExtractorFactory { + /** * Create the {@link ValueExtractor} for specific column. * diff --git a/pinot-core/src/main/java/org/apache/pinot/core/startree/StarTreeUtils.java b/pinot-core/src/main/java/org/apache/pinot/core/startree/StarTreeUtils.java index 049da07..12c2ee5 100644 --- a/pinot-core/src/main/java/org/apache/pinot/core/startree/StarTreeUtils.java +++ b/pinot-core/src/main/java/org/apache/pinot/core/startree/StarTreeUtils.java @@ -121,6 +121,9 @@ public class StarTreeUtils { .add(new CompositePredicateEvaluator(predicateEvaluators)); } break; + case NOT: + // TODO: Support NOT in star-tree + return null; case PREDICATE: Predicate predicate = filterNode.getPredicate(); PredicateEvaluator predicateEvaluator = getPredicateEvaluator(indexSegment, predicate, predicateEvaluatorMap); @@ -188,7 +191,9 @@ public class StarTreeUtils { assert filter.getType() == FilterContext.Type.OR; List<Predicate> predicates = new ArrayList<>(); - extractOrClausePredicates(filter, predicates); + if (!extractOrClausePredicates(filter, predicates)) { + return null; + } String identifier = null; List<PredicateEvaluator> predicateEvaluators = new ArrayList<>(); @@ -219,7 +224,9 @@ public class StarTreeUtils { } /** - * Extracts the predicates under the given OR clause, returns {@code false} if there is nested AND under OR clause. + * Extracts the predicates under the given OR clause, returns {@code false} if there is nested AND or NOT under OR + * clause. + * TODO: Support NOT in star-tree */ private static boolean extractOrClausePredicates(FilterContext filter, List<Predicate> predicates) { assert filter.getType() == FilterContext.Type.OR; @@ -227,6 +234,7 @@ public class StarTreeUtils { for (FilterContext child : filter.getChildren()) { switch (child.getType()) { case AND: + case NOT: return false; case OR: if (!extractOrClausePredicates(child, predicates)) { diff --git a/pinot-integration-test-base/src/test/java/org/apache/pinot/integration/tests/ClusterIntegrationTestUtils.java b/pinot-integration-test-base/src/test/java/org/apache/pinot/integration/tests/ClusterIntegrationTestUtils.java index fc78bf4..b67399f 100644 --- a/pinot-integration-test-base/src/test/java/org/apache/pinot/integration/tests/ClusterIntegrationTestUtils.java +++ b/pinot-integration-test-base/src/test/java/org/apache/pinot/integration/tests/ClusterIntegrationTestUtils.java @@ -19,7 +19,6 @@ package org.apache.pinot.integration.tests; import com.fasterxml.jackson.databind.JsonNode; -import com.google.common.collect.Lists; import com.google.common.math.DoubleMath; import com.google.common.primitives.Longs; import java.io.ByteArrayOutputStream; @@ -65,9 +64,11 @@ import org.apache.pinot.common.request.PinotQuery; import org.apache.pinot.common.request.SelectionSort; import org.apache.pinot.common.utils.StringUtil; import org.apache.pinot.common.utils.TarGzCompressionUtils; +import org.apache.pinot.core.query.request.context.QueryContext; +import org.apache.pinot.core.query.request.context.utils.QueryContextConverterUtils; +import org.apache.pinot.core.query.request.context.utils.QueryContextUtils; import org.apache.pinot.core.requesthandler.PinotQueryParserFactory; import org.apache.pinot.plugin.inputformat.avro.AvroUtils; -import org.apache.pinot.pql.parsers.PinotQuery2BrokerRequestConverter; import org.apache.pinot.segment.local.segment.creator.impl.SegmentIndexCreationDriverImpl; import org.apache.pinot.segment.spi.creator.SegmentGeneratorConfig; import org.apache.pinot.segment.spi.creator.SegmentIndexCreationDriver; @@ -782,14 +783,14 @@ public class ClusterIntegrationTestUtils { * @param pinotQuery Pinot sql query * @param brokerUrl Pinot broker URL * @param pinotConnection Pinot connection - * @param sqlQueries H2 SQL query + * @param h2Queries H2 queries * @param h2Connection H2 connection * @throws Exception */ static void testSqlQuery(String pinotQuery, String brokerUrl, org.apache.pinot.client.Connection pinotConnection, - @Nullable List<String> sqlQueries, @Nullable Connection h2Connection) + List<String> h2Queries, Connection h2Connection) throws Exception { - testSqlQuery(pinotQuery, brokerUrl, pinotConnection, sqlQueries, h2Connection, null); + testSqlQuery(pinotQuery, brokerUrl, pinotConnection, h2Queries, h2Connection, null); } /** @@ -798,28 +799,14 @@ public class ClusterIntegrationTestUtils { * @param pinotQuery Pinot sql query * @param brokerUrl Pinot broker URL * @param pinotConnection Pinot connection - * @param sqlQueries H2 SQL query + * @param h2Queries H2 queries * @param h2Connection H2 connection * @param headers headers * @throws Exception */ static void testSqlQuery(String pinotQuery, String brokerUrl, org.apache.pinot.client.Connection pinotConnection, - @Nullable List<String> sqlQueries, @Nullable Connection h2Connection, @Nullable Map<String, String> headers) + List<String> h2Queries, Connection h2Connection, @Nullable Map<String, String> headers) throws Exception { - if (pinotQuery == null || sqlQueries == null) { - return; - } - - // TODO: Use PinotQuery instead of BrokerRequest here - BrokerRequest brokerRequest = - new PinotQuery2BrokerRequestConverter().convert(CalciteSqlParser.compileToPinotQuery(pinotQuery)); - - List<String> orderByColumns = new ArrayList<>(); - if (isSelectionQuery(brokerRequest) && brokerRequest.getOrderBy() != null - && !brokerRequest.getOrderBy().isEmpty()) { - orderByColumns.addAll(CalciteSqlParser.extractIdentifiers(brokerRequest.getPinotQuery().getOrderByList(), false)); - } - // broker response JsonNode pinotResponse = ClusterTest.postSqlQuery(pinotQuery, brokerUrl, headers); if (!pinotResponse.get("exceptions").isEmpty()) { @@ -836,23 +823,34 @@ public class ClusterIntegrationTestUtils { int numColumns = resultTableResultSet.getColumnCount(); // h2 response - String sqlQuery = sqlQueries.get(0); + String h2Query = h2Queries.get(0); Assert.assertNotNull(h2Connection); Statement h2statement = h2Connection.createStatement(ResultSet.TYPE_FORWARD_ONLY, ResultSet.CONCUR_READ_ONLY); - h2statement.execute(sqlQuery); + h2statement.execute(h2Query); ResultSet h2ResultSet = h2statement.getResultSet(); // compare results - if (isSelectionQuery(brokerRequest)) { // selection + QueryContext queryContext = QueryContextConverterUtils.getQueryContextFromSQL(pinotQuery); + if (!QueryContextUtils.isAggregationQuery(queryContext)) { + // selection/distinct + + List<String> orderByColumns = new ArrayList<>(); + if (queryContext.getOrderByExpressions() != null) { + orderByColumns.addAll( + CalciteSqlParser.extractIdentifiers(queryContext.getBrokerRequest().getPinotQuery().getOrderByList(), + false)); + } Set<String> expectedValues = new HashSet<>(); List<String> expectedOrderByValues = new ArrayList<>(); int h2NumRows = getH2ExpectedValues(expectedValues, expectedOrderByValues, h2ResultSet, h2ResultSet.getMetaData(), orderByColumns); comparePinotResultsWithExpectedValues(expectedValues, expectedOrderByValues, resultTableResultSet, orderByColumns, - pinotQuery, sqlQueries, h2NumRows, pinotNumRecordsSelected); - } else { // aggregation - if (!brokerRequest.isSetGroupBy()) { // aggregation only + pinotQuery, h2Queries, h2NumRows, pinotNumRecordsSelected); + } else { + if (queryContext.getGroupByExpressions() == null) { + // aggregation only + // compare the single row h2ResultSet.first(); for (int c = 0; c < numColumns; c++) { @@ -864,7 +862,7 @@ public class ClusterIntegrationTestUtils { if (pinotNumRecordsSelected != 0) { String failureMessage = "No record selected in H2 but " + pinotNumRecordsSelected + " records selected in Pinot"; - failure(pinotQuery, Lists.newArrayList(sqlQuery), failureMessage); + failure(pinotQuery, h2Queries, failureMessage); } // Skip further comparison @@ -880,16 +878,17 @@ public class ClusterIntegrationTestUtils { String failureMessage = "Value: " + c + " does not match, expected: " + h2Value + ", got broker value: " + brokerValue + ", got client value:" + connectionValue; - failure(pinotQuery, Lists.newArrayList(sqlQuery), failureMessage); + failure(pinotQuery, h2Queries, failureMessage); } } - } else { // aggregation group by + } else { + // aggregation group-by // TODO: compare results for aggregation group by queries w/o order by // Compare results for aggregation group by queries with order by - if (brokerRequest.getOrderBy() != null && !brokerRequest.getOrderBy().isEmpty()) { + if (queryContext.getOrderByExpressions() != null) { // don't compare query with multi-value column. - if (sqlQuery.contains("_MV")) { + if (h2Query.contains("_MV")) { return; } if (h2ResultSet.first()) { @@ -903,7 +902,7 @@ public class ClusterIntegrationTestUtils { String failureMessage = "Value: " + c + " does not match, expected: " + h2Value + ", got broker value: " + brokerValue + ", got client value:" + connectionValue; - failure(pinotQuery, Lists.newArrayList(sqlQuery), failureMessage); + failure(pinotQuery, h2Queries, failureMessage); } } if (!h2ResultSet.next()) { @@ -916,23 +915,6 @@ public class ClusterIntegrationTestUtils { } } - private static boolean isSelectionQuery(BrokerRequest brokerRequest) { - if (brokerRequest.getSelections() != null) { - return true; - } - if (brokerRequest.getAggregationsInfo() != null && brokerRequest.getAggregationsInfo().get(0).getAggregationType() - .equalsIgnoreCase("DISTINCT")) { - return true; - } - return false; - } - - private static void convertToUpperCase(List<String> columns) { - for (int i = 0; i < columns.size(); i++) { - columns.set(i, columns.get(i).toUpperCase()); - } - } - private static int getH2ExpectedValues(Set<String> expectedValues, List<String> expectedOrderByValues, ResultSet h2ResultSet, ResultSetMetaData h2MetaData, Collection<String> orderByColumns) throws SQLException { @@ -960,8 +942,8 @@ public class ClusterIntegrationTestUtils { // Handle multi-value columns int length = columnName.length(); - if (length > H2_MULTI_VALUE_SUFFIX_LENGTH && columnName - .substring(length - H2_MULTI_VALUE_SUFFIX_LENGTH, length - 1).equals("__MV")) { + if (length > H2_MULTI_VALUE_SUFFIX_LENGTH && columnName.substring(length - H2_MULTI_VALUE_SUFFIX_LENGTH, + length - 1).equals("__MV")) { // Multi-value column String multiValueColumnName = columnName.substring(0, length - H2_MULTI_VALUE_SUFFIX_LENGTH); List<String> multiValue = reusableMultiValuesMap.get(multiValueColumnName); @@ -1136,8 +1118,8 @@ public class ClusterIntegrationTestUtils { double expectedValue = Double.parseDouble(h2Value); double actualValueBroker = Double.parseDouble(brokerValue); double actualValueConnection = Double.parseDouble(connectionValue); - if (!DoubleMath.fuzzyEquals(actualValueBroker, expectedValue, 1.0) || !DoubleMath - .fuzzyEquals(actualValueConnection, expectedValue, 1.0)) { + if (!DoubleMath.fuzzyEquals(actualValueBroker, expectedValue, 1.0) || !DoubleMath.fuzzyEquals( + actualValueConnection, expectedValue, 1.0)) { error = true; } } else { diff --git a/pinot-integration-tests/src/test/java/org/apache/pinot/integration/tests/BaseClusterIntegrationTestSet.java b/pinot-integration-tests/src/test/java/org/apache/pinot/integration/tests/BaseClusterIntegrationTestSet.java index eba1dd1..78588dd 100644 --- a/pinot-integration-tests/src/test/java/org/apache/pinot/integration/tests/BaseClusterIntegrationTestSet.java +++ b/pinot-integration-tests/src/test/java/org/apache/pinot/integration/tests/BaseClusterIntegrationTestSet.java @@ -240,6 +240,24 @@ public abstract class BaseClusterIntegrationTestSet extends BaseClusterIntegrati query = "SELECT COUNT(*) AS \"date\", MAX(ArrTime) AS \"group\", MIN(ArrTime) AS min FROM myTable"; testSqlQuery(query, Collections.singletonList(query)); + // LIKE + query = "SELECT count(*) FROM mytable WHERE OriginState LIKE 'A_'"; + testSqlQuery(query, Collections.singletonList(query)); + query = "SELECT count(*) FROM mytable WHERE DestCityName LIKE 'C%'"; + testSqlQuery(query, Collections.singletonList(query)); + query = "SELECT count(*) FROM mytable WHERE DestCityName LIKE '_h%'"; + testSqlQuery(query, Collections.singletonList(query)); + + // NOT + query = "SELECT count(*) FROM mytable WHERE OriginState NOT BETWEEN 'DE' AND 'PA'"; + testSqlQuery(query, Collections.singletonList(query)); + query = "SELECT count(*) FROM mytable WHERE OriginState NOT LIKE 'A_'"; + testSqlQuery(query, Collections.singletonList(query)); + query = "SELECT count(*) FROM mytable WHERE NOT (DaysSinceEpoch = 16312 AND Carrier = 'DL')"; + testSqlQuery(query, Collections.singletonList(query)); + query = "SELECT count(*) FROM mytable WHERE (NOT DaysSinceEpoch = 16312) AND Carrier = 'DL'"; + testSqlQuery(query, Collections.singletonList(query)); + // Post-aggregation in ORDER-BY query = "SELECT MAX(ArrTime) FROM mytable GROUP BY DaysSinceEpoch ORDER BY MAX(ArrTime) - MIN(ArrTime)"; testSqlQuery(query, Collections.singletonList(query)); diff --git a/pinot-segment-local/src/main/java/org/apache/pinot/segment/local/realtime/impl/json/MutableJsonIndexImpl.java b/pinot-segment-local/src/main/java/org/apache/pinot/segment/local/realtime/impl/json/MutableJsonIndexImpl.java index 2eb37ea..4958835 100644 --- a/pinot-segment-local/src/main/java/org/apache/pinot/segment/local/realtime/impl/json/MutableJsonIndexImpl.java +++ b/pinot-segment-local/src/main/java/org/apache/pinot/segment/local/realtime/impl/json/MutableJsonIndexImpl.java @@ -44,8 +44,6 @@ import org.roaringbitmap.IntConsumer; import org.roaringbitmap.RoaringBitmap; import org.roaringbitmap.buffer.MutableRoaringBitmap; -import static org.apache.pinot.common.request.context.FilterContext.Type.PREDICATE; - /** * Json index for mutable segment. @@ -122,7 +120,7 @@ public class MutableJsonIndexImpl implements MutableJsonIndex { _readLock.lock(); try { - if (filter.getType() == PREDICATE && isExclusive(filter.getPredicate().getType())) { + if (filter.getType() == FilterContext.Type.PREDICATE && isExclusive(filter.getPredicate().getType())) { // Handle exclusive predicate separately because the flip can only be applied to the unflattened doc ids in // order to get the correct result, and it cannot be nested RoaringBitmap matchingFlattenedDocIds = getMatchingFlattenedDocIds(filter.getPredicate()); diff --git a/pinot-segment-local/src/main/java/org/apache/pinot/segment/local/segment/index/readers/json/ImmutableJsonIndexReader.java b/pinot-segment-local/src/main/java/org/apache/pinot/segment/local/segment/index/readers/json/ImmutableJsonIndexReader.java index cd0d144..1adc20e 100644 --- a/pinot-segment-local/src/main/java/org/apache/pinot/segment/local/segment/index/readers/json/ImmutableJsonIndexReader.java +++ b/pinot-segment-local/src/main/java/org/apache/pinot/segment/local/segment/index/readers/json/ImmutableJsonIndexReader.java @@ -41,8 +41,6 @@ import org.roaringbitmap.IntConsumer; import org.roaringbitmap.buffer.ImmutableRoaringBitmap; import org.roaringbitmap.buffer.MutableRoaringBitmap; -import static org.apache.pinot.common.request.context.FilterContext.Type.PREDICATE; - /** * Reader for json index. @@ -87,7 +85,7 @@ public class ImmutableJsonIndexReader implements JsonIndexReader { throw new BadQueryRequestException("Invalid json match filter: " + filterString); } - if (filter.getType() == PREDICATE && isExclusive(filter.getPredicate().getType())) { + if (filter.getType() == FilterContext.Type.PREDICATE && isExclusive(filter.getPredicate().getType())) { // Handle exclusive predicate separately because the flip can only be applied to the unflattened doc ids in order // to get the correct result, and it cannot be nested MutableRoaringBitmap matchingFlattenedDocIds = getMatchingFlattenedDocIds(filter.getPredicate()); --------------------------------------------------------------------- To unsubscribe, e-mail: commits-unsubscr...@pinot.apache.org For additional commands, e-mail: commits-h...@pinot.apache.org