mirror of https://github.com/apache/druid.git
Fix queries filtering for the same condition with both an IN and EQUALS to not return empty results (#16597)
temp fix until CALCITE-6435 gets fixed (released&upgraded to) added a custom rule (FixIncorrectInExpansionTypes) to fix-up types of the affected literals added a testcase which will alert on upgrade
This commit is contained in:
parent
09e0eefdc3
commit
a9bd0eea2a
|
@ -61,6 +61,7 @@ import org.apache.druid.sql.calcite.rule.ExtensionCalciteRuleProvider;
|
|||
import org.apache.druid.sql.calcite.rule.FilterDecomposeCoalesceRule;
|
||||
import org.apache.druid.sql.calcite.rule.FilterDecomposeConcatRule;
|
||||
import org.apache.druid.sql.calcite.rule.FilterJoinExcludePushToChildRule;
|
||||
import org.apache.druid.sql.calcite.rule.FixIncorrectInExpansionTypes;
|
||||
import org.apache.druid.sql.calcite.rule.FlattenConcatRule;
|
||||
import org.apache.druid.sql.calcite.rule.ProjectAggregatePruneUnusedCallRule;
|
||||
import org.apache.druid.sql.calcite.rule.ReverseLookupRule;
|
||||
|
@ -71,6 +72,7 @@ import org.apache.druid.sql.calcite.rule.logical.DruidLogicalRules;
|
|||
import org.apache.druid.sql.calcite.run.EngineFeature;
|
||||
|
||||
import java.util.ArrayList;
|
||||
import java.util.Collections;
|
||||
import java.util.List;
|
||||
import java.util.Set;
|
||||
|
||||
|
@ -291,6 +293,7 @@ public class CalciteRulesManager
|
|||
// Program that pre-processes the tree before letting the full-on VolcanoPlanner loose.
|
||||
final List<Program> prePrograms = new ArrayList<>();
|
||||
prePrograms.add(new LoggingProgram("Start", isDebug));
|
||||
prePrograms.add(sqlToRelWorkaroundProgram());
|
||||
prePrograms.add(Programs.subQuery(DefaultRelMetadataProvider.INSTANCE));
|
||||
prePrograms.add(new LoggingProgram("Finished subquery program", isDebug));
|
||||
prePrograms.add(DecorrelateAndTrimFieldsProgram.INSTANCE);
|
||||
|
@ -306,6 +309,12 @@ public class CalciteRulesManager
|
|||
return Programs.sequence(prePrograms.toArray(new Program[0]));
|
||||
}
|
||||
|
||||
private Program sqlToRelWorkaroundProgram()
|
||||
{
|
||||
Set<RelOptRule> rules = Collections.singleton(new FixIncorrectInExpansionTypes());
|
||||
return Programs.hep(rules, true, DefaultRelMetadataProvider.INSTANCE);
|
||||
}
|
||||
|
||||
/**
|
||||
* Program to perform manipulations on the logical tree prior to starting the cost-based planner. Mainly this
|
||||
* helps the cost-based planner finish faster, and helps the decoupled planner generate the same plans as the
|
||||
|
|
|
@ -0,0 +1,97 @@
|
|||
/*
|
||||
* 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.druid.sql.calcite.rule;
|
||||
|
||||
import org.apache.calcite.plan.RelOptRule;
|
||||
import org.apache.calcite.plan.RelOptRuleCall;
|
||||
import org.apache.calcite.rel.RelNode;
|
||||
import org.apache.calcite.rel.rules.SubstitutionRule;
|
||||
import org.apache.calcite.rex.RexBuilder;
|
||||
import org.apache.calcite.rex.RexCall;
|
||||
import org.apache.calcite.rex.RexNode;
|
||||
import org.apache.calcite.rex.RexShuttle;
|
||||
import org.apache.calcite.rex.RexUtil;
|
||||
import org.apache.calcite.sql.SqlKind;
|
||||
import org.apache.calcite.sql.type.SqlTypeName;
|
||||
|
||||
/**
|
||||
* Rewrites comparisions to avoid bug FIXME.
|
||||
*
|
||||
* Rewrites RexCall::VARCHAR = RexLiteral::CHAR to RexCall::VARCHAR =
|
||||
* RexLiteral::VARCHAR
|
||||
*
|
||||
* needed until CALCITE-6435 is fixed & released.
|
||||
*/
|
||||
public class FixIncorrectInExpansionTypes extends RelOptRule implements SubstitutionRule
|
||||
{
|
||||
public FixIncorrectInExpansionTypes()
|
||||
{
|
||||
super(operand(RelNode.class, any()));
|
||||
}
|
||||
|
||||
@Override
|
||||
public void onMatch(RelOptRuleCall call)
|
||||
{
|
||||
final RelNode oldNode = call.rel(0);
|
||||
final RewriteShuttle shuttle = new RewriteShuttle(oldNode.getCluster().getRexBuilder());
|
||||
final RelNode newNode = oldNode.accept(shuttle);
|
||||
|
||||
// noinspection ObjectEquality
|
||||
if (newNode != oldNode) {
|
||||
call.transformTo(newNode);
|
||||
call.getPlanner().prune(oldNode);
|
||||
}
|
||||
}
|
||||
|
||||
private static class RewriteShuttle extends RexShuttle
|
||||
{
|
||||
private final RexBuilder rexBuilder;
|
||||
|
||||
public RewriteShuttle(RexBuilder rexBuilder)
|
||||
{
|
||||
this.rexBuilder = rexBuilder;
|
||||
}
|
||||
|
||||
@Override
|
||||
public RexNode visitCall(RexCall call)
|
||||
{
|
||||
RexNode newNode = super.visitCall(call);
|
||||
if (newNode.getKind() == SqlKind.EQUALS || newNode.getKind() == SqlKind.NOT_EQUALS) {
|
||||
RexCall newCall = (RexCall) newNode;
|
||||
RexNode op0 = newCall.getOperands().get(0);
|
||||
RexNode op1 = newCall.getOperands().get(1);
|
||||
if (RexUtil.isLiteral(op1, false)) {
|
||||
|
||||
if (op1.getType().getSqlTypeName() == SqlTypeName.CHAR
|
||||
&& op0.getType().getSqlTypeName() == SqlTypeName.VARCHAR) {
|
||||
|
||||
RexNode newLiteral = rexBuilder.ensureType(op0.getType(), op1, true);
|
||||
return rexBuilder.makeCall(
|
||||
newCall.getOperator(),
|
||||
op0,
|
||||
newLiteral
|
||||
);
|
||||
}
|
||||
}
|
||||
}
|
||||
return newNode;
|
||||
}
|
||||
}
|
||||
}
|
|
@ -21,6 +21,7 @@ package org.apache.druid.sql.calcite;
|
|||
|
||||
import com.google.common.collect.ImmutableList;
|
||||
import com.google.common.collect.ImmutableMap;
|
||||
import org.apache.calcite.rel.RelNode;
|
||||
import org.apache.druid.common.config.NullHandling;
|
||||
import org.apache.druid.error.DruidExceptionMatcher;
|
||||
import org.apache.druid.java.util.common.DateTimes;
|
||||
|
@ -61,6 +62,8 @@ import java.util.HashMap;
|
|||
import java.util.List;
|
||||
import java.util.Map;
|
||||
|
||||
import static org.junit.jupiter.api.Assertions.assertEquals;
|
||||
|
||||
public class CalciteSelectQueryTest extends BaseCalciteQueryTest
|
||||
{
|
||||
@Test
|
||||
|
@ -2175,4 +2178,28 @@ public class CalciteSelectQueryTest extends BaseCalciteQueryTest
|
|||
)
|
||||
.run();
|
||||
}
|
||||
|
||||
@Test
|
||||
public void testSqlToRelInConversion()
|
||||
{
|
||||
assertEquals(
|
||||
"1.37.0",
|
||||
RelNode.class.getPackage().getImplementationVersion(),
|
||||
"Calcite version changed; check if CALCITE-6435 is fixed and remove:\n * method CalciteRulesManager#sqlToRelWorkaroundProgram\n * FixIncorrectInExpansionTypes class\n* this assertion"
|
||||
);
|
||||
|
||||
testBuilder()
|
||||
.sql(
|
||||
"SELECT channel FROM wikipedia\n"
|
||||
+ "WHERE channel in ('#en.wikipedia') and channel = '#en.wikipedia' and\n"
|
||||
+ "isRobot = 'false'\n"
|
||||
+ "LIMIT 1"
|
||||
)
|
||||
.expectedResults(
|
||||
ImmutableList.of(
|
||||
new Object[] {"#en.wikipedia"}
|
||||
)
|
||||
)
|
||||
.run();
|
||||
}
|
||||
}
|
||||
|
|
|
@ -26,13 +26,13 @@ LogicalSort(sort0=[$0], dir0=[ASC])
|
|||
LogicalSort(sort0=[$0], dir0=[ASC])
|
||||
LogicalAggregate(group=[{0}], cnt=[COUNT($1) FILTER $2], aall=[COUNT()])
|
||||
LogicalProject(cityName=[$2], channel=[$1], $f3=[IS TRUE(>($17, 0))])
|
||||
LogicalFilter(condition=[SEARCH($2, Sarg['Aarhus':VARCHAR(8), 'New York':VARCHAR(8)]:VARCHAR(8))])
|
||||
LogicalFilter(condition=[SEARCH($2, Sarg['Aarhus':VARCHAR, 'New York':VARCHAR]:VARCHAR)])
|
||||
LogicalTableScan(table=[[druid, wikipedia]])
|
||||
|
||||
!logicalPlan
|
||||
DruidAggregate(group=[{0}], cnt=[COUNT($1) FILTER $2], aall=[COUNT()], druid=[logical])
|
||||
DruidProject(cityName=[$2], channel=[$1], $f3=[IS TRUE(>($17, 0))], druid=[logical])
|
||||
DruidFilter(condition=[SEARCH($2, Sarg['Aarhus':VARCHAR(8), 'New York':VARCHAR(8)]:VARCHAR(8))])
|
||||
DruidFilter(condition=[SEARCH($2, Sarg['Aarhus':VARCHAR, 'New York':VARCHAR]:VARCHAR)])
|
||||
DruidTableScan(table=[[druid, wikipedia]], druid=[logical])
|
||||
|
||||
!druidPlan
|
||||
|
|
Loading…
Reference in New Issue