You are viewing a plain text version of this content. The canonical link for it is here.
Posted to notifications@ignite.apache.org by GitBox <gi...@apache.org> on 2021/12/20 14:14:59 UTC

[GitHub] [ignite] korlov42 commented on a change in pull request #9647: IGNITE-16107 Fix projects and filters merge into scan

korlov42 commented on a change in pull request #9647:
URL: https://github.com/apache/ignite/pull/9647#discussion_r771441235



##########
File path: modules/calcite/src/main/java/org/apache/ignite/internal/processors/query/calcite/rel/logical/IgniteLogicalIndexScan.java
##########
@@ -58,8 +56,6 @@ public static IgniteLogicalIndexScan create(
             Mappings.TargetMapping targetMapping = Commons.mapping(requiredColumns,

Review comment:
       `rowType` on the line 52 could be removed too

##########
File path: modules/calcite/src/test/java/org/apache/ignite/internal/processors/query/calcite/planner/ProjectFilterScanMergePlannerTest.java
##########
@@ -0,0 +1,212 @@
+/*
+ * 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.ignite.internal.processors.query.calcite.planner;
+
+import org.apache.calcite.rel.RelCollations;
+import org.apache.calcite.rel.type.RelDataType;
+import org.apache.calcite.rel.type.RelDataTypeFactory;
+import org.apache.calcite.sql.type.SqlTypeName;
+import org.apache.calcite.util.ImmutableBitSet;
+import org.apache.ignite.internal.processors.query.calcite.rel.IgniteAggregate;
+import org.apache.ignite.internal.processors.query.calcite.rel.IgniteIndexScan;
+import org.apache.ignite.internal.processors.query.calcite.rel.IgniteTableScan;
+import org.apache.ignite.internal.processors.query.calcite.schema.IgniteIndex;
+import org.apache.ignite.internal.processors.query.calcite.schema.IgniteSchema;
+import org.apache.ignite.internal.processors.query.calcite.trait.IgniteDistributions;
+import org.apache.ignite.internal.processors.query.calcite.type.IgniteTypeFactory;
+import org.apache.ignite.internal.processors.query.calcite.type.IgniteTypeSystem;
+import org.junit.Before;
+import org.junit.Test;
+
+/**
+ * Tests ProjectScanMergeRule and FilterScanMergeRule.
+ */
+public class ProjectFilterScanMergePlannerTest extends AbstractPlannerTest {
+    /** Public schema. */
+    private IgniteSchema publicSchema;
+
+    /** {@inheritDoc} */
+    @Before
+    @Override public void setup() {
+        super.setup();
+
+        publicSchema = new IgniteSchema("PUBLIC");
+
+        IgniteTypeFactory f = new IgniteTypeFactory(IgniteTypeSystem.INSTANCE);
+
+        RelDataType type = new RelDataTypeFactory.Builder(f)
+            .add("A", f.createSqlType(SqlTypeName.INTEGER))
+            .add("B", f.createSqlType(SqlTypeName.INTEGER))
+            .add("C", f.createSqlType(SqlTypeName.INTEGER))
+            .build();
+
+        createTable(publicSchema, "TBL", type, IgniteDistributions.single(), null);

Review comment:
       ```suggestion
           publicSchema = createSchema(
               createTable("TBL", IgniteDistributions.single(),
                   "A", Integer.class,
                   "B", Integer.class,
                   "C", Integer.class
               )
           );
   ```
   
   what do you think about such an syntax?

##########
File path: modules/calcite/src/main/java/org/apache/ignite/internal/processors/query/calcite/rule/logical/ProjectScanMergeRule.java
##########
@@ -92,53 +80,73 @@ private ProjectScanMergeRule(Config config) {
         RelOptCluster cluster = scan.getCluster();
         List<RexNode> projects = relProject.getProjects();
         RexNode cond = scan.condition();
+        ImmutableBitSet requiredColumns = scan.requiredColumns();
+        List<RexNode> scanProjects = scan.projects();
 
-        // projection changes input collation and distribution.
-        RelTraitSet traits = scan.getTraitSet();
-
-        traits = traits.replace(TraitUtils.projectCollation(
-            TraitUtils.collation(traits), projects, scan.getRowType()));
-
-        traits = traits.replace(TraitUtils.projectDistribution(
-            TraitUtils.distribution(traits), projects, scan.getRowType()));
+        // Set default traits, real traits will be calculated for physical node.
+        RelTraitSet traits = cluster.traitSet();
 
         IgniteTable tbl = scan.getTable().unwrap(IgniteTable.class);
         IgniteTypeFactory typeFactory = Commons.typeFactory(cluster);
-        ImmutableBitSet.Builder builder = ImmutableBitSet.builder();
-
-        new RexShuttle() {
-            @Override public RexNode visitInputRef(RexInputRef ref) {
-                builder.set(ref.getIndex());
-                return ref;
-            }
-        }.apply(projects);
 
-        new RexShuttle() {
-            @Override public RexNode visitLocalRef(RexLocalRef inputRef) {
-                builder.set(inputRef.getIndex());
-                return inputRef;
-            }
-        }.apply(cond);
-
-        ImmutableBitSet requiredColumns = builder.build();
-
-        Mappings.TargetMapping targetMapping = Commons.mapping(requiredColumns,
-            tbl.getRowType(typeFactory).getFieldCount());
-
-        projects = new RexShuttle() {
-            @Override public RexNode visitInputRef(RexInputRef ref) {
-                return new RexLocalRef(targetMapping.getTarget(ref.getIndex()), ref.getType());
-            }
-        }.apply(projects);
+        if (requiredColumns == null) {
+            assert scanProjects == null;
+
+            ImmutableBitSet.Builder builder = ImmutableBitSet.builder();
+
+            new RexShuttle() {
+                @Override public RexNode visitInputRef(RexInputRef ref) {
+                    builder.set(ref.getIndex());
+                    return ref;
+                }
+            }.apply(projects);
+
+            new RexShuttle() {
+                @Override public RexNode visitLocalRef(RexLocalRef inputRef) {
+                    builder.set(inputRef.getIndex());
+                    return inputRef;
+                }
+            }.apply(cond);
+
+            requiredColumns = builder.build();
+
+            Mappings.TargetMapping targetMapping = Commons.mapping(requiredColumns,
+                tbl.getRowType(typeFactory).getFieldCount());
+
+            projects = new RexShuttle() {
+                @Override public RexNode visitInputRef(RexInputRef ref) {
+                    return new RexLocalRef(targetMapping.getTarget(ref.getIndex()), ref.getType());
+                }
+            }.apply(projects);
+
+            cond = new RexShuttle() {
+                @Override public RexNode visitLocalRef(RexLocalRef ref) {
+                    return new RexLocalRef(targetMapping.getTarget(ref.getIndex()), ref.getType());
+                }
+            }.apply(cond);
+        }
+        else
+            projects = RexUtils.replaceInputRefs(projects);
 
         if (RexUtils.isIdentity(projects, tbl.getRowType(typeFactory, requiredColumns), true))
             projects = null;
 
-        cond = new RexShuttle() {
-            @Override public RexNode visitLocalRef(RexLocalRef ref) {
-                return new RexLocalRef(targetMapping.getTarget(ref.getIndex()), ref.getType());
+        if (scanProjects != null) {
+            if (projects != null) {
+                // Merge projects.
+                projects = new RexShuttle() {
+                    @Override public RexNode visitLocalRef(RexLocalRef ref) {
+                        return scanProjects.get(ref.getIndex());
+                    }
+                }.apply(projects);
+
+                // Check again after merge.
+                if (RexUtils.isIdentity(projects, tbl.getRowType(typeFactory, requiredColumns), true))
+                    projects = null;
             }
-        }.apply(cond);
+            else
+                projects = scanProjects;

Review comment:
       Such transformation is not valid because it could possibly change the row type of the resulting rel. Please assume the follow case:
   
   ```
   Project($0, $1, $2)
       Scan(proj=[$0, $1, $2, <some constant or constant expression like 2 * 2>])
   ```
   
   So you should first merge projects and then discard the result if it is identity




-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: notifications-unsubscribe@ignite.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org