You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@pinot.apache.org by ro...@apache.org on 2022/09/14 22:53:58 UTC

[pinot] branch master updated: [Feature] Support IsDistinctFrom and IsNotDistinctFrom (#9312)

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

rongr 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 d2b65db6df [Feature]  Support IsDistinctFrom and IsNotDistinctFrom (#9312)
d2b65db6df is described below

commit d2b65db6df9d313ce17ff0bb7bdbc8c2898f8828
Author: Yao Liu <ya...@startree.ai>
AuthorDate: Wed Sep 14 15:53:49 2022 -0700

    [Feature]  Support IsDistinctFrom and IsNotDistinctFrom (#9312)
    
    The operators IsDistinctFrom and IsNotDistinctFrom only supports column names as argument for now.
    When null option is disabled, the row is considered as not null by default.
    Expected value:
    `Null is IsDistinctFrom ValueA`: True
    `Null is IsDistinctFrom Null`: False
    `ValueA is IsDistinctFrom ValueB`: `NotEquals(ValueA, ValueB)`
    `Null is IsNotDistinctFrom ValueA`: False
    `Null is IsNotDistinctFrom Null`: True
    `ValueA is IsNotDistinctFrom ValueB`: Equals(ValueA, ValueB)`
    
    Example Usage:
    `ColumnA IsDistinctFrom ColumnB`
    `ColumnA IsNotDistinctFrom ColumnB`
---
 .../common/function/TransformFunctionType.java     |   3 +
 .../function/BinaryOperatorTransformFunction.java  |  14 +-
 .../function/DistinctFromTransformFunction.java    | 124 ++++++++
 .../function/IsDistinctFromTransformFunction.java  |  42 +++
 .../IsNotDistinctFromTransformFunction.java        |  42 +++
 .../function/TransformFunctionFactory.java         |   2 +
 .../DistinctFromTransformFunctionTest.java         | 313 +++++++++++++++++++++
 .../tests/NullHandlingIntegrationTest.java         |  18 ++
 8 files changed, 551 insertions(+), 7 deletions(-)

diff --git a/pinot-common/src/main/java/org/apache/pinot/common/function/TransformFunctionType.java b/pinot-common/src/main/java/org/apache/pinot/common/function/TransformFunctionType.java
index d058ea3169..75e339c6d5 100644
--- a/pinot-common/src/main/java/org/apache/pinot/common/function/TransformFunctionType.java
+++ b/pinot-common/src/main/java/org/apache/pinot/common/function/TransformFunctionType.java
@@ -63,6 +63,9 @@ public enum TransformFunctionType {
   IS_NULL("is_null"),
   IS_NOT_NULL("is_not_null"),
 
+  IS_DISTINCT_FROM("is_distinct_from"),
+  IS_NOT_DISTINCT_FROM("is_not_distinct_from"),
+
   AND("and"),
   OR("or"),
   NOT("not"),   // NOT operator doesn't cover the transform for NOT IN and NOT LIKE
diff --git a/pinot-core/src/main/java/org/apache/pinot/core/operator/transform/function/BinaryOperatorTransformFunction.java b/pinot-core/src/main/java/org/apache/pinot/core/operator/transform/function/BinaryOperatorTransformFunction.java
index cd764cdadd..d1531558ef 100644
--- a/pinot-core/src/main/java/org/apache/pinot/core/operator/transform/function/BinaryOperatorTransformFunction.java
+++ b/pinot-core/src/main/java/org/apache/pinot/core/operator/transform/function/BinaryOperatorTransformFunction.java
@@ -43,13 +43,13 @@ public abstract class BinaryOperatorTransformFunction extends BaseTransformFunct
   private static final int LESS_THAN_OR_EQUAL = 4;
   private static final int NOT_EQUAL = 5;
 
-  private final int _op;
-  private final TransformFunctionType _transformFunctionType;
-  private TransformFunction _leftTransformFunction;
-  private TransformFunction _rightTransformFunction;
-  private DataType _leftStoredType;
-  private DataType _rightStoredType;
-  private int[] _results;
+  protected final int _op;
+  protected final TransformFunctionType _transformFunctionType;
+  protected TransformFunction _leftTransformFunction;
+  protected TransformFunction _rightTransformFunction;
+  protected DataType _leftStoredType;
+  protected DataType _rightStoredType;
+  protected int[] _results;
 
   protected BinaryOperatorTransformFunction(TransformFunctionType transformFunctionType) {
     // translate to integer in [0, 5] for guaranteed tableswitch
diff --git a/pinot-core/src/main/java/org/apache/pinot/core/operator/transform/function/DistinctFromTransformFunction.java b/pinot-core/src/main/java/org/apache/pinot/core/operator/transform/function/DistinctFromTransformFunction.java
new file mode 100644
index 0000000000..1f89423100
--- /dev/null
+++ b/pinot-core/src/main/java/org/apache/pinot/core/operator/transform/function/DistinctFromTransformFunction.java
@@ -0,0 +1,124 @@
+/**
+ * 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.operator.transform.function;
+
+import java.util.List;
+import java.util.Map;
+import javax.annotation.Nullable;
+import org.apache.pinot.common.function.TransformFunctionType;
+import org.apache.pinot.core.operator.blocks.ProjectionBlock;
+import org.apache.pinot.core.operator.transform.TransformResultMetadata;
+import org.apache.pinot.segment.spi.datasource.DataSource;
+import org.roaringbitmap.IntConsumer;
+import org.roaringbitmap.RoaringBitmap;
+
+
+/**
+ * <code>DistinctFromTransformFunction</code> abstracts the transform needed for IsDistinctFrom and IsNotDistinctFrom.
+ * Null value is considered as distinct from non-null value.
+ * When both values are not null, this function calls equal transform function to determined whether two values are
+ * distinct.
+ * This function only supports two arguments which are both column names.
+ */
+public class DistinctFromTransformFunction extends BinaryOperatorTransformFunction {
+  // Result value to save when two values are distinct.
+  // 1 for isDistinct, 0 for isNotDistinct
+  private final int _distinctResult;
+  // Result value to save when two values are not distinct.
+  // 0 for isDistinct, 1 for isNotDistinct
+  private final int _notDistinctResult;
+
+  /**
+   * Returns a bit map of corresponding column.
+   * Returns null by default if null option is disabled.
+   */
+  @Nullable
+  private static RoaringBitmap getNullBitMap(ProjectionBlock projectionBlock, TransformFunction transformFunction) {
+    String columnName = ((IdentifierTransformFunction) transformFunction).getColumnName();
+    return projectionBlock.getBlockValueSet(columnName).getNullBitmap();
+  }
+
+  /**
+   * Returns true when bitmap is null (null option is disabled) or bitmap is empty.
+   */
+  private static boolean isEmpty(RoaringBitmap bitmap) {
+    return bitmap == null || bitmap.isEmpty();
+  }
+
+  /**
+   * @param distinct is set to true for IsDistinctFrom, otherwise it is for IsNotDistinctFrom.
+   */
+  protected DistinctFromTransformFunction(boolean distinct) {
+    super(distinct ? TransformFunctionType.NOT_EQUALS : TransformFunctionType.EQUALS);
+    _distinctResult = distinct ? 1 : 0;
+    _notDistinctResult = distinct ? 0 : 1;
+  }
+
+  @Override
+  public String getName() {
+    if (_distinctResult == 1) {
+      return TransformFunctionType.IS_DISTINCT_FROM.getName();
+    }
+    return TransformFunctionType.IS_NOT_DISTINCT_FROM.getName();
+  }
+
+  @Override
+  public void init(List<TransformFunction> arguments, Map<String, DataSource> dataSourceMap) {
+    super.init(arguments, dataSourceMap);
+    if (!(_leftTransformFunction instanceof IdentifierTransformFunction)
+        || !(_rightTransformFunction instanceof IdentifierTransformFunction)) {
+      throw new IllegalArgumentException("Only column names are supported in DistinctFrom transformation.");
+    }
+  }
+
+  @Override
+  public TransformResultMetadata getResultMetadata() {
+    return BOOLEAN_SV_NO_DICTIONARY_METADATA;
+  }
+
+  @Override
+  public int[] transformToIntValuesSV(ProjectionBlock projectionBlock) {
+    _results = super.transformToIntValuesSV(projectionBlock);
+    RoaringBitmap leftNull = getNullBitMap(projectionBlock, _leftTransformFunction);
+    RoaringBitmap rightNull = getNullBitMap(projectionBlock, _rightTransformFunction);
+    // Both sides are not null.
+    if (isEmpty(leftNull) && isEmpty(rightNull)) {
+      return _results;
+    }
+    // Left side is not null.
+    if (isEmpty(leftNull)) {
+      // Mark right null rows as distinct.
+      rightNull.forEach((IntConsumer) i -> _results[i] = _distinctResult);
+      return _results;
+    }
+    // Right side is not null.
+    if (isEmpty(rightNull)) {
+      // Mark left null rows as distinct.
+      leftNull.forEach((IntConsumer) i -> _results[i] = _distinctResult);
+      return _results;
+    }
+    RoaringBitmap xorNull = RoaringBitmap.xor(leftNull, rightNull);
+    // For rows that with one null and one not null, mark them as distinct
+    xorNull.forEach((IntConsumer) i -> _results[i] = _distinctResult);
+    RoaringBitmap andNull = RoaringBitmap.and(leftNull, rightNull);
+    // For rows that are both null, mark them as not distinct.
+    andNull.forEach((IntConsumer) i -> _results[i] = _notDistinctResult);
+    return _results;
+  }
+}
diff --git a/pinot-core/src/main/java/org/apache/pinot/core/operator/transform/function/IsDistinctFromTransformFunction.java b/pinot-core/src/main/java/org/apache/pinot/core/operator/transform/function/IsDistinctFromTransformFunction.java
new file mode 100644
index 0000000000..d02392c91d
--- /dev/null
+++ b/pinot-core/src/main/java/org/apache/pinot/core/operator/transform/function/IsDistinctFromTransformFunction.java
@@ -0,0 +1,42 @@
+/**
+ * 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.operator.transform.function;
+
+/**
+ * The <code>IsDistinctFromTransformFunction</code> extends <code>DistinctFromTransformFunction</code> to implement the
+ * IS_DISTINCT_FROM operator.
+ *
+ * The results are in boolean format and stored as an integer array with 1 represents true and 0 represents false.
+ * Expected result:
+ * NUll IS_DISTINCT_FROM Value: 1
+ * NUll IS_DISTINCT_FROM Null: 0
+ * ValueA IS_DISTINCT_FROM ValueB: NotEQUALS(ValueA, ValueB)
+ *
+ * Note this operator only takes column names for now.
+ * SQL Syntax:
+ *    columnA IS DISTINCT FROM columnB
+ *
+ * Sample Usage:
+ *    IS_DISTINCT_FROM(columnA, columnB)
+ */
+public class IsDistinctFromTransformFunction extends DistinctFromTransformFunction {
+  public IsDistinctFromTransformFunction() {
+    super(true);
+  }
+}
diff --git a/pinot-core/src/main/java/org/apache/pinot/core/operator/transform/function/IsNotDistinctFromTransformFunction.java b/pinot-core/src/main/java/org/apache/pinot/core/operator/transform/function/IsNotDistinctFromTransformFunction.java
new file mode 100644
index 0000000000..44b7439e88
--- /dev/null
+++ b/pinot-core/src/main/java/org/apache/pinot/core/operator/transform/function/IsNotDistinctFromTransformFunction.java
@@ -0,0 +1,42 @@
+/**
+ * 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.operator.transform.function;
+
+/**
+ * The <code>IsNotDistinctFromTransformFunction</code> extends <code>DistinctFromTransformFunction</code> to
+ * implement the IS_NOT_DISTINCT_FROM operator.
+ *
+ * The results are in boolean format and stored as an integer array with 1 represents true and 0 represents false.
+ * Expected result:
+ * NUll IS_NOT_DISTINCT_FROM Value: 0
+ * NUll IS_NOT_DISTINCT_FROM Null: 1
+ * ValueA IS_NOT_DISTINCT_FROM ValueB: EQUALS(ValueA, ValueB)
+ *
+ * Note this operator only takes column names for now.
+ * SQL Syntax:
+ *    columnA IS_NOT_DISTINCT_FROM columnB
+ *
+ * Sample Usage:
+ *    IS_NOT_DISTINCT_FROM(columnA, columnB)
+ */
+public class IsNotDistinctFromTransformFunction extends DistinctFromTransformFunction {
+  public IsNotDistinctFromTransformFunction() {
+    super(false);
+  }
+}
diff --git a/pinot-core/src/main/java/org/apache/pinot/core/operator/transform/function/TransformFunctionFactory.java b/pinot-core/src/main/java/org/apache/pinot/core/operator/transform/function/TransformFunctionFactory.java
index 672dd4f4a4..dee41efde3 100644
--- a/pinot-core/src/main/java/org/apache/pinot/core/operator/transform/function/TransformFunctionFactory.java
+++ b/pinot-core/src/main/java/org/apache/pinot/core/operator/transform/function/TransformFunctionFactory.java
@@ -202,6 +202,8 @@ public class TransformFunctionFactory {
     typeToImplementation.put(TransformFunctionType.IS_NULL, IsNullTransformFunction.class);
     typeToImplementation.put(TransformFunctionType.IS_NOT_NULL,
         IsNotNullTransformFunction.class);
+    typeToImplementation.put(TransformFunctionType.IS_DISTINCT_FROM, IsDistinctFromTransformFunction.class);
+    typeToImplementation.put(TransformFunctionType.IS_NOT_DISTINCT_FROM, IsNotDistinctFromTransformFunction.class);
 
     // Trignometric functions
     typeToImplementation.put(TransformFunctionType.SIN, SinTransformFunction.class);
diff --git a/pinot-core/src/test/java/org/apache/pinot/core/operator/transform/function/DistinctFromTransformFunctionTest.java b/pinot-core/src/test/java/org/apache/pinot/core/operator/transform/function/DistinctFromTransformFunctionTest.java
new file mode 100644
index 0000000000..71dc9d2132
--- /dev/null
+++ b/pinot-core/src/test/java/org/apache/pinot/core/operator/transform/function/DistinctFromTransformFunctionTest.java
@@ -0,0 +1,313 @@
+/**
+ * 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.operator.transform.function;
+
+import java.io.File;
+import java.util.ArrayList;
+import java.util.HashMap;
+import java.util.List;
+import java.util.Map;
+import java.util.Random;
+import java.util.Set;
+import org.apache.commons.io.FileUtils;
+import org.apache.pinot.common.request.context.ExpressionContext;
+import org.apache.pinot.common.request.context.RequestContextUtils;
+import org.apache.pinot.core.operator.DocIdSetOperator;
+import org.apache.pinot.core.operator.ProjectionOperator;
+import org.apache.pinot.core.operator.blocks.ProjectionBlock;
+import org.apache.pinot.core.operator.filter.MatchAllFilterOperator;
+import org.apache.pinot.core.plan.DocIdSetPlanNode;
+import org.apache.pinot.segment.local.indexsegment.immutable.ImmutableSegmentLoader;
+import org.apache.pinot.segment.local.segment.creator.impl.SegmentIndexCreationDriverImpl;
+import org.apache.pinot.segment.local.segment.readers.GenericRowRecordReader;
+import org.apache.pinot.segment.spi.IndexSegment;
+import org.apache.pinot.segment.spi.creator.SegmentGeneratorConfig;
+import org.apache.pinot.segment.spi.datasource.DataSource;
+import org.apache.pinot.spi.config.table.TableConfig;
+import org.apache.pinot.spi.config.table.TableType;
+import org.apache.pinot.spi.data.FieldSpec;
+import org.apache.pinot.spi.data.Schema;
+import org.apache.pinot.spi.data.readers.GenericRow;
+import org.apache.pinot.spi.utils.ReadMode;
+import org.apache.pinot.spi.utils.builder.TableConfigBuilder;
+import org.testng.Assert;
+import org.testng.annotations.BeforeClass;
+import org.testng.annotations.Test;
+
+
+public class DistinctFromTransformFunctionTest {
+  private static final String ENABLE_NULL_SEGMENT_NAME = "testSegment1";
+  private static final String DISABLE_NULL_SEGMENT_NAME = "testSegment2";
+  private static final String IS_DISTINCT_FROM_EXPR = "%s IS DISTINCT FROM %s";
+  private static final String IS_NOT_DISTINCT_FROM_EXPR = "%s IS NOT DISTINCT FROM %s";
+  private static final Random RANDOM = new Random();
+
+  private static final int NUM_ROWS = 1000;
+  private static final String INT_SV_COLUMN = "intSV";
+  private static final String INT_SV_NULL_COLUMN = "intSV2";
+  private final int[] _intSVValues = new int[NUM_ROWS];
+  private Map<String, DataSource> _enableNullDataSourceMap;
+  private Map<String, DataSource> _disableNullDataSourceMap;
+  private ProjectionBlock _enableNullProjectionBlock;
+  private ProjectionBlock _disableNullProjectionBlock;
+  protected static final int VALUE_MOD = 3;
+
+  private static String getIndexDirPath(String segmentName) {
+    return FileUtils.getTempDirectoryPath() + File.separator + segmentName;
+  }
+
+  private static Map<String, DataSource> getDataSourceMap(Schema schema, List<GenericRow> rows, String segmentName)
+      throws Exception {
+    TableConfig tableConfig =
+        new TableConfigBuilder(TableType.OFFLINE).setTableName(segmentName).setNullHandlingEnabled(true).build();
+    SegmentGeneratorConfig config = new SegmentGeneratorConfig(tableConfig, schema);
+    config.setOutDir(getIndexDirPath(segmentName));
+    config.setSegmentName(segmentName);
+    SegmentIndexCreationDriverImpl driver = new SegmentIndexCreationDriverImpl();
+    driver.init(config, new GenericRowRecordReader(rows));
+    driver.build();
+    IndexSegment indexSegment =
+        ImmutableSegmentLoader.load(new File(getIndexDirPath(segmentName), segmentName), ReadMode.heap);
+    Set<String> columnNames = indexSegment.getPhysicalColumnNames();
+    Map<String, DataSource> enableNullDataSourceMap = new HashMap<>(columnNames.size());
+    for (String columnName : columnNames) {
+      enableNullDataSourceMap.put(columnName, indexSegment.getDataSource(columnName));
+    }
+    return enableNullDataSourceMap;
+  }
+
+  private static ProjectionBlock getProjectionBlock(Map<String, DataSource> dataSourceMap) {
+    return new ProjectionOperator(dataSourceMap,
+        new DocIdSetOperator(new MatchAllFilterOperator(NUM_ROWS), DocIdSetPlanNode.MAX_DOC_PER_CALL)).nextBlock();
+  }
+
+  private static boolean isEqualRow(int i) {
+    return i % VALUE_MOD == 0;
+  }
+
+  private static boolean isNotEqualRow(int i) {
+    return i % VALUE_MOD == 1;
+  }
+
+  private static boolean isNullRow(int i) {
+    return i % VALUE_MOD == 2;
+  }
+
+  @BeforeClass
+  public void setup()
+      throws Exception {
+    // Set up two tables: one with null option enable, the other with null option disable.
+    // Each table has two int columns.
+    // One column with every row filled in with random integer number.
+    // The other column has 1/3 rows equal to first column, 1/3 rows not equal to first column and 1/3 null rows.
+    FileUtils.deleteQuietly(new File(getIndexDirPath(DISABLE_NULL_SEGMENT_NAME)));
+    FileUtils.deleteQuietly(new File(getIndexDirPath(ENABLE_NULL_SEGMENT_NAME)));
+    for (int i = 0; i < NUM_ROWS; i++) {
+      _intSVValues[i] = RANDOM.nextInt();
+    }
+    List<GenericRow> rows = new ArrayList<>(NUM_ROWS);
+    for (int i = 0; i < NUM_ROWS; i++) {
+      Map<String, Object> map = new HashMap<>();
+      map.put(INT_SV_COLUMN, _intSVValues[i]);
+      if (isEqualRow(i)) {
+        map.put(INT_SV_NULL_COLUMN, _intSVValues[i]);
+      } else if (isNotEqualRow(i)) {
+        map.put(INT_SV_NULL_COLUMN, _intSVValues[i] + 1);
+      } else if (isNullRow(i)) {
+        map.put(INT_SV_NULL_COLUMN, null);
+      }
+      GenericRow row = new GenericRow();
+      row.init(map);
+      rows.add(row);
+    }
+
+    Schema schema = new Schema.SchemaBuilder().addSingleValueDimension(INT_SV_COLUMN, FieldSpec.DataType.INT)
+        .addSingleValueDimension(INT_SV_NULL_COLUMN, FieldSpec.DataType.INT).build();
+    _enableNullDataSourceMap = getDataSourceMap(schema, rows, ENABLE_NULL_SEGMENT_NAME);
+    _enableNullProjectionBlock = getProjectionBlock(_enableNullDataSourceMap);
+    _disableNullDataSourceMap = getDataSourceMap(schema, rows, DISABLE_NULL_SEGMENT_NAME);
+    _disableNullProjectionBlock = getProjectionBlock(_disableNullDataSourceMap);
+  }
+
+  protected void testTransformFunction(ExpressionContext expression, boolean[] expectedValues,
+      ProjectionBlock projectionBlock, Map<String, DataSource> dataSourceMap)
+      throws Exception {
+    int[] intValues = getTransformFunctionInstance(expression, dataSourceMap).transformToIntValuesSV(projectionBlock);
+    long[] longValues =
+        getTransformFunctionInstance(expression, dataSourceMap).transformToLongValuesSV(projectionBlock);
+    float[] floatValues =
+        getTransformFunctionInstance(expression, dataSourceMap).transformToFloatValuesSV(projectionBlock);
+    double[] doubleValues =
+        getTransformFunctionInstance(expression, dataSourceMap).transformToDoubleValuesSV(projectionBlock);
+    String[] stringValues =
+        getTransformFunctionInstance(expression, dataSourceMap).transformToStringValuesSV(projectionBlock);
+    for (int i = 0; i < NUM_ROWS; i++) {
+      Assert.assertEquals(intValues[i] == 1, expectedValues[i]);
+      Assert.assertEquals(longValues[i] == 1, expectedValues[i]);
+      Assert.assertEquals(floatValues[i] == 1, expectedValues[i]);
+      Assert.assertEquals(doubleValues[i] == 1, expectedValues[i]);
+      Assert.assertEquals(stringValues[i], Boolean.toString(expectedValues[i]));
+    }
+  }
+
+  private TransformFunction getTransformFunctionInstance(ExpressionContext expression,
+      Map<String, DataSource> dataSourceMap) {
+    return TransformFunctionFactory.get(expression, dataSourceMap);
+  }
+
+  // Test that left column of the operator has null values and right column is not null.
+  @Test
+  public void testDistinctFromLeftNull()
+      throws Exception {
+    ExpressionContext isDistinctFromExpression =
+        RequestContextUtils.getExpression(String.format(IS_DISTINCT_FROM_EXPR, INT_SV_NULL_COLUMN, INT_SV_COLUMN));
+    TransformFunction isDistinctFromTransformFunction =
+        TransformFunctionFactory.get(isDistinctFromExpression, _enableNullDataSourceMap);
+    Assert.assertEquals(isDistinctFromTransformFunction.getName(), "is_distinct_from");
+    ExpressionContext isNotDistinctFromExpression =
+        RequestContextUtils.getExpression(String.format(IS_NOT_DISTINCT_FROM_EXPR, INT_SV_NULL_COLUMN, INT_SV_COLUMN));
+    TransformFunction isNotDistinctFromTransformFunction =
+        TransformFunctionFactory.get(isNotDistinctFromExpression, _enableNullDataSourceMap);
+    Assert.assertEquals(isNotDistinctFromTransformFunction.getName(), "is_not_distinct_from");
+    boolean[] isDistinctFromExpectedIntValues = new boolean[NUM_ROWS];
+    boolean[] isNotDistinctFromExpectedIntValues = new boolean[NUM_ROWS];
+    for (int i = 0; i < NUM_ROWS; i++) {
+      if (isEqualRow(i)) {
+        isDistinctFromExpectedIntValues[i] = false;
+        isNotDistinctFromExpectedIntValues[i] = true;
+      } else if (isNotEqualRow(i)) {
+        isDistinctFromExpectedIntValues[i] = true;
+        isNotDistinctFromExpectedIntValues[i] = false;
+      } else if (isNullRow(i)) {
+        isDistinctFromExpectedIntValues[i] = true;
+        isNotDistinctFromExpectedIntValues[i] = false;
+      }
+    }
+    testTransformFunction(isDistinctFromExpression, isDistinctFromExpectedIntValues, _enableNullProjectionBlock,
+        _enableNullDataSourceMap);
+    testTransformFunction(isNotDistinctFromExpression, isNotDistinctFromExpectedIntValues, _enableNullProjectionBlock,
+        _enableNullDataSourceMap);
+    testTransformFunction(isDistinctFromExpression, isDistinctFromExpectedIntValues, _disableNullProjectionBlock,
+        _disableNullDataSourceMap);
+    testTransformFunction(isNotDistinctFromExpression, isNotDistinctFromExpectedIntValues, _disableNullProjectionBlock,
+        _disableNullDataSourceMap);
+  }
+
+  // Test that right column of the operator has null values and left column is not null.
+  @Test
+  public void testDistinctFromRightNull()
+      throws Exception {
+    ExpressionContext isDistinctFromExpression =
+        RequestContextUtils.getExpression(String.format(IS_DISTINCT_FROM_EXPR, INT_SV_COLUMN, INT_SV_NULL_COLUMN));
+    TransformFunction isDistinctFromTransformFunction =
+        TransformFunctionFactory.get(isDistinctFromExpression, _enableNullDataSourceMap);
+    Assert.assertEquals(isDistinctFromTransformFunction.getName(), "is_distinct_from");
+    ExpressionContext isNotDistinctFromExpression =
+        RequestContextUtils.getExpression(String.format(IS_NOT_DISTINCT_FROM_EXPR, INT_SV_COLUMN, INT_SV_NULL_COLUMN));
+    TransformFunction isNotDistinctFromTransformFunction =
+        TransformFunctionFactory.get(isNotDistinctFromExpression, _enableNullDataSourceMap);
+    Assert.assertEquals(isNotDistinctFromTransformFunction.getName(), "is_not_distinct_from");
+    boolean[] isDistinctFromExpectedIntValues = new boolean[NUM_ROWS];
+    boolean[] isNotDistinctFromExpectedIntValues = new boolean[NUM_ROWS];
+    for (int i = 0; i < NUM_ROWS; i++) {
+      if (isEqualRow(i)) {
+        isDistinctFromExpectedIntValues[i] = false;
+        isNotDistinctFromExpectedIntValues[i] = true;
+      } else if (isNotEqualRow(i)) {
+        isDistinctFromExpectedIntValues[i] = true;
+        isNotDistinctFromExpectedIntValues[i] = false;
+      } else if (isNullRow(i)) {
+        isDistinctFromExpectedIntValues[i] = true;
+        isNotDistinctFromExpectedIntValues[i] = false;
+      }
+    }
+    testTransformFunction(isDistinctFromExpression, isDistinctFromExpectedIntValues, _enableNullProjectionBlock,
+        _enableNullDataSourceMap);
+    testTransformFunction(isNotDistinctFromExpression, isNotDistinctFromExpectedIntValues, _enableNullProjectionBlock,
+        _enableNullDataSourceMap);
+    testTransformFunction(isDistinctFromExpression, isDistinctFromExpectedIntValues, _disableNullProjectionBlock,
+        _disableNullDataSourceMap);
+    testTransformFunction(isNotDistinctFromExpression, isNotDistinctFromExpectedIntValues, _disableNullProjectionBlock,
+        _disableNullDataSourceMap);
+  }
+
+  // Test the cases where both left and right columns of th operator has null values.
+  @Test
+  public void testDistinctFromBothNull()
+      throws Exception {
+    ExpressionContext isDistinctFromExpression =
+        RequestContextUtils.getExpression(String.format(IS_DISTINCT_FROM_EXPR, INT_SV_NULL_COLUMN, INT_SV_NULL_COLUMN));
+    TransformFunction isDistinctFromTransformFunction =
+        TransformFunctionFactory.get(isDistinctFromExpression, _enableNullDataSourceMap);
+    Assert.assertEquals(isDistinctFromTransformFunction.getName(), "is_distinct_from");
+    ExpressionContext isNotDistinctFromExpression = RequestContextUtils.getExpression(
+        String.format(IS_NOT_DISTINCT_FROM_EXPR, INT_SV_NULL_COLUMN, INT_SV_NULL_COLUMN));
+    TransformFunction isNotDistinctFromTransformFunction =
+        TransformFunctionFactory.get(isNotDistinctFromExpression, _enableNullDataSourceMap);
+    Assert.assertEquals(isNotDistinctFromTransformFunction.getName(), "is_not_distinct_from");
+    boolean[] isDistinctFromExpectedIntValues = new boolean[NUM_ROWS];
+    boolean[] isNotDistinctFromExpectedIntValues = new boolean[NUM_ROWS];
+    for (int i = 0; i < NUM_ROWS; i++) {
+      isDistinctFromExpectedIntValues[i] = false;
+      isNotDistinctFromExpectedIntValues[i] = true;
+    }
+    testTransformFunction(isDistinctFromExpression, isDistinctFromExpectedIntValues, _enableNullProjectionBlock,
+        _enableNullDataSourceMap);
+    testTransformFunction(isNotDistinctFromExpression, isNotDistinctFromExpectedIntValues, _enableNullProjectionBlock,
+        _enableNullDataSourceMap);
+    testTransformFunction(isDistinctFromExpression, isDistinctFromExpectedIntValues, _disableNullProjectionBlock,
+        _disableNullDataSourceMap);
+    testTransformFunction(isNotDistinctFromExpression, isNotDistinctFromExpectedIntValues, _disableNullProjectionBlock,
+        _disableNullDataSourceMap);
+  }
+
+  // Test that non-column-names appear in one side of the operator.
+  @Test
+  public void testIllegalColumnName()
+      throws Exception {
+    ExpressionContext isDistinctFromExpression =
+        RequestContextUtils.getExpression(String.format(IS_DISTINCT_FROM_EXPR, _intSVValues[0], INT_SV_NULL_COLUMN));
+    ExpressionContext isNotDistinctFromExpression = RequestContextUtils.getExpression(
+        String.format(IS_NOT_DISTINCT_FROM_EXPR, _intSVValues[0], INT_SV_NULL_COLUMN));
+
+    Assert.assertThrows(RuntimeException.class, () -> {
+      TransformFunctionFactory.get(isDistinctFromExpression, _enableNullDataSourceMap);
+    });
+    Assert.assertThrows(RuntimeException.class, () -> {
+      TransformFunctionFactory.get(isNotDistinctFromExpression, _enableNullDataSourceMap);
+    });
+  }
+
+  // Test that more than 2 arguments appear for the operator.
+  @Test
+  public void testIllegalNumArgs()
+      throws Exception {
+    ExpressionContext isDistinctFromExpression = RequestContextUtils.getExpression(
+        String.format("is_distinct_from(%s, %s, %s)", INT_SV_COLUMN, INT_SV_NULL_COLUMN, INT_SV_COLUMN));
+    ExpressionContext isNotDistinctFromExpression = RequestContextUtils.getExpression(
+        String.format("is_not_distinct_from(%s, %s, %s)", INT_SV_COLUMN, INT_SV_NULL_COLUMN, INT_SV_COLUMN));
+
+    Assert.assertThrows(RuntimeException.class, () -> {
+      TransformFunctionFactory.get(isDistinctFromExpression, _enableNullDataSourceMap);
+    });
+    Assert.assertThrows(RuntimeException.class, () -> {
+      TransformFunctionFactory.get(isNotDistinctFromExpression, _enableNullDataSourceMap);
+    });
+  }
+}
diff --git a/pinot-integration-tests/src/test/java/org/apache/pinot/integration/tests/NullHandlingIntegrationTest.java b/pinot-integration-tests/src/test/java/org/apache/pinot/integration/tests/NullHandlingIntegrationTest.java
index b409362439..c74e9ade80 100644
--- a/pinot-integration-tests/src/test/java/org/apache/pinot/integration/tests/NullHandlingIntegrationTest.java
+++ b/pinot-integration-tests/src/test/java/org/apache/pinot/integration/tests/NullHandlingIntegrationTest.java
@@ -168,4 +168,22 @@ public class NullHandlingIntegrationTest extends BaseClusterIntegrationTestSet {
     String query = "SELECT CASE WHEN description IS NOT NULL THEN 1 ELSE 0 END FROM " + getTableName();
     testQuery(query);
   }
+
+  @Test
+  public void testCaseWithIsDistinctFrom()
+      throws Exception {
+    String query = "SELECT salary IS DISTINCT FROM salary FROM " + getTableName();
+    testQuery(query);
+    query = "SELECT salary FROM " + getTableName() + " where salary IS DISTINCT FROM salary";
+    testQuery(query);
+  }
+
+  @Test
+  public void testCaseWithIsNotDistinctFrom()
+      throws Exception {
+    String query = "SELECT description IS NOT DISTINCT FROM description FROM " + getTableName();
+    testQuery(query);
+    query = "SELECT description FROM " + getTableName() + " where description IS NOT DISTINCT FROM description";
+    testQuery(query);
+  }
 }


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