You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@pinot.apache.org by si...@apache.org on 2021/05/03 03:23:38 UTC

[incubator-pinot] branch master updated: Normalize LHS and RHS numerical types for = and != operator. (#6811)

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

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


The following commit(s) were added to refs/heads/master by this push:
     new d033b3f  Normalize LHS and RHS numerical types for = and != operator. (#6811)
d033b3f is described below

commit d033b3f6ba370b3d1cd582e44e9c0818bf50d5ba
Author: Amrish Lal <am...@gmail.com>
AuthorDate: Sun May 2 20:23:19 2021 -0700

    Normalize LHS and RHS numerical types for = and != operator. (#6811)
    
    * Normalize LHS and RHS numerical types for = and != operator.
    
    * Fix formatting.
    
    * Fix formatting.
    
    * fix test case.
    
    * fix test case.
    
    * Rebuild.
    
    * Cleanup.
    
    * Cleanup.
    
    * Code review changes and cleanup.
    
    * Fix test cases.
    
    * Fix formatting.
    
    * Fix formatting
    
    * null check
    
    * Rebuild.
    
    * code review changes.
    
    * code review changes.
    
    * code review changes.
    
    * Revert test case changes.
    
    * code review changes.
    
    * set boolean constant value to expression.
---
 .../requesthandler/BaseBrokerRequestHandler.java   |  55 +++-
 .../pinot/common/utils/request/RequestUtils.java   |   6 +
 .../pinot/core/query/optimizer/QueryOptimizer.java |   3 +-
 .../optimizer/filter/MergeEqInFilterOptimizer.java |   3 +-
 .../filter/MergeRangeFilterOptimizer.java          |   2 +-
 .../optimizer/filter/NumericalFilterOptimizer.java | 277 ++++++++++++++++++
 .../core/query/optimizer/QueryOptimizerTest.java   |   8 +-
 .../filter/NumericalFilterOptimizerTest.java       | 316 +++++++++++++++++++++
 8 files changed, 662 insertions(+), 8 deletions(-)

diff --git a/pinot-broker/src/main/java/org/apache/pinot/broker/requesthandler/BaseBrokerRequestHandler.java b/pinot-broker/src/main/java/org/apache/pinot/broker/requesthandler/BaseBrokerRequestHandler.java
index 49f24c4..f0d4fdc 100644
--- a/pinot-broker/src/main/java/org/apache/pinot/broker/requesthandler/BaseBrokerRequestHandler.java
+++ b/pinot-broker/src/main/java/org/apache/pinot/broker/requesthandler/BaseBrokerRequestHandler.java
@@ -102,6 +102,8 @@ import org.slf4j.LoggerFactory;
 public abstract class BaseBrokerRequestHandler implements BrokerRequestHandler {
   private static final Logger LOGGER = LoggerFactory.getLogger(BaseBrokerRequestHandler.class);
   private static final String IN_SUBQUERY = "inSubquery";
+  private static final Expression FALSE = RequestUtils.getLiteralExpression(false);
+  private static final Expression TRUE = RequestUtils.getLiteralExpression(true);
 
   protected final PinotConfiguration _config;
   protected final RoutingManager _routingManager;
@@ -360,6 +362,40 @@ public abstract class BaseBrokerRequestHandler implements BrokerRequestHandler {
       requestStatistics.setRealtimeServerTenant(getServerTenant(realtimeTableName));
     }
 
+    // Check if response can be send without server query evaluation.
+    if (offlineBrokerRequest == null || offlineBrokerRequest.getPinotQuery() == null || isFilterAlwaysFalse(
+        offlineBrokerRequest)) {
+      // We don't need to evaluate offline request
+      offlineBrokerRequest = null;
+    }
+
+    if (realtimeBrokerRequest == null || realtimeBrokerRequest.getPinotQuery() == null || isFilterAlwaysFalse(
+        realtimeBrokerRequest)) {
+      // We don't need to evaluate realtime request
+      realtimeBrokerRequest = null;
+    }
+
+    if (offlineBrokerRequest == null && realtimeBrokerRequest == null) {
+      // Send empty response since we don't need to evaluate either offline or realtime request.
+      BrokerResponseNative brokerResponse = BrokerResponseNative.empty();
+      logBrokerResponse(requestId, query, requestStatistics, brokerRequest, 0, new ServerStats(), brokerResponse,
+          System.nanoTime());
+
+      return brokerResponse;
+    }
+
+    if (offlineBrokerRequest != null && offlineBrokerRequest.getPinotQuery() != null && isFilterAlwaysTrue(
+        offlineBrokerRequest)) {
+      // Drop offline request filter since it is always true
+      offlineBrokerRequest.getPinotQuery().setFilterExpression(null);
+    }
+
+    if (realtimeBrokerRequest != null && realtimeBrokerRequest.getPinotQuery() != null && isFilterAlwaysTrue(
+        realtimeBrokerRequest)) {
+      // Drop realtime request filter since it is always true
+      realtimeBrokerRequest.getPinotQuery().setFilterExpression(null);
+    }
+
     // Calculate routing table for the query
     long routingStartTimeNs = System.nanoTime();
     Map<ServerInstance, List<String>> offlineRoutingTable = null;
@@ -450,6 +486,24 @@ public abstract class BaseBrokerRequestHandler implements BrokerRequestHandler {
     requestStatistics.setQueryProcessingTime(totalTimeMs);
     requestStatistics.setStatistics(brokerResponse);
 
+    logBrokerResponse(requestId, query, requestStatistics, brokerRequest, numUnavailableSegments, serverStats,
+        brokerResponse, totalTimeMs);
+    return brokerResponse;
+  }
+
+  /** Given a {@link BrokerRequest}, check if the WHERE clause will always evaluate to false. */
+  private boolean isFilterAlwaysFalse(BrokerRequest brokerRequest) {
+    return FALSE.equals(brokerRequest.getPinotQuery().getFilterExpression());
+  }
+
+  /** Given a {@link BrokerRequest}, check if the WHERE clause will always evaluate to true. */
+  private boolean isFilterAlwaysTrue(BrokerRequest brokerRequest) {
+    return TRUE.equals(brokerRequest.getPinotQuery().getFilterExpression());
+  }
+
+  private void logBrokerResponse(long requestId, String query, RequestStatistics requestStatistics,
+      BrokerRequest brokerRequest, int numUnavailableSegments, ServerStats serverStats,
+      BrokerResponseNative brokerResponse, long totalTimeMs) {
     LOGGER.debug("Broker Response: {}", brokerResponse);
 
     // Please keep the format as name=value comma-separated with no spaces
@@ -486,7 +540,6 @@ public abstract class BaseBrokerRequestHandler implements BrokerRequestHandler {
       // Increment the count for dropped log
       _numDroppedLog.incrementAndGet();
     }
-    return brokerResponse;
   }
 
   private String getServerTenant(String tableNameWithType) {
diff --git a/pinot-common/src/main/java/org/apache/pinot/common/utils/request/RequestUtils.java b/pinot-common/src/main/java/org/apache/pinot/common/utils/request/RequestUtils.java
index da4f53a..f23e670 100644
--- a/pinot-common/src/main/java/org/apache/pinot/common/utils/request/RequestUtils.java
+++ b/pinot-common/src/main/java/org/apache/pinot/common/utils/request/RequestUtils.java
@@ -126,6 +126,12 @@ public class RequestUtils {
     return expression;
   }
 
+  public static Expression getLiteralExpression(boolean value) {
+    Expression expression = createNewLiteralExpression();
+    expression.getLiteral().setBoolValue(value);
+    return expression;
+  }
+
   public static Expression getLiteralExpression(long value) {
     Expression expression = createNewLiteralExpression();
     expression.getLiteral().setLongValue(value);
diff --git a/pinot-core/src/main/java/org/apache/pinot/core/query/optimizer/QueryOptimizer.java b/pinot-core/src/main/java/org/apache/pinot/core/query/optimizer/QueryOptimizer.java
index 922fcc2..c91baa8 100644
--- a/pinot-core/src/main/java/org/apache/pinot/core/query/optimizer/QueryOptimizer.java
+++ b/pinot-core/src/main/java/org/apache/pinot/core/query/optimizer/QueryOptimizer.java
@@ -31,12 +31,13 @@ import org.apache.pinot.core.query.optimizer.filter.FilterOptimizer;
 import org.apache.pinot.core.query.optimizer.filter.FlattenAndOrFilterOptimizer;
 import org.apache.pinot.core.query.optimizer.filter.MergeEqInFilterOptimizer;
 import org.apache.pinot.core.query.optimizer.filter.MergeRangeFilterOptimizer;
+import org.apache.pinot.core.query.optimizer.filter.NumericalFilterOptimizer;
 import org.apache.pinot.spi.data.Schema;
 
 
 public class QueryOptimizer {
   private static final List<FilterOptimizer> FILTER_OPTIMIZERS = Arrays
-      .asList(new FlattenAndOrFilterOptimizer(), new MergeEqInFilterOptimizer(), new MergeRangeFilterOptimizer());
+      .asList(new FlattenAndOrFilterOptimizer(), new NumericalFilterOptimizer(), new MergeEqInFilterOptimizer(), new MergeRangeFilterOptimizer());
 
   /**
    * Optimizes the given PQL query.
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 a0eb71d..8b89d2d 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
@@ -27,6 +27,7 @@ import java.util.Map;
 import java.util.Set;
 import javax.annotation.Nullable;
 import org.apache.pinot.common.request.Expression;
+import org.apache.pinot.common.request.ExpressionType;
 import org.apache.pinot.common.request.FilterOperator;
 import org.apache.pinot.common.request.Function;
 import org.apache.pinot.common.utils.request.FilterQueryTree;
@@ -150,7 +151,7 @@ public class MergeEqInFilterOptimizer implements FilterOptimizer {
 
   @Override
   public Expression optimize(Expression filterExpression, @Nullable Schema schema) {
-    return optimize(filterExpression);
+    return filterExpression.getType() == ExpressionType.FUNCTION ? optimize(filterExpression) : filterExpression;
   }
 
   private Expression optimize(Expression filterExpression) {
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 65e46a5..e5148a2 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
@@ -143,7 +143,7 @@ public class MergeRangeFilterOptimizer implements FilterOptimizer {
 
   @Override
   public Expression optimize(Expression filterExpression, @Nullable Schema schema) {
-    if (schema == null) {
+    if (schema == null || filterExpression.getType() != ExpressionType.FUNCTION) {
       return filterExpression;
     }
     Function function = filterExpression.getFunctionCall();
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
new file mode 100644
index 0000000..88ff24c
--- /dev/null
+++ b/pinot-core/src/main/java/org/apache/pinot/core/query/optimizer/filter/NumericalFilterOptimizer.java
@@ -0,0 +1,277 @@
+/**
+ * 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.pinot.core.query.optimizer.filter;
+
+import java.math.BigDecimal;
+import java.util.List;
+import javax.annotation.Nullable;
+import org.apache.pinot.common.request.Expression;
+import org.apache.pinot.common.request.ExpressionType;
+import org.apache.pinot.common.request.Function;
+import org.apache.pinot.common.request.Literal;
+import org.apache.pinot.common.utils.request.FilterQueryTree;
+import org.apache.pinot.common.utils.request.RequestUtils;
+import org.apache.pinot.pql.parsers.pql2.ast.FilterKind;
+import org.apache.pinot.spi.data.FieldSpec;
+import org.apache.pinot.spi.data.Schema;
+
+
+/**
+ * Numerical expressions of form "column = literal" or "column != literal" can compare a column of one datatype
+ * (say INT) with a literal of different datatype (say DOUBLE). These expressions can not be evaluated on the Server.
+ * Hence, we rewrite such expressions into an equivalent expression whose LHS and RHS are of the same datatype.
+ *
+ * Simple predicate examples:
+ *  1) WHERE "intColumn = 5.0"  gets rewritten to "WHERE intColumn = 5"
+ *  2) WHERE "intColumn != 5.0" gets rewritten to "WHERE intColumn != 5"
+ *  3) WHERE "intColumn = 5.5"  gets rewritten to "WHERE false" because INT values can not match 5.5.
+ *  4) WHERE "intColumn = 3000000000 gets rewritten to "WHERE false" because INT values can not match 3000000000.
+ *  5) WHERE "intColumn != 3000000000 gets rewritten to "WHERE true" becuase INT values always not equal to 3000000000.
+ *
+ * Compound predicate examples:
+ *  6) WHERE "intColumn1 = 5.5 AND intColumn2 = intColumn3"
+ *       rewrite to "WHERE false AND intColumn2 = intColumn3"
+ *       rewrite to "WHERE intColumn2 = intColumn3"
+ *  7) WHERE "intColumn1 != 5.5 OR intColumn2 = 5000000000" (5000000000 is out of bounds for integer column)
+ *       rewrite to "WHERE true OR false"
+ *       rewrite to "WHERE true"
+ *       rewrite to query without any WHERE clause.
+ *
+ * When entire predicate gets rewritten to false (Example 3 above), the query will not return any data. Hence, it is
+ * better for the Broker itself to return an empty response rather than sending the query to servers for further
+ * evaluation.
+ */
+public class NumericalFilterOptimizer implements FilterOptimizer {
+
+  private static final Expression TRUE = RequestUtils.getLiteralExpression(true);
+  private static final Expression FALSE = RequestUtils.getLiteralExpression(false);
+
+  @Override
+  public FilterQueryTree optimize(FilterQueryTree filterQueryTree, @Nullable Schema schema) {
+    // Don't do anything here since this is for PQL queries which we no longer support.
+    return filterQueryTree;
+  }
+
+  @Override
+  public Expression optimize(Expression expression, @Nullable Schema schema) {
+    ExpressionType type = expression.getType();
+    if (type != ExpressionType.FUNCTION || schema == null) {
+      // We have nothing to rewrite if expression is not a function or schema is null
+      return expression;
+    }
+
+    Function function = expression.getFunctionCall();
+    List<Expression> operands = function.getOperands();
+    String operator = function.getOperator();
+    if (operator.equals(FilterKind.AND.name()) || operator.equals(FilterKind.OR.name())) {
+      // One of the operands may be an EQUALS or NOT_EQUALS function so recursively traverse the expression tree to see
+      // if we find an EQUALS or NOT_EQUALS function to rewrite.
+      operands.forEach(operand -> optimize(operand, schema));
+
+      // We have rewritten the child operands, so rewrite the parent if needed.
+      return optimizeCurrent(expression);
+    } else if (operator.equals(FilterKind.EQUALS.name()) || operator.equals(FilterKind.NOT_EQUALS.name())) {
+      // Verify that LHS is a numeric column and RHS is a numeric literal before rewriting.
+      Expression lhs = operands.get(0);
+      Expression rhs = operands.get(1);
+      if (isNumericColumn(lhs, schema) && isNumericLiteral(rhs)) {
+        // Rewrite the expression.
+        return rewrite(expression, lhs, rhs, schema);
+      }
+    }
+
+    return expression;
+  }
+
+  /**
+   * If any of the operands of AND function is "false", then the AND function itself is false and can be replaced with
+   * "false" literal. Otherwise, remove all the "true" operands of the AND function. Similarly, if any of the operands
+   * of OR function is "true", then the OR function itself is true and can be replaced with "true" literal. Otherwise,
+   * remove all the "false" operands of the OR function.
+   */
+  private static Expression optimizeCurrent(Expression expression) {
+    Function function = expression.getFunctionCall();
+    List<Expression> operands = function.getOperands();
+    if (function.getOperator().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);
+        }
+      }
+
+      // Remove all Literal operands that are TRUE.
+      operands.removeIf(x -> x.equals(TRUE));
+      if (operands.size() == 0) {
+        return setExpressionToBoolean(expression, true);
+      }
+    } else if (function.getOperator().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);
+        }
+      }
+
+      // Remove all Literal operands that are FALSE.
+      operands.removeIf(x -> x.equals(FALSE));
+      if (operands.size() == 0) {
+        return setExpressionToBoolean(expression, false);
+      }
+    }
+
+    return expression;
+  }
+
+  private boolean isNumericColumn(Expression expression, Schema schema) {
+    if (expression.getType() != ExpressionType.IDENTIFIER) {
+      // Expression can not be a column.
+      return false;
+    }
+
+    String column = expression.getIdentifier().getName();
+    FieldSpec fieldSpec = schema.getFieldSpecFor(column);
+    if (fieldSpec == null || !fieldSpec.isSingleValueField()) {
+      // Expression can not be a column name.
+      return false;
+    }
+
+    return schema.getFieldSpecFor(column).getDataType().isNumeric();
+  }
+
+  private boolean isNumericLiteral(Expression expression) {
+    if (expression.getType() == ExpressionType.LITERAL) {
+      Literal._Fields type = expression.getLiteral().getSetField();
+      switch (type) {
+        case SHORT_VALUE:
+        case INT_VALUE:
+        case LONG_VALUE:
+        case DOUBLE_VALUE:
+          return true;
+      }
+    }
+    return false;
+  }
+
+  /** Change the expression value to boolean literal with given value. */
+  private static Expression setExpressionToBoolean(Expression expression, boolean value) {
+    expression.unsetFunctionCall();
+    expression.setType(ExpressionType.LITERAL);
+    expression.setLiteral(Literal.boolValue(value));
+
+    return expression;
+  }
+
+  /**
+   * Rewrite expressions of form "column = literal" or "column != literal" to ensure that RHS literal is the same
+   * datatype as LHS column.
+   */
+  private Expression rewrite(Expression equals, Expression lhs, Expression rhs, Schema schema) {
+    // Get expression operator
+    boolean result = equals.getFunctionCall().getOperator().equals(FilterKind.NOT_EQUALS.name());
+
+    // Get column data type.
+    FieldSpec.DataType dataType = schema.getFieldSpecFor(lhs.getIdentifier().getName()).getDataType();
+
+    switch (rhs.getLiteral().getSetField()) {
+      case SHORT_VALUE:
+      case INT_VALUE:
+        // No rewrites needed since SHORT and INT conversion to numeric column types (INT, LONG, FLOAT, and DOUBLE) is
+        // lossless and will be implicitly handled on the server side.
+        break;
+      case LONG_VALUE: {
+        long actual = rhs.getLiteral().getLongValue();
+        switch (dataType) {
+          case INT: {
+            int converted = (int) actual;
+            if (converted != actual) {
+              // Long value does not fall within the bounds of INT column.
+              setExpressionToBoolean(equals, result);
+            } else {
+              // Replace long value with converted int value.
+              rhs.getLiteral().setLongValue(converted);
+            }
+            break;
+          }
+          case FLOAT: {
+            float converted = (float) actual;
+            if (BigDecimal.valueOf(actual).compareTo(BigDecimal.valueOf(converted)) != 0) {
+              // Long to float conversion is lossy.
+              setExpressionToBoolean(equals, result);
+            } else {
+              // Replace long value with converted float value.
+              rhs.getLiteral().setDoubleValue(converted);
+            }
+            break;
+          }
+          case DOUBLE: {
+            double converted = (double) actual;
+            if (BigDecimal.valueOf(actual).compareTo(BigDecimal.valueOf(converted)) != 0) {
+              // Long to double conversion is lossy.
+              setExpressionToBoolean(equals, result);
+            } else {
+              // Replace long value with converted double value.
+              rhs.getLiteral().setDoubleValue(converted);
+            }
+            break;
+          }
+        }
+        break;
+      }
+      case DOUBLE_VALUE:
+        double actual = rhs.getLiteral().getDoubleValue();
+        switch (dataType) {
+          case INT: {
+            int converted = (int) actual;
+            if (converted != actual) {
+              // Double value does not fall within the bounds of INT column.
+              setExpressionToBoolean(equals, result);
+            } else {
+              // Replace double value with converted int value.
+              rhs.getLiteral().setLongValue(converted);
+            }
+            break;
+          }
+          case LONG: {
+            long converted = (long) actual;
+            if (BigDecimal.valueOf(actual).compareTo(BigDecimal.valueOf(converted)) != 0) {
+              // Double to long conversion is lossy.
+              setExpressionToBoolean(equals, result);
+            } else {
+              // Replace double value with converted long value.
+              rhs.getLiteral().setLongValue(converted);
+            }
+            break;
+          }
+          case FLOAT: {
+            float converted = (float) actual;
+            if (converted != actual) {
+              // Double to float conversion is lossy.
+              setExpressionToBoolean(equals, result);
+            } else {
+              // Replace double value with converted float value.
+              rhs.getLiteral().setDoubleValue(converted);
+            }
+            break;
+          }
+        }
+    }
+    return equals;
+  }
+}
diff --git a/pinot-core/src/test/java/org/apache/pinot/core/query/optimizer/QueryOptimizerTest.java b/pinot-core/src/test/java/org/apache/pinot/core/query/optimizer/QueryOptimizerTest.java
index ddab506..0c0b7ab 100644
--- a/pinot-core/src/test/java/org/apache/pinot/core/query/optimizer/QueryOptimizerTest.java
+++ b/pinot-core/src/test/java/org/apache/pinot/core/query/optimizer/QueryOptimizerTest.java
@@ -104,8 +104,8 @@ public class QueryOptimizerTest {
     assertEquals(secondChildFunction.getOperator(), FilterKind.AND.name());
     List<Expression> secondChildChildren = secondChildFunction.getOperands();
     assertEquals(secondChildChildren.size(), 3);
-    assertEquals(secondChildChildren.get(0), getEqFilterExpression("long", 5));
-    assertEquals(secondChildChildren.get(1), getEqFilterExpression("float", 9));
+    assertEquals(secondChildChildren.get(0), getEqFilterExpression("long", 5l));
+    assertEquals(secondChildChildren.get(1), getEqFilterExpression("float", 9f));
     assertEquals(secondChildChildren.get(2), getEqFilterExpression("double", 7.5));
   }
 
@@ -166,7 +166,7 @@ public class QueryOptimizerTest {
     List<Expression> children = filterFunction.getOperands();
     assertEquals(children.size(), 3);
     assertEquals(children.get(0), getEqFilterExpression("int", 1));
-    checkInFilterFunction(children.get(1).getFunctionCall(), "long", Arrays.asList(2, 3, 4));
+    checkInFilterFunction(children.get(1).getFunctionCall(), "long", Arrays.asList(2l, 3l, 4l));
 
     Function thirdChildFunction = children.get(2).getFunctionCall();
     assertEquals(thirdChildFunction.getOperator(), FilterKind.OR.name());
@@ -248,7 +248,7 @@ public class QueryOptimizerTest {
     assertEquals(secondChildFunction.getOperator(), FilterKind.AND.name());
     List<Expression> secondChildChildren = secondChildFunction.getOperands();
     assertEquals(secondChildChildren.size(), 2);
-    assertEquals(secondChildChildren.get(0), getEqFilterExpression("float", 6));
+    assertEquals(secondChildChildren.get(0), getEqFilterExpression("float", 6f));
     assertEquals(secondChildChildren.get(1), getRangeFilterExpression("float", "[6.0\0006.5)"));
 
     // Range filter on multi-value column should not be merged ([-5, 10] can match this filter)
diff --git a/pinot-core/src/test/java/org/apache/pinot/core/query/optimizer/filter/NumericalFilterOptimizerTest.java b/pinot-core/src/test/java/org/apache/pinot/core/query/optimizer/filter/NumericalFilterOptimizerTest.java
new file mode 100644
index 0000000..e75aa5f
--- /dev/null
+++ b/pinot-core/src/test/java/org/apache/pinot/core/query/optimizer/filter/NumericalFilterOptimizerTest.java
@@ -0,0 +1,316 @@
+/**
+ * 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.pinot.core.query.optimizer.filter;
+
+import org.apache.pinot.common.request.BrokerRequest;
+import org.apache.pinot.common.request.PinotQuery;
+import org.apache.pinot.core.query.optimizer.QueryOptimizer;
+import org.apache.pinot.spi.data.FieldSpec;
+import org.apache.pinot.spi.data.Schema;
+import org.apache.pinot.sql.parsers.CalciteSqlCompiler;
+import org.testng.Assert;
+import org.testng.annotations.Test;
+
+
+public class NumericalFilterOptimizerTest {
+  private static final QueryOptimizer OPTIMIZER = new QueryOptimizer();
+  private static final CalciteSqlCompiler SQL_COMPILER = new CalciteSqlCompiler();
+  private static final Schema SCHEMA =
+      new Schema.SchemaBuilder().setSchemaName("testTable").addSingleValueDimension("intColumn", FieldSpec.DataType.INT)
+          .addSingleValueDimension("longColumn", FieldSpec.DataType.LONG)
+          .addSingleValueDimension("floatColumn", FieldSpec.DataType.FLOAT)
+          .addSingleValueDimension("doubleColumn", FieldSpec.DataType.DOUBLE)
+          .addSingleValueDimension("stringColumn", FieldSpec.DataType.STRING)
+          .addSingleValueDimension("bytesColumn", FieldSpec.DataType.BYTES)
+          .addMultiValueDimension("mvIntColumn", FieldSpec.DataType.INT).build();
+
+  @Test
+  public void testEqualsIntColumnWithValidDecimalValue() {
+    BrokerRequest sqlBrokerRequest =
+        SQL_COMPILER.compileToBrokerRequest("SELECT * FROM testTable WHERE intColumn = 5.0");
+    PinotQuery pinotQuery = sqlBrokerRequest.getPinotQuery();
+    OPTIMIZER.optimize(pinotQuery, SCHEMA);
+
+    // "intColumn = 5.0" should have been rewritten to "intColumn = 5" since 5.0 converts to integer value without any
+    // loss of information.
+    Assert.assertEquals(pinotQuery.getFilterExpression().toString(),
+        "Expression(type:FUNCTION, functionCall:Function(operator:EQUALS, operands:[Expression(type:IDENTIFIER, identifier:Identifier(name:intColumn)), Expression(type:LITERAL, literal:<Literal longValue:5>)]))");
+  }
+
+  @Test
+  public void testEqualsIntColumnWithInvalidDecimalValue() {
+    BrokerRequest sqlBrokerRequest =
+        SQL_COMPILER.compileToBrokerRequest("SELECT * FROM testTable WHERE intColumn = 5.5");
+    PinotQuery pinotQuery = sqlBrokerRequest.getPinotQuery();
+    OPTIMIZER.optimize(pinotQuery, SCHEMA);
+
+    Assert.assertEquals(pinotQuery.getFilterExpression().toString(),
+        "Expression(type:LITERAL, literal:<Literal boolValue:false>)");
+  }
+
+  @Test
+  public void testNotEqualsIntColumnWithValidDecimalValue() {
+    BrokerRequest sqlBrokerRequest =
+        SQL_COMPILER.compileToBrokerRequest("SELECT * FROM testTable WHERE intColumn != 5.0");
+    PinotQuery pinotQuery = sqlBrokerRequest.getPinotQuery();
+    OPTIMIZER.optimize(pinotQuery, SCHEMA);
+
+    Assert.assertEquals(pinotQuery.getFilterExpression().toString(),
+        "Expression(type:FUNCTION, functionCall:Function(operator:NOT_EQUALS, operands:[Expression(type:IDENTIFIER, identifier:Identifier(name:intColumn)), Expression(type:LITERAL, literal:<Literal longValue:5>)]))");
+  }
+
+  @Test
+  public void testNotEqualsIntColumnWithInvalidDecimalValue() {
+    BrokerRequest sqlBrokerRequest =
+        SQL_COMPILER.compileToBrokerRequest("SELECT * FROM testTable WHERE intColumn != 5.5");
+    PinotQuery pinotQuery = sqlBrokerRequest.getPinotQuery();
+    OPTIMIZER.optimize(pinotQuery, SCHEMA);
+
+    Assert.assertEquals(pinotQuery.getFilterExpression().toString(),
+        "Expression(type:LITERAL, literal:<Literal boolValue:true>)");
+  }
+
+  @Test
+  public void testEqualsIntColumnWithOutOfDomainValue() {
+    BrokerRequest sqlBrokerRequest =
+        SQL_COMPILER.compileToBrokerRequest("SELECT * FROM testTable WHERE intColumn = 5000000000");
+    PinotQuery pinotQuery = sqlBrokerRequest.getPinotQuery();
+    OPTIMIZER.optimize(pinotQuery, SCHEMA);
+
+    Assert.assertEquals(pinotQuery.getFilterExpression().toString(),
+        "Expression(type:LITERAL, literal:<Literal boolValue:false>)");
+  }
+
+  @Test
+  public void testNotEqualsIntColumnWithOutOfDomainValue() {
+    BrokerRequest sqlBrokerRequest =
+        SQL_COMPILER.compileToBrokerRequest("SELECT * FROM testTable WHERE intColumn != 5000000000");
+    PinotQuery pinotQuery = sqlBrokerRequest.getPinotQuery();
+    OPTIMIZER.optimize(pinotQuery, SCHEMA);
+
+    Assert.assertEquals(pinotQuery.getFilterExpression().toString(),
+        "Expression(type:LITERAL, literal:<Literal boolValue:true>)");
+  }
+
+  @Test
+  public void testEqualsLongColumnWithValidDecimalValue() {
+    BrokerRequest sqlBrokerRequest =
+        SQL_COMPILER.compileToBrokerRequest("SELECT * FROM testTable WHERE longColumn = 5.0");
+    PinotQuery pinotQuery = sqlBrokerRequest.getPinotQuery();
+    OPTIMIZER.optimize(pinotQuery, SCHEMA);
+
+    // "intColumn = 5.0" should have been rewritten to "intColumn = 5" since 5.0 converts to integer value without any
+    // loss of information.
+    Assert.assertEquals(pinotQuery.getFilterExpression().toString(),
+        "Expression(type:FUNCTION, functionCall:Function(operator:EQUALS, operands:[Expression(type:IDENTIFIER, identifier:Identifier(name:longColumn)), Expression(type:LITERAL, literal:<Literal longValue:5>)]))");
+  }
+
+  @Test
+  public void testEqualsLongColumnWithInvalidDecimalValue() {
+    BrokerRequest sqlBrokerRequest =
+        SQL_COMPILER.compileToBrokerRequest("SELECT * FROM testTable WHERE longColumn = 5.5");
+    PinotQuery pinotQuery = sqlBrokerRequest.getPinotQuery();
+    OPTIMIZER.optimize(pinotQuery, SCHEMA);
+
+    // "intColumn = 5.5" should have been rewritten to "false" literal since an INT column can never have a decimal
+    // value.
+    Assert.assertEquals(pinotQuery.getFilterExpression().toString(),
+        "Expression(type:LITERAL, literal:<Literal boolValue:false>)");
+  }
+
+  @Test
+  public void testNotEqualsLongColumnWithValidDecimalValue() {
+    BrokerRequest sqlBrokerRequest =
+        SQL_COMPILER.compileToBrokerRequest("SELECT * FROM testTable WHERE longColumn != 5.0");
+    PinotQuery pinotQuery = sqlBrokerRequest.getPinotQuery();
+    OPTIMIZER.optimize(pinotQuery, SCHEMA);
+
+    // "intColumn = 5.5" should have been rewritten to "false" literal since an INT column can never have a decimal
+    // value.
+    Assert.assertEquals(pinotQuery.getFilterExpression().toString(),
+        "Expression(type:FUNCTION, functionCall:Function(operator:NOT_EQUALS, operands:[Expression(type:IDENTIFIER, identifier:Identifier(name:longColumn)), Expression(type:LITERAL, literal:<Literal longValue:5>)]))");
+  }
+
+  @Test
+  public void testNotEqualsLongColumnWithInvalidDecimalValue() {
+    BrokerRequest sqlBrokerRequest =
+        SQL_COMPILER.compileToBrokerRequest("SELECT * FROM testTable WHERE longColumn != 5.5");
+    PinotQuery pinotQuery = sqlBrokerRequest.getPinotQuery();
+    OPTIMIZER.optimize(pinotQuery, SCHEMA);
+
+    Assert.assertEquals(pinotQuery.getFilterExpression().toString(),
+        "Expression(type:LITERAL, literal:<Literal boolValue:true>)");
+
+    System.out.println(Float.MAX_VALUE);
+  }
+
+  @Test
+  public void testEqualsFloatColumnWithValidLongValue() {
+    BrokerRequest sqlBrokerRequest =
+        SQL_COMPILER.compileToBrokerRequest("SELECT * FROM testTable WHERE floatColumn = 5");
+    PinotQuery pinotQuery = sqlBrokerRequest.getPinotQuery();
+    OPTIMIZER.optimize(pinotQuery, SCHEMA);
+
+    Assert.assertEquals(pinotQuery.getFilterExpression().toString(),
+        "Expression(type:FUNCTION, functionCall:Function(operator:EQUALS, operands:[Expression(type:IDENTIFIER, identifier:Identifier(name:floatColumn)), Expression(type:LITERAL, literal:<Literal doubleValue:5.0>)]))");
+  }
+
+  @Test
+  public void testEqualsFloatColumnWithInvalidLongValue() {
+    BrokerRequest sqlBrokerRequest = SQL_COMPILER
+        .compileToBrokerRequest("SELECT * FROM testTable WHERE floatColumn = " + String.valueOf(Long.MAX_VALUE));
+    PinotQuery pinotQuery = sqlBrokerRequest.getPinotQuery();
+    OPTIMIZER.optimize(pinotQuery, SCHEMA);
+
+    Assert.assertEquals(pinotQuery.getFilterExpression().toString(),
+        "Expression(type:LITERAL, literal:<Literal boolValue:false>)");
+  }
+
+  @Test
+  public void testNotEqualsFloatColumnWithValidLongValue() {
+    BrokerRequest sqlBrokerRequest = SQL_COMPILER
+        .compileToBrokerRequest("SELECT * FROM testTable WHERE floatColumn != " + String.valueOf(Long.MAX_VALUE));
+    PinotQuery pinotQuery = sqlBrokerRequest.getPinotQuery();
+    OPTIMIZER.optimize(pinotQuery, SCHEMA);
+
+    Assert.assertEquals(pinotQuery.getFilterExpression().toString(),
+        "Expression(type:LITERAL, literal:<Literal boolValue:true>)");
+  }
+
+  @Test
+  public void testNotEqualsFloatColumnWithInvalidLongValue() {
+    BrokerRequest sqlBrokerRequest =
+        SQL_COMPILER.compileToBrokerRequest("SELECT * FROM testTable WHERE floatColumn != 5");
+    PinotQuery pinotQuery = sqlBrokerRequest.getPinotQuery();
+    OPTIMIZER.optimize(pinotQuery, SCHEMA);
+
+    Assert.assertEquals(pinotQuery.getFilterExpression().toString(),
+        "Expression(type:FUNCTION, functionCall:Function(operator:NOT_EQUALS, operands:[Expression(type:IDENTIFIER, identifier:Identifier(name:floatColumn)), Expression(type:LITERAL, literal:<Literal doubleValue:5.0>)]))");
+  }
+
+  @Test
+  public void testEqualsDoubleColumnWithValidLongValue() {
+    BrokerRequest sqlBrokerRequest =
+        SQL_COMPILER.compileToBrokerRequest("SELECT * FROM testTable WHERE doubleColumn = 5");
+    PinotQuery pinotQuery = sqlBrokerRequest.getPinotQuery();
+    OPTIMIZER.optimize(pinotQuery, SCHEMA);
+
+    Assert.assertEquals(pinotQuery.getFilterExpression().toString(),
+        "Expression(type:FUNCTION, functionCall:Function(operator:EQUALS, operands:[Expression(type:IDENTIFIER, identifier:Identifier(name:doubleColumn)), Expression(type:LITERAL, literal:<Literal doubleValue:5.0>)]))");
+  }
+
+  @Test
+  public void testEqualsDoubleColumnWithInvalidLongValue() {
+    BrokerRequest sqlBrokerRequest = SQL_COMPILER
+        .compileToBrokerRequest("SELECT * FROM testTable WHERE doubleColumn = " + String.valueOf(Long.MAX_VALUE));
+    PinotQuery pinotQuery = sqlBrokerRequest.getPinotQuery();
+    OPTIMIZER.optimize(pinotQuery, SCHEMA);
+
+    Assert.assertEquals(pinotQuery.getFilterExpression().toString(),
+        "Expression(type:LITERAL, literal:<Literal boolValue:false>)");
+  }
+
+  @Test
+  public void testNotEqualsDoubleColumnWithValidLongValue() {
+    BrokerRequest sqlBrokerRequest = SQL_COMPILER
+        .compileToBrokerRequest("SELECT * FROM testTable WHERE doubleColumn != " + String.valueOf(Long.MAX_VALUE));
+    PinotQuery pinotQuery = sqlBrokerRequest.getPinotQuery();
+    OPTIMIZER.optimize(pinotQuery, SCHEMA);
+
+    Assert.assertEquals(pinotQuery.getFilterExpression().toString(),
+        "Expression(type:LITERAL, literal:<Literal boolValue:true>)");
+  }
+
+  @Test
+  public void testNotEqualsDoubleColumnWithInvalidLongValue() {
+    BrokerRequest sqlBrokerRequest =
+        SQL_COMPILER.compileToBrokerRequest("SELECT * FROM testTable WHERE doubleColumn != 5");
+    PinotQuery pinotQuery = sqlBrokerRequest.getPinotQuery();
+    OPTIMIZER.optimize(pinotQuery, SCHEMA);
+
+    Assert.assertEquals(pinotQuery.getFilterExpression().toString(),
+        "Expression(type:FUNCTION, functionCall:Function(operator:NOT_EQUALS, operands:[Expression(type:IDENTIFIER, identifier:Identifier(name:doubleColumn)), Expression(type:LITERAL, literal:<Literal doubleValue:5.0>)]))");
+  }
+
+  @Test
+  public void testAndWithAllTrueOperands() {
+    BrokerRequest sqlBrokerRequest =
+        SQL_COMPILER.compileToBrokerRequest("SELECT * FROM testTable WHERE intColumn != 5.5 AND longColumn != 6.4");
+    PinotQuery pinotQuery = sqlBrokerRequest.getPinotQuery();
+    OPTIMIZER.optimize(pinotQuery, SCHEMA);
+
+    Assert.assertEquals(pinotQuery.getFilterExpression().toString(),
+        "Expression(type:LITERAL, literal:<Literal boolValue:true>)");
+  }
+
+  @Test
+  public void testAndWithAllFalseOperands() {
+    BrokerRequest sqlBrokerRequest =
+        SQL_COMPILER.compileToBrokerRequest("SELECT * FROM testTable WHERE intColumn = 5.5 AND longColumn = 6.4");
+    PinotQuery pinotQuery = sqlBrokerRequest.getPinotQuery();
+    OPTIMIZER.optimize(pinotQuery, SCHEMA);
+
+    Assert.assertEquals(pinotQuery.getFilterExpression().toString(),
+        "Expression(type:LITERAL, literal:<Literal boolValue:false>)");
+  }
+
+  @Test
+  public void testAndWithOneFalseOperands() {
+    BrokerRequest sqlBrokerRequest =
+        SQL_COMPILER.compileToBrokerRequest("SELECT * FROM testTable WHERE intColumn = 5.5 AND longColumn != 6.4");
+    PinotQuery pinotQuery = sqlBrokerRequest.getPinotQuery();
+    OPTIMIZER.optimize(pinotQuery, SCHEMA);
+
+    Assert.assertEquals(pinotQuery.getFilterExpression().toString(),
+        "Expression(type:LITERAL, literal:<Literal boolValue:false>)");
+  }
+
+  @Test
+  public void testOrWithAllTrueOperands() {
+    BrokerRequest sqlBrokerRequest =
+        SQL_COMPILER.compileToBrokerRequest("SELECT * FROM testTable WHERE intColumn != 5.5 OR longColumn != 6.4");
+    PinotQuery pinotQuery = sqlBrokerRequest.getPinotQuery();
+    OPTIMIZER.optimize(pinotQuery, SCHEMA);
+
+    Assert.assertEquals(pinotQuery.getFilterExpression().toString(),
+        "Expression(type:LITERAL, literal:<Literal boolValue:true>)");
+  }
+
+  @Test
+  public void testOrWithAllFalseOperands() {
+    BrokerRequest sqlBrokerRequest =
+        SQL_COMPILER.compileToBrokerRequest("SELECT * FROM testTable WHERE intColumn = 5.5 OR longColumn = 6.4");
+    PinotQuery pinotQuery = sqlBrokerRequest.getPinotQuery();
+    OPTIMIZER.optimize(pinotQuery, SCHEMA);
+
+    Assert.assertEquals(pinotQuery.getFilterExpression().toString(),
+        "Expression(type:LITERAL, literal:<Literal boolValue:false>)");
+  }
+
+  @Test
+  public void testOrWithOneFalseOperands() {
+    BrokerRequest sqlBrokerRequest =
+        SQL_COMPILER.compileToBrokerRequest("SELECT * FROM testTable WHERE intColumn != 5.5 OR longColumn = 6.4");
+    PinotQuery pinotQuery = sqlBrokerRequest.getPinotQuery();
+    OPTIMIZER.optimize(pinotQuery, SCHEMA);
+
+    Assert.assertEquals(pinotQuery.getFilterExpression().toString(),
+        "Expression(type:LITERAL, literal:<Literal boolValue:true>)");
+  }
+}

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