You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@iotdb.apache.org by ja...@apache.org on 2024/04/18 02:03:16 UTC

(iotdb) branch ty/TableModelGrammar updated (4e3e0fdb240 -> eff3915bcfd)

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

jackietien pushed a change to branch ty/TableModelGrammar
in repository https://gitbox.apache.org/repos/asf/iotdb.git


    from 4e3e0fdb240 add temp index scan, prune column, distribute query planner
     new 64bd103f54e add some filter converter
     new d14e3f21fd3 partial ok
     new 7ffa882b6fc partial ok
     new a02b95265c2 Merge branch 'ty/TableModelGrammar' of https://github.com/apache/iotdb into ty/TableModelGrammar
     new eff3915bcfd resolve conflicts

The 5 revisions listed above as "new" are entirely new to this
repository and will be described in separate emails.  The revisions
listed as "add" were already present in the repository and have only
been added to this reference.


Summary of changes:
 .../iotdb/db/protocol/session/IClientSession.java  |  34 +-
 .../iotdb/db/protocol/session/SessionManager.java  |   3 +-
 .../iotdb/db/queryengine/common/SessionInfo.java   |  37 +-
 .../operator/process/FilterAndProjectOperator.java |   9 +
 .../source/relational/TableScanOperator.java       |  42 +-
 .../relational/ColumnTransformerBuilder.java       | 674 +++++++++++++++++++++
 .../queryengine/plan/analyze/PredicateUtils.java   |  35 ++
 .../plan/planner/LocalExecutionPlanner.java        |  14 +-
 .../plan/planner/TableOperatorGenerator.java       | 341 ++++++++++-
 ...Predicate.java => TableModelTimePredicate.java} |  11 +-
 .../predicate/ConvertPredicateToFilterVisitor.java | 371 ++++++++++++
 .../ConvertPredicateToTimeFilterVisitor.java       | 218 +++++++
 .../predicate/PredicatePushIntoScanChecker.java    | 131 ++++
 .../analyzer/predicate/PredicateVisitor.java       |  87 +++
 .../plan/relational/planner/node/LimitNode.java    |   4 +
 .../plan/relational/planner/node/OffsetNode.java   |   4 +
 .../plan/relational/planner/node/ProjectNode.java  |   4 +
 .../relational/planner/node/TableScanNode.java     |   8 +-
 .../LogicalAndMultiColumnTransformer.java}         |  55 +-
 .../LogicalMultiColumnTransformer.java}            |  22 +-
 .../LogicalOrMultiColumnTransformer.java}          |  50 +-
 .../MultiColumnTransformer.java}                   |  43 +-
 .../dag/column/unary/InColumnTransformer.java      |  80 +++
 .../plan/relational/analyzer/AnalyzerTest.java     |   5 +-
 .../tsfile/read/filter/factory/FilterFactory.java  |  18 +
 25 files changed, 2175 insertions(+), 125 deletions(-)
 create mode 100644 iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/execution/relational/ColumnTransformerBuilder.java
 copy iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/planner/plan/{TreeModelTimePredicate.java => TableModelTimePredicate.java} (83%)
 create mode 100644 iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/relational/analyzer/predicate/ConvertPredicateToFilterVisitor.java
 create mode 100644 iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/relational/analyzer/predicate/ConvertPredicateToTimeFilterVisitor.java
 create mode 100644 iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/relational/analyzer/predicate/PredicatePushIntoScanChecker.java
 create mode 100644 iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/relational/analyzer/predicate/PredicateVisitor.java
 copy iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/transformation/dag/column/{binary/LogicOrColumnTransformer.java => multi/LogicalAndMultiColumnTransformer.java} (56%)
 copy iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/transformation/dag/column/{binary/LogicBinaryColumnTransformer.java => multi/LogicalMultiColumnTransformer.java} (67%)
 copy iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/transformation/dag/column/{binary/ArithmeticBinaryColumnTransformer.java => multi/LogicalOrMultiColumnTransformer.java} (56%)
 copy iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/transformation/dag/column/{binary/BinaryColumnTransformer.java => multi/MultiColumnTransformer.java} (60%)


(iotdb) 02/05: partial ok

Posted by ja...@apache.org.
This is an automated email from the ASF dual-hosted git repository.

jackietien pushed a commit to branch ty/TableModelGrammar
in repository https://gitbox.apache.org/repos/asf/iotdb.git

commit d14e3f21fd382e70c16c56326a1860b8fd590c65
Author: JackieTien97 <ja...@gmail.com>
AuthorDate: Wed Apr 17 21:41:36 2024 +0800

    partial ok
---
 .../operator/process/FilterAndProjectOperator.java |   9 +
 .../source/relational/TableScanOperator.java       |  26 +-
 .../relational/ColumnTransformerBuilder.java       | 674 +++++++++++++++++++++
 .../queryengine/plan/analyze/PredicateUtils.java   |  12 +-
 .../plan/planner/TableOperatorGenerator.java       | 192 ++++--
 .../predicate/ConvertPredicateToFilterVisitor.java | 351 ++++++++++-
 .../ConvertPredicateToTimeFilterVisitor.java       |   2 +-
 .../predicate/PredicatePushIntoScanChecker.java    |   2 +-
 .../plan/relational/planner/node/ProjectNode.java  |   4 +
 .../multi/LogicalAndMultiColumnTransformer.java    |  64 ++
 .../multi/LogicalMultiColumnTransformer.java}      |  25 +-
 .../multi/LogicalOrMultiColumnTransformer.java     |  63 ++
 .../dag/column/multi/MultiColumnTransformer.java   |  66 ++
 .../dag/column/unary/InColumnTransformer.java      |  80 +++
 14 files changed, 1510 insertions(+), 60 deletions(-)

diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/execution/operator/process/FilterAndProjectOperator.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/execution/operator/process/FilterAndProjectOperator.java
index f62658c68de..3c1c45036c3 100644
--- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/execution/operator/process/FilterAndProjectOperator.java
+++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/execution/operator/process/FilterAndProjectOperator.java
@@ -27,6 +27,7 @@ import org.apache.iotdb.db.queryengine.transformation.dag.column.binary.BinaryCo
 import org.apache.iotdb.db.queryengine.transformation.dag.column.leaf.IdentityColumnTransformer;
 import org.apache.iotdb.db.queryengine.transformation.dag.column.leaf.LeafColumnTransformer;
 import org.apache.iotdb.db.queryengine.transformation.dag.column.multi.MappableUDFColumnTransformer;
+import org.apache.iotdb.db.queryengine.transformation.dag.column.multi.MultiColumnTransformer;
 import org.apache.iotdb.db.queryengine.transformation.dag.column.ternary.TernaryColumnTransformer;
 import org.apache.iotdb.db.queryengine.transformation.dag.column.unary.UnaryColumnTransformer;
 import org.apache.iotdb.tsfile.common.conf.TSFileDescriptor;
@@ -43,6 +44,7 @@ import com.google.common.util.concurrent.ListenableFuture;
 
 import java.util.ArrayList;
 import java.util.List;
+import java.util.OptionalInt;
 
 public class FilterAndProjectOperator implements ProcessOperator {
 
@@ -357,6 +359,13 @@ public class FilterAndProjectOperator implements ProcessOperator {
                   ((CaseWhenThenColumnTransformer) columnTransformer).getElseTransformer()));
       childMaxLevel = Math.max(childMaxLevel, childCount + 2);
       return childMaxLevel;
+    } else if (columnTransformer instanceof MultiColumnTransformer) {
+      int childrenCount = ((MultiColumnTransformer) columnTransformer).getChildren().size();
+      OptionalInt childMaxLevel =
+          ((MultiColumnTransformer) columnTransformer)
+              .getChildren().stream().mapToInt(this::getMaxLevelOfColumnTransformerTree).max();
+
+      return Math.max(childrenCount + 1, childMaxLevel.orElse(childrenCount + 1));
     } else {
       throw new UnsupportedOperationException("Unsupported ColumnTransformer");
     }
diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/execution/operator/source/relational/TableScanOperator.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/execution/operator/source/relational/TableScanOperator.java
index 1e26fa5d636..a8f3a6da567 100644
--- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/execution/operator/source/relational/TableScanOperator.java
+++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/execution/operator/source/relational/TableScanOperator.java
@@ -315,7 +315,7 @@ public class TableScanOperator extends AbstractDataSourceOperator {
   }
 
   @Override
-  protected List<TSDataType> getResultDataTypes() {
+  public List<TSDataType> getResultDataTypes() {
     List<TSDataType> resultDataTypes = new ArrayList<>(columnSchemas.size());
     for (ColumnSchema columnSchema : columnSchemas) {
       resultDataTypes.add(getTSDataType(columnSchema.getType()));
@@ -343,7 +343,22 @@ public class TableScanOperator extends AbstractDataSourceOperator {
   }
 
   private AlignedSeriesScanUtil constructAlignedSeriesScanUtil(DeviceEntry deviceEntry) {
+    AlignedPath alignedPath =
+        constructAlignedPath(deviceEntry, measurementColumnNames, measurementSchemas);
 
+    return new AlignedSeriesScanUtil(
+        alignedPath,
+        scanOrder,
+        seriesScanOptions,
+        operatorContext.getInstanceContext(),
+        true,
+        measurementColumnTSDataTypes);
+  }
+
+  public static AlignedPath constructAlignedPath(
+      DeviceEntry deviceEntry,
+      List<String> measurementColumnNames,
+      List<IMeasurementSchema> measurementSchemas) {
     String[] devicePath = new String[1 + deviceEntry.getDeviceID().segmentNum()];
     devicePath[0] = "root";
     for (int i = 1; i < devicePath.length; i++) {
@@ -353,13 +368,6 @@ public class TableScanOperator extends AbstractDataSourceOperator {
 
     alignedPath.setMeasurementList(measurementColumnNames);
     alignedPath.setSchemaList(measurementSchemas);
-
-    return new AlignedSeriesScanUtil(
-        alignedPath,
-        scanOrder,
-        seriesScanOptions,
-        operatorContext.getInstanceContext(),
-        true,
-        measurementColumnTSDataTypes);
+    return alignedPath;
   }
 }
diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/execution/relational/ColumnTransformerBuilder.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/execution/relational/ColumnTransformerBuilder.java
new file mode 100644
index 00000000000..2685376f87a
--- /dev/null
+++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/execution/relational/ColumnTransformerBuilder.java
@@ -0,0 +1,674 @@
+/*
+ * 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.iotdb.db.queryengine.execution.relational;
+
+import org.apache.iotdb.db.queryengine.common.SessionInfo;
+import org.apache.iotdb.db.queryengine.plan.analyze.TypeProvider;
+import org.apache.iotdb.db.queryengine.plan.planner.plan.parameter.InputLocation;
+import org.apache.iotdb.db.queryengine.plan.relational.planner.Symbol;
+import org.apache.iotdb.db.queryengine.transformation.dag.column.ColumnTransformer;
+import org.apache.iotdb.db.queryengine.transformation.dag.column.binary.ArithmeticAdditionColumnTransformer;
+import org.apache.iotdb.db.queryengine.transformation.dag.column.binary.ArithmeticDivisionColumnTransformer;
+import org.apache.iotdb.db.queryengine.transformation.dag.column.binary.ArithmeticModuloColumnTransformer;
+import org.apache.iotdb.db.queryengine.transformation.dag.column.binary.ArithmeticMultiplicationColumnTransformer;
+import org.apache.iotdb.db.queryengine.transformation.dag.column.binary.ArithmeticSubtractionColumnTransformer;
+import org.apache.iotdb.db.queryengine.transformation.dag.column.binary.CompareEqualToColumnTransformer;
+import org.apache.iotdb.db.queryengine.transformation.dag.column.binary.CompareGreaterEqualColumnTransformer;
+import org.apache.iotdb.db.queryengine.transformation.dag.column.binary.CompareGreaterThanColumnTransformer;
+import org.apache.iotdb.db.queryengine.transformation.dag.column.binary.CompareLessEqualColumnTransformer;
+import org.apache.iotdb.db.queryengine.transformation.dag.column.binary.CompareLessThanColumnTransformer;
+import org.apache.iotdb.db.queryengine.transformation.dag.column.binary.CompareNonEqualColumnTransformer;
+import org.apache.iotdb.db.queryengine.transformation.dag.column.leaf.ConstantColumnTransformer;
+import org.apache.iotdb.db.queryengine.transformation.dag.column.leaf.IdentityColumnTransformer;
+import org.apache.iotdb.db.queryengine.transformation.dag.column.leaf.LeafColumnTransformer;
+import org.apache.iotdb.db.queryengine.transformation.dag.column.leaf.NullColumnTransformer;
+import org.apache.iotdb.db.queryengine.transformation.dag.column.multi.LogicalAndMultiColumnTransformer;
+import org.apache.iotdb.db.queryengine.transformation.dag.column.multi.LogicalOrMultiColumnTransformer;
+import org.apache.iotdb.db.queryengine.transformation.dag.column.unary.ArithmeticNegationColumnTransformer;
+import org.apache.iotdb.db.queryengine.transformation.dag.column.unary.InColumnTransformer;
+import org.apache.iotdb.db.queryengine.transformation.dag.column.unary.IsNullColumnTransformer;
+import org.apache.iotdb.db.queryengine.transformation.dag.column.unary.LogicNotColumnTransformer;
+import org.apache.iotdb.db.queryengine.transformation.dag.column.unary.RegularColumnTransformer;
+import org.apache.iotdb.db.relational.sql.tree.ArithmeticBinaryExpression;
+import org.apache.iotdb.db.relational.sql.tree.ArithmeticUnaryExpression;
+import org.apache.iotdb.db.relational.sql.tree.AstVisitor;
+import org.apache.iotdb.db.relational.sql.tree.BetweenPredicate;
+import org.apache.iotdb.db.relational.sql.tree.BinaryLiteral;
+import org.apache.iotdb.db.relational.sql.tree.BooleanLiteral;
+import org.apache.iotdb.db.relational.sql.tree.Cast;
+import org.apache.iotdb.db.relational.sql.tree.CoalesceExpression;
+import org.apache.iotdb.db.relational.sql.tree.ComparisonExpression;
+import org.apache.iotdb.db.relational.sql.tree.CurrentDatabase;
+import org.apache.iotdb.db.relational.sql.tree.CurrentTime;
+import org.apache.iotdb.db.relational.sql.tree.CurrentUser;
+import org.apache.iotdb.db.relational.sql.tree.DecimalLiteral;
+import org.apache.iotdb.db.relational.sql.tree.DoubleLiteral;
+import org.apache.iotdb.db.relational.sql.tree.Expression;
+import org.apache.iotdb.db.relational.sql.tree.FunctionCall;
+import org.apache.iotdb.db.relational.sql.tree.IfExpression;
+import org.apache.iotdb.db.relational.sql.tree.InListExpression;
+import org.apache.iotdb.db.relational.sql.tree.InPredicate;
+import org.apache.iotdb.db.relational.sql.tree.IsNotNullPredicate;
+import org.apache.iotdb.db.relational.sql.tree.IsNullPredicate;
+import org.apache.iotdb.db.relational.sql.tree.LikePredicate;
+import org.apache.iotdb.db.relational.sql.tree.Literal;
+import org.apache.iotdb.db.relational.sql.tree.LogicalExpression;
+import org.apache.iotdb.db.relational.sql.tree.LongLiteral;
+import org.apache.iotdb.db.relational.sql.tree.NotExpression;
+import org.apache.iotdb.db.relational.sql.tree.NullIfExpression;
+import org.apache.iotdb.db.relational.sql.tree.NullLiteral;
+import org.apache.iotdb.db.relational.sql.tree.SearchedCaseExpression;
+import org.apache.iotdb.db.relational.sql.tree.SimpleCaseExpression;
+import org.apache.iotdb.db.relational.sql.tree.StringLiteral;
+import org.apache.iotdb.db.relational.sql.tree.SymbolReference;
+import org.apache.iotdb.db.relational.sql.tree.Trim;
+import org.apache.iotdb.tsfile.common.conf.TSFileConfig;
+import org.apache.iotdb.tsfile.file.metadata.enums.TSDataType;
+import org.apache.iotdb.tsfile.read.common.block.column.BinaryColumn;
+import org.apache.iotdb.tsfile.read.common.block.column.BooleanColumn;
+import org.apache.iotdb.tsfile.read.common.block.column.DoubleColumn;
+import org.apache.iotdb.tsfile.read.common.block.column.LongColumn;
+import org.apache.iotdb.tsfile.read.common.type.Type;
+import org.apache.iotdb.tsfile.utils.Binary;
+
+import java.util.ArrayList;
+import java.util.List;
+import java.util.Map;
+import java.util.Optional;
+import java.util.stream.Collectors;
+
+import static com.google.common.base.Preconditions.checkArgument;
+import static org.apache.iotdb.tsfile.read.common.type.BinaryType.TEXT;
+import static org.apache.iotdb.tsfile.read.common.type.BooleanType.BOOLEAN;
+import static org.apache.iotdb.tsfile.read.common.type.DoubleType.DOUBLE;
+import static org.apache.iotdb.tsfile.read.common.type.LongType.INT64;
+import static org.apache.iotdb.tsfile.utils.RegexUtils.compileRegex;
+import static org.apache.iotdb.tsfile.utils.RegexUtils.parseLikePatternToRegex;
+
+public class ColumnTransformerBuilder
+    extends AstVisitor<ColumnTransformer, ColumnTransformerBuilder.Context> {
+
+  private static final String UNSUPPORTED_EXPRESSION = "Unsupported expression: %s";
+
+  @Override
+  public ColumnTransformer visitExpression(Expression expression, Context context) {
+    throw new IllegalArgumentException(
+        String.format(UNSUPPORTED_EXPRESSION, expression.getClass().getSimpleName()));
+  }
+
+  @Override
+  protected ColumnTransformer visitArithmeticBinary(
+      ArithmeticBinaryExpression node, Context context) {
+    ColumnTransformer res =
+        context.cache.computeIfAbsent(
+            node,
+            n -> {
+              if (context.hasSeen.containsKey(node)) {
+                IdentityColumnTransformer identity =
+                    new IdentityColumnTransformer(
+                        DOUBLE, context.originSize + context.commonTransformerList.size());
+                ColumnTransformer columnTransformer = context.hasSeen.get(node);
+                columnTransformer.addReferenceCount();
+                context.commonTransformerList.add(columnTransformer);
+                context.leafList.add(identity);
+                context.inputDataTypes.add(TSDataType.DOUBLE);
+                return identity;
+              } else {
+                ColumnTransformer left = process(node.getLeft(), context);
+                ColumnTransformer right = process(node.getRight(), context);
+                switch (node.getOperator()) {
+                  case ADD:
+                    return new ArithmeticAdditionColumnTransformer(DOUBLE, left, right);
+                  case SUBTRACT:
+                    return new ArithmeticSubtractionColumnTransformer(DOUBLE, left, right);
+                  case MULTIPLY:
+                    return new ArithmeticMultiplicationColumnTransformer(DOUBLE, left, right);
+                  case DIVIDE:
+                    return new ArithmeticDivisionColumnTransformer(DOUBLE, left, right);
+                  case MODULUS:
+                    return new ArithmeticModuloColumnTransformer(DOUBLE, left, right);
+                  default:
+                    throw new UnsupportedOperationException(
+                        String.format(UNSUPPORTED_EXPRESSION, node.getOperator()));
+                }
+              }
+            });
+    res.addReferenceCount();
+    return res;
+  }
+
+  @Override
+  protected ColumnTransformer visitArithmeticUnary(
+      ArithmeticUnaryExpression node, Context context) {
+    switch (node.getSign()) {
+      case PLUS:
+        return process(node.getValue(), context);
+      case MINUS:
+        ColumnTransformer res =
+            context.cache.computeIfAbsent(
+                node,
+                n -> {
+                  if (context.hasSeen.containsKey(node)) {
+                    IdentityColumnTransformer identity =
+                        new IdentityColumnTransformer(
+                            DOUBLE, context.originSize + context.commonTransformerList.size());
+                    ColumnTransformer columnTransformer = context.hasSeen.get(node);
+                    columnTransformer.addReferenceCount();
+                    context.commonTransformerList.add(columnTransformer);
+                    context.leafList.add(identity);
+                    context.inputDataTypes.add(TSDataType.DOUBLE);
+                    return identity;
+                  } else {
+                    ColumnTransformer childColumnTransformer = process(node.getValue(), context);
+                    return new ArithmeticNegationColumnTransformer(DOUBLE, childColumnTransformer);
+                  }
+                });
+        res.addReferenceCount();
+        return res;
+      default:
+        throw new UnsupportedOperationException("Unknown sign: " + node.getSign());
+    }
+  }
+
+  @Override
+  protected ColumnTransformer visitBetweenPredicate(BetweenPredicate node, Context context) {
+    throw new UnsupportedOperationException(String.format(UNSUPPORTED_EXPRESSION, node));
+  }
+
+  @Override
+  protected ColumnTransformer visitCast(Cast node, Context context) {
+    throw new UnsupportedOperationException(String.format(UNSUPPORTED_EXPRESSION, node));
+  }
+
+  @Override
+  protected ColumnTransformer visitBooleanLiteral(BooleanLiteral node, Context context) {
+    ColumnTransformer res =
+        context.cache.computeIfAbsent(
+            node,
+            e -> {
+              ConstantColumnTransformer columnTransformer =
+                  new ConstantColumnTransformer(
+                      BOOLEAN,
+                      new BooleanColumn(1, Optional.empty(), new boolean[] {node.getValue()}));
+              context.leafList.add(columnTransformer);
+              return columnTransformer;
+            });
+    res.addReferenceCount();
+    return res;
+  }
+
+  @Override
+  protected ColumnTransformer visitBinaryLiteral(BinaryLiteral node, Context context) {
+    throw new UnsupportedOperationException();
+  }
+
+  @Override
+  protected ColumnTransformer visitStringLiteral(StringLiteral node, Context context) {
+    ColumnTransformer res =
+        context.cache.computeIfAbsent(
+            node,
+            e -> {
+              ConstantColumnTransformer columnTransformer =
+                  new ConstantColumnTransformer(
+                      TEXT,
+                      new BinaryColumn(
+                          1,
+                          Optional.empty(),
+                          new Binary[] {new Binary(node.getValue(), TSFileConfig.STRING_CHARSET)}));
+              context.leafList.add(columnTransformer);
+              return columnTransformer;
+            });
+    res.addReferenceCount();
+    return res;
+  }
+
+  @Override
+  protected ColumnTransformer visitLongLiteral(LongLiteral node, Context context) {
+    ColumnTransformer res =
+        context.cache.computeIfAbsent(
+            node,
+            e -> {
+              ConstantColumnTransformer columnTransformer =
+                  new ConstantColumnTransformer(
+                      INT64,
+                      new LongColumn(1, Optional.empty(), new long[] {node.getParsedValue()}));
+              context.leafList.add(columnTransformer);
+              return columnTransformer;
+            });
+    res.addReferenceCount();
+    return res;
+  }
+
+  @Override
+  protected ColumnTransformer visitDoubleLiteral(DoubleLiteral node, Context context) {
+    ColumnTransformer res =
+        context.cache.computeIfAbsent(
+            node,
+            e -> {
+              ConstantColumnTransformer columnTransformer =
+                  new ConstantColumnTransformer(
+                      DOUBLE,
+                      new DoubleColumn(1, Optional.empty(), new double[] {node.getValue()}));
+              context.leafList.add(columnTransformer);
+              return columnTransformer;
+            });
+    res.addReferenceCount();
+    return res;
+  }
+
+  @Override
+  protected ColumnTransformer visitDecimalLiteral(DecimalLiteral node, Context context) {
+    throw new UnsupportedOperationException();
+  }
+
+  @Override
+  protected ColumnTransformer visitNullLiteral(NullLiteral node, Context context) {
+    ColumnTransformer res =
+        context.cache.computeIfAbsent(
+            node,
+            e -> {
+              NullColumnTransformer columnTransformer = new NullColumnTransformer();
+              context.leafList.add(columnTransformer);
+              return columnTransformer;
+            });
+    res.addReferenceCount();
+    return res;
+  }
+
+  @Override
+  protected ColumnTransformer visitComparisonExpression(
+      ComparisonExpression node, Context context) {
+    ColumnTransformer res =
+        context.cache.computeIfAbsent(
+            node,
+            n -> {
+              if (context.hasSeen.containsKey(node)) {
+                IdentityColumnTransformer identity =
+                    new IdentityColumnTransformer(
+                        BOOLEAN, context.originSize + context.commonTransformerList.size());
+                ColumnTransformer columnTransformer = context.hasSeen.get(node);
+                columnTransformer.addReferenceCount();
+                context.commonTransformerList.add(columnTransformer);
+                context.leafList.add(identity);
+                context.inputDataTypes.add(TSDataType.BOOLEAN);
+                return identity;
+              } else {
+                ColumnTransformer left = process(node.getLeft(), context);
+                ColumnTransformer right = process(node.getRight(), context);
+                switch (node.getOperator()) {
+                  case EQUAL:
+                    return new CompareEqualToColumnTransformer(BOOLEAN, left, right);
+                  case NOT_EQUAL:
+                    return new CompareNonEqualColumnTransformer(BOOLEAN, left, right);
+                  case GREATER_THAN:
+                    return new CompareGreaterThanColumnTransformer(BOOLEAN, left, right);
+                  case GREATER_THAN_OR_EQUAL:
+                    return new CompareGreaterEqualColumnTransformer(BOOLEAN, left, right);
+                  case LESS_THAN:
+                    return new CompareLessThanColumnTransformer(BOOLEAN, left, right);
+                  case LESS_THAN_OR_EQUAL:
+                    return new CompareLessEqualColumnTransformer(BOOLEAN, left, right);
+                  default:
+                    throw new UnsupportedOperationException(
+                        String.format(UNSUPPORTED_EXPRESSION, node.getOperator()));
+                }
+              }
+            });
+    res.addReferenceCount();
+    return res;
+  }
+
+  @Override
+  protected ColumnTransformer visitCurrentDatabase(CurrentDatabase node, Context context) {
+    Optional<String> currentDatabase = context.sessionInfo.getDatabaseName();
+    ColumnTransformer res;
+    res =
+        currentDatabase
+            .map(
+                s ->
+                    context.cache.computeIfAbsent(
+                        node,
+                        e -> {
+                          ConstantColumnTransformer columnTransformer =
+                              new ConstantColumnTransformer(
+                                  TEXT,
+                                  new BinaryColumn(
+                                      1,
+                                      Optional.empty(),
+                                      new Binary[] {new Binary(s, TSFileConfig.STRING_CHARSET)}));
+                          context.leafList.add(columnTransformer);
+                          return columnTransformer;
+                        }))
+            .orElseGet(
+                () ->
+                    context.cache.computeIfAbsent(
+                        node,
+                        e -> {
+                          NullColumnTransformer columnTransformer = new NullColumnTransformer();
+                          context.leafList.add(columnTransformer);
+                          return columnTransformer;
+                        }));
+    res.addReferenceCount();
+    return res;
+  }
+
+  @Override
+  protected ColumnTransformer visitCurrentTime(CurrentTime node, Context context) {
+    throw new UnsupportedOperationException(String.format(UNSUPPORTED_EXPRESSION, node));
+  }
+
+  @Override
+  protected ColumnTransformer visitCurrentUser(CurrentUser node, Context context) {
+    String currentUser = context.sessionInfo.getUserName();
+    ColumnTransformer res =
+        context.cache.computeIfAbsent(
+            node,
+            e -> {
+              ConstantColumnTransformer columnTransformer =
+                  new ConstantColumnTransformer(
+                      TEXT,
+                      new BinaryColumn(
+                          1,
+                          Optional.empty(),
+                          new Binary[] {new Binary(currentUser, TSFileConfig.STRING_CHARSET)}));
+              context.leafList.add(columnTransformer);
+              return columnTransformer;
+            });
+    res.addReferenceCount();
+    return res;
+  }
+
+  @Override
+  protected ColumnTransformer visitFunctionCall(FunctionCall node, Context context) {
+    throw new UnsupportedOperationException(String.format(UNSUPPORTED_EXPRESSION, node));
+  }
+
+  @Override
+  protected ColumnTransformer visitInPredicate(InPredicate node, Context context) {
+    ColumnTransformer res =
+        context.cache.computeIfAbsent(
+            node,
+            n -> {
+              if (context.hasSeen.containsKey(node)) {
+                IdentityColumnTransformer identity =
+                    new IdentityColumnTransformer(
+                        BOOLEAN, context.originSize + context.commonTransformerList.size());
+                ColumnTransformer columnTransformer = context.hasSeen.get(node);
+                columnTransformer.addReferenceCount();
+                context.commonTransformerList.add(columnTransformer);
+                context.leafList.add(identity);
+                context.inputDataTypes.add(TSDataType.BOOLEAN);
+                return identity;
+              } else {
+                ColumnTransformer childColumnTransformer = process(node.getValue(), context);
+                InListExpression inListExpression = (InListExpression) node.getValueList();
+                List<Expression> expressionList = inListExpression.getValues();
+                List<Literal> values = new ArrayList<>();
+                for (Expression expression : expressionList) {
+                  checkArgument(expression instanceof Literal);
+                  values.add((Literal) expression);
+                }
+                return new InColumnTransformer(BOOLEAN, childColumnTransformer, values);
+              }
+            });
+    res.addReferenceCount();
+    return res;
+  }
+
+  @Override
+  protected ColumnTransformer visitNotExpression(NotExpression node, Context context) {
+    ColumnTransformer res =
+        context.cache.computeIfAbsent(
+            node,
+            n -> {
+              if (context.hasSeen.containsKey(node)) {
+                IdentityColumnTransformer identity =
+                    new IdentityColumnTransformer(
+                        BOOLEAN, context.originSize + context.commonTransformerList.size());
+                ColumnTransformer columnTransformer = context.hasSeen.get(node);
+                columnTransformer.addReferenceCount();
+                context.commonTransformerList.add(columnTransformer);
+                context.leafList.add(identity);
+                context.inputDataTypes.add(TSDataType.BOOLEAN);
+                return identity;
+              } else {
+                ColumnTransformer childColumnTransformer = process(node.getValue(), context);
+                return new LogicNotColumnTransformer(BOOLEAN, childColumnTransformer);
+              }
+            });
+    res.addReferenceCount();
+    return res;
+  }
+
+  @Override
+  protected ColumnTransformer visitLikePredicate(LikePredicate node, Context context) {
+    ColumnTransformer res =
+        context.cache.computeIfAbsent(
+            node,
+            n -> {
+              if (context.hasSeen.containsKey(node)) {
+                IdentityColumnTransformer identity =
+                    new IdentityColumnTransformer(
+                        BOOLEAN, context.originSize + context.commonTransformerList.size());
+                ColumnTransformer columnTransformer = context.hasSeen.get(node);
+                columnTransformer.addReferenceCount();
+                context.commonTransformerList.add(columnTransformer);
+                context.leafList.add(identity);
+                context.inputDataTypes.add(TSDataType.BOOLEAN);
+                return identity;
+              } else {
+                ColumnTransformer childColumnTransformer = process(node.getValue(), context);
+                return new RegularColumnTransformer(
+                    BOOLEAN,
+                    childColumnTransformer,
+                    compileRegex(
+                        parseLikePatternToRegex(((StringLiteral) node.getPattern()).getValue())));
+              }
+            });
+    res.addReferenceCount();
+    return res;
+  }
+
+  @Override
+  protected ColumnTransformer visitIsNotNullPredicate(IsNotNullPredicate node, Context context) {
+    ColumnTransformer res =
+        context.cache.computeIfAbsent(
+            node,
+            n -> {
+              if (context.hasSeen.containsKey(node)) {
+                IdentityColumnTransformer identity =
+                    new IdentityColumnTransformer(
+                        BOOLEAN, context.originSize + context.commonTransformerList.size());
+                ColumnTransformer columnTransformer = context.hasSeen.get(node);
+                columnTransformer.addReferenceCount();
+                context.commonTransformerList.add(columnTransformer);
+                context.leafList.add(identity);
+                context.inputDataTypes.add(TSDataType.BOOLEAN);
+                return identity;
+              } else {
+                ColumnTransformer childColumnTransformer = process(node.getValue(), context);
+                return new IsNullColumnTransformer(BOOLEAN, childColumnTransformer, true);
+              }
+            });
+    res.addReferenceCount();
+    return res;
+  }
+
+  @Override
+  protected ColumnTransformer visitIsNullPredicate(IsNullPredicate node, Context context) {
+    ColumnTransformer res =
+        context.cache.computeIfAbsent(
+            node,
+            n -> {
+              if (context.hasSeen.containsKey(node)) {
+                IdentityColumnTransformer identity =
+                    new IdentityColumnTransformer(
+                        BOOLEAN, context.originSize + context.commonTransformerList.size());
+                ColumnTransformer columnTransformer = context.hasSeen.get(node);
+                columnTransformer.addReferenceCount();
+                context.commonTransformerList.add(columnTransformer);
+                context.leafList.add(identity);
+                context.inputDataTypes.add(TSDataType.BOOLEAN);
+                return identity;
+              } else {
+                ColumnTransformer childColumnTransformer = process(node.getValue(), context);
+                return new IsNullColumnTransformer(BOOLEAN, childColumnTransformer, false);
+              }
+            });
+    res.addReferenceCount();
+    return res;
+  }
+
+  @Override
+  protected ColumnTransformer visitLogicalExpression(LogicalExpression node, Context context) {
+    ColumnTransformer res =
+        context.cache.computeIfAbsent(
+            node,
+            n -> {
+              if (context.hasSeen.containsKey(node)) {
+                IdentityColumnTransformer identity =
+                    new IdentityColumnTransformer(
+                        BOOLEAN, context.originSize + context.commonTransformerList.size());
+                ColumnTransformer columnTransformer = context.hasSeen.get(node);
+                columnTransformer.addReferenceCount();
+                context.commonTransformerList.add(columnTransformer);
+                context.leafList.add(identity);
+                context.inputDataTypes.add(TSDataType.BOOLEAN);
+                return identity;
+              } else {
+                List<ColumnTransformer> children =
+                    node.getChildren().stream()
+                        .map(c -> process(c, context))
+                        .collect(Collectors.toList());
+                switch (node.getOperator()) {
+                  case OR:
+                    return new LogicalOrMultiColumnTransformer(BOOLEAN, children);
+                  case AND:
+                    return new LogicalAndMultiColumnTransformer(BOOLEAN, children);
+                  default:
+                    throw new UnsupportedOperationException(
+                        String.format(UNSUPPORTED_EXPRESSION, node.getOperator()));
+                }
+              }
+            });
+    res.addReferenceCount();
+    return res;
+  }
+
+  @Override
+  protected ColumnTransformer visitSymbolReference(SymbolReference node, Context context) {
+    ColumnTransformer res =
+        context.cache.computeIfAbsent(
+            node,
+            e -> {
+              IdentityColumnTransformer identity =
+                  new IdentityColumnTransformer(
+                      context.getType(node),
+                      context.inputLocations.get(Symbol.from(node)).get(0).getValueColumnIndex());
+              context.leafList.add(identity);
+              return identity;
+            });
+    res.addReferenceCount();
+    return res;
+  }
+
+  @Override
+  protected ColumnTransformer visitSimpleCaseExpression(
+      SimpleCaseExpression node, Context context) {
+    throw new UnsupportedOperationException(String.format(UNSUPPORTED_EXPRESSION, node));
+  }
+
+  @Override
+  protected ColumnTransformer visitSearchedCaseExpression(
+      SearchedCaseExpression node, Context context) {
+    throw new UnsupportedOperationException(String.format(UNSUPPORTED_EXPRESSION, node));
+  }
+
+  @Override
+  protected ColumnTransformer visitTrim(Trim node, Context context) {
+    throw new UnsupportedOperationException(String.format(UNSUPPORTED_EXPRESSION, node));
+  }
+
+  @Override
+  protected ColumnTransformer visitIfExpression(IfExpression node, Context context) {
+    throw new UnsupportedOperationException(String.format(UNSUPPORTED_EXPRESSION, node));
+  }
+
+  @Override
+  protected ColumnTransformer visitNullIfExpression(NullIfExpression node, Context context) {
+    throw new UnsupportedOperationException(String.format(UNSUPPORTED_EXPRESSION, node));
+  }
+
+  @Override
+  protected ColumnTransformer visitCoalesceExpression(CoalesceExpression node, Context context) {
+    throw new UnsupportedOperationException(String.format(UNSUPPORTED_EXPRESSION, node));
+  }
+
+  public static class Context {
+
+    private final SessionInfo sessionInfo;
+
+    // LeafColumnTransformer for LeafOperand
+    private final List<LeafColumnTransformer> leafList;
+
+    // Index of input column
+    private final Map<Symbol, List<InputLocation>> inputLocations;
+
+    // cache for constructing ColumnTransformer tree
+    private final Map<Expression, ColumnTransformer> cache;
+
+    // Sub expressions that has been seen in filter
+    private final Map<Expression, ColumnTransformer> hasSeen;
+
+    // Common Transformer between filter and project
+    private final List<ColumnTransformer> commonTransformerList;
+
+    private final List<TSDataType> inputDataTypes;
+
+    private final int originSize;
+
+    private final TypeProvider typeProvider;
+
+    public Context(
+        SessionInfo sessionInfo,
+        List<LeafColumnTransformer> leafList,
+        Map<Symbol, List<InputLocation>> inputLocations,
+        Map<Expression, ColumnTransformer> cache,
+        Map<Expression, ColumnTransformer> hasSeen,
+        List<ColumnTransformer> commonTransformerList,
+        List<TSDataType> inputDataTypes,
+        int originSize,
+        TypeProvider typeProvider) {
+      this.sessionInfo = sessionInfo;
+      this.leafList = leafList;
+      this.inputLocations = inputLocations;
+      this.cache = cache;
+      this.hasSeen = hasSeen;
+      this.commonTransformerList = commonTransformerList;
+      this.inputDataTypes = inputDataTypes;
+      this.originSize = originSize;
+      this.typeProvider = typeProvider;
+    }
+
+    public Type getType(SymbolReference symbolReference) {
+      return typeProvider.get(Symbol.from(symbolReference));
+    }
+  }
+}
diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/analyze/PredicateUtils.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/analyze/PredicateUtils.java
index d635f074789..dc2d8653b2c 100644
--- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/analyze/PredicateUtils.java
+++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/analyze/PredicateUtils.java
@@ -40,6 +40,8 @@ import org.apache.iotdb.db.queryengine.plan.expression.visitor.predicate.Convert
 import org.apache.iotdb.db.queryengine.plan.expression.visitor.predicate.PredicatePushIntoScanChecker;
 import org.apache.iotdb.db.queryengine.plan.expression.visitor.predicate.PredicateSimplifier;
 import org.apache.iotdb.db.queryengine.plan.expression.visitor.predicate.ReversePredicateVisitor;
+import org.apache.iotdb.db.queryengine.plan.relational.metadata.ColumnSchema;
+import org.apache.iotdb.db.queryengine.plan.relational.planner.Symbol;
 import org.apache.iotdb.tsfile.file.metadata.enums.TSDataType;
 import org.apache.iotdb.tsfile.read.filter.basic.Filter;
 import org.apache.iotdb.tsfile.utils.Pair;
@@ -48,6 +50,7 @@ import java.util.ArrayList;
 import java.util.HashSet;
 import java.util.Iterator;
 import java.util.List;
+import java.util.Map;
 import java.util.Set;
 import java.util.stream.Collectors;
 
@@ -293,14 +296,15 @@ public class PredicateUtils {
   public static Filter convertPredicateToFilter(
       org.apache.iotdb.db.relational.sql.tree.Expression predicate,
       List<String> allMeasurements,
-      TypeProvider typeProvider) {
+      Map<Symbol, ColumnSchema> schemaMap) {
     if (predicate == null) {
       return null;
     }
     return predicate.accept(
-        new ConvertPredicateToFilterVisitor(),
-        new ConvertPredicateToFilterVisitor.Context(
-            allMeasurements, isBuildPlanUseTemplate, typeProvider));
+        new org.apache.iotdb.db.queryengine.plan.relational.analyzer.predicate
+            .ConvertPredicateToFilterVisitor(),
+        new org.apache.iotdb.db.queryengine.plan.relational.analyzer.predicate
+            .ConvertPredicateToFilterVisitor.Context(allMeasurements, schemaMap));
   }
 
   /**
diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/planner/TableOperatorGenerator.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/planner/TableOperatorGenerator.java
index f5e22768ce5..273e949b37f 100644
--- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/planner/TableOperatorGenerator.java
+++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/planner/TableOperatorGenerator.java
@@ -23,11 +23,15 @@ import org.apache.iotdb.commons.path.AlignedPath;
 import org.apache.iotdb.db.queryengine.execution.driver.DataDriverContext;
 import org.apache.iotdb.db.queryengine.execution.operator.Operator;
 import org.apache.iotdb.db.queryengine.execution.operator.OperatorContext;
+import org.apache.iotdb.db.queryengine.execution.operator.process.FilterAndProjectOperator;
 import org.apache.iotdb.db.queryengine.execution.operator.source.AlignedSeriesScanOperator;
 import org.apache.iotdb.db.queryengine.execution.operator.source.relational.TableScanOperator;
+import org.apache.iotdb.db.queryengine.execution.relational.ColumnTransformerBuilder;
 import org.apache.iotdb.db.queryengine.plan.analyze.PredicateUtils;
 import org.apache.iotdb.db.queryengine.plan.planner.plan.node.PlanNode;
+import org.apache.iotdb.db.queryengine.plan.planner.plan.node.PlanNodeId;
 import org.apache.iotdb.db.queryengine.plan.planner.plan.node.PlanVisitor;
+import org.apache.iotdb.db.queryengine.plan.planner.plan.parameter.InputLocation;
 import org.apache.iotdb.db.queryengine.plan.planner.plan.parameter.SeriesScanOptions;
 import org.apache.iotdb.db.queryengine.plan.relational.metadata.ColumnSchema;
 import org.apache.iotdb.db.queryengine.plan.relational.planner.Symbol;
@@ -40,6 +44,8 @@ import org.apache.iotdb.db.queryengine.plan.relational.planner.node.ProjectNode;
 import org.apache.iotdb.db.queryengine.plan.relational.planner.node.SortNode;
 import org.apache.iotdb.db.queryengine.plan.relational.planner.node.TableScanNode;
 import org.apache.iotdb.db.queryengine.plan.relational.planner.node.TopKNode;
+import org.apache.iotdb.db.queryengine.transformation.dag.column.ColumnTransformer;
+import org.apache.iotdb.db.queryengine.transformation.dag.column.leaf.LeafColumnTransformer;
 import org.apache.iotdb.db.relational.sql.tree.Expression;
 import org.apache.iotdb.tsfile.common.conf.TSFileDescriptor;
 import org.apache.iotdb.tsfile.file.metadata.enums.TSDataType;
@@ -47,14 +53,21 @@ import org.apache.iotdb.tsfile.read.filter.basic.Filter;
 import org.apache.iotdb.tsfile.write.schema.IMeasurementSchema;
 import org.apache.iotdb.tsfile.write.schema.MeasurementSchema;
 
+import com.google.common.collect.ImmutableList;
+import com.google.common.collect.ImmutableMap;
+
 import java.util.ArrayList;
 import java.util.Collections;
+import java.util.HashMap;
 import java.util.HashSet;
+import java.util.LinkedHashMap;
 import java.util.List;
 import java.util.Map;
 
 import static java.util.Objects.requireNonNull;
+import static org.apache.iotdb.db.queryengine.execution.operator.source.relational.TableScanOperator.constructAlignedPath;
 import static org.apache.iotdb.db.queryengine.plan.analyze.PredicateUtils.convertPredicateToFilter;
+import static org.apache.iotdb.db.queryengine.plan.expression.leaf.TimestampOperand.TIMESTAMP_EXPRESSION_STRING;
 import static org.apache.iotdb.db.queryengine.plan.relational.type.InternalTypeManager.getTSDataType;
 
 /** This Visitor is responsible for transferring Table PlanNode Tree to Table Operator Tree. */
@@ -102,7 +115,7 @@ public class TableOperatorGenerator extends PlanVisitor<Operator, LocalExecution
       }
     }
 
-    SeriesScanOptions.Builder scanOptionsBuilder = getSeriesScanOptionsBuilder(node, context);
+    SeriesScanOptions.Builder scanOptionsBuilder = getSeriesScanOptionsBuilder(context);
     scanOptionsBuilder.withPushDownLimit(node.getPushDownLimit());
     scanOptionsBuilder.withPushDownOffset(node.getPushDownOffset());
     scanOptionsBuilder.withAllSensors(new HashSet<>(measurementColumnNames));
@@ -111,11 +124,7 @@ public class TableOperatorGenerator extends PlanVisitor<Operator, LocalExecution
     boolean predicateCanPushIntoScan = canPushIntoScan(pushDownPredicate);
     if (pushDownPredicate != null && predicateCanPushIntoScan) {
       scanOptionsBuilder.withPushDownFilter(
-          convertPredicateToFilter(
-              pushDownPredicate,
-              node.getAlignedPath().getMeasurementList(),
-              context.getTypeProvider().getTemplatedInfo() != null,
-              context.getTypeProvider()));
+          convertPredicateToFilter(pushDownPredicate, measurementColumnNames, columnSchemaMap));
     }
 
     OperatorContext operatorContext =
@@ -138,58 +147,63 @@ public class TableOperatorGenerator extends PlanVisitor<Operator, LocalExecution
         new TableScanOperator(
             operatorContext,
             node.getPlanNodeId(),
-            seriesPath,
+            columnSchemas,
+            columnsIndexArray,
+            measurementColumnCount,
+            node.getDeviceEntries(),
             node.getScanOrder(),
             scanOptionsBuilder.build(),
-            node.isQueryAllSensors(),
-            context.getTypeProvider().getTemplatedInfo() != null
-                ? context.getTypeProvider().getTemplatedInfo().getDataTypes()
-                : null,
+            measurementColumnNames,
+            measurementSchemas,
             maxTsBlockLineNum);
 
-    ((DataDriverContext) context.getDriverContext()).addSourceOperator(seriesScanOperator);
-    ((DataDriverContext) context.getDriverContext()).addPath(seriesPath);
+    ((DataDriverContext) context.getDriverContext()).addSourceOperator(tableScanOperator);
+
+    for (int i = 0, size = node.getDeviceEntries().size(); i < size; i++) {
+      AlignedPath alignedPath =
+          constructAlignedPath(
+              node.getDeviceEntries().get(i), measurementColumnNames, measurementSchemas);
+      ((DataDriverContext) context.getDriverContext()).addPath(alignedPath);
+    }
+
     context.getDriverContext().setInputDriver(true);
 
     if (!predicateCanPushIntoScan) {
-      if (context.isBuildPlanUseTemplate()) {
-        TemplatedInfo templatedInfo = context.getTemplatedInfo();
-        return constructFilterOperator(
-            pushDownPredicate,
-            seriesScanOperator,
-            templatedInfo.getProjectExpressions(),
-            templatedInfo.getDataTypes(),
-            templatedInfo.getLayoutMap(),
-            templatedInfo.isKeepNull(),
-            node.getPlanNodeId(),
-            templatedInfo.getScanOrder(),
-            context);
-      }
-
-      AlignedPath alignedPath = node.getAlignedPath();
-      List<Expression> expressions = new ArrayList<>();
-      List<TSDataType> dataTypes = new ArrayList<>();
-      for (int i = 0; i < alignedPath.getMeasurementList().size(); i++) {
-        expressions.add(ExpressionFactory.timeSeries(alignedPath.getSubMeasurementPath(i)));
-        dataTypes.add(alignedPath.getSubMeasurementDataType(i));
-      }
 
       return constructFilterOperator(
           pushDownPredicate,
-          seriesScanOperator,
-          expressions.toArray(new Expression[0]),
-          dataTypes,
+          tableScanOperator,
+          node.getOutputSymbols().stream()
+              .map(Symbol::toSymbolReference)
+              .toArray(Expression[]::new),
+          tableScanOperator.getResultDataTypes(),
           makeLayout(Collections.singletonList(node)),
-          false,
           node.getPlanNodeId(),
-          node.getScanOrder(),
           context);
     }
-    return seriesScanOperator;
+    return tableScanOperator;
   }
 
-  private SeriesScanOptions.Builder getSeriesScanOptionsBuilder(
-      TableScanNode node, LocalExecutionPlanContext context) {
+  private Map<Symbol, List<InputLocation>> makeLayout(List<PlanNode> children) {
+    Map<Symbol, List<InputLocation>> outputMappings = new LinkedHashMap<>();
+    int tsBlockIndex = 0;
+    for (PlanNode childNode : children) {
+      outputMappings
+          .computeIfAbsent(new Symbol(TIMESTAMP_EXPRESSION_STRING), key -> new ArrayList<>())
+          .add(new InputLocation(tsBlockIndex, -1));
+      int valueColumnIndex = 0;
+      for (Symbol columnName : childNode.getOutputSymbols()) {
+        outputMappings
+            .computeIfAbsent(columnName, key -> new ArrayList<>())
+            .add(new InputLocation(tsBlockIndex, valueColumnIndex));
+        valueColumnIndex++;
+      }
+      tsBlockIndex++;
+    }
+    return outputMappings;
+  }
+
+  private SeriesScanOptions.Builder getSeriesScanOptionsBuilder(LocalExecutionPlanContext context) {
     SeriesScanOptions.Builder scanOptionsBuilder = new SeriesScanOptions.Builder();
 
     Filter globalTimeFilter = context.getGlobalTimeFilter();
@@ -210,8 +224,102 @@ public class TableOperatorGenerator extends PlanVisitor<Operator, LocalExecution
     return super.visitFilter(node, context);
   }
 
+  private Operator constructFilterOperator(
+      Expression predicate,
+      Operator inputOperator,
+      Expression[] projectExpressions,
+      List<TSDataType> inputDataTypes,
+      Map<Symbol, List<InputLocation>> inputLocations,
+      PlanNodeId planNodeId,
+      LocalExecutionPlanContext context) {
+
+    final List<TSDataType> filterOutputDataTypes = new ArrayList<>(inputDataTypes);
+
+    // records LeafColumnTransformer of filter
+    List<LeafColumnTransformer> filterLeafColumnTransformerList = new ArrayList<>();
+
+    // records common ColumnTransformer between filter and project expressions
+    List<ColumnTransformer> commonTransformerList = new ArrayList<>();
+
+    // records LeafColumnTransformer of project expressions
+    List<LeafColumnTransformer> projectLeafColumnTransformerList = new ArrayList<>();
+
+    // records subexpression -> ColumnTransformer for filter
+    Map<Expression, ColumnTransformer> filterExpressionColumnTransformerMap = new HashMap<>();
+
+    ColumnTransformerBuilder visitor = new ColumnTransformerBuilder();
+
+    ColumnTransformerBuilder.Context filterColumnTransformerContext =
+        new ColumnTransformerBuilder.Context(
+            context.getDriverContext().getFragmentInstanceContext().getSessionInfo(),
+            filterLeafColumnTransformerList,
+            inputLocations,
+            filterExpressionColumnTransformerMap,
+            ImmutableMap.of(),
+            ImmutableList.of(),
+            ImmutableList.of(),
+            0,
+            context.getTypeProvider());
+
+    ColumnTransformer filterOutputTransformer =
+        visitor.process(predicate, filterColumnTransformerContext);
+
+    List<ColumnTransformer> projectOutputTransformerList = new ArrayList<>();
+
+    Map<Expression, ColumnTransformer> projectExpressionColumnTransformerMap = new HashMap<>();
+
+    ColumnTransformerBuilder.Context projectColumnTransformerContext =
+        new ColumnTransformerBuilder.Context(
+            context.getDriverContext().getFragmentInstanceContext().getSessionInfo(),
+            projectLeafColumnTransformerList,
+            inputLocations,
+            projectExpressionColumnTransformerMap,
+            filterExpressionColumnTransformerMap,
+            commonTransformerList,
+            filterOutputDataTypes,
+            inputLocations.size() - 1,
+            context.getTypeProvider());
+
+    for (Expression expression : projectExpressions) {
+      projectOutputTransformerList.add(
+          visitor.process(expression, projectColumnTransformerContext));
+    }
+
+    final OperatorContext operatorContext =
+        context
+            .getDriverContext()
+            .addOperatorContext(
+                context.getNextOperatorId(),
+                planNodeId,
+                FilterAndProjectOperator.class.getSimpleName());
+
+    // Project expressions don't contain Non-Mappable UDF, TransformOperator is not needed
+    return new FilterAndProjectOperator(
+        operatorContext,
+        inputOperator,
+        filterOutputDataTypes,
+        filterLeafColumnTransformerList,
+        filterOutputTransformer,
+        commonTransformerList,
+        projectLeafColumnTransformerList,
+        projectOutputTransformerList,
+        false,
+        true);
+  }
+
   @Override
   public Operator visitProject(ProjectNode node, LocalExecutionPlanContext context) {
+    if (node.getChild() instanceof FilterNode) {
+      FilterNode filterNode = (FilterNode) node.getChild();
+      return constructFilterOperator(
+          filterNode.getPredicate(),
+          filterNode.getChild().accept(this, context),
+          node.getAssignments().getExpressions().toArray(new Expression[0]),
+          tableScanOperator.getResultDataTypes(),
+          makeLayout(Collections.singletonList(filterNode.getChild())),
+          node.getPlanNodeId(),
+          context);
+    }
     return super.visitProject(node, context);
   }
 
diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/relational/analyzer/predicate/ConvertPredicateToFilterVisitor.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/relational/analyzer/predicate/ConvertPredicateToFilterVisitor.java
index 7ab2448ded1..32cc28b9637 100644
--- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/relational/analyzer/predicate/ConvertPredicateToFilterVisitor.java
+++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/relational/analyzer/predicate/ConvertPredicateToFilterVisitor.java
@@ -19,4 +19,353 @@
 
 package org.apache.iotdb.db.queryengine.plan.relational.analyzer.predicate;
 
-public class ConvertPredicateToFilterVisitor {}
+import org.apache.iotdb.commons.schema.table.column.TsTableColumnCategory;
+import org.apache.iotdb.db.queryengine.plan.relational.metadata.ColumnSchema;
+import org.apache.iotdb.db.queryengine.plan.relational.planner.Symbol;
+import org.apache.iotdb.db.relational.sql.tree.BetweenPredicate;
+import org.apache.iotdb.db.relational.sql.tree.BooleanLiteral;
+import org.apache.iotdb.db.relational.sql.tree.ComparisonExpression;
+import org.apache.iotdb.db.relational.sql.tree.DoubleLiteral;
+import org.apache.iotdb.db.relational.sql.tree.Expression;
+import org.apache.iotdb.db.relational.sql.tree.IfExpression;
+import org.apache.iotdb.db.relational.sql.tree.InListExpression;
+import org.apache.iotdb.db.relational.sql.tree.InPredicate;
+import org.apache.iotdb.db.relational.sql.tree.IsNotNullPredicate;
+import org.apache.iotdb.db.relational.sql.tree.IsNullPredicate;
+import org.apache.iotdb.db.relational.sql.tree.LikePredicate;
+import org.apache.iotdb.db.relational.sql.tree.Literal;
+import org.apache.iotdb.db.relational.sql.tree.LogicalExpression;
+import org.apache.iotdb.db.relational.sql.tree.NotExpression;
+import org.apache.iotdb.db.relational.sql.tree.NullIfExpression;
+import org.apache.iotdb.db.relational.sql.tree.SearchedCaseExpression;
+import org.apache.iotdb.db.relational.sql.tree.SimpleCaseExpression;
+import org.apache.iotdb.db.relational.sql.tree.StringLiteral;
+import org.apache.iotdb.db.relational.sql.tree.SymbolReference;
+import org.apache.iotdb.tsfile.common.conf.TSFileConfig;
+import org.apache.iotdb.tsfile.read.common.type.Type;
+import org.apache.iotdb.tsfile.read.filter.basic.Filter;
+import org.apache.iotdb.tsfile.read.filter.factory.FilterFactory;
+import org.apache.iotdb.tsfile.read.filter.factory.ValueFilterApi;
+import org.apache.iotdb.tsfile.read.filter.operator.ValueFilterOperators;
+import org.apache.iotdb.tsfile.utils.Binary;
+
+import java.util.HashMap;
+import java.util.HashSet;
+import java.util.List;
+import java.util.Map;
+import java.util.Set;
+import java.util.stream.Collectors;
+
+import static com.google.common.base.Preconditions.checkArgument;
+import static org.apache.iotdb.db.queryengine.plan.relational.analyzer.predicate.ConvertPredicateToTimeFilterVisitor.getLongValue;
+import static org.apache.iotdb.db.queryengine.plan.relational.analyzer.predicate.ConvertPredicateToTimeFilterVisitor.isTimeColumn;
+import static org.apache.iotdb.db.queryengine.plan.relational.analyzer.predicate.PredicatePushIntoScanChecker.isLiteral;
+import static org.apache.iotdb.db.queryengine.plan.relational.analyzer.predicate.PredicatePushIntoScanChecker.isSymbolReference;
+
+public class ConvertPredicateToFilterVisitor
+    extends PredicateVisitor<Filter, ConvertPredicateToFilterVisitor.Context> {
+
+  private static final ConvertPredicateToTimeFilterVisitor TIME_FILTER_VISITOR =
+      new ConvertPredicateToTimeFilterVisitor();
+
+  @Override
+  protected Filter visitInPredicate(InPredicate node, Context context) {
+    Expression operand = node.getValue();
+    if (isTimeColumn(operand)) {
+      return TIME_FILTER_VISITOR.process(node, null);
+    }
+
+    checkArgument(isSymbolReference(operand));
+
+    Expression valueList = node.getValueList();
+    checkArgument(valueList instanceof InListExpression);
+    List<Expression> values = ((InListExpression) valueList).getValues();
+    for (Expression value : values) {
+      checkArgument(value instanceof Literal);
+    }
+
+    if (values.size() == 1) {
+      return constructCompareFilter(
+          ComparisonExpression.Operator.EQUAL,
+          (SymbolReference) operand,
+          (Literal) values.get(0),
+          context);
+    }
+
+    return constructInFilter(
+        (SymbolReference) operand,
+        values.stream().map(v -> (Literal) v).collect(Collectors.toList()),
+        context);
+  }
+
+  private <T extends Comparable<T>> ValueFilterOperators.ValueIn<T> constructInFilter(
+      SymbolReference operand, List<Literal> values, Context context) {
+    int measurementIndex = context.getMeasurementIndex((operand).getName());
+    Set<T> inSet = constructInSet(values, context.getType(Symbol.from(operand)));
+    return ValueFilterApi.in(measurementIndex, inSet);
+  }
+
+  private <T extends Comparable<T>> Set<T> constructInSet(List<Literal> literals, Type dataType) {
+    Set<T> values = new HashSet<>();
+    for (Literal literal : literals) {
+      values.add(getValue(literal, dataType));
+    }
+    return values;
+  }
+
+  public static <T extends Comparable<T>> Filter constructCompareFilter(
+      ComparisonExpression.Operator operator,
+      SymbolReference symbolReference,
+      Literal literal,
+      Context context) {
+
+    if (!context.isMeasurementColumn(symbolReference)) {
+      throw new IllegalStateException(
+          String.format("Only support measurement column in filter: %s", symbolReference));
+    }
+
+    int measurementIndex = context.getMeasurementIndex(symbolReference.getName());
+    Type type = context.getType(Symbol.from(symbolReference));
+
+    T value = getValue(literal, type);
+
+    switch (operator) {
+      case EQUAL:
+        return ValueFilterApi.eq(measurementIndex, value);
+      case NOT_EQUAL:
+        return ValueFilterApi.notEq(measurementIndex, value);
+      case GREATER_THAN:
+        return ValueFilterApi.gt(measurementIndex, value);
+      case GREATER_THAN_OR_EQUAL:
+        return ValueFilterApi.gtEq(measurementIndex, value);
+      case LESS_THAN:
+        return ValueFilterApi.lt(measurementIndex, value);
+      case LESS_THAN_OR_EQUAL:
+        return ValueFilterApi.ltEq(measurementIndex, value);
+      default:
+        throw new IllegalArgumentException(
+            String.format("Unsupported comparison operator %s", operator));
+    }
+  }
+
+  @SuppressWarnings("unchecked")
+  public static <T extends Comparable<T>> T getValue(Literal value, Type dataType) {
+    try {
+      switch (dataType.getTypeEnum()) {
+        case INT32:
+          return (T) Integer.valueOf(Long.valueOf(getLongValue(value)).intValue());
+        case INT64:
+          return (T) Long.valueOf(getLongValue(value));
+        case FLOAT:
+          return (T) Float.valueOf((float) getDoubleValue(value));
+        case DOUBLE:
+          return (T) Double.valueOf(getDoubleValue(value));
+        case BOOLEAN:
+          return (T) Boolean.valueOf(getBooleanValue(value));
+        case TEXT:
+          return (T) new Binary(getStringValue(value), TSFileConfig.STRING_CHARSET);
+        default:
+          throw new UnsupportedOperationException(
+              String.format("Unsupported data type %s", dataType));
+      }
+    } catch (NumberFormatException e) {
+      throw new IllegalArgumentException(
+          String.format("\"%s\" cannot be cast to [%s]", value, dataType));
+    }
+  }
+
+  @Override
+  protected Filter visitIsNullPredicate(IsNullPredicate node, Context context) {
+    throw new IllegalArgumentException("IS NULL cannot be pushed down");
+  }
+
+  @Override
+  protected Filter visitIsNotNullPredicate(IsNotNullPredicate node, Context context) {
+    checkArgument(isSymbolReference(node.getValue()));
+    SymbolReference operand = (SymbolReference) node.getValue();
+    checkArgument(context.isMeasurementColumn(operand));
+    int measurementIndex = context.getMeasurementIndex(operand.getName());
+    return ValueFilterApi.isNotNull(measurementIndex);
+  }
+
+  @Override
+  protected Filter visitLikePredicate(LikePredicate node, Context context) {
+    checkArgument(isSymbolReference(node.getValue()));
+    SymbolReference operand = (SymbolReference) node.getValue();
+    checkArgument(context.isMeasurementColumn(operand));
+    int measurementIndex = context.getMeasurementIndex(operand.getName());
+    Expression pattern = node.getPattern();
+    return ValueFilterApi.like(measurementIndex, getStringValue(pattern));
+  }
+
+  @Override
+  protected Filter visitLogicalExpression(LogicalExpression node, Context context) {
+    switch (node.getOperator()) {
+      case OR:
+        return FilterFactory.or(
+            node.getTerms().stream().map(n -> process(n, context)).collect(Collectors.toList()));
+      case AND:
+        return FilterFactory.and(
+            node.getTerms().stream().map(n -> process(n, context)).collect(Collectors.toList()));
+      default:
+        throw new IllegalArgumentException(
+            String.format("Unsupported logical operator %s", node.getOperator()));
+    }
+  }
+
+  @Override
+  protected Filter visitNotExpression(NotExpression node, Context context) {
+    return FilterFactory.not(process(node.getValue(), context));
+  }
+
+  @Override
+  protected Filter visitComparisonExpression(ComparisonExpression node, Context context) {
+    if (isTimeColumn(node.getLeft()) || isTimeColumn(node.getRight())) {
+      return TIME_FILTER_VISITOR.process(node, null);
+    }
+
+    Expression left = node.getLeft();
+    Expression right = node.getRight();
+
+    if (isSymbolReference(left)
+        && context.isMeasurementColumn((SymbolReference) left)
+        && isLiteral(right)) {
+      return constructCompareFilter(
+          node.getOperator(), (SymbolReference) left, (Literal) right, context);
+    } else if (isLiteral(left)
+        && isSymbolReference(right)
+        && context.isMeasurementColumn((SymbolReference) right)) {
+      return constructCompareFilter(
+          node.getOperator(), (SymbolReference) right, (Literal) left, context);
+    } else {
+      throw new IllegalStateException(
+          String.format("%s is not supported in value push down", node));
+    }
+  }
+
+  @Override
+  protected Filter visitSimpleCaseExpression(SimpleCaseExpression node, Context context) {
+    throw new UnsupportedOperationException("Filter push down does not support CASE WHEN");
+  }
+
+  @Override
+  protected Filter visitSearchedCaseExpression(SearchedCaseExpression node, Context context) {
+    throw new UnsupportedOperationException("Filter push down does not support CASE WHEN");
+  }
+
+  @Override
+  protected Filter visitIfExpression(IfExpression node, Context context) {
+    throw new UnsupportedOperationException("Filter push down does not support IF");
+  }
+
+  @Override
+  protected Filter visitNullIfExpression(NullIfExpression node, Context context) {
+    throw new UnsupportedOperationException("Filter push down does not support NULLIF");
+  }
+
+  @Override
+  protected Filter visitBetweenPredicate(BetweenPredicate node, Context context) {
+    Expression firstExpression = node.getValue();
+    Expression secondExpression = node.getMin();
+    Expression thirdExpression = node.getMax();
+
+    if (isTimeColumn(firstExpression)
+        || isTimeColumn(secondExpression)
+        || isTimeColumn(thirdExpression)) {
+      return TIME_FILTER_VISITOR.process(node, null);
+    }
+
+    if (isSymbolReference(firstExpression)
+        && context.isMeasurementColumn((SymbolReference) firstExpression)) {
+      return constructBetweenFilter(
+          (SymbolReference) firstExpression, secondExpression, thirdExpression, context);
+    } else if (isSymbolReference(secondExpression)
+        && context.isMeasurementColumn((SymbolReference) secondExpression)) {
+      checkArgument(isLiteral(firstExpression));
+      return constructCompareFilter(
+          ComparisonExpression.Operator.LESS_THAN_OR_EQUAL,
+          (SymbolReference) secondExpression,
+          (Literal) firstExpression,
+          context);
+    } else if (isSymbolReference(thirdExpression)
+        && context.isMeasurementColumn((SymbolReference) thirdExpression)) {
+      checkArgument(isLiteral(firstExpression));
+      return constructCompareFilter(
+          ComparisonExpression.Operator.GREATER_THAN_OR_EQUAL,
+          (SymbolReference) thirdExpression,
+          (Literal) firstExpression,
+          context);
+    } else {
+      throw new IllegalStateException(
+          String.format("%s is not supported in value push down", node));
+    }
+  }
+
+  private <T extends Comparable<T>> Filter constructBetweenFilter(
+      SymbolReference measurementReference,
+      Expression minValue,
+      Expression maxValue,
+      ConvertPredicateToFilterVisitor.Context context) {
+    int measurementIndex = context.getMeasurementIndex(measurementReference.getName());
+    Type dataType = context.getType(Symbol.from(measurementReference));
+
+    checkArgument(isLiteral(minValue) && isLiteral(maxValue));
+
+    T min = getValue((Literal) minValue, dataType);
+    T max = getValue((Literal) maxValue, dataType);
+
+    if (min.compareTo(max) == 0) {
+      return ValueFilterApi.eq(measurementIndex, min);
+    }
+    return ValueFilterApi.between(measurementIndex, min, max);
+  }
+
+  public static double getDoubleValue(Expression expression) {
+    return ((DoubleLiteral) expression).getValue();
+  }
+
+  public static boolean getBooleanValue(Expression expression) {
+    return ((BooleanLiteral) expression).getValue();
+  }
+
+  public static String getStringValue(Expression expression) {
+    return ((StringLiteral) expression).getValue();
+  }
+
+  public static class Context {
+
+    private final Map<String, Integer> measuremrntsMap;
+    private final Map<Symbol, ColumnSchema> schemaMap;
+
+    public Context(List<String> allMeasurements, Map<Symbol, ColumnSchema> schemaMap) {
+      this.measuremrntsMap = new HashMap<>(allMeasurements.size());
+      for (int i = 0, size = allMeasurements.size(); i < size; i++) {
+        measuremrntsMap.put(allMeasurements.get(i), i);
+      }
+      this.schemaMap = schemaMap;
+    }
+
+    public int getMeasurementIndex(String measurement) {
+      Integer index = measuremrntsMap.get(measurement);
+      if (index == null) {
+        throw new IllegalArgumentException(
+            String.format("Measurement %s does not exist", measurement));
+      }
+      return index;
+    }
+
+    public Type getType(Symbol symbol) {
+      Type type = schemaMap.get(symbol).getType();
+      if (type == null) {
+        throw new IllegalArgumentException(
+            String.format("ColumnSchema of Symbol %s isn't saved in schemaMap", symbol));
+      }
+      return type;
+    }
+
+    public boolean isMeasurementColumn(SymbolReference symbolReference) {
+      ColumnSchema schema = schemaMap.get(Symbol.from(symbolReference));
+      return schema != null && schema.getColumnCategory() == TsTableColumnCategory.MEASUREMENT;
+    }
+  }
+}
diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/relational/analyzer/predicate/ConvertPredicateToTimeFilterVisitor.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/relational/analyzer/predicate/ConvertPredicateToTimeFilterVisitor.java
index f48a78151df..0deedfcefe4 100644
--- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/relational/analyzer/predicate/ConvertPredicateToTimeFilterVisitor.java
+++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/relational/analyzer/predicate/ConvertPredicateToTimeFilterVisitor.java
@@ -59,7 +59,7 @@ public class ConvertPredicateToTimeFilterVisitor extends PredicateVisitor<Filter
       checkArgument(value instanceof LongLiteral);
     }
     if (values.size() == 1) {
-      TimeFilterApi.eq(((LongLiteral) values.get(0)).getParsedValue());
+      TimeFilterApi.eq(getLongValue(values.get(0)));
     }
     Set<Long> longValues = new HashSet<>();
     for (Expression value : values) {
diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/relational/analyzer/predicate/PredicatePushIntoScanChecker.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/relational/analyzer/predicate/PredicatePushIntoScanChecker.java
index 519027bf832..7885deb71dd 100644
--- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/relational/analyzer/predicate/PredicatePushIntoScanChecker.java
+++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/relational/analyzer/predicate/PredicatePushIntoScanChecker.java
@@ -63,7 +63,7 @@ public class PredicatePushIntoScanChecker extends PredicateVisitor<Boolean, Void
   protected Boolean visitLikePredicate(LikePredicate node, Void context) {
     return isSymbolReference(node.getValue())
         && isLiteral(node.getPattern())
-        && node.getEscape().map(this::isLiteral).orElse(true);
+        && node.getEscape().map(PredicatePushIntoScanChecker::isLiteral).orElse(true);
   }
 
   @Override
diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/relational/planner/node/ProjectNode.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/relational/planner/node/ProjectNode.java
index ed65602cf9e..96f5da07379 100644
--- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/relational/planner/node/ProjectNode.java
+++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/relational/planner/node/ProjectNode.java
@@ -35,6 +35,10 @@ public class ProjectNode extends SingleChildProcessNode {
     return null;
   }
 
+  public Assignments getAssignments() {
+    return assignments;
+  }
+
   @Override
   protected void serializeAttributes(ByteBuffer byteBuffer) {}
 
diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/transformation/dag/column/multi/LogicalAndMultiColumnTransformer.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/transformation/dag/column/multi/LogicalAndMultiColumnTransformer.java
new file mode 100644
index 00000000000..af14cb2204e
--- /dev/null
+++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/transformation/dag/column/multi/LogicalAndMultiColumnTransformer.java
@@ -0,0 +1,64 @@
+/*
+ * 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.iotdb.db.queryengine.transformation.dag.column.multi;
+
+import org.apache.iotdb.db.queryengine.transformation.dag.column.ColumnTransformer;
+import org.apache.iotdb.tsfile.read.common.block.column.Column;
+import org.apache.iotdb.tsfile.read.common.block.column.ColumnBuilder;
+import org.apache.iotdb.tsfile.read.common.type.Type;
+
+import java.util.List;
+
+public class LogicalAndMultiColumnTransformer extends LogicalMultiColumnTransformer {
+  public LogicalAndMultiColumnTransformer(
+      Type returnType, List<ColumnTransformer> columnTransformerList) {
+    super(returnType, columnTransformerList);
+  }
+
+  @Override
+  protected void doTransform(
+      List<Column> childrenColumns, ColumnBuilder builder, int positionCount) {
+    for (int i = 0; i < positionCount; i++) {
+      boolean result = true;
+      boolean hasNull = false;
+      for (Column column : childrenColumns) {
+        if (column.isNull(i)) {
+          hasNull = true;
+        } else if (!column.getBoolean(i)) {
+          result = false;
+          break;
+        }
+      }
+      // have no null, all is true, result will be true
+      // have no null, and also have false, result will be false
+      // have null, and others are all true, result will be null
+      // have null, and also have false, result will be false
+      if (!result) {
+        returnType.writeBoolean(builder, false);
+      } else {
+        if (hasNull) {
+          builder.appendNull();
+        } else {
+          returnType.writeBoolean(builder, true);
+        }
+      }
+    }
+  }
+}
diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/relational/analyzer/predicate/ConvertPredicateToFilterVisitor.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/transformation/dag/column/multi/LogicalMultiColumnTransformer.java
similarity index 50%
copy from iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/relational/analyzer/predicate/ConvertPredicateToFilterVisitor.java
copy to iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/transformation/dag/column/multi/LogicalMultiColumnTransformer.java
index 7ab2448ded1..6824af8ce97 100644
--- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/relational/analyzer/predicate/ConvertPredicateToFilterVisitor.java
+++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/transformation/dag/column/multi/LogicalMultiColumnTransformer.java
@@ -17,6 +17,27 @@
  * under the License.
  */
 
-package org.apache.iotdb.db.queryengine.plan.relational.analyzer.predicate;
+package org.apache.iotdb.db.queryengine.transformation.dag.column.multi;
 
-public class ConvertPredicateToFilterVisitor {}
+import org.apache.iotdb.db.queryengine.transformation.dag.column.ColumnTransformer;
+import org.apache.iotdb.tsfile.read.common.type.Type;
+import org.apache.iotdb.tsfile.read.common.type.TypeEnum;
+
+import java.util.List;
+
+public abstract class LogicalMultiColumnTransformer extends MultiColumnTransformer {
+
+  protected LogicalMultiColumnTransformer(
+      Type returnType, List<ColumnTransformer> columnTransformerList) {
+    super(returnType, columnTransformerList);
+  }
+
+  @Override
+  protected void checkType() {
+    for (ColumnTransformer child : columnTransformerList) {
+      if (!child.typeEquals(TypeEnum.BOOLEAN)) {
+        throw new UnsupportedOperationException("Unsupported Type");
+      }
+    }
+  }
+}
diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/transformation/dag/column/multi/LogicalOrMultiColumnTransformer.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/transformation/dag/column/multi/LogicalOrMultiColumnTransformer.java
new file mode 100644
index 00000000000..0d49f0bce91
--- /dev/null
+++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/transformation/dag/column/multi/LogicalOrMultiColumnTransformer.java
@@ -0,0 +1,63 @@
+/*
+ * 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.iotdb.db.queryengine.transformation.dag.column.multi;
+
+import org.apache.iotdb.db.queryengine.transformation.dag.column.ColumnTransformer;
+import org.apache.iotdb.tsfile.read.common.block.column.Column;
+import org.apache.iotdb.tsfile.read.common.block.column.ColumnBuilder;
+import org.apache.iotdb.tsfile.read.common.type.Type;
+
+import java.util.List;
+
+public class LogicalOrMultiColumnTransformer extends LogicalMultiColumnTransformer {
+  public LogicalOrMultiColumnTransformer(
+      Type returnType, List<ColumnTransformer> columnTransformerList) {
+    super(returnType, columnTransformerList);
+  }
+
+  @Override
+  protected void doTransform(
+      List<Column> childrenColumns, ColumnBuilder builder, int positionCount) {
+    for (int i = 0; i < positionCount; i++) {
+      boolean result = false;
+      boolean hasNull = false;
+      for (Column column : childrenColumns) {
+        if (column.isNull(i)) {
+          hasNull = true;
+        } else if (column.getBoolean(i)) {
+          result = true;
+          break;
+        }
+      }
+      // have any true, result will be true
+      // have no true, and have both false and null, result will be null
+      // have all false, result will be false
+      if (result) {
+        returnType.writeBoolean(builder, true);
+      } else {
+        if (hasNull) {
+          builder.appendNull();
+        } else {
+          returnType.writeBoolean(builder, false);
+        }
+      }
+    }
+  }
+}
diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/transformation/dag/column/multi/MultiColumnTransformer.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/transformation/dag/column/multi/MultiColumnTransformer.java
new file mode 100644
index 00000000000..4bb874c6c07
--- /dev/null
+++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/transformation/dag/column/multi/MultiColumnTransformer.java
@@ -0,0 +1,66 @@
+/*
+ * 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.iotdb.db.queryengine.transformation.dag.column.multi;
+
+import org.apache.iotdb.db.queryengine.transformation.dag.column.ColumnTransformer;
+import org.apache.iotdb.tsfile.read.common.block.column.Column;
+import org.apache.iotdb.tsfile.read.common.block.column.ColumnBuilder;
+import org.apache.iotdb.tsfile.read.common.type.Type;
+
+import java.util.List;
+import java.util.stream.Collectors;
+
+public abstract class MultiColumnTransformer extends ColumnTransformer {
+
+  protected final List<ColumnTransformer> columnTransformerList;
+
+  protected MultiColumnTransformer(Type returnType, List<ColumnTransformer> columnTransformerList) {
+    super(returnType);
+    this.columnTransformerList = columnTransformerList;
+    checkType();
+  }
+
+  @Override
+  public void evaluate() {
+
+    for (ColumnTransformer child : columnTransformerList) {
+      child.tryEvaluate();
+    }
+
+    // attention: get positionCount before calling getColumn
+    int positionCount = columnTransformerList.get(0).getColumnCachePositionCount();
+
+    ColumnBuilder builder = returnType.createColumnBuilder(positionCount);
+    doTransform(
+        columnTransformerList.stream()
+            .map(ColumnTransformer::getColumn)
+            .collect(Collectors.toList()),
+        builder,
+        positionCount);
+    initializeColumnCache(builder.build());
+  }
+
+  protected abstract void doTransform(
+      List<Column> childrenColumns, ColumnBuilder builder, int positionCount);
+
+  public List<ColumnTransformer> getChildren() {
+    return columnTransformerList;
+  }
+}
diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/transformation/dag/column/unary/InColumnTransformer.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/transformation/dag/column/unary/InColumnTransformer.java
index 6aa7e0797c8..7f197802f09 100644
--- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/transformation/dag/column/unary/InColumnTransformer.java
+++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/transformation/dag/column/unary/InColumnTransformer.java
@@ -21,6 +21,11 @@ package org.apache.iotdb.db.queryengine.transformation.dag.column.unary;
 
 import org.apache.iotdb.db.exception.sql.SemanticException;
 import org.apache.iotdb.db.queryengine.transformation.dag.column.ColumnTransformer;
+import org.apache.iotdb.db.relational.sql.tree.BooleanLiteral;
+import org.apache.iotdb.db.relational.sql.tree.DoubleLiteral;
+import org.apache.iotdb.db.relational.sql.tree.Literal;
+import org.apache.iotdb.db.relational.sql.tree.LongLiteral;
+import org.apache.iotdb.db.relational.sql.tree.StringLiteral;
 import org.apache.iotdb.tsfile.common.conf.TSFileConfig;
 import org.apache.iotdb.tsfile.read.common.block.column.Column;
 import org.apache.iotdb.tsfile.read.common.block.column.ColumnBuilder;
@@ -28,6 +33,7 @@ import org.apache.iotdb.tsfile.read.common.type.Type;
 import org.apache.iotdb.tsfile.read.common.type.TypeEnum;
 
 import java.util.HashSet;
+import java.util.List;
 import java.util.Set;
 
 public class InColumnTransformer extends UnaryColumnTransformer {
@@ -56,6 +62,17 @@ public class InColumnTransformer extends UnaryColumnTransformer {
     initTypedSet(values);
   }
 
+  public InColumnTransformer(
+      Type returnType, ColumnTransformer childColumnTransformer, List<Literal> values) {
+    super(returnType, childColumnTransformer);
+    satisfy = new InSatisfy();
+    this.childType =
+        childColumnTransformer.getType() == null
+            ? null
+            : childColumnTransformer.getType().getTypeEnum();
+    initTypedSet(values);
+  }
+
   @Override
   protected void doTransform(Column column, ColumnBuilder columnBuilder) {
     for (int i = 0, n = column.getPositionCount(); i < n; i++) {
@@ -150,6 +167,69 @@ public class InColumnTransformer extends UnaryColumnTransformer {
     }
   }
 
+  private void initTypedSet(List<Literal> values) {
+    if (childType == null) {
+      return;
+    }
+    String errorMsg = "\"%s\" cannot be cast to [%s]";
+    switch (childType) {
+      case INT32:
+        intSet = new HashSet<>();
+        for (Literal value : values) {
+          try {
+            intSet.add((int) ((LongLiteral) value).getParsedValue());
+          } catch (IllegalArgumentException e) {
+            throw new SemanticException(String.format(errorMsg, value, childType));
+          }
+        }
+        break;
+      case INT64:
+        longSet = new HashSet<>();
+        for (Literal value : values) {
+          try {
+            longSet.add((((LongLiteral) value).getParsedValue()));
+          } catch (IllegalArgumentException e) {
+            throw new SemanticException(String.format(errorMsg, value, childType));
+          }
+        }
+        break;
+      case FLOAT:
+        floatSet = new HashSet<>();
+        for (Literal value : values) {
+          try {
+            floatSet.add((float) ((DoubleLiteral) value).getValue());
+          } catch (IllegalArgumentException e) {
+            throw new SemanticException(String.format(errorMsg, value, childType));
+          }
+        }
+        break;
+      case DOUBLE:
+        doubleSet = new HashSet<>();
+        for (Literal value : values) {
+          try {
+            doubleSet.add(((DoubleLiteral) value).getValue());
+          } catch (IllegalArgumentException e) {
+            throw new SemanticException(String.format(errorMsg, value, childType));
+          }
+        }
+        break;
+      case BOOLEAN:
+        booleanSet = new HashSet<>();
+        for (Literal value : values) {
+          booleanSet.add(((BooleanLiteral) value).getValue());
+        }
+        break;
+      case TEXT:
+        stringSet = new HashSet<>();
+        for (Literal value : values) {
+          stringSet.add(((StringLiteral) value).getValue());
+        }
+        break;
+      default:
+        throw new UnsupportedOperationException("unsupported data type: " + childType);
+    }
+  }
+
   private boolean strictCastToBool(String s) {
     if ("true".equalsIgnoreCase(s)) {
       return true;


(iotdb) 03/05: partial ok

Posted by ja...@apache.org.
This is an automated email from the ASF dual-hosted git repository.

jackietien pushed a commit to branch ty/TableModelGrammar
in repository https://gitbox.apache.org/repos/asf/iotdb.git

commit 7ffa882b6fcb0a14870e523bdad10a4e6b7900be
Author: JackieTien97 <ja...@gmail.com>
AuthorDate: Thu Apr 18 10:00:49 2024 +0800

    partial ok
---
 .../iotdb/db/protocol/session/IClientSession.java  |  34 +++++-
 .../iotdb/db/protocol/session/SessionManager.java  |   3 +-
 .../iotdb/db/queryengine/common/SessionInfo.java   |  37 +++++-
 .../queryengine/plan/analyze/PredicateUtils.java   |  11 ++
 .../plan/planner/LocalExecutionPlanner.java        |  14 ++-
 .../plan/planner/TableOperatorGenerator.java       | 135 +++++++++++++++------
 .../plan/planner/plan/TableModelTimePredicate.java |  65 ++++++++++
 .../plan/relational/planner/node/LimitNode.java    |   4 +
 .../plan/relational/planner/node/OffsetNode.java   |   4 +
 .../plan/relational/analyzer/AnalyzerTest.java     |   5 +-
 10 files changed, 265 insertions(+), 47 deletions(-)

diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/protocol/session/IClientSession.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/protocol/session/IClientSession.java
index ff3040a34ec..6ef98c4308a 100644
--- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/protocol/session/IClientSession.java
+++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/protocol/session/IClientSession.java
@@ -22,9 +22,13 @@ package org.apache.iotdb.db.protocol.session;
 import org.apache.iotdb.commons.conf.IoTDBConstant.ClientVersion;
 import org.apache.iotdb.service.rpc.thrift.TSConnectionInfo;
 import org.apache.iotdb.service.rpc.thrift.TSConnectionType;
+import org.apache.iotdb.tsfile.utils.ReadWriteIOUtils;
 
 import javax.annotation.Nullable;
 
+import java.io.DataOutputStream;
+import java.io.IOException;
+import java.nio.ByteBuffer;
 import java.time.ZoneId;
 import java.util.Set;
 import java.util.TimeZone;
@@ -159,7 +163,33 @@ public abstract class IClientSession {
   }
 
   public enum SqlDialect {
-    TREE,
-    TABLE
+    TREE((byte) 0),
+    TABLE((byte) 1);
+
+    private final byte dialect;
+
+    SqlDialect(byte dialect) {
+      this.dialect = dialect;
+    }
+
+    public byte getDialect() {
+      return dialect;
+    }
+
+    public void serialize(DataOutputStream stream) throws IOException {
+      ReadWriteIOUtils.write(dialect, stream);
+    }
+
+    public static SqlDialect deserializeFrom(ByteBuffer buffer) {
+      byte b = ReadWriteIOUtils.readByte(buffer);
+      switch (b) {
+        case 0:
+          return TREE;
+        case 1:
+          return TABLE;
+        default:
+          throw new IllegalArgumentException(String.format("Unknown sql dialect: %s", b));
+      }
+    }
   }
 }
diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/protocol/session/SessionManager.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/protocol/session/SessionManager.java
index a3f2c4358ce..6f13111731d 100644
--- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/protocol/session/SessionManager.java
+++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/protocol/session/SessionManager.java
@@ -378,7 +378,8 @@ public class SessionManager implements SessionManagerMBean {
         session.getUsername(),
         session.getZoneId(),
         session.getClientVersion(),
-        session.getDatabaseName());
+        session.getDatabaseName(),
+        session.getSqlDialect());
   }
 
   @Override
diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/common/SessionInfo.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/common/SessionInfo.java
index 958d39abafa..2cc9aed732a 100644
--- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/common/SessionInfo.java
+++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/common/SessionInfo.java
@@ -20,7 +20,7 @@
 package org.apache.iotdb.db.queryengine.common;
 
 import org.apache.iotdb.commons.conf.IoTDBConstant.ClientVersion;
-import org.apache.iotdb.commons.utils.TestOnly;
+import org.apache.iotdb.db.protocol.session.IClientSession;
 import org.apache.iotdb.db.queryengine.plan.relational.security.Identity;
 import org.apache.iotdb.tsfile.utils.ReadWriteIOUtils;
 
@@ -40,6 +40,8 @@ public class SessionInfo {
 
   @Nullable private final String databaseName;
 
+  private final IClientSession.SqlDialect sqlDialect;
+
   private ClientVersion version = ClientVersion.V_1_0;
 
   public SessionInfo(long sessionId, String userName, ZoneId zoneId) {
@@ -47,12 +49,16 @@ public class SessionInfo {
     this.userName = userName;
     this.zoneId = zoneId;
     this.databaseName = null;
+    this.sqlDialect = IClientSession.SqlDialect.TREE;
   }
 
-  @TestOnly
   public SessionInfo(
-      long sessionId, String userName, ZoneId zoneId, @Nullable String databaseName) {
-    this(sessionId, userName, zoneId, ClientVersion.V_1_0, databaseName);
+      long sessionId,
+      String userName,
+      ZoneId zoneId,
+      @Nullable String databaseName,
+      IClientSession.SqlDialect sqlDialect) {
+    this(sessionId, userName, zoneId, ClientVersion.V_1_0, databaseName, sqlDialect);
   }
 
   public SessionInfo(
@@ -60,12 +66,14 @@ public class SessionInfo {
       String userName,
       ZoneId zoneId,
       ClientVersion version,
-      @Nullable String databaseName) {
+      @Nullable String databaseName,
+      IClientSession.SqlDialect sqlDialect) {
     this.sessionId = sessionId;
     this.userName = userName;
     this.zoneId = zoneId;
     this.version = version;
     this.databaseName = databaseName;
+    this.sqlDialect = sqlDialect;
   }
 
   public long getSessionId() {
@@ -92,16 +100,33 @@ public class SessionInfo {
     return Optional.ofNullable(databaseName);
   }
 
+  public IClientSession.SqlDialect getSqlDialect() {
+    return sqlDialect;
+  }
+
   public static SessionInfo deserializeFrom(ByteBuffer buffer) {
     long sessionId = ReadWriteIOUtils.readLong(buffer);
     String userName = ReadWriteIOUtils.readString(buffer);
     ZoneId zoneId = ZoneId.of(Objects.requireNonNull(ReadWriteIOUtils.readString(buffer)));
-    return new SessionInfo(sessionId, userName, zoneId);
+    boolean hasDatabaseName = ReadWriteIOUtils.readBool(buffer);
+    String databaseName = null;
+    if (hasDatabaseName) {
+      databaseName = ReadWriteIOUtils.readString(buffer);
+    }
+    IClientSession.SqlDialect sqlDialect1 = IClientSession.SqlDialect.deserializeFrom(buffer);
+    return new SessionInfo(sessionId, userName, zoneId, databaseName, sqlDialect1);
   }
 
   public void serialize(DataOutputStream stream) throws IOException {
     ReadWriteIOUtils.write(sessionId, stream);
     ReadWriteIOUtils.write(userName, stream);
     ReadWriteIOUtils.write(zoneId.getId(), stream);
+    if (databaseName == null) {
+      ReadWriteIOUtils.write((byte) 0, stream);
+    } else {
+      ReadWriteIOUtils.write((byte) 1, stream);
+      ReadWriteIOUtils.write(databaseName, stream);
+    }
+    sqlDialect.serialize(stream);
   }
 }
diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/analyze/PredicateUtils.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/analyze/PredicateUtils.java
index dc2d8653b2c..872f80217f9 100644
--- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/analyze/PredicateUtils.java
+++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/analyze/PredicateUtils.java
@@ -279,6 +279,17 @@ public class PredicateUtils {
     return predicate.accept(new ConvertPredicateToTimeFilterVisitor(), null);
   }
 
+  public static Filter convertPredicateToTimeFilter(
+      org.apache.iotdb.db.relational.sql.tree.Expression predicate) {
+    if (predicate == null) {
+      return null;
+    }
+    return predicate.accept(
+        new org.apache.iotdb.db.queryengine.plan.relational.analyzer.predicate
+            .ConvertPredicateToTimeFilterVisitor(),
+        null);
+  }
+
   public static Filter convertPredicateToFilter(
       Expression predicate,
       List<String> allMeasurements,
diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/planner/LocalExecutionPlanner.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/planner/LocalExecutionPlanner.java
index 6acd4e1b716..2894ac2b432 100644
--- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/planner/LocalExecutionPlanner.java
+++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/planner/LocalExecutionPlanner.java
@@ -21,6 +21,7 @@ package org.apache.iotdb.db.queryengine.plan.planner;
 import org.apache.iotdb.commons.path.PartialPath;
 import org.apache.iotdb.db.conf.IoTDBConfig;
 import org.apache.iotdb.db.conf.IoTDBDescriptor;
+import org.apache.iotdb.db.protocol.session.IClientSession;
 import org.apache.iotdb.db.queryengine.exception.MemoryNotEnoughException;
 import org.apache.iotdb.db.queryengine.execution.driver.DataDriverContext;
 import org.apache.iotdb.db.queryengine.execution.fragment.DataNodeQueryContext;
@@ -80,7 +81,18 @@ public class LocalExecutionPlanner {
 
     // Generate pipelines, return the last pipeline data structure
     // TODO Replace operator with operatorFactory to build multiple driver for one pipeline
-    Operator root = plan.accept(new OperatorTreeGenerator(), context);
+    Operator root;
+    IClientSession.SqlDialect sqlDialect = instanceContext.getSessionInfo().getSqlDialect();
+    switch (sqlDialect) {
+      case TREE:
+        root = plan.accept(new OperatorTreeGenerator(), context);
+        break;
+      case TABLE:
+        root = plan.accept(new TableOperatorGenerator(), context);
+        break;
+      default:
+        throw new IllegalArgumentException(String.format("Unknown sql dialect: %s", sqlDialect));
+    }
 
     // check whether current free memory is enough to execute current query
     long estimatedMemorySize = checkMemory(root, instanceContext.getStateMachine());
diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/planner/TableOperatorGenerator.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/planner/TableOperatorGenerator.java
index 273e949b37f..bd699c18b5e 100644
--- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/planner/TableOperatorGenerator.java
+++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/planner/TableOperatorGenerator.java
@@ -24,10 +24,13 @@ import org.apache.iotdb.db.queryengine.execution.driver.DataDriverContext;
 import org.apache.iotdb.db.queryengine.execution.operator.Operator;
 import org.apache.iotdb.db.queryengine.execution.operator.OperatorContext;
 import org.apache.iotdb.db.queryengine.execution.operator.process.FilterAndProjectOperator;
+import org.apache.iotdb.db.queryengine.execution.operator.process.LimitOperator;
+import org.apache.iotdb.db.queryengine.execution.operator.process.OffsetOperator;
 import org.apache.iotdb.db.queryengine.execution.operator.source.AlignedSeriesScanOperator;
 import org.apache.iotdb.db.queryengine.execution.operator.source.relational.TableScanOperator;
 import org.apache.iotdb.db.queryengine.execution.relational.ColumnTransformerBuilder;
 import org.apache.iotdb.db.queryengine.plan.analyze.PredicateUtils;
+import org.apache.iotdb.db.queryengine.plan.analyze.TypeProvider;
 import org.apache.iotdb.db.queryengine.plan.planner.plan.node.PlanNode;
 import org.apache.iotdb.db.queryengine.plan.planner.plan.node.PlanNodeId;
 import org.apache.iotdb.db.queryengine.plan.planner.plan.node.PlanVisitor;
@@ -63,6 +66,8 @@ import java.util.HashSet;
 import java.util.LinkedHashMap;
 import java.util.List;
 import java.util.Map;
+import java.util.Optional;
+import java.util.stream.Collectors;
 
 import static java.util.Objects.requireNonNull;
 import static org.apache.iotdb.db.queryengine.execution.operator.source.relational.TableScanOperator.constructAlignedPath;
@@ -170,8 +175,8 @@ public class TableOperatorGenerator extends PlanVisitor<Operator, LocalExecution
 
     if (!predicateCanPushIntoScan) {
 
-      return constructFilterOperator(
-          pushDownPredicate,
+      return constructFilterAndProjectOperator(
+          Optional.of(pushDownPredicate),
           tableScanOperator,
           node.getOutputSymbols().stream()
               .map(Symbol::toSymbolReference)
@@ -221,11 +226,24 @@ public class TableOperatorGenerator extends PlanVisitor<Operator, LocalExecution
 
   @Override
   public Operator visitFilter(FilterNode node, LocalExecutionPlanContext context) {
-    return super.visitFilter(node, context);
+    TypeProvider typeProvider = context.getTypeProvider();
+    Optional<Expression> predicate = Optional.of(node.getPredicate());
+    Operator inputOperator = node.getChild().accept(this, context);
+    List<TSDataType> inputDataTypes = getInputColumnTypes(node, typeProvider);
+    Map<Symbol, List<InputLocation>> inputLocations = makeLayout(node.getChildren());
+
+    return constructFilterAndProjectOperator(
+        predicate,
+        inputOperator,
+        node.getOutputSymbols().stream().map(Symbol::toSymbolReference).toArray(Expression[]::new),
+        inputDataTypes,
+        inputLocations,
+        node.getPlanNodeId(),
+        context);
   }
 
-  private Operator constructFilterOperator(
-      Expression predicate,
+  private Operator constructFilterAndProjectOperator(
+      Optional<Expression> predicate,
       Operator inputOperator,
       Expression[] projectExpressions,
       List<TSDataType> inputDataTypes,
@@ -238,36 +256,41 @@ public class TableOperatorGenerator extends PlanVisitor<Operator, LocalExecution
     // records LeafColumnTransformer of filter
     List<LeafColumnTransformer> filterLeafColumnTransformerList = new ArrayList<>();
 
-    // records common ColumnTransformer between filter and project expressions
-    List<ColumnTransformer> commonTransformerList = new ArrayList<>();
-
-    // records LeafColumnTransformer of project expressions
-    List<LeafColumnTransformer> projectLeafColumnTransformerList = new ArrayList<>();
-
     // records subexpression -> ColumnTransformer for filter
     Map<Expression, ColumnTransformer> filterExpressionColumnTransformerMap = new HashMap<>();
 
     ColumnTransformerBuilder visitor = new ColumnTransformerBuilder();
 
-    ColumnTransformerBuilder.Context filterColumnTransformerContext =
-        new ColumnTransformerBuilder.Context(
-            context.getDriverContext().getFragmentInstanceContext().getSessionInfo(),
-            filterLeafColumnTransformerList,
-            inputLocations,
-            filterExpressionColumnTransformerMap,
-            ImmutableMap.of(),
-            ImmutableList.of(),
-            ImmutableList.of(),
-            0,
-            context.getTypeProvider());
-
     ColumnTransformer filterOutputTransformer =
-        visitor.process(predicate, filterColumnTransformerContext);
+        predicate
+            .map(
+                p -> {
+                  ColumnTransformerBuilder.Context filterColumnTransformerContext =
+                      new ColumnTransformerBuilder.Context(
+                          context.getDriverContext().getFragmentInstanceContext().getSessionInfo(),
+                          filterLeafColumnTransformerList,
+                          inputLocations,
+                          filterExpressionColumnTransformerMap,
+                          ImmutableMap.of(),
+                          ImmutableList.of(),
+                          ImmutableList.of(),
+                          0,
+                          context.getTypeProvider());
+
+                  return visitor.process(p, filterColumnTransformerContext);
+                })
+            .orElse(null);
+
+    // records LeafColumnTransformer of project expressions
+    List<LeafColumnTransformer> projectLeafColumnTransformerList = new ArrayList<>();
 
     List<ColumnTransformer> projectOutputTransformerList = new ArrayList<>();
 
     Map<Expression, ColumnTransformer> projectExpressionColumnTransformerMap = new HashMap<>();
 
+    // records common ColumnTransformer between filter and project expressions
+    List<ColumnTransformer> commonTransformerList = new ArrayList<>();
+
     ColumnTransformerBuilder.Context projectColumnTransformerContext =
         new ColumnTransformerBuilder.Context(
             context.getDriverContext().getFragmentInstanceContext().getSessionInfo(),
@@ -304,33 +327,73 @@ public class TableOperatorGenerator extends PlanVisitor<Operator, LocalExecution
         projectLeafColumnTransformerList,
         projectOutputTransformerList,
         false,
-        true);
+        predicate.isPresent());
   }
 
   @Override
   public Operator visitProject(ProjectNode node, LocalExecutionPlanContext context) {
+    TypeProvider typeProvider = context.getTypeProvider();
+    Optional<Expression> predicate;
+    Operator inputOperator;
+    List<TSDataType> inputDataTypes;
+    Map<Symbol, List<InputLocation>> inputLocations;
     if (node.getChild() instanceof FilterNode) {
       FilterNode filterNode = (FilterNode) node.getChild();
-      return constructFilterOperator(
-          filterNode.getPredicate(),
-          filterNode.getChild().accept(this, context),
-          node.getAssignments().getExpressions().toArray(new Expression[0]),
-          tableScanOperator.getResultDataTypes(),
-          makeLayout(Collections.singletonList(filterNode.getChild())),
-          node.getPlanNodeId(),
-          context);
+      predicate = Optional.of(filterNode.getPredicate());
+      inputOperator = filterNode.getChild().accept(this, context);
+      inputDataTypes = getInputColumnTypes(filterNode, typeProvider);
+      inputLocations = makeLayout(filterNode.getChildren());
+    } else {
+      predicate = Optional.empty();
+      inputOperator = node.getChild().accept(this, context);
+      inputDataTypes = getInputColumnTypes(node, typeProvider);
+      inputLocations = makeLayout(node.getChildren());
     }
-    return super.visitProject(node, context);
+
+    return constructFilterAndProjectOperator(
+        predicate,
+        inputOperator,
+        node.getAssignments().getExpressions().toArray(new Expression[0]),
+        inputDataTypes,
+        inputLocations,
+        node.getPlanNodeId(),
+        context);
+  }
+
+  private List<TSDataType> getInputColumnTypes(PlanNode node, TypeProvider typeProvider) {
+    return node.getChildren().stream()
+        .map(PlanNode::getOutputSymbols)
+        .flatMap(List::stream)
+        .map(s -> getTSDataType(typeProvider.get(s)))
+        .collect(Collectors.toList());
   }
 
   @Override
   public Operator visitLimit(LimitNode node, LocalExecutionPlanContext context) {
-    return super.visitLimit(node, context);
+    Operator child = node.getChild().accept(this, context);
+    OperatorContext operatorContext =
+        context
+            .getDriverContext()
+            .addOperatorContext(
+                context.getNextOperatorId(),
+                node.getPlanNodeId(),
+                LimitOperator.class.getSimpleName());
+
+    return new LimitOperator(operatorContext, node.getCount(), child);
   }
 
   @Override
   public Operator visitOffset(OffsetNode node, LocalExecutionPlanContext context) {
-    return super.visitOffset(node, context);
+    Operator child = node.getChild().accept(this, context);
+    OperatorContext operatorContext =
+        context
+            .getDriverContext()
+            .addOperatorContext(
+                context.getNextOperatorId(),
+                node.getPlanNodeId(),
+                OffsetOperator.class.getSimpleName());
+
+    return new OffsetOperator(operatorContext, node.getCount(), child);
   }
 
   @Override
diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/planner/plan/TableModelTimePredicate.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/planner/plan/TableModelTimePredicate.java
new file mode 100644
index 00000000000..5a19fb08a71
--- /dev/null
+++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/planner/plan/TableModelTimePredicate.java
@@ -0,0 +1,65 @@
+/*
+ * 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.iotdb.db.queryengine.plan.planner.plan;
+
+import org.apache.iotdb.db.queryengine.plan.analyze.PredicateUtils;
+import org.apache.iotdb.db.relational.sql.tree.Expression;
+import org.apache.iotdb.tsfile.read.filter.basic.Filter;
+
+import java.io.DataOutputStream;
+import java.io.IOException;
+import java.io.UnsupportedEncodingException;
+import java.util.Objects;
+
+public class TableModelTimePredicate implements TimePredicate {
+
+  private final Expression timePredicate;
+
+  public TableModelTimePredicate(Expression timePredicate) {
+    this.timePredicate = timePredicate;
+  }
+
+  @Override
+  public void serialize(DataOutputStream stream) throws IOException {
+    throw new UnsupportedEncodingException();
+  }
+
+  @Override
+  public Filter convertPredicateToTimeFilter() {
+    return PredicateUtils.convertPredicateToTimeFilter(timePredicate);
+  }
+
+  @Override
+  public boolean equals(Object o) {
+    if (this == o) {
+      return true;
+    }
+    if (o == null || getClass() != o.getClass()) {
+      return false;
+    }
+    TableModelTimePredicate that = (TableModelTimePredicate) o;
+    return Objects.equals(timePredicate, that.timePredicate);
+  }
+
+  @Override
+  public int hashCode() {
+    return Objects.hash(timePredicate);
+  }
+}
diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/relational/planner/node/LimitNode.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/relational/planner/node/LimitNode.java
index 17481074932..dbc82fcb5b9 100644
--- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/relational/planner/node/LimitNode.java
+++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/relational/planner/node/LimitNode.java
@@ -46,4 +46,8 @@ public class LimitNode extends SingleChildProcessNode {
   public List<Symbol> getOutputSymbols() {
     return child.getOutputSymbols();
   }
+
+  public long getCount() {
+    return count;
+  }
 }
diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/relational/planner/node/OffsetNode.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/relational/planner/node/OffsetNode.java
index 69ac2517e83..b1fa2db71ee 100644
--- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/relational/planner/node/OffsetNode.java
+++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/relational/planner/node/OffsetNode.java
@@ -38,4 +38,8 @@ public class OffsetNode extends SingleChildProcessNode {
   public List<Symbol> getOutputSymbols() {
     return child.getOutputSymbols();
   }
+
+  public long getCount() {
+    return count;
+  }
 }
diff --git a/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/queryengine/plan/relational/analyzer/AnalyzerTest.java b/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/queryengine/plan/relational/analyzer/AnalyzerTest.java
index 8bb2cecd3bc..583f1ec64ca 100644
--- a/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/queryengine/plan/relational/analyzer/AnalyzerTest.java
+++ b/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/queryengine/plan/relational/analyzer/AnalyzerTest.java
@@ -20,6 +20,7 @@
 package org.apache.iotdb.db.queryengine.plan.relational.analyzer;
 
 import org.apache.iotdb.commons.exception.IoTDBException;
+import org.apache.iotdb.db.protocol.session.IClientSession;
 import org.apache.iotdb.db.queryengine.common.MPPQueryContext;
 import org.apache.iotdb.db.queryengine.common.QueryId;
 import org.apache.iotdb.db.queryengine.common.SessionInfo;
@@ -154,7 +155,9 @@ public class AnalyzerTest {
   private Analysis analyzeSQL(String sql, Metadata metadata) {
     try {
       Statement statement = sqlParser.createStatement(sql);
-      SessionInfo session = new SessionInfo(0, "test", ZoneId.systemDefault(), "testdb");
+      SessionInfo session =
+          new SessionInfo(
+              0, "test", ZoneId.systemDefault(), "testdb", IClientSession.SqlDialect.TABLE);
       StatementAnalyzerFactory statementAnalyzerFactory =
           new StatementAnalyzerFactory(metadata, sqlParser, nopAccessControl);
 


(iotdb) 05/05: resolve conflicts

Posted by ja...@apache.org.
This is an automated email from the ASF dual-hosted git repository.

jackietien pushed a commit to branch ty/TableModelGrammar
in repository https://gitbox.apache.org/repos/asf/iotdb.git

commit eff3915bcfdcb3e27a45c6b0e3a8bedcef389743
Author: JackieTien97 <ja...@gmail.com>
AuthorDate: Thu Apr 18 10:03:08 2024 +0800

    resolve conflicts
---
 .../iotdb/db/queryengine/plan/planner/TableOperatorGenerator.java | 2 +-
 .../queryengine/plan/relational/planner/node/TableScanNode.java   | 8 --------
 2 files changed, 1 insertion(+), 9 deletions(-)

diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/planner/TableOperatorGenerator.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/planner/TableOperatorGenerator.java
index bd699c18b5e..ea34df6f712 100644
--- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/planner/TableOperatorGenerator.java
+++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/planner/TableOperatorGenerator.java
@@ -91,7 +91,7 @@ public class TableOperatorGenerator extends PlanVisitor<Operator, LocalExecution
     List<ColumnSchema> columnSchemas = new ArrayList<>(outputColumnCount);
     int[] columnsIndexArray = new int[outputColumnCount];
     Map<Symbol, ColumnSchema> columnSchemaMap = node.getAssignments();
-    Map<Symbol, Integer> idAndAttributeColumnsIndexMap = node.getAttributesMap();
+    Map<Symbol, Integer> idAndAttributeColumnsIndexMap = node.getIdAndAttributeIndexMap();
     List<String> measurementColumnNames = new ArrayList<>();
     List<IMeasurementSchema> measurementSchemas = new ArrayList<>();
     int measurementColumnCount = 0;
diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/relational/planner/node/TableScanNode.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/relational/planner/node/TableScanNode.java
index f62896907cc..ee599da87e3 100644
--- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/relational/planner/node/TableScanNode.java
+++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/relational/planner/node/TableScanNode.java
@@ -125,10 +125,6 @@ public class TableScanNode extends PlanNode {
     return this.qualifiedTableName;
   }
 
-  public List<DeviceEntry> getDeviceEntries() {
-    return this.deviceEntries;
-  }
-
   public void setDeviceEntries(List<DeviceEntry> deviceEntries) {
     this.deviceEntries = deviceEntries;
   }
@@ -149,10 +145,6 @@ public class TableScanNode extends PlanNode {
     return deviceEntries;
   }
 
-  public Map<Symbol, Integer> getAttributesMap() {
-    return attributesMap;
-  }
-
   public Expression getPushDownPredicate() {
     return this.pushDownPredicate;
   }


(iotdb) 04/05: Merge branch 'ty/TableModelGrammar' of https://github.com/apache/iotdb into ty/TableModelGrammar

Posted by ja...@apache.org.
This is an automated email from the ASF dual-hosted git repository.

jackietien pushed a commit to branch ty/TableModelGrammar
in repository https://gitbox.apache.org/repos/asf/iotdb.git

commit a02b95265c21c919b9b9128a05f891eb48d59f64
Merge: 7ffa882b6fc 4e3e0fdb240
Author: JackieTien97 <ja...@gmail.com>
AuthorDate: Thu Apr 18 10:01:27 2024 +0800

    Merge branch 'ty/TableModelGrammar' of https://github.com/apache/iotdb into ty/TableModelGrammar

 .../planner/distribution/DistributionPlanner.java  |   3 +
 .../SimpleFragmentParallelPlanner.java             |   4 +-
 .../plan/relational/analyzer/Analysis.java         |   9 +
 .../plan/relational/planner/LogicalPlanner.java    |  11 +-
 .../plan/relational/planner/PlanBuilder.java       |  41 +-
 .../plan/relational/planner/PredicateUtils.java    | 415 +++++++++++++++++++++
 .../plan/relational/planner/QueryPlanner.java      |  22 +-
 .../plan/relational/planner/RelationPlanner.java   |  22 +-
 .../relational/planner/RelationalModelPlanner.java |   4 +-
 .../ExchangeNodeGenerator.java}                    |  20 +-
 .../distribute/FragmentInstanceGenerator.java}     | 179 +++------
 .../distribute/RelationalDistributionPlanner.java  |  31 +-
 .../planner/distribute/SimplePlanRewriter.java     |  44 +++
 .../planner/distribute/SubPlanGenerator.java       |  77 ++++
 .../relational/planner/node/TableScanNode.java     |  18 +-
 .../planner/optimizations/IndexScan.java           |  93 ++++-
 ...Expressions.java => PruneTableScanColumns.java} |  53 ++-
 .../optimizations/RelationalPlanOptimizer.java     |   9 +-
 .../RemoveRedundantIdentityProjections.java        |  12 +-
 .../planner/optimizations/SimplifyExpressions.java |   9 +-
 .../plan/relational/analyzer/AnalyzerTest.java     |  19 +-
 .../relational/planner/PredicateUtilsTest.java     |  52 +++
 .../db/relational/sql/tree/LogicalExpression.java  |   6 +-
 23 files changed, 972 insertions(+), 181 deletions(-)

diff --cc iotdb-core/datanode/src/test/java/org/apache/iotdb/db/queryengine/plan/relational/analyzer/AnalyzerTest.java
index 583f1ec64ca,8e562f742e6..2ca8cbe947c
--- a/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/queryengine/plan/relational/analyzer/AnalyzerTest.java
+++ b/iotdb-core/datanode/src/test/java/org/apache/iotdb/db/queryengine/plan/relational/analyzer/AnalyzerTest.java
@@@ -148,16 -146,20 +147,22 @@@ public class AnalyzerTest 
      WarningCollector warningCollector = WarningCollector.NOOP;
      LogicalPlanner logicalPlanner =
          new LogicalPlanner(context, metadata, sessionInfo, warningCollector);
-     LogicalQueryPlan result = logicalPlanner.plan(actualAnalysis);
-     System.out.println(result);
+     LogicalQueryPlan logicalQueryPlan = logicalPlanner.plan(actualAnalysis);
+     System.out.println(logicalQueryPlan);
+ 
+     RelationalDistributionPlanner distributionPlanner =
+         new RelationalDistributionPlanner(actualAnalysis, logicalQueryPlan, context);
+     // DistributedQueryPlan distributedQueryPlan = distributionPlanner.plan();
+     // System.out.println(distributedQueryPlan);
    }
  
-   private Analysis analyzeSQL(String sql, Metadata metadata) {
+   public static Analysis analyzeSQL(String sql, Metadata metadata) {
      try {
+       SqlParser sqlParser = new SqlParser();
        Statement statement = sqlParser.createStatement(sql);
 -      SessionInfo session = new SessionInfo(0, "test", ZoneId.systemDefault(), "testdb");
 +      SessionInfo session =
 +          new SessionInfo(
 +              0, "test", ZoneId.systemDefault(), "testdb", IClientSession.SqlDialect.TABLE);
        StatementAnalyzerFactory statementAnalyzerFactory =
            new StatementAnalyzerFactory(metadata, sqlParser, nopAccessControl);
  


(iotdb) 01/05: add some filter converter

Posted by ja...@apache.org.
This is an automated email from the ASF dual-hosted git repository.

jackietien pushed a commit to branch ty/TableModelGrammar
in repository https://gitbox.apache.org/repos/asf/iotdb.git

commit 64bd103f54e1a9996e4810dea9c3030ddc0f8baa
Author: JackieTien97 <ja...@gmail.com>
AuthorDate: Tue Apr 16 21:32:24 2024 +0800

    add some filter converter
---
 .../source/relational/TableScanOperator.java       |  16 +-
 .../queryengine/plan/analyze/PredicateUtils.java   |  20 ++
 .../plan/planner/TableOperatorGenerator.java       | 162 ++++++++++++++-
 .../predicate/ConvertPredicateToFilterVisitor.java |  22 +++
 .../ConvertPredicateToTimeFilterVisitor.java       | 218 +++++++++++++++++++++
 .../predicate/PredicatePushIntoScanChecker.java    | 131 +++++++++++++
 .../analyzer/predicate/PredicateVisitor.java       |  87 ++++++++
 .../relational/planner/node/TableScanNode.java     |   8 +
 .../tsfile/read/filter/factory/FilterFactory.java  |  18 ++
 9 files changed, 671 insertions(+), 11 deletions(-)

diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/execution/operator/source/relational/TableScanOperator.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/execution/operator/source/relational/TableScanOperator.java
index 97ffe2b2282..1e26fa5d636 100644
--- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/execution/operator/source/relational/TableScanOperator.java
+++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/execution/operator/source/relational/TableScanOperator.java
@@ -28,7 +28,6 @@ import org.apache.iotdb.db.queryengine.plan.planner.plan.node.PlanNodeId;
 import org.apache.iotdb.db.queryengine.plan.planner.plan.parameter.SeriesScanOptions;
 import org.apache.iotdb.db.queryengine.plan.relational.metadata.ColumnSchema;
 import org.apache.iotdb.db.queryengine.plan.relational.metadata.DeviceEntry;
-import org.apache.iotdb.db.queryengine.plan.relational.planner.Symbol;
 import org.apache.iotdb.db.queryengine.plan.statement.component.Ordering;
 import org.apache.iotdb.db.storageengine.dataregion.read.QueryDataSource;
 import org.apache.iotdb.tsfile.common.conf.TSFileConfig;
@@ -56,8 +55,6 @@ import static org.apache.iotdb.db.queryengine.plan.relational.type.InternalTypeM
 
 public class TableScanOperator extends AbstractDataSourceOperator {
 
-  private final List<Symbol> outputColumnNames;
-
   private final List<ColumnSchema> columnSchemas;
 
   private final int[] columnsIndexArray;
@@ -78,18 +75,18 @@ public class TableScanOperator extends AbstractDataSourceOperator {
   private final List<TSDataType> measurementColumnTSDataTypes;
 
   private final TsBlockBuilder measurementDataBuilder;
+
+  private final int maxTsBlockLineNum;
+
   private TsBlock measurementDataBlock;
 
   private QueryDataSource queryDataSource;
 
-  private int currentDeviceIndex = 0;
-
-  private int maxTsBlockLineNum = -1;
+  private int currentDeviceIndex;
 
   public TableScanOperator(
       OperatorContext context,
       PlanNodeId sourceId,
-      List<Symbol> outputColumnNames,
       List<ColumnSchema> columnSchemas,
       int[] columnsIndexArray,
       int measurementColumnCount,
@@ -101,7 +98,6 @@ public class TableScanOperator extends AbstractDataSourceOperator {
       int maxTsBlockLineNum) {
     this.sourceId = sourceId;
     this.operatorContext = context;
-    this.outputColumnNames = outputColumnNames;
     this.columnSchemas = columnSchemas;
     this.columnsIndexArray = columnsIndexArray;
     this.measurementColumnCount = measurementColumnCount;
@@ -118,7 +114,7 @@ public class TableScanOperator extends AbstractDataSourceOperator {
     this.maxReturnSize =
         Math.min(
             maxReturnSize,
-            (1L + outputColumnNames.size())
+            (1L + columnsIndexArray.length)
                 * TSFileDescriptor.getInstance().getConfig().getPageSizeInByte());
     this.maxTsBlockLineNum = maxTsBlockLineNum;
 
@@ -303,7 +299,7 @@ public class TableScanOperator extends AbstractDataSourceOperator {
 
   @Override
   public long calculateMaxPeekMemory() {
-    return (1L + outputColumnNames.size())
+    return (1L + columnsIndexArray.length)
         * TSFileDescriptor.getInstance().getConfig().getPageSizeInByte()
         * 3L;
   }
diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/analyze/PredicateUtils.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/analyze/PredicateUtils.java
index ce41726768b..d635f074789 100644
--- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/analyze/PredicateUtils.java
+++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/analyze/PredicateUtils.java
@@ -290,6 +290,19 @@ public class PredicateUtils {
             allMeasurements, isBuildPlanUseTemplate, typeProvider));
   }
 
+  public static Filter convertPredicateToFilter(
+      org.apache.iotdb.db.relational.sql.tree.Expression predicate,
+      List<String> allMeasurements,
+      TypeProvider typeProvider) {
+    if (predicate == null) {
+      return null;
+    }
+    return predicate.accept(
+        new ConvertPredicateToFilterVisitor(),
+        new ConvertPredicateToFilterVisitor.Context(
+            allMeasurements, isBuildPlanUseTemplate, typeProvider));
+  }
+
   /**
    * Combine the given conjuncts into a single expression using "and".
    *
@@ -396,4 +409,11 @@ public class PredicateUtils {
   public static boolean predicateCanPushIntoScan(Expression predicate) {
     return new PredicatePushIntoScanChecker().process(predicate, null);
   }
+
+  public static boolean predicateCanPushIntoScan(
+      org.apache.iotdb.db.relational.sql.tree.Expression predicate) {
+    return new org.apache.iotdb.db.queryengine.plan.relational.analyzer.predicate
+            .PredicatePushIntoScanChecker()
+        .process(predicate, null);
+  }
 }
diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/planner/TableOperatorGenerator.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/planner/TableOperatorGenerator.java
index 9899ce6922e..f5e22768ce5 100644
--- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/planner/TableOperatorGenerator.java
+++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/planner/TableOperatorGenerator.java
@@ -19,9 +19,18 @@
 
 package org.apache.iotdb.db.queryengine.plan.planner;
 
+import org.apache.iotdb.commons.path.AlignedPath;
+import org.apache.iotdb.db.queryengine.execution.driver.DataDriverContext;
 import org.apache.iotdb.db.queryengine.execution.operator.Operator;
+import org.apache.iotdb.db.queryengine.execution.operator.OperatorContext;
+import org.apache.iotdb.db.queryengine.execution.operator.source.AlignedSeriesScanOperator;
+import org.apache.iotdb.db.queryengine.execution.operator.source.relational.TableScanOperator;
+import org.apache.iotdb.db.queryengine.plan.analyze.PredicateUtils;
 import org.apache.iotdb.db.queryengine.plan.planner.plan.node.PlanNode;
 import org.apache.iotdb.db.queryengine.plan.planner.plan.node.PlanVisitor;
+import org.apache.iotdb.db.queryengine.plan.planner.plan.parameter.SeriesScanOptions;
+import org.apache.iotdb.db.queryengine.plan.relational.metadata.ColumnSchema;
+import org.apache.iotdb.db.queryengine.plan.relational.planner.Symbol;
 import org.apache.iotdb.db.queryengine.plan.relational.planner.node.FilterNode;
 import org.apache.iotdb.db.queryengine.plan.relational.planner.node.LimitNode;
 import org.apache.iotdb.db.queryengine.plan.relational.planner.node.MergeSortNode;
@@ -31,6 +40,22 @@ import org.apache.iotdb.db.queryengine.plan.relational.planner.node.ProjectNode;
 import org.apache.iotdb.db.queryengine.plan.relational.planner.node.SortNode;
 import org.apache.iotdb.db.queryengine.plan.relational.planner.node.TableScanNode;
 import org.apache.iotdb.db.queryengine.plan.relational.planner.node.TopKNode;
+import org.apache.iotdb.db.relational.sql.tree.Expression;
+import org.apache.iotdb.tsfile.common.conf.TSFileDescriptor;
+import org.apache.iotdb.tsfile.file.metadata.enums.TSDataType;
+import org.apache.iotdb.tsfile.read.filter.basic.Filter;
+import org.apache.iotdb.tsfile.write.schema.IMeasurementSchema;
+import org.apache.iotdb.tsfile.write.schema.MeasurementSchema;
+
+import java.util.ArrayList;
+import java.util.Collections;
+import java.util.HashSet;
+import java.util.List;
+import java.util.Map;
+
+import static java.util.Objects.requireNonNull;
+import static org.apache.iotdb.db.queryengine.plan.analyze.PredicateUtils.convertPredicateToFilter;
+import static org.apache.iotdb.db.queryengine.plan.relational.type.InternalTypeManager.getTSDataType;
 
 /** This Visitor is responsible for transferring Table PlanNode Tree to Table Operator Tree. */
 public class TableOperatorGenerator extends PlanVisitor<Operator, LocalExecutionPlanContext> {
@@ -42,7 +67,142 @@ public class TableOperatorGenerator extends PlanVisitor<Operator, LocalExecution
 
   @Override
   public Operator visitTableScan(TableScanNode node, LocalExecutionPlanContext context) {
-    return super.visitTableScan(node, context);
+
+    List<Symbol> outputColumnNames = node.getOutputSymbols();
+    int outputColumnCount = outputColumnNames.size();
+    List<ColumnSchema> columnSchemas = new ArrayList<>(outputColumnCount);
+    int[] columnsIndexArray = new int[outputColumnCount];
+    Map<Symbol, ColumnSchema> columnSchemaMap = node.getAssignments();
+    Map<Symbol, Integer> idAndAttributeColumnsIndexMap = node.getAttributesMap();
+    List<String> measurementColumnNames = new ArrayList<>();
+    List<IMeasurementSchema> measurementSchemas = new ArrayList<>();
+    int measurementColumnCount = 0;
+    for (int i = 0; i < outputColumnCount; i++) {
+      Symbol columnName = outputColumnNames.get(i);
+      ColumnSchema schema =
+          requireNonNull(columnSchemaMap.get(columnName), columnName + " is null");
+      columnSchemas.add(schema);
+      switch (schema.getColumnCategory()) {
+        case ID:
+        case ATTRIBUTE:
+          columnsIndexArray[i] =
+              requireNonNull(
+                  idAndAttributeColumnsIndexMap.get(columnName), columnName + " is null");
+          break;
+        case MEASUREMENT:
+          columnsIndexArray[i] = measurementColumnCount;
+          measurementColumnCount++;
+          measurementColumnNames.add(columnName.getName());
+          measurementSchemas.add(
+              new MeasurementSchema(schema.getName(), getTSDataType(schema.getType())));
+          break;
+        default:
+          throw new IllegalArgumentException(
+              "Unexpected column category: " + schema.getColumnCategory());
+      }
+    }
+
+    SeriesScanOptions.Builder scanOptionsBuilder = getSeriesScanOptionsBuilder(node, context);
+    scanOptionsBuilder.withPushDownLimit(node.getPushDownLimit());
+    scanOptionsBuilder.withPushDownOffset(node.getPushDownOffset());
+    scanOptionsBuilder.withAllSensors(new HashSet<>(measurementColumnNames));
+
+    Expression pushDownPredicate = node.getPushDownPredicate();
+    boolean predicateCanPushIntoScan = canPushIntoScan(pushDownPredicate);
+    if (pushDownPredicate != null && predicateCanPushIntoScan) {
+      scanOptionsBuilder.withPushDownFilter(
+          convertPredicateToFilter(
+              pushDownPredicate,
+              node.getAlignedPath().getMeasurementList(),
+              context.getTypeProvider().getTemplatedInfo() != null,
+              context.getTypeProvider()));
+    }
+
+    OperatorContext operatorContext =
+        context
+            .getDriverContext()
+            .addOperatorContext(
+                context.getNextOperatorId(),
+                node.getPlanNodeId(),
+                AlignedSeriesScanOperator.class.getSimpleName());
+
+    int maxTsBlockLineNum = TSFileDescriptor.getInstance().getConfig().getMaxTsBlockLineNumber();
+    if (context.getTypeProvider().getTemplatedInfo() != null) {
+      maxTsBlockLineNum =
+          (int)
+              Math.min(
+                  context.getTypeProvider().getTemplatedInfo().getLimitValue(), maxTsBlockLineNum);
+    }
+
+    TableScanOperator tableScanOperator =
+        new TableScanOperator(
+            operatorContext,
+            node.getPlanNodeId(),
+            seriesPath,
+            node.getScanOrder(),
+            scanOptionsBuilder.build(),
+            node.isQueryAllSensors(),
+            context.getTypeProvider().getTemplatedInfo() != null
+                ? context.getTypeProvider().getTemplatedInfo().getDataTypes()
+                : null,
+            maxTsBlockLineNum);
+
+    ((DataDriverContext) context.getDriverContext()).addSourceOperator(seriesScanOperator);
+    ((DataDriverContext) context.getDriverContext()).addPath(seriesPath);
+    context.getDriverContext().setInputDriver(true);
+
+    if (!predicateCanPushIntoScan) {
+      if (context.isBuildPlanUseTemplate()) {
+        TemplatedInfo templatedInfo = context.getTemplatedInfo();
+        return constructFilterOperator(
+            pushDownPredicate,
+            seriesScanOperator,
+            templatedInfo.getProjectExpressions(),
+            templatedInfo.getDataTypes(),
+            templatedInfo.getLayoutMap(),
+            templatedInfo.isKeepNull(),
+            node.getPlanNodeId(),
+            templatedInfo.getScanOrder(),
+            context);
+      }
+
+      AlignedPath alignedPath = node.getAlignedPath();
+      List<Expression> expressions = new ArrayList<>();
+      List<TSDataType> dataTypes = new ArrayList<>();
+      for (int i = 0; i < alignedPath.getMeasurementList().size(); i++) {
+        expressions.add(ExpressionFactory.timeSeries(alignedPath.getSubMeasurementPath(i)));
+        dataTypes.add(alignedPath.getSubMeasurementDataType(i));
+      }
+
+      return constructFilterOperator(
+          pushDownPredicate,
+          seriesScanOperator,
+          expressions.toArray(new Expression[0]),
+          dataTypes,
+          makeLayout(Collections.singletonList(node)),
+          false,
+          node.getPlanNodeId(),
+          node.getScanOrder(),
+          context);
+    }
+    return seriesScanOperator;
+  }
+
+  private SeriesScanOptions.Builder getSeriesScanOptionsBuilder(
+      TableScanNode node, LocalExecutionPlanContext context) {
+    SeriesScanOptions.Builder scanOptionsBuilder = new SeriesScanOptions.Builder();
+
+    Filter globalTimeFilter = context.getGlobalTimeFilter();
+    if (globalTimeFilter != null) {
+      // time filter may be stateful, so we need to copy it
+      scanOptionsBuilder.withGlobalTimeFilter(globalTimeFilter.copy());
+    }
+
+    return scanOptionsBuilder;
+  }
+
+  private boolean canPushIntoScan(Expression pushDownPredicate) {
+    return pushDownPredicate == null || PredicateUtils.predicateCanPushIntoScan(pushDownPredicate);
   }
 
   @Override
diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/relational/analyzer/predicate/ConvertPredicateToFilterVisitor.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/relational/analyzer/predicate/ConvertPredicateToFilterVisitor.java
new file mode 100644
index 00000000000..7ab2448ded1
--- /dev/null
+++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/relational/analyzer/predicate/ConvertPredicateToFilterVisitor.java
@@ -0,0 +1,22 @@
+/*
+ * 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.iotdb.db.queryengine.plan.relational.analyzer.predicate;
+
+public class ConvertPredicateToFilterVisitor {}
diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/relational/analyzer/predicate/ConvertPredicateToTimeFilterVisitor.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/relational/analyzer/predicate/ConvertPredicateToTimeFilterVisitor.java
new file mode 100644
index 00000000000..f48a78151df
--- /dev/null
+++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/relational/analyzer/predicate/ConvertPredicateToTimeFilterVisitor.java
@@ -0,0 +1,218 @@
+/*
+ * 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.iotdb.db.queryengine.plan.relational.analyzer.predicate;
+
+import org.apache.iotdb.db.relational.sql.tree.BetweenPredicate;
+import org.apache.iotdb.db.relational.sql.tree.ComparisonExpression;
+import org.apache.iotdb.db.relational.sql.tree.Expression;
+import org.apache.iotdb.db.relational.sql.tree.IfExpression;
+import org.apache.iotdb.db.relational.sql.tree.InListExpression;
+import org.apache.iotdb.db.relational.sql.tree.InPredicate;
+import org.apache.iotdb.db.relational.sql.tree.IsNotNullPredicate;
+import org.apache.iotdb.db.relational.sql.tree.IsNullPredicate;
+import org.apache.iotdb.db.relational.sql.tree.LikePredicate;
+import org.apache.iotdb.db.relational.sql.tree.LogicalExpression;
+import org.apache.iotdb.db.relational.sql.tree.LongLiteral;
+import org.apache.iotdb.db.relational.sql.tree.NotExpression;
+import org.apache.iotdb.db.relational.sql.tree.NullIfExpression;
+import org.apache.iotdb.db.relational.sql.tree.SearchedCaseExpression;
+import org.apache.iotdb.db.relational.sql.tree.SimpleCaseExpression;
+import org.apache.iotdb.db.relational.sql.tree.SymbolReference;
+import org.apache.iotdb.tsfile.read.filter.basic.Filter;
+import org.apache.iotdb.tsfile.read.filter.factory.FilterFactory;
+import org.apache.iotdb.tsfile.read.filter.factory.TimeFilterApi;
+
+import java.util.HashSet;
+import java.util.List;
+import java.util.Objects;
+import java.util.Set;
+import java.util.stream.Collectors;
+
+import static com.google.common.base.Preconditions.checkArgument;
+
+public class ConvertPredicateToTimeFilterVisitor extends PredicateVisitor<Filter, Void> {
+
+  @Override
+  protected Filter visitInPredicate(InPredicate node, Void context) {
+    checkArgument(isTimeColumn(node.getValue()));
+    Expression valueList = node.getValueList();
+    checkArgument(valueList instanceof InListExpression);
+    List<Expression> values = ((InListExpression) valueList).getValues();
+    for (Expression value : values) {
+      checkArgument(value instanceof LongLiteral);
+    }
+    if (values.size() == 1) {
+      TimeFilterApi.eq(((LongLiteral) values.get(0)).getParsedValue());
+    }
+    Set<Long> longValues = new HashSet<>();
+    for (Expression value : values) {
+      longValues.add(((LongLiteral) value).getParsedValue());
+    }
+    return TimeFilterApi.in(longValues);
+  }
+
+  @Override
+  protected Filter visitIsNullPredicate(IsNullPredicate node, Void context) {
+    throw new UnsupportedOperationException("TIMESTAMP does not support IS NULL");
+  }
+
+  @Override
+  protected Filter visitIsNotNullPredicate(IsNotNullPredicate node, Void context) {
+    throw new UnsupportedOperationException("TIMESTAMP does not support IS NOT NULL");
+  }
+
+  @Override
+  protected Filter visitLikePredicate(LikePredicate node, Void context) {
+    throw new UnsupportedOperationException("TIMESTAMP does not support LIKE");
+  }
+
+  @Override
+  protected Filter visitLogicalExpression(LogicalExpression node, Void context) {
+    List<Filter> filterList =
+        node.getTerms().stream()
+            .map(n -> n.accept(this, context))
+            .filter(Objects::nonNull)
+            .collect(Collectors.toList());
+    switch (node.getOperator()) {
+      case OR:
+        return FilterFactory.or(filterList);
+      case AND:
+        return FilterFactory.and(filterList);
+      default:
+        throw new IllegalArgumentException("Unsupported operator: " + node.getOperator());
+    }
+  }
+
+  @Override
+  protected Filter visitNotExpression(NotExpression node, Void context) {
+    return FilterFactory.not(node.getValue().accept(this, context));
+  }
+
+  @Override
+  protected Filter visitComparisonExpression(ComparisonExpression node, Void context) {
+    long value;
+    if (node.getLeft() instanceof LongLiteral) {
+      value = getLongValue(node.getLeft());
+    } else if (node.getRight() instanceof LongLiteral) {
+      value = getLongValue(node.getRight());
+    } else {
+      throw new IllegalStateException(
+          "Either left or right operand of Time ComparisonExpression should be LongLiteral");
+    }
+    switch (node.getOperator()) {
+      case EQUAL:
+        return TimeFilterApi.eq(value);
+      case NOT_EQUAL:
+        return TimeFilterApi.notEq(value);
+      case GREATER_THAN:
+        return TimeFilterApi.gt(value);
+      case GREATER_THAN_OR_EQUAL:
+        return TimeFilterApi.gtEq(value);
+      case LESS_THAN:
+        return TimeFilterApi.lt(value);
+      case LESS_THAN_OR_EQUAL:
+        return TimeFilterApi.ltEq(value);
+      default:
+        throw new IllegalArgumentException("Unsupported operator: " + node.getOperator());
+    }
+  }
+
+  @Override
+  protected Filter visitSimpleCaseExpression(SimpleCaseExpression node, Void context) {
+    throw new UnsupportedOperationException("TIMESTAMP does not CASE WHEN");
+  }
+
+  @Override
+  protected Filter visitSearchedCaseExpression(SearchedCaseExpression node, Void context) {
+    throw new UnsupportedOperationException("TIMESTAMP does not CASE WHEN");
+  }
+
+  @Override
+  protected Filter visitIfExpression(IfExpression node, Void context) {
+    throw new UnsupportedOperationException("TIMESTAMP does not IF");
+  }
+
+  @Override
+  protected Filter visitNullIfExpression(NullIfExpression node, Void context) {
+    throw new UnsupportedOperationException("TIMESTAMP does not NULLIF");
+  }
+
+  @Override
+  protected Filter visitBetweenPredicate(BetweenPredicate node, Void context) {
+    Expression firstExpression = node.getValue();
+    Expression secondExpression = node.getMin();
+    Expression thirdExpression = node.getMax();
+
+    if (isTimeColumn(firstExpression)) {
+      // firstExpression is TIMESTAMP
+      long minValue = getLongValue(secondExpression);
+      long maxValue = getLongValue(thirdExpression);
+
+      if (minValue == maxValue) {
+        return TimeFilterApi.eq(minValue);
+      }
+      return TimeFilterApi.between(minValue, maxValue);
+    } else if (isTimeColumn(secondExpression)) {
+      // secondExpression is TIMESTAMP
+      long value = getLongValue(firstExpression);
+      long maxValue = getLongValue(thirdExpression);
+
+      // cases:
+      // 1 BETWEEN time AND 2 => time <= 1
+      // 1 BETWEEN time AND 1 => time <= 1
+      // 1 BETWEEN time AND 0 => FALSE
+      // 1 NOT BETWEEN time AND 2 => time > 1
+      // 1 NOT BETWEEN time AND 1 => time > 1
+      // 1 NOT BETWEEN time AND 0 => TRUE
+      checkArgument(
+          value <= maxValue,
+          String.format("Predicate [%s] should be simplified in previous step", node));
+      return TimeFilterApi.ltEq(value);
+    } else if (isTimeColumn(thirdExpression)) {
+      // thirdExpression is TIMESTAMP
+      long value = getLongValue(firstExpression);
+      long minValue = getLongValue(secondExpression);
+
+      // cases:
+      // 1 BETWEEN 2 AND time => FALSE
+      // 1 BETWEEN 1 AND time => time >= 1
+      // 1 BETWEEN 0 AND time => time >= 1
+      // 1 NOT BETWEEN 2 AND time => TRUE
+      // 1 NOT BETWEEN 1 AND time => time < 1
+      // 1 NOT BETWEEN 0 AND time => time < 1
+      checkArgument(
+          value >= minValue,
+          String.format("Predicate [%s] should be simplified in previous step", node));
+      return TimeFilterApi.gtEq(value);
+    } else {
+      throw new IllegalStateException(
+          "Three operand of between expression should have time column.");
+    }
+  }
+
+  public static boolean isTimeColumn(Expression expression) {
+    return expression instanceof SymbolReference
+        && "time".equalsIgnoreCase(((SymbolReference) expression).getName());
+  }
+
+  public static long getLongValue(Expression expression) {
+    return ((LongLiteral) expression).getParsedValue();
+  }
+}
diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/relational/analyzer/predicate/PredicatePushIntoScanChecker.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/relational/analyzer/predicate/PredicatePushIntoScanChecker.java
new file mode 100644
index 00000000000..519027bf832
--- /dev/null
+++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/relational/analyzer/predicate/PredicatePushIntoScanChecker.java
@@ -0,0 +1,131 @@
+/*
+ * 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.iotdb.db.queryengine.plan.relational.analyzer.predicate;
+
+import org.apache.iotdb.db.relational.sql.tree.BetweenPredicate;
+import org.apache.iotdb.db.relational.sql.tree.ComparisonExpression;
+import org.apache.iotdb.db.relational.sql.tree.Expression;
+import org.apache.iotdb.db.relational.sql.tree.IfExpression;
+import org.apache.iotdb.db.relational.sql.tree.InPredicate;
+import org.apache.iotdb.db.relational.sql.tree.IsNotNullPredicate;
+import org.apache.iotdb.db.relational.sql.tree.IsNullPredicate;
+import org.apache.iotdb.db.relational.sql.tree.LikePredicate;
+import org.apache.iotdb.db.relational.sql.tree.Literal;
+import org.apache.iotdb.db.relational.sql.tree.LogicalExpression;
+import org.apache.iotdb.db.relational.sql.tree.NotExpression;
+import org.apache.iotdb.db.relational.sql.tree.NullIfExpression;
+import org.apache.iotdb.db.relational.sql.tree.SearchedCaseExpression;
+import org.apache.iotdb.db.relational.sql.tree.SimpleCaseExpression;
+import org.apache.iotdb.db.relational.sql.tree.SymbolReference;
+
+import java.util.List;
+
+public class PredicatePushIntoScanChecker extends PredicateVisitor<Boolean, Void> {
+
+  @Override
+  public Boolean visitExpression(Expression expression, Void context) {
+    return Boolean.FALSE;
+  }
+
+  @Override
+  protected Boolean visitInPredicate(InPredicate node, Void context) {
+    return isSymbolReference(node.getValue());
+  }
+
+  @Override
+  protected Boolean visitIsNullPredicate(IsNullPredicate node, Void context) {
+    throw new IllegalArgumentException("IS NULL Expression can't be pushed down");
+  }
+
+  @Override
+  protected Boolean visitIsNotNullPredicate(IsNotNullPredicate node, Void context) {
+    return isSymbolReference(node.getValue());
+  }
+
+  @Override
+  protected Boolean visitLikePredicate(LikePredicate node, Void context) {
+    return isSymbolReference(node.getValue())
+        && isLiteral(node.getPattern())
+        && node.getEscape().map(this::isLiteral).orElse(true);
+  }
+
+  @Override
+  protected Boolean visitLogicalExpression(LogicalExpression node, Void context) {
+    List<Expression> children = node.getTerms();
+    for (Expression child : children) {
+      if (!process(child, context)) {
+        return Boolean.FALSE;
+      }
+    }
+    return Boolean.TRUE;
+  }
+
+  @Override
+  protected Boolean visitNotExpression(NotExpression node, Void context) {
+    throw new IllegalArgumentException("Not Expression can't be pushed down");
+  }
+
+  @Override
+  protected Boolean visitComparisonExpression(ComparisonExpression node, Void context) {
+    return (isSymbolReference(node.getLeft()) && isLiteral(node.getRight()))
+        || (isSymbolReference(node.getRight()) && isLiteral(node.getLeft()));
+  }
+
+  @Override
+  protected Boolean visitSimpleCaseExpression(SimpleCaseExpression node, Void context) {
+    return Boolean.FALSE;
+  }
+
+  @Override
+  protected Boolean visitSearchedCaseExpression(SearchedCaseExpression node, Void context) {
+    return Boolean.FALSE;
+  }
+
+  @Override
+  protected Boolean visitIfExpression(IfExpression node, Void context) {
+    return Boolean.FALSE;
+  }
+
+  @Override
+  protected Boolean visitNullIfExpression(NullIfExpression node, Void context) {
+    return Boolean.FALSE;
+  }
+
+  @Override
+  protected Boolean visitBetweenPredicate(BetweenPredicate node, Void context) {
+    return (isSymbolReference(node.getValue())
+            && isLiteral(node.getMin())
+            && isLiteral(node.getMax()))
+        || (isLiteral(node.getValue())
+            && isSymbolReference(node.getMin())
+            && isLiteral(node.getMax()))
+        || (isLiteral(node.getValue())
+            && isLiteral(node.getMin())
+            && isSymbolReference(node.getMax()));
+  }
+
+  public static boolean isLiteral(Expression expression) {
+    return expression instanceof Literal;
+  }
+
+  public static boolean isSymbolReference(Expression expression) {
+    return expression instanceof SymbolReference;
+  }
+}
diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/relational/analyzer/predicate/PredicateVisitor.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/relational/analyzer/predicate/PredicateVisitor.java
new file mode 100644
index 00000000000..70d1ae11d0b
--- /dev/null
+++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/relational/analyzer/predicate/PredicateVisitor.java
@@ -0,0 +1,87 @@
+/*
+ * 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.iotdb.db.queryengine.plan.relational.analyzer.predicate;
+
+import org.apache.iotdb.db.relational.sql.tree.AstVisitor;
+import org.apache.iotdb.db.relational.sql.tree.BetweenPredicate;
+import org.apache.iotdb.db.relational.sql.tree.ComparisonExpression;
+import org.apache.iotdb.db.relational.sql.tree.Expression;
+import org.apache.iotdb.db.relational.sql.tree.IfExpression;
+import org.apache.iotdb.db.relational.sql.tree.InPredicate;
+import org.apache.iotdb.db.relational.sql.tree.IsNotNullPredicate;
+import org.apache.iotdb.db.relational.sql.tree.IsNullPredicate;
+import org.apache.iotdb.db.relational.sql.tree.LikePredicate;
+import org.apache.iotdb.db.relational.sql.tree.LogicalExpression;
+import org.apache.iotdb.db.relational.sql.tree.NotExpression;
+import org.apache.iotdb.db.relational.sql.tree.NullIfExpression;
+import org.apache.iotdb.db.relational.sql.tree.SearchedCaseExpression;
+import org.apache.iotdb.db.relational.sql.tree.SimpleCaseExpression;
+
+/**
+ * This class provides a visitor of {@link Expression}, which can be extended to create a visitor
+ * which only needs to handle a subset of the available methods.
+ *
+ * @param <R> The return type of the visit operation.
+ * @param <C> The context information during visiting.
+ */
+public abstract class PredicateVisitor<R, C> extends AstVisitor<R, C> {
+
+  @Override
+  public R visitExpression(Expression expression, C context) {
+    throw new IllegalArgumentException(
+        "Unsupported expression: " + expression.getClass().getSimpleName());
+  }
+
+  @Override
+  protected abstract R visitInPredicate(InPredicate node, C context);
+
+  @Override
+  protected abstract R visitIsNullPredicate(IsNullPredicate node, C context);
+
+  @Override
+  protected abstract R visitIsNotNullPredicate(IsNotNullPredicate node, C context);
+
+  @Override
+  protected abstract R visitLikePredicate(LikePredicate node, C context);
+
+  @Override
+  protected abstract R visitLogicalExpression(LogicalExpression node, C context);
+
+  @Override
+  protected abstract R visitNotExpression(NotExpression node, C context);
+
+  @Override
+  protected abstract R visitComparisonExpression(ComparisonExpression node, C context);
+
+  @Override
+  protected abstract R visitSimpleCaseExpression(SimpleCaseExpression node, C context);
+
+  @Override
+  protected abstract R visitSearchedCaseExpression(SearchedCaseExpression node, C context);
+
+  @Override
+  protected abstract R visitIfExpression(IfExpression node, C context);
+
+  @Override
+  protected abstract R visitNullIfExpression(NullIfExpression node, C context);
+
+  @Override
+  protected abstract R visitBetweenPredicate(BetweenPredicate node, C context);
+}
diff --git a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/relational/planner/node/TableScanNode.java b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/relational/planner/node/TableScanNode.java
index 8849b3ddb18..521c4749b13 100644
--- a/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/relational/planner/node/TableScanNode.java
+++ b/iotdb-core/datanode/src/main/java/org/apache/iotdb/db/queryengine/plan/relational/planner/node/TableScanNode.java
@@ -131,6 +131,14 @@ public class TableScanNode extends PlanNode {
     return this.scanOrder;
   }
 
+  public List<DeviceEntry> getDeviceEntries() {
+    return deviceEntries;
+  }
+
+  public Map<Symbol, Integer> getAttributesMap() {
+    return attributesMap;
+  }
+
   public Expression getPushDownPredicate() {
     return this.pushDownPredicate;
   }
diff --git a/iotdb-core/tsfile/src/main/java/org/apache/iotdb/tsfile/read/filter/factory/FilterFactory.java b/iotdb-core/tsfile/src/main/java/org/apache/iotdb/tsfile/read/filter/factory/FilterFactory.java
index 23546767987..14f8d43bac4 100644
--- a/iotdb-core/tsfile/src/main/java/org/apache/iotdb/tsfile/read/filter/factory/FilterFactory.java
+++ b/iotdb-core/tsfile/src/main/java/org/apache/iotdb/tsfile/read/filter/factory/FilterFactory.java
@@ -24,6 +24,8 @@ import org.apache.iotdb.tsfile.read.filter.operator.And;
 import org.apache.iotdb.tsfile.read.filter.operator.Not;
 import org.apache.iotdb.tsfile.read.filter.operator.Or;
 
+import java.util.List;
+
 import static org.apache.iotdb.tsfile.utils.Preconditions.checkArgument;
 
 public class FilterFactory {
@@ -43,6 +45,14 @@ public class FilterFactory {
     return new And(left, right);
   }
 
+  public static Filter and(List<Filter> filterList) {
+    And result = new And(filterList.get(0), filterList.get(1));
+    for (int i = 2, size = filterList.size(); i < size; i++) {
+      result = new And(result, filterList.get(i));
+    }
+    return result;
+  }
+
   public static Filter or(Filter left, Filter right) {
     if (left == null && right == null) {
       return null;
@@ -54,6 +64,14 @@ public class FilterFactory {
     return new Or(left, right);
   }
 
+  public static Filter or(List<Filter> filterList) {
+    Or result = new Or(filterList.get(0), filterList.get(1));
+    for (int i = 2, size = filterList.size(); i < size; i++) {
+      result = new Or(result, filterList.get(i));
+    }
+    return result;
+  }
+
   public static Not not(Filter filter) {
     checkArgument(filter != null, "filter cannot be null");
     return new Not(filter);