mirror of https://github.com/apache/druid.git
Fallback vectorization for FunctionExpr and BaseMacroFunctionExpr. (#16366)
* Fallback vectorization for FunctionExpr and BaseMacroFunctionExpr. This patch adds FallbackVectorProcessor, a processor that adapts non-vectorizable operations into vectorizable ones. It is used in FunctionExpr and BaseMacroFunctionExpr. In addition: - Identifiers are updated to offer getObjectVector for ARRAY and COMPLEX in addition to STRING. ExprEvalObjectVector is updated to offer ARRAY and COMPLEX as well. - In SQL tests, cannotVectorize now fails tests if an exception is not thrown. This makes it easier to identify tests that can now vectorize. - Fix a null-matcher bug in StringObjectVectorValueMatcher. * Fix tests. * Fixes. * Fix tests. * Fix test. * Fix test.
This commit is contained in:
parent
2534a42539
commit
277006446d
|
@ -307,9 +307,6 @@ public class HllSketchSqlAggregatorTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testApproxCountDistinctHllSketch()
|
||||
{
|
||||
// Can't vectorize due to SUBSTRING expression.
|
||||
cannotVectorize();
|
||||
|
||||
final String sql = "SELECT\n"
|
||||
+ " SUM(cnt),\n"
|
||||
+ " APPROX_COUNT_DISTINCT_DS_HLL(dim2),\n" // uppercase
|
||||
|
@ -1099,7 +1096,6 @@ public class HllSketchSqlAggregatorTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testHllEstimateAsVirtualColumnWithGroupByOrderBy()
|
||||
{
|
||||
cannotVectorize();
|
||||
testQuery(
|
||||
"SELECT"
|
||||
+ " HLL_SKETCH_ESTIMATE(hllsketch_dim1), count(*)"
|
||||
|
@ -1197,7 +1193,6 @@ public class HllSketchSqlAggregatorTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testResultCacheWithWindowing()
|
||||
{
|
||||
cannotVectorize();
|
||||
for (int i = 0; i < 2; i++) {
|
||||
testBuilder()
|
||||
.queryContext(ImmutableMap.of(PlannerContext.CTX_ENABLE_WINDOW_FNS, true))
|
||||
|
|
|
@ -165,9 +165,6 @@ public class ThetaSketchSqlAggregatorTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testApproxCountDistinctThetaSketch()
|
||||
{
|
||||
// Cannot vectorize due to SUBSTRING.
|
||||
cannotVectorize();
|
||||
|
||||
final String sql = "SELECT\n"
|
||||
+ " SUM(cnt),\n"
|
||||
+ " APPROX_COUNT_DISTINCT_DS_THETA(dim2),\n"
|
||||
|
@ -1121,7 +1118,6 @@ public class ThetaSketchSqlAggregatorTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testThetaEstimateAsVirtualColumnWithGroupByOrderBy()
|
||||
{
|
||||
cannotVectorize();
|
||||
testQuery(
|
||||
"SELECT"
|
||||
+ " THETA_SKETCH_ESTIMATE(thetasketch_dim1), count(*)"
|
||||
|
|
|
@ -98,7 +98,6 @@ public class BloomDimFilterSqlTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testBloomFilterExprFilter() throws IOException
|
||||
{
|
||||
cannotVectorize();
|
||||
BloomKFilter filter = new BloomKFilter(1500);
|
||||
filter.addString("a-foo");
|
||||
filter.addString("-foo");
|
||||
|
|
|
@ -25,6 +25,8 @@ import com.google.common.collect.ImmutableList;
|
|||
import com.google.common.collect.ImmutableSet;
|
||||
import com.google.common.collect.Maps;
|
||||
import org.apache.druid.java.util.common.StringUtils;
|
||||
import org.apache.druid.math.expr.vector.ExprVectorProcessor;
|
||||
import org.apache.druid.math.expr.vector.FallbackVectorProcessor;
|
||||
|
||||
import javax.annotation.Nullable;
|
||||
import java.util.Collections;
|
||||
|
@ -158,6 +160,18 @@ public class ExprMacroTable
|
|||
return analyzeInputsSupplier.get();
|
||||
}
|
||||
|
||||
@Override
|
||||
public boolean canVectorize(InputBindingInspector inspector)
|
||||
{
|
||||
return getOutputType(inspector) != null && inspector.canVectorize(args);
|
||||
}
|
||||
|
||||
@Override
|
||||
public <T> ExprVectorProcessor<T> asVectorProcessor(VectorInputBindingInspector inspector)
|
||||
{
|
||||
return FallbackVectorProcessor.create(macro, args, inspector);
|
||||
}
|
||||
|
||||
/**
|
||||
* Implemented by subclasses to provide the value for {@link #analyzeInputs()}, which uses a memoized supplier.
|
||||
*/
|
||||
|
|
|
@ -24,6 +24,7 @@ import com.google.common.collect.ImmutableList;
|
|||
import org.apache.druid.error.DruidException;
|
||||
import org.apache.druid.java.util.common.StringUtils;
|
||||
import org.apache.druid.math.expr.vector.ExprVectorProcessor;
|
||||
import org.apache.druid.math.expr.vector.FallbackVectorProcessor;
|
||||
import org.apache.druid.segment.column.Types;
|
||||
|
||||
import javax.annotation.Nullable;
|
||||
|
@ -223,13 +224,18 @@ class FunctionExpr implements Expr
|
|||
@Override
|
||||
public boolean canVectorize(InputBindingInspector inspector)
|
||||
{
|
||||
return function.canVectorize(inspector, args);
|
||||
return function.canVectorize(inspector, args)
|
||||
|| (getOutputType(inspector) != null && inspector.canVectorize(args));
|
||||
}
|
||||
|
||||
@Override
|
||||
public ExprVectorProcessor<?> asVectorProcessor(VectorInputBindingInspector inspector)
|
||||
{
|
||||
if (function.canVectorize(inspector, args)) {
|
||||
return function.asVectorProcessor(inspector, args);
|
||||
} else {
|
||||
return FallbackVectorProcessor.create(function, args, inspector);
|
||||
}
|
||||
}
|
||||
|
||||
@Override
|
||||
|
|
|
@ -39,7 +39,7 @@ public final class CastToStringVectorProcessor extends CastToTypeVectorProcessor
|
|||
for (int i = 0; i < objects.length; i++) {
|
||||
output[i] = Evals.asString(objects[i]);
|
||||
}
|
||||
return new ExprEvalObjectVector(output);
|
||||
return new ExprEvalObjectVector(output, ExpressionType.STRING);
|
||||
}
|
||||
|
||||
@Override
|
||||
|
|
|
@ -20,8 +20,10 @@
|
|||
package org.apache.druid.math.expr.vector;
|
||||
|
||||
import org.apache.druid.common.config.NullHandling;
|
||||
import org.apache.druid.error.DruidException;
|
||||
import org.apache.druid.math.expr.Evals;
|
||||
import org.apache.druid.math.expr.ExprEval;
|
||||
import org.apache.druid.math.expr.ExprType;
|
||||
import org.apache.druid.math.expr.ExpressionType;
|
||||
|
||||
import javax.annotation.Nullable;
|
||||
|
@ -36,9 +38,17 @@ public final class ExprEvalObjectVector extends ExprEvalVector<Object[]>
|
|||
@Nullable
|
||||
private boolean[] numericNulls;
|
||||
|
||||
public ExprEvalObjectVector(Object[] values)
|
||||
private final ExpressionType type;
|
||||
|
||||
public ExprEvalObjectVector(Object[] values, ExpressionType type)
|
||||
{
|
||||
super(values, null);
|
||||
this.type = type;
|
||||
|
||||
if (type.isNumeric()) {
|
||||
// Cannot use ExprEvalObjectSelector on types that are innately numbers.
|
||||
throw DruidException.defensive("Expression of type[%s] is numeric", type);
|
||||
}
|
||||
}
|
||||
|
||||
private void computeNumbers()
|
||||
|
@ -47,7 +57,9 @@ public final class ExprEvalObjectVector extends ExprEvalVector<Object[]>
|
|||
longs = new long[values.length];
|
||||
doubles = new double[values.length];
|
||||
numericNulls = new boolean[values.length];
|
||||
boolean isString = type.is(ExprType.STRING);
|
||||
for (int i = 0; i < values.length; i++) {
|
||||
if (isString) {
|
||||
Number n = ExprEval.computeNumber(Evals.asString(values[i]));
|
||||
if (n != null) {
|
||||
longs[i] = n.longValue();
|
||||
|
@ -58,6 +70,13 @@ public final class ExprEvalObjectVector extends ExprEvalVector<Object[]>
|
|||
doubles[i] = 0.0;
|
||||
numericNulls[i] = NullHandling.sqlCompatible();
|
||||
}
|
||||
} else {
|
||||
// ARRAY, COMPLEX
|
||||
final ExprEval<?> valueEval = ExprEval.bestEffortOf(values[i]);
|
||||
longs[i] = valueEval.asLong();
|
||||
doubles[i] = valueEval.asDouble();
|
||||
numericNulls[i] = valueEval.isNumericNull();
|
||||
}
|
||||
}
|
||||
}
|
||||
}
|
||||
|
@ -73,7 +92,7 @@ public final class ExprEvalObjectVector extends ExprEvalVector<Object[]>
|
|||
@Override
|
||||
public ExpressionType getType()
|
||||
{
|
||||
return ExpressionType.STRING;
|
||||
return type;
|
||||
}
|
||||
|
||||
@Override
|
||||
|
|
|
@ -0,0 +1,422 @@
|
|||
/*
|
||||
* 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.math.expr.vector;
|
||||
|
||||
import org.apache.druid.error.DruidException;
|
||||
import org.apache.druid.math.expr.Expr;
|
||||
import org.apache.druid.math.expr.ExprEval;
|
||||
import org.apache.druid.math.expr.ExprMacroTable;
|
||||
import org.apache.druid.math.expr.ExprType;
|
||||
import org.apache.druid.math.expr.ExpressionType;
|
||||
import org.apache.druid.math.expr.Function;
|
||||
|
||||
import javax.annotation.Nullable;
|
||||
import java.util.ArrayList;
|
||||
import java.util.List;
|
||||
import java.util.Objects;
|
||||
import java.util.function.Supplier;
|
||||
|
||||
/**
|
||||
* Implementation of {@link ExprVectorProcessor} that adapts non-vectorized {@link Expr#eval(Expr.ObjectBinding)}.
|
||||
* This allows non-vectorized expressions to participate in vectorized queries.
|
||||
*/
|
||||
public abstract class FallbackVectorProcessor<T> implements ExprVectorProcessor<T>
|
||||
{
|
||||
final Supplier<ExprEval<?>> fn;
|
||||
final List<AdaptedExpr> adaptedArgs;
|
||||
|
||||
private final ExpressionType outputType;
|
||||
|
||||
private FallbackVectorProcessor(
|
||||
final Supplier<ExprEval<?>> fn,
|
||||
final List<AdaptedExpr> adaptedArgs,
|
||||
final ExpressionType outputType
|
||||
)
|
||||
{
|
||||
this.fn = fn;
|
||||
this.adaptedArgs = adaptedArgs;
|
||||
this.outputType = outputType;
|
||||
}
|
||||
|
||||
/**
|
||||
* Create a processor for a non-vectorizable {@link Function}.
|
||||
*/
|
||||
public static <T> FallbackVectorProcessor<T> create(
|
||||
final Function function,
|
||||
final List<Expr> args,
|
||||
final Expr.VectorInputBindingInspector inspector
|
||||
)
|
||||
{
|
||||
final List<Expr> adaptedArgs = makeAdaptedArgs(args, inspector);
|
||||
return makeFallbackProcessor(
|
||||
() -> function.apply(adaptedArgs, UnusedBinding.INSTANCE),
|
||||
adaptedArgs,
|
||||
function.getOutputType(inspector, args),
|
||||
inspector
|
||||
);
|
||||
}
|
||||
|
||||
/**
|
||||
* Create a processor for a non-vectorizable {@link ExprMacroTable.ExprMacro}.
|
||||
*/
|
||||
public static <T> FallbackVectorProcessor<T> create(
|
||||
final ExprMacroTable.ExprMacro macro,
|
||||
final List<Expr> args,
|
||||
final Expr.VectorInputBindingInspector inspector
|
||||
)
|
||||
{
|
||||
final List<Expr> adaptedArgs = makeAdaptedArgs(args, inspector);
|
||||
final Expr adaptedExpr = macro.apply(adaptedArgs);
|
||||
return makeFallbackProcessor(
|
||||
() -> adaptedExpr.eval(UnusedBinding.INSTANCE),
|
||||
adaptedArgs,
|
||||
adaptedExpr.getOutputType(inspector),
|
||||
inspector
|
||||
);
|
||||
}
|
||||
|
||||
/**
|
||||
* Helper for the two {@link #create} methods. Makes {@link AdaptedExpr} that can replace the original args to
|
||||
* the {@link Expr} that requires fallback.
|
||||
*
|
||||
* @param args args to the original expr
|
||||
* @param inspector binding inspector
|
||||
*
|
||||
* @return list of {@link AdaptedExpr}
|
||||
*/
|
||||
private static List<Expr> makeAdaptedArgs(
|
||||
final List<Expr> args,
|
||||
final Expr.VectorInputBindingInspector inspector
|
||||
)
|
||||
{
|
||||
final List<Expr> adaptedArgs = new ArrayList<>(args.size());
|
||||
|
||||
for (final Expr arg : args) {
|
||||
adaptedArgs.add(new AdaptedExpr(arg.asVectorProcessor(inspector), arg));
|
||||
}
|
||||
|
||||
return adaptedArgs;
|
||||
}
|
||||
|
||||
/**
|
||||
* Helper for the two {@link #create} methods.
|
||||
*
|
||||
* @param fn eval function that uses the "adaptedArgs" as inputs
|
||||
* @param adaptedArgs adapted args from {@link #makeAdaptedArgs(List, Expr.VectorInputBindingInspector)}
|
||||
* @param outputType output type of the eval from "fn"
|
||||
* @param inspector binding inspector
|
||||
*/
|
||||
@SuppressWarnings({"unchecked", "rawtypes"})
|
||||
private static <T> FallbackVectorProcessor<T> makeFallbackProcessor(
|
||||
final Supplier<ExprEval<?>> fn,
|
||||
final List<Expr> adaptedArgs,
|
||||
final ExpressionType outputType,
|
||||
final Expr.VectorInputBindingInspector inspector
|
||||
)
|
||||
{
|
||||
if (outputType == null) {
|
||||
throw DruidException.defensive("Plan has null outputType");
|
||||
} else if (outputType.equals(ExpressionType.LONG)) {
|
||||
return (FallbackVectorProcessor<T>) new OfLong(fn, (List) adaptedArgs, outputType, inspector);
|
||||
} else if (outputType.equals(ExpressionType.DOUBLE)) {
|
||||
return (FallbackVectorProcessor<T>) new OfDouble(fn, (List) adaptedArgs, outputType, inspector);
|
||||
} else {
|
||||
return (FallbackVectorProcessor<T>) new OfObject(fn, (List) adaptedArgs, outputType, inspector);
|
||||
}
|
||||
}
|
||||
|
||||
@Override
|
||||
public ExpressionType getOutputType()
|
||||
{
|
||||
return outputType;
|
||||
}
|
||||
|
||||
/**
|
||||
* Specialization for non-numeric types.
|
||||
*/
|
||||
private static class OfObject extends FallbackVectorProcessor<Object[]>
|
||||
{
|
||||
private final Object[] outValues;
|
||||
|
||||
public OfObject(
|
||||
final Supplier<ExprEval<?>> fn,
|
||||
final List<AdaptedExpr> args,
|
||||
final ExpressionType outputType,
|
||||
final Expr.VectorInputBindingInspector inspector
|
||||
)
|
||||
{
|
||||
super(fn, args, outputType);
|
||||
this.outValues = new Object[inspector.getMaxVectorSize()];
|
||||
}
|
||||
|
||||
@Override
|
||||
public ExprEvalVector<Object[]> evalVector(Expr.VectorInputBinding vectorBindings)
|
||||
{
|
||||
for (final AdaptedExpr adaptedArg : adaptedArgs) {
|
||||
adaptedArg.populate(vectorBindings);
|
||||
}
|
||||
|
||||
final int sz = vectorBindings.getCurrentVectorSize();
|
||||
for (int i = 0; i < sz; i++) {
|
||||
for (final AdaptedExpr adaptedArg : adaptedArgs) {
|
||||
adaptedArg.setRowNumber(i);
|
||||
}
|
||||
|
||||
outValues[i] = fn.get().value();
|
||||
}
|
||||
|
||||
return new ExprEvalObjectVector(outValues, getOutputType());
|
||||
}
|
||||
}
|
||||
|
||||
/**
|
||||
* Specialization for {@link ExpressionType#LONG}.
|
||||
*/
|
||||
private static class OfLong extends FallbackVectorProcessor<long[]>
|
||||
{
|
||||
private final long[] outValues;
|
||||
private final boolean[] outNulls;
|
||||
|
||||
public OfLong(
|
||||
final Supplier<ExprEval<?>> fn,
|
||||
final List<AdaptedExpr> args,
|
||||
final ExpressionType outputType,
|
||||
final Expr.VectorInputBindingInspector inspector
|
||||
)
|
||||
{
|
||||
super(fn, args, outputType);
|
||||
this.outValues = new long[inspector.getMaxVectorSize()];
|
||||
this.outNulls = new boolean[inspector.getMaxVectorSize()];
|
||||
}
|
||||
|
||||
@Override
|
||||
public ExprEvalVector<long[]> evalVector(Expr.VectorInputBinding vectorBindings)
|
||||
{
|
||||
for (final AdaptedExpr adaptedArg : adaptedArgs) {
|
||||
adaptedArg.populate(vectorBindings);
|
||||
}
|
||||
|
||||
final int sz = vectorBindings.getCurrentVectorSize();
|
||||
boolean anyNulls = false;
|
||||
|
||||
for (int i = 0; i < sz; i++) {
|
||||
for (final AdaptedExpr adaptedArg : adaptedArgs) {
|
||||
adaptedArg.setRowNumber(i);
|
||||
}
|
||||
|
||||
final ExprEval<?> eval = fn.get();
|
||||
outValues[i] = eval.asLong();
|
||||
outNulls[i] = eval.isNumericNull();
|
||||
anyNulls = anyNulls || outNulls[i];
|
||||
}
|
||||
|
||||
return new ExprEvalLongVector(outValues, anyNulls ? outNulls : null);
|
||||
}
|
||||
}
|
||||
|
||||
/**
|
||||
* Specialization for {@link ExpressionType#DOUBLE}.
|
||||
*/
|
||||
private static class OfDouble extends FallbackVectorProcessor<double[]>
|
||||
{
|
||||
private final double[] outValues;
|
||||
private final boolean[] outNulls;
|
||||
|
||||
public OfDouble(
|
||||
final Supplier<ExprEval<?>> fn,
|
||||
final List<AdaptedExpr> args,
|
||||
final ExpressionType outputType,
|
||||
final Expr.VectorInputBindingInspector inspector
|
||||
)
|
||||
{
|
||||
super(fn, args, outputType);
|
||||
this.outValues = new double[inspector.getMaxVectorSize()];
|
||||
this.outNulls = new boolean[inspector.getMaxVectorSize()];
|
||||
}
|
||||
|
||||
@Override
|
||||
public ExprEvalVector<double[]> evalVector(Expr.VectorInputBinding vectorBindings)
|
||||
{
|
||||
for (final AdaptedExpr adaptedArg : adaptedArgs) {
|
||||
adaptedArg.populate(vectorBindings);
|
||||
}
|
||||
|
||||
final int sz = vectorBindings.getCurrentVectorSize();
|
||||
boolean anyNulls = false;
|
||||
|
||||
for (int i = 0; i < sz; i++) {
|
||||
for (final AdaptedExpr adaptedArg : adaptedArgs) {
|
||||
adaptedArg.setRowNumber(i);
|
||||
}
|
||||
|
||||
final ExprEval<?> eval = fn.get();
|
||||
outValues[i] = eval.asDouble();
|
||||
outNulls[i] = eval.isNumericNull();
|
||||
anyNulls = anyNulls || outNulls[i];
|
||||
}
|
||||
|
||||
return new ExprEvalDoubleVector(outValues, anyNulls ? outNulls : null);
|
||||
}
|
||||
}
|
||||
|
||||
/**
|
||||
* Wrapper around {@link Expr} that pulls results from a {@link ExprVectorProcessor} rather than calling
|
||||
* {@link Expr#eval(ObjectBinding)}. When using {@link FallbackVectorProcessor}, adapters of this class replace
|
||||
* the arguments of the original {@link Expr}.
|
||||
*/
|
||||
private static class AdaptedExpr implements Expr
|
||||
{
|
||||
private final ExprVectorProcessor<?> processor;
|
||||
private final Expr originalExpr;
|
||||
private final ExpressionType type;
|
||||
|
||||
private ExprEvalVector<?> results;
|
||||
private int rowNum;
|
||||
|
||||
public AdaptedExpr(final ExprVectorProcessor<?> processor, final Expr originalExpr)
|
||||
{
|
||||
this.processor = processor;
|
||||
this.originalExpr = originalExpr;
|
||||
this.type = processor.getOutputType();
|
||||
}
|
||||
|
||||
/**
|
||||
* Populate the {@link #results} vector. Called once per vector.
|
||||
*/
|
||||
public void populate(final Expr.VectorInputBinding vectorBindings)
|
||||
{
|
||||
this.results = processor.evalVector(vectorBindings);
|
||||
}
|
||||
|
||||
/**
|
||||
* Set {@link #rowNum}, which controls which row of {@link #results} is returned by {@link #eval(ObjectBinding)}.
|
||||
*/
|
||||
public void setRowNumber(final int rowNum)
|
||||
{
|
||||
this.rowNum = rowNum;
|
||||
}
|
||||
|
||||
@Override
|
||||
public ExprEval eval(ObjectBinding bindings)
|
||||
{
|
||||
if (results == null) {
|
||||
// "results" can be null if eval is called during ExprMacro#apply.
|
||||
return originalExpr.eval(bindings);
|
||||
}
|
||||
|
||||
// In all other cases, ignore the provided bindings and use the computed "results" instead.
|
||||
if (type.is(ExprType.LONG)) {
|
||||
final boolean isNull = results.getNullVector() != null && results.getNullVector()[rowNum];
|
||||
return ExprEval.ofLong(isNull ? null : results.getLongVector()[rowNum]);
|
||||
} else if (type.is(ExprType.DOUBLE)) {
|
||||
final boolean isNull = results.getNullVector() != null && results.getNullVector()[rowNum];
|
||||
return ExprEval.ofDouble(isNull ? null : results.getDoubleVector()[rowNum]);
|
||||
} else {
|
||||
return ExprEval.ofType(type, results.getObjectVector()[rowNum]);
|
||||
}
|
||||
}
|
||||
|
||||
@Override
|
||||
public String stringify()
|
||||
{
|
||||
throw DruidException.defensive(
|
||||
"Unexpected call to stringify in fallback processor for expr[%s]",
|
||||
originalExpr.stringify()
|
||||
);
|
||||
}
|
||||
|
||||
@Override
|
||||
public Expr visit(Shuttle shuttle)
|
||||
{
|
||||
throw DruidException.defensive(
|
||||
"Unexpected call to visit in fallback processor for expr[%s]",
|
||||
originalExpr.stringify()
|
||||
);
|
||||
}
|
||||
|
||||
@Override
|
||||
public BindingAnalysis analyzeInputs()
|
||||
{
|
||||
return originalExpr.analyzeInputs();
|
||||
}
|
||||
|
||||
@Override
|
||||
public boolean isLiteral()
|
||||
{
|
||||
return originalExpr.isLiteral();
|
||||
}
|
||||
|
||||
@Override
|
||||
public boolean isNullLiteral()
|
||||
{
|
||||
return originalExpr.isNullLiteral();
|
||||
}
|
||||
|
||||
@Nullable
|
||||
@Override
|
||||
public Object getLiteralValue()
|
||||
{
|
||||
return originalExpr.getLiteralValue();
|
||||
}
|
||||
|
||||
@Override
|
||||
public boolean equals(Object o)
|
||||
{
|
||||
if (this == o) {
|
||||
return true;
|
||||
}
|
||||
if (o == null || getClass() != o.getClass()) {
|
||||
return false;
|
||||
}
|
||||
AdaptedExpr that = (AdaptedExpr) o;
|
||||
return Objects.equals(originalExpr, that.originalExpr) && Objects.equals(type, that.type);
|
||||
}
|
||||
|
||||
@Override
|
||||
public int hashCode()
|
||||
{
|
||||
return Objects.hash(originalExpr, type);
|
||||
}
|
||||
}
|
||||
|
||||
/**
|
||||
* Implementation of {@link Expr.ObjectBinding} where we do not actually expect any methods to be called. This is
|
||||
* because bindings should only be used by identifiers, and this fallback processor is never used to
|
||||
* implement identifiers.
|
||||
*/
|
||||
private static class UnusedBinding implements Expr.ObjectBinding
|
||||
{
|
||||
public static final UnusedBinding INSTANCE = new UnusedBinding();
|
||||
|
||||
@Nullable
|
||||
@Override
|
||||
public Object get(String name)
|
||||
{
|
||||
throw DruidException.defensive("Unexpected binding.get call for field[%s]", name);
|
||||
}
|
||||
|
||||
@Nullable
|
||||
@Override
|
||||
public ExpressionType getType(String name)
|
||||
{
|
||||
throw DruidException.defensive("Unexpected binding.getType call for field[%s]", name);
|
||||
}
|
||||
}
|
||||
}
|
|
@ -61,7 +61,7 @@ public abstract class ObjectOutMultiObjectInVectorProcessor implements ExprVecto
|
|||
for (int i = 0; i < currentSize; i++) {
|
||||
processIndex(in, i);
|
||||
}
|
||||
return new ExprEvalObjectVector(outValues);
|
||||
return new ExprEvalObjectVector(outValues, expressionType);
|
||||
}
|
||||
|
||||
abstract void processIndex(Object[][] in, int i);
|
||||
|
|
|
@ -61,7 +61,7 @@ public abstract class ObjectOutObjectsInFunctionVectorProcessor
|
|||
@Override
|
||||
ExprEvalVector<Object[]> asEval()
|
||||
{
|
||||
return new ExprEvalObjectVector(outValues);
|
||||
return new ExprEvalObjectVector(outValues, expressionType);
|
||||
}
|
||||
|
||||
@Override
|
||||
|
|
|
@ -89,7 +89,7 @@ public class VectorProcessors
|
|||
{
|
||||
final Object[] strings = new Object[maxVectorSize];
|
||||
Arrays.fill(strings, constant);
|
||||
final ExprEvalObjectVector eval = new ExprEvalObjectVector(strings);
|
||||
final ExprEvalObjectVector eval = new ExprEvalObjectVector(strings, ExpressionType.STRING);
|
||||
return new ExprVectorProcessor<T>()
|
||||
{
|
||||
@Override
|
||||
|
@ -200,7 +200,7 @@ public class VectorProcessors
|
|||
@Override
|
||||
public ExprEvalVector<Object[]> evalVector(Expr.VectorInputBinding bindings)
|
||||
{
|
||||
return new ExprEvalObjectVector(bindings.getObjectVector(binding));
|
||||
return new ExprEvalObjectVector(bindings.getObjectVector(binding), ExpressionType.STRING);
|
||||
}
|
||||
};
|
||||
}
|
||||
|
@ -223,17 +223,15 @@ public class VectorProcessors
|
|||
return new ExprEvalDoubleVector(bindings.getDoubleVector(binding), bindings.getNullVector(binding));
|
||||
}
|
||||
};
|
||||
case STRING:
|
||||
default:
|
||||
return new IdentifierVectorProcessor<Object[]>(inputType)
|
||||
{
|
||||
@Override
|
||||
public ExprEvalVector<Object[]> evalVector(Expr.VectorInputBinding bindings)
|
||||
{
|
||||
return new ExprEvalObjectVector(bindings.getObjectVector(binding));
|
||||
return new ExprEvalObjectVector(bindings.getObjectVector(binding), ExpressionType.STRING);
|
||||
}
|
||||
};
|
||||
default:
|
||||
throw Exprs.cannotVectorize("[" + binding + "]");
|
||||
}
|
||||
}
|
||||
|
||||
|
@ -619,7 +617,7 @@ public class VectorProcessors
|
|||
@Override
|
||||
public ExprEvalVector<Object[]> asEval()
|
||||
{
|
||||
return new ExprEvalObjectVector(output);
|
||||
return new ExprEvalObjectVector(output, getOutputType());
|
||||
}
|
||||
}
|
||||
);
|
||||
|
|
|
@ -58,7 +58,7 @@ public class StringObjectVectorValueMatcher implements VectorValueMatcherFactory
|
|||
|
||||
for (int i = 0; i < mask.getSelectionSize(); i++) {
|
||||
final int rowNum = mask.getSelection()[i];
|
||||
if ((value == null && includeUnknown) || Objects.equals(value, vector[rowNum])) {
|
||||
if ((vector[rowNum] == null && includeUnknown) || Objects.equals(value, vector[rowNum])) {
|
||||
selection[numRows++] = rowNum;
|
||||
}
|
||||
}
|
||||
|
|
|
@ -171,7 +171,7 @@ public class ExpressionVectorSelectors
|
|||
default:
|
||||
binding.addObjectSelector(
|
||||
columnName,
|
||||
ExpressionType.STRING,
|
||||
ExpressionType.fromColumnType(columnCapabilities.toColumnType()),
|
||||
vectorColumnSelectorFactory.makeObjectSelector(columnName)
|
||||
);
|
||||
}
|
||||
|
|
|
@ -0,0 +1,295 @@
|
|||
/*
|
||||
* 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.math.expr.vector;
|
||||
|
||||
import com.google.common.collect.ImmutableList;
|
||||
import com.google.common.collect.ImmutableMap;
|
||||
import org.apache.druid.common.config.NullHandling;
|
||||
import org.apache.druid.java.util.common.ISE;
|
||||
import org.apache.druid.math.expr.Expr;
|
||||
import org.apache.druid.math.expr.ExprMacroTable;
|
||||
import org.apache.druid.math.expr.ExpressionType;
|
||||
import org.apache.druid.math.expr.Function;
|
||||
import org.apache.druid.math.expr.Parser;
|
||||
import org.apache.druid.segment.column.ColumnType;
|
||||
import org.apache.druid.segment.column.RowSignature;
|
||||
import org.apache.druid.testing.InitializedNullHandlingTest;
|
||||
import org.junit.Assert;
|
||||
import org.junit.Before;
|
||||
import org.junit.Test;
|
||||
|
||||
import javax.annotation.Nullable;
|
||||
import java.util.Arrays;
|
||||
import java.util.List;
|
||||
import java.util.Map;
|
||||
|
||||
public class FallbackVectorProcessorTest extends InitializedNullHandlingTest
|
||||
{
|
||||
private static final int VECTOR_ID = 0;
|
||||
private static final int VECTOR_SIZE = 3;
|
||||
private static final RowSignature SIGNATURE =
|
||||
RowSignature.builder()
|
||||
.add("long", ColumnType.LONG)
|
||||
.add("double", ColumnType.DOUBLE)
|
||||
.add("doubleNoNulls", ColumnType.DOUBLE)
|
||||
.add("str", ColumnType.STRING)
|
||||
.add("arr", ColumnType.LONG_ARRAY)
|
||||
.build();
|
||||
|
||||
private static final Map<String, List<Object>> DATA =
|
||||
ImmutableMap.<String, List<Object>>builder()
|
||||
.put("long", Arrays.asList(101L, null, 103L))
|
||||
.put("double", Arrays.asList(1.1, null, 1.3))
|
||||
.put("doubleNoNulls", Arrays.asList(1.1, 1.2, 1.3))
|
||||
.put("str", Arrays.asList("foo", null, "baz"))
|
||||
.put(
|
||||
"arr",
|
||||
Arrays.asList(
|
||||
new Object[]{201L, null, 203L},
|
||||
null,
|
||||
new Object[]{301L, null, 303L}
|
||||
)
|
||||
)
|
||||
.build();
|
||||
|
||||
private Expr.VectorInputBinding binding;
|
||||
|
||||
@Before
|
||||
public void setUp()
|
||||
{
|
||||
binding = new Expr.VectorInputBinding()
|
||||
{
|
||||
@Override
|
||||
public Object[] getObjectVector(String name)
|
||||
{
|
||||
if (getType(name).isNumeric()) {
|
||||
throw new ISE("Incorrect call for field[%s] of type[%s]", name, getType(name));
|
||||
}
|
||||
|
||||
return DATA.get(name).toArray();
|
||||
}
|
||||
|
||||
@Override
|
||||
public long[] getLongVector(String name)
|
||||
{
|
||||
if (!getType(name).equals(ExpressionType.LONG)) {
|
||||
throw new ISE("Incorrect call for field[%s] of type[%s]", name, getType(name));
|
||||
}
|
||||
|
||||
final List<Object> longs = DATA.get(name);
|
||||
final long[] longArray = new long[VECTOR_SIZE];
|
||||
|
||||
for (int i = 0; i < longs.size(); i++) {
|
||||
Object o = longs.get(i);
|
||||
longArray[i] = o == null ? 0 : (long) o;
|
||||
}
|
||||
|
||||
return longArray;
|
||||
}
|
||||
|
||||
@Override
|
||||
public double[] getDoubleVector(String name)
|
||||
{
|
||||
if (!getType(name).equals(ExpressionType.DOUBLE)) {
|
||||
throw new ISE("Incorrect call for field[%s] of type[%s]", name, getType(name));
|
||||
}
|
||||
|
||||
final List<Object> doubles = DATA.get(name);
|
||||
final double[] doubleArray = new double[VECTOR_SIZE];
|
||||
|
||||
for (int i = 0; i < doubles.size(); i++) {
|
||||
Object o = doubles.get(i);
|
||||
doubleArray[i] = o == null ? 0 : (double) o;
|
||||
}
|
||||
|
||||
return doubleArray;
|
||||
}
|
||||
|
||||
@Nullable
|
||||
@Override
|
||||
public boolean[] getNullVector(String name)
|
||||
{
|
||||
final List<Object> objects = DATA.get(name);
|
||||
final boolean[] nullArray = new boolean[VECTOR_SIZE];
|
||||
|
||||
boolean anyNulls = false;
|
||||
for (int i = 0; i < objects.size(); i++) {
|
||||
Object o = objects.get(i);
|
||||
nullArray[i] = o == null;
|
||||
anyNulls = anyNulls || o == null;
|
||||
}
|
||||
|
||||
return anyNulls ? nullArray : null;
|
||||
}
|
||||
|
||||
@Override
|
||||
public int getCurrentVectorSize()
|
||||
{
|
||||
return VECTOR_SIZE;
|
||||
}
|
||||
|
||||
@Override
|
||||
public int getCurrentVectorId()
|
||||
{
|
||||
return VECTOR_ID;
|
||||
}
|
||||
|
||||
@Override
|
||||
public int getMaxVectorSize()
|
||||
{
|
||||
return VECTOR_SIZE;
|
||||
}
|
||||
|
||||
@Nullable
|
||||
@Override
|
||||
public ExpressionType getType(String name)
|
||||
{
|
||||
return SIGNATURE.getColumnType(name).map(ExpressionType::fromColumnType).orElse(null);
|
||||
}
|
||||
};
|
||||
}
|
||||
|
||||
@Test
|
||||
public void test_case_long_double()
|
||||
{
|
||||
final FallbackVectorProcessor<Object[]> processor = FallbackVectorProcessor.create(
|
||||
new Function.CaseSimpleFunc(),
|
||||
ImmutableList.of(
|
||||
Parser.parse("long + double", ExprMacroTable.nil()),
|
||||
Parser.parse("102.1", ExprMacroTable.nil()),
|
||||
Parser.parse("long", ExprMacroTable.nil()),
|
||||
Parser.parse("double", ExprMacroTable.nil())
|
||||
),
|
||||
binding
|
||||
);
|
||||
|
||||
final ExprEvalVector<Object[]> eval = processor.evalVector(binding);
|
||||
|
||||
Assert.assertEquals(ExpressionType.DOUBLE, eval.getType());
|
||||
Assert.assertArrayEquals(
|
||||
new Object[]{101.0, NullHandling.defaultDoubleValue(), 1.3},
|
||||
eval.getObjectVector()
|
||||
);
|
||||
Assert.assertArrayEquals(
|
||||
new double[]{101.0, 0L, 1.3},
|
||||
eval.getDoubleVector(),
|
||||
0
|
||||
);
|
||||
Assert.assertArrayEquals(
|
||||
NullHandling.sqlCompatible() ? new boolean[]{false, true, false} : null,
|
||||
eval.getNullVector()
|
||||
);
|
||||
}
|
||||
|
||||
@Test
|
||||
public void test_upper_string()
|
||||
{
|
||||
final FallbackVectorProcessor<Object[]> processor = FallbackVectorProcessor.create(
|
||||
new Function.UpperFunc(),
|
||||
ImmutableList.of(
|
||||
Parser.parse("str", ExprMacroTable.nil())
|
||||
),
|
||||
binding
|
||||
);
|
||||
|
||||
final ExprEvalVector<Object[]> eval = processor.evalVector(binding);
|
||||
|
||||
Assert.assertEquals(ExpressionType.STRING, eval.getType());
|
||||
Assert.assertArrayEquals(
|
||||
new Object[]{"FOO", null, "BAZ"},
|
||||
eval.getObjectVector()
|
||||
);
|
||||
}
|
||||
|
||||
@Test
|
||||
public void test_concat_string_doubleNoNulls()
|
||||
{
|
||||
final FallbackVectorProcessor<Object[]> processor = FallbackVectorProcessor.create(
|
||||
new Function.ConcatFunc(),
|
||||
ImmutableList.of(
|
||||
Parser.parse("str", ExprMacroTable.nil()),
|
||||
Parser.parse("doubleNoNulls + 2", ExprMacroTable.nil())
|
||||
),
|
||||
binding
|
||||
);
|
||||
|
||||
final ExprEvalVector<Object[]> eval = processor.evalVector(binding);
|
||||
|
||||
Assert.assertEquals(ExpressionType.STRING, eval.getType());
|
||||
Assert.assertArrayEquals(
|
||||
new Object[]{"foo3.1", NullHandling.sqlCompatible() ? null : "3.2", "baz3.3"},
|
||||
eval.getObjectVector()
|
||||
);
|
||||
}
|
||||
|
||||
@Test
|
||||
public void test_array_length()
|
||||
{
|
||||
final FallbackVectorProcessor<Object[]> processor = FallbackVectorProcessor.create(
|
||||
new Function.ArrayLengthFunction(),
|
||||
ImmutableList.of(
|
||||
Parser.parse("arr", ExprMacroTable.nil())
|
||||
),
|
||||
binding
|
||||
);
|
||||
|
||||
final ExprEvalVector<Object[]> eval = processor.evalVector(binding);
|
||||
|
||||
Assert.assertEquals(ExpressionType.LONG, eval.getType());
|
||||
Assert.assertArrayEquals(
|
||||
new Object[]{3L, null, 3L},
|
||||
eval.getObjectVector()
|
||||
);
|
||||
Assert.assertArrayEquals(
|
||||
new long[]{3L, 0L, 3L},
|
||||
eval.getLongVector()
|
||||
);
|
||||
Assert.assertArrayEquals(
|
||||
new boolean[]{false, true, false},
|
||||
eval.getNullVector()
|
||||
);
|
||||
}
|
||||
|
||||
@Test
|
||||
public void test_array_concat()
|
||||
{
|
||||
final FallbackVectorProcessor<Object[]> processor = FallbackVectorProcessor.create(
|
||||
new Function.ArrayConcatFunction(),
|
||||
ImmutableList.of(
|
||||
Parser.parse("arr", ExprMacroTable.nil()),
|
||||
Parser.parse("long", ExprMacroTable.nil())
|
||||
),
|
||||
binding
|
||||
);
|
||||
|
||||
final ExprEvalVector<Object[]> eval = processor.evalVector(binding);
|
||||
|
||||
Assert.assertEquals(ExpressionType.LONG_ARRAY, eval.getType());
|
||||
Assert.assertArrayEquals(
|
||||
new Object[]{201L, null, 203L, 101L},
|
||||
(Object[]) eval.getObjectVector()[0]
|
||||
);
|
||||
Assert.assertNull(eval.getObjectVector()[1]);
|
||||
Assert.assertArrayEquals(
|
||||
new Object[]{301L, null, 303L, 103L},
|
||||
(Object[]) eval.getObjectVector()[2]
|
||||
);
|
||||
}
|
||||
}
|
|
@ -2268,8 +2268,6 @@ public class TimeseriesQueryRunnerTest extends InitializedNullHandlingTest
|
|||
@Test
|
||||
public void testTimeSeriesWithFilteredAggAndExpressionFilteredAgg()
|
||||
{
|
||||
// can't vectorize if expression
|
||||
cannotVectorize();
|
||||
TimeseriesQuery query = Druids
|
||||
.newTimeseriesQueryBuilder()
|
||||
.dataSource(QueryRunnerTestHelper.DATA_SOURCE)
|
||||
|
|
|
@ -20,6 +20,7 @@
|
|||
package org.apache.druid.segment.virtual;
|
||||
|
||||
import org.apache.druid.math.expr.Expr;
|
||||
import org.apache.druid.math.expr.ExpressionType;
|
||||
import org.apache.druid.math.expr.vector.ExprEvalObjectVector;
|
||||
import org.apache.druid.math.expr.vector.ExprEvalVector;
|
||||
import org.apache.druid.math.expr.vector.ExprVectorProcessor;
|
||||
|
@ -57,7 +58,7 @@ public class ExpressionVectorObjectSelectorTest
|
|||
public void testSelectObject()
|
||||
{
|
||||
final String[] vector = new String[]{"1", "2", null, "3"};
|
||||
ExprEvalVector vectorEval = new ExprEvalObjectVector(vector);
|
||||
ExprEvalVector vectorEval = new ExprEvalObjectVector(vector, ExpressionType.STRING);
|
||||
EasyMock.expect(binding.getCurrentVectorId()).andReturn(1).anyTimes();
|
||||
EasyMock.expect(vectorProcessor.evalVector(binding)).andReturn(vectorEval).once();
|
||||
EasyMock.replay(binding, vectorProcessor);
|
||||
|
|
|
@ -1203,7 +1203,6 @@ public class CalciteArraysQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testArrayContainsArrayStringColumns()
|
||||
{
|
||||
cannotVectorize();
|
||||
testQuery(
|
||||
"SELECT ARRAY_CONTAINS(arrayStringNulls, ARRAY['a', 'b']), ARRAY_CONTAINS(arrayStringNulls, arrayString) FROM druid.arrays LIMIT 5",
|
||||
ImmutableList.of(
|
||||
|
@ -1548,7 +1547,7 @@ public class CalciteArraysQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testArrayLength()
|
||||
{
|
||||
// Cannot vectorize due to usage of expressions.
|
||||
// Cannot vectorize due to array expressions.
|
||||
cannotVectorize();
|
||||
|
||||
testQuery(
|
||||
|
@ -1591,7 +1590,7 @@ public class CalciteArraysQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testArrayLengthArrayColumn()
|
||||
{
|
||||
// Cannot vectorize due to usage of expressions.
|
||||
// Cannot vectorize due to array expressions.
|
||||
cannotVectorize();
|
||||
|
||||
testQuery(
|
||||
|
@ -1649,7 +1648,7 @@ public class CalciteArraysQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testArrayAppend()
|
||||
{
|
||||
// Cannot vectorize due to usage of expressions.
|
||||
// Cannot vectorize due to array expressions.
|
||||
cannotVectorize();
|
||||
|
||||
ImmutableList<Object[]> results;
|
||||
|
@ -1706,7 +1705,7 @@ public class CalciteArraysQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testArrayPrepend()
|
||||
{
|
||||
// Cannot vectorize due to usage of expressions.
|
||||
// Cannot vectorize due to array expressions.
|
||||
cannotVectorize();
|
||||
|
||||
ImmutableList<Object[]> results;
|
||||
|
@ -1763,7 +1762,7 @@ public class CalciteArraysQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testArrayPrependAppend()
|
||||
{
|
||||
// Cannot vectorize due to usage of expressions.
|
||||
// Cannot vectorize due to array expressions.
|
||||
cannotVectorize();
|
||||
|
||||
ImmutableList<Object[]> results;
|
||||
|
@ -1827,7 +1826,7 @@ public class CalciteArraysQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testArrayConcat()
|
||||
{
|
||||
// Cannot vectorize due to usage of expressions.
|
||||
// Cannot vectorize due to array expressions.
|
||||
cannotVectorize();
|
||||
|
||||
ImmutableList<Object[]> results;
|
||||
|
@ -1884,7 +1883,7 @@ public class CalciteArraysQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testArrayOffset()
|
||||
{
|
||||
// Cannot vectorize due to usage of expressions.
|
||||
// Cannot vectorize due to array expressions.
|
||||
cannotVectorize();
|
||||
|
||||
testQuery(
|
||||
|
@ -2164,7 +2163,7 @@ public class CalciteArraysQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testArrayGroupAsArrayWithFunction()
|
||||
{
|
||||
// Cannot vectorize due to usage of expressions.
|
||||
// Cannot vectorize due to array expressions.
|
||||
cannotVectorize();
|
||||
testQuery(
|
||||
"SELECT ARRAY[ARRAY_ORDINAL(dim3, 2)], SUM(cnt) FROM druid.numfoo GROUP BY 1 ORDER BY 2 DESC",
|
||||
|
@ -2208,7 +2207,7 @@ public class CalciteArraysQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testArrayOrdinal()
|
||||
{
|
||||
// Cannot vectorize due to usage of expressions.
|
||||
// Cannot vectorize due to array expressions.
|
||||
cannotVectorize();
|
||||
|
||||
testQuery(
|
||||
|
@ -2251,7 +2250,7 @@ public class CalciteArraysQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testArrayOffsetOf()
|
||||
{
|
||||
// Cannot vectorize due to usage of expressions.
|
||||
// Cannot vectorize due to array expressions.
|
||||
cannotVectorize();
|
||||
|
||||
testQuery(
|
||||
|
@ -2300,7 +2299,7 @@ public class CalciteArraysQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testArrayOrdinalOf()
|
||||
{
|
||||
// Cannot vectorize due to usage of expressions.
|
||||
// Cannot vectorize due to array expressions.
|
||||
cannotVectorize();
|
||||
|
||||
testQuery(
|
||||
|
@ -2350,7 +2349,7 @@ public class CalciteArraysQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testArrayToString()
|
||||
{
|
||||
// Cannot vectorize due to usage of expressions.
|
||||
// Cannot vectorize due to array expressions.
|
||||
cannotVectorize();
|
||||
|
||||
ImmutableList<Object[]> results;
|
||||
|
@ -2406,7 +2405,7 @@ public class CalciteArraysQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testArrayToStringToMultiValueString()
|
||||
{
|
||||
// Cannot vectorize due to usage of expressions.
|
||||
// Cannot vectorize due to array expressions.
|
||||
cannotVectorize();
|
||||
|
||||
ImmutableList<Object[]> results;
|
||||
|
@ -3505,8 +3504,6 @@ public class CalciteArraysQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testArrayAggGroupByArrayAggFromSubquery()
|
||||
{
|
||||
cannotVectorize();
|
||||
|
||||
testQuery(
|
||||
"SELECT dim2, arr, COUNT(*) FROM (SELECT dim2, ARRAY_AGG(DISTINCT dim1) as arr FROM foo WHERE dim1 is not null GROUP BY 1 LIMIT 5) GROUP BY 1,2",
|
||||
QUERY_CONTEXT_NO_STRINGIFY_ARRAY,
|
||||
|
@ -3936,7 +3933,6 @@ public class CalciteArraysQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testUnnestInline()
|
||||
{
|
||||
cannotVectorize();
|
||||
testQuery(
|
||||
"SELECT * FROM UNNEST(ARRAY[1,2,3])",
|
||||
QUERY_CONTEXT_UNNEST,
|
||||
|
@ -4002,7 +3998,6 @@ public class CalciteArraysQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testUnnest()
|
||||
{
|
||||
cannotVectorize();
|
||||
testQuery(
|
||||
"SELECT d3 FROM druid.numfoo, UNNEST(MV_TO_ARRAY(dim3)) as unnested (d3)",
|
||||
QUERY_CONTEXT_UNNEST,
|
||||
|
@ -4047,7 +4042,6 @@ public class CalciteArraysQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testUnnestArrayColumnsString()
|
||||
{
|
||||
cannotVectorize();
|
||||
testQuery(
|
||||
"SELECT a FROM druid.arrays, UNNEST(arrayString) as unnested (a)",
|
||||
QUERY_CONTEXT_UNNEST,
|
||||
|
@ -4096,7 +4090,6 @@ public class CalciteArraysQueryTest extends BaseCalciteQueryTest
|
|||
public void testUnnestArrayColumnsStringThenFunction()
|
||||
{
|
||||
// Regresson test for https://github.com/apache/druid/issues/16543.
|
||||
cannotVectorize();
|
||||
testQuery(
|
||||
"SELECT a || '.txt' FROM druid.arrays, UNNEST(arrayString) as unnested (a)",
|
||||
QUERY_CONTEXT_UNNEST,
|
||||
|
@ -4143,7 +4136,6 @@ public class CalciteArraysQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testUnnestArrayColumnsStringNulls()
|
||||
{
|
||||
cannotVectorize();
|
||||
testQuery(
|
||||
"SELECT a FROM druid.arrays, UNNEST(arrayStringNulls) as unnested (a)",
|
||||
QUERY_CONTEXT_UNNEST,
|
||||
|
@ -4190,7 +4182,6 @@ public class CalciteArraysQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testUnnestArrayColumnsLong()
|
||||
{
|
||||
cannotVectorize();
|
||||
testQuery(
|
||||
"SELECT a FROM druid.arrays, UNNEST(arrayLong) as unnested (a)",
|
||||
QUERY_CONTEXT_UNNEST,
|
||||
|
@ -4244,7 +4235,6 @@ public class CalciteArraysQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testUnnestArrayColumnsLongNulls()
|
||||
{
|
||||
cannotVectorize();
|
||||
testQuery(
|
||||
"SELECT a FROM druid.arrays, UNNEST(arrayLongNulls) as unnested (a)",
|
||||
QUERY_CONTEXT_UNNEST,
|
||||
|
@ -4294,7 +4284,6 @@ public class CalciteArraysQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testUnnestArrayColumnsDouble()
|
||||
{
|
||||
cannotVectorize();
|
||||
testQuery(
|
||||
"SELECT a FROM druid.arrays, UNNEST(arrayDouble) as unnested (a)",
|
||||
QUERY_CONTEXT_UNNEST,
|
||||
|
@ -4348,7 +4337,6 @@ public class CalciteArraysQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testUnnestArrayColumnsDoubleNulls()
|
||||
{
|
||||
cannotVectorize();
|
||||
testQuery(
|
||||
"SELECT a FROM druid.arrays, UNNEST(arrayDoubleNulls) as unnested (a)",
|
||||
QUERY_CONTEXT_UNNEST,
|
||||
|
@ -4401,7 +4389,6 @@ public class CalciteArraysQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testUnnestTwice()
|
||||
{
|
||||
cannotVectorize();
|
||||
testQuery(
|
||||
"SELECT dim1, MV_TO_ARRAY(dim3), STRING_TO_ARRAY(dim1, U&'\\005C.') AS dim1_split, dim1_split_unnest, dim3_unnest\n"
|
||||
+ "FROM\n"
|
||||
|
@ -4478,7 +4465,6 @@ public class CalciteArraysQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testUnnestTwiceArrayColumns()
|
||||
{
|
||||
cannotVectorize();
|
||||
testQuery(
|
||||
"SELECT arrayStringNulls, arrayLongNulls, usn, uln"
|
||||
+ " FROM\n"
|
||||
|
@ -4561,7 +4547,6 @@ public class CalciteArraysQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testUnnestTwiceWithFiltersAndExpressions()
|
||||
{
|
||||
cannotVectorize();
|
||||
testQuery(
|
||||
"SELECT dim1, MV_TO_ARRAY(dim3), STRING_TO_ARRAY(dim1, U&'\\005C.') AS dim1_split, dim1_split_unnest, dim3_unnest || 'xx'\n"
|
||||
+ "FROM\n"
|
||||
|
@ -4627,7 +4612,6 @@ public class CalciteArraysQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testUnnestThriceWithFiltersOnDimAndUnnestCol()
|
||||
{
|
||||
cannotVectorize();
|
||||
String sql = " SELECT dimZipf, dim3_unnest1, dim3_unnest2, dim3_unnest3 FROM \n"
|
||||
+ " ( SELECT * FROM \n"
|
||||
+ " ( SELECT * FROM lotsocolumns, UNNEST(MV_TO_ARRAY(dimMultivalEnumerated)) as ut(dim3_unnest1) )"
|
||||
|
@ -4725,7 +4709,6 @@ public class CalciteArraysQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testUnnestThriceWithFiltersOnDimAndAllUnnestColumns()
|
||||
{
|
||||
cannotVectorize();
|
||||
String sql = " SELECT dimZipf, dim3_unnest1, dim3_unnest2, dim3_unnest3 FROM \n"
|
||||
+ " ( SELECT * FROM \n"
|
||||
+ " ( SELECT * FROM lotsocolumns, UNNEST(MV_TO_ARRAY(dimMultivalEnumerated)) as ut(dim3_unnest1) )"
|
||||
|
@ -4794,7 +4777,6 @@ public class CalciteArraysQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testUnnestThriceWithFiltersOnDimAndAllUnnestColumnsArrayColumns()
|
||||
{
|
||||
cannotVectorize();
|
||||
String sql = " SELECT arrayString, uln, udn, usn FROM \n"
|
||||
+ " ( SELECT * FROM \n"
|
||||
+ " ( SELECT * FROM arrays, UNNEST(arrayLongNulls) as ut(uln))"
|
||||
|
@ -4862,7 +4844,6 @@ public class CalciteArraysQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testUnnestThriceWithFiltersOnDimAndUnnestColumnsORCombinations()
|
||||
{
|
||||
cannotVectorize();
|
||||
String sql = " SELECT dimZipf, dim3_unnest1, dim3_unnest2, dim3_unnest3 FROM \n"
|
||||
+ " ( SELECT * FROM \n"
|
||||
+ " ( SELECT * FROM lotsocolumns, UNNEST(MV_TO_ARRAY(dimMultivalEnumerated)) as ut(dim3_unnest1) )"
|
||||
|
@ -4942,7 +4923,6 @@ public class CalciteArraysQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testUnnestThriceWithFiltersOnDimAndAllUnnestColumnsArrayColumnsOrFilters()
|
||||
{
|
||||
cannotVectorize();
|
||||
String sql = " SELECT arrayString, uln, udn, usn FROM \n"
|
||||
+ " ( SELECT * FROM \n"
|
||||
+ " ( SELECT * FROM arrays, UNNEST(arrayLongNulls) as ut(uln))"
|
||||
|
@ -5137,7 +5117,6 @@ public class CalciteArraysQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testUnnestWithGroupByOrderByWithLimit()
|
||||
{
|
||||
cannotVectorize();
|
||||
testQuery(
|
||||
"SELECT d3, COUNT(*) FROM druid.numfoo, UNNEST(MV_TO_ARRAY(dim3)) AS unnested(d3) GROUP BY d3 ORDER BY d3 ASC LIMIT 4 ",
|
||||
QUERY_CONTEXT_UNNEST,
|
||||
|
@ -5213,7 +5192,6 @@ public class CalciteArraysQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testUnnestWithLimit()
|
||||
{
|
||||
cannotVectorize();
|
||||
testQuery(
|
||||
"SELECT d3 FROM druid.numfoo, UNNEST(MV_TO_ARRAY(dim3)) as unnested (d3) LIMIT 3",
|
||||
QUERY_CONTEXT_UNNEST,
|
||||
|
@ -5243,7 +5221,6 @@ public class CalciteArraysQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testUnnestFirstQueryOnSelect()
|
||||
{
|
||||
cannotVectorize();
|
||||
testQuery(
|
||||
"SELECT d3 FROM (select dim1, dim2, dim3 from druid.numfoo), UNNEST(MV_TO_ARRAY(dim3)) as unnested (d3)",
|
||||
QUERY_CONTEXT_UNNEST,
|
||||
|
@ -5288,7 +5265,6 @@ public class CalciteArraysQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testUnnestVirtualWithColumns1()
|
||||
{
|
||||
cannotVectorize();
|
||||
testQuery(
|
||||
"SELECT strings, m1 FROM druid.numfoo, UNNEST(MV_TO_ARRAY(dim3)) as unnested (strings) where (strings='a' and (m1<=10 or strings='b'))",
|
||||
QUERY_CONTEXT_UNNEST,
|
||||
|
@ -5329,7 +5305,6 @@ public class CalciteArraysQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testUnnestVirtualWithColumns2()
|
||||
{
|
||||
cannotVectorize();
|
||||
testQuery(
|
||||
"SELECT strings, m1 FROM druid.numfoo, UNNEST(MV_TO_ARRAY(dim3)) as unnested (strings) where (strings='a' or (m1=2 and strings='b'))",
|
||||
QUERY_CONTEXT_UNNEST,
|
||||
|
@ -5367,7 +5342,6 @@ public class CalciteArraysQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testUnnestWithFilters()
|
||||
{
|
||||
cannotVectorize();
|
||||
testQuery(
|
||||
"SELECT d3 FROM (select * from druid.numfoo where dim2='a'), UNNEST(MV_TO_ARRAY(dim3)) as unnested (d3)",
|
||||
QUERY_CONTEXT_UNNEST,
|
||||
|
@ -5399,7 +5373,6 @@ public class CalciteArraysQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testUnnestWithFiltersWithExpressionInInnerQuery()
|
||||
{
|
||||
cannotVectorize();
|
||||
testQuery(
|
||||
"SELECT t,d3 FROM (select FLOOR(__time to hour) t, dim3 from druid.numfoo where dim2='a'), UNNEST(MV_TO_ARRAY(dim3)) as unnested (d3)",
|
||||
QUERY_CONTEXT_UNNEST,
|
||||
|
@ -5436,7 +5409,6 @@ public class CalciteArraysQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testUnnestWithInFiltersWithExpressionInInnerQuery()
|
||||
{
|
||||
cannotVectorize();
|
||||
testQuery(
|
||||
"SELECT t,d3 FROM (select FLOOR(__time to hour) t, dim3 from druid.numfoo where dim2 IN ('a','b')), UNNEST(MV_TO_ARRAY(dim3)) as unnested (d3)",
|
||||
QUERY_CONTEXT_UNNEST,
|
||||
|
@ -5471,7 +5443,6 @@ public class CalciteArraysQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testUnnestWithFiltersInnerLimit()
|
||||
{
|
||||
cannotVectorize();
|
||||
testQuery(
|
||||
"SELECT d3 FROM (select dim2,dim3 from druid.numfoo where dim2='a' LIMIT 2), UNNEST(MV_TO_ARRAY(dim3)) as unnested (d3)",
|
||||
QUERY_CONTEXT_UNNEST,
|
||||
|
@ -5518,7 +5489,6 @@ public class CalciteArraysQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testUnnestWithFiltersInsideAndOutside()
|
||||
{
|
||||
cannotVectorize();
|
||||
testQuery(
|
||||
"SELECT d3 FROM\n"
|
||||
+ " (select * from druid.numfoo where dim2='a') as t,\n"
|
||||
|
@ -5556,7 +5526,6 @@ public class CalciteArraysQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testUnnestWithFiltersInsideAndOutside1()
|
||||
{
|
||||
cannotVectorize();
|
||||
testQuery(
|
||||
"SELECT d3 FROM\n"
|
||||
+ " (select * from druid.numfoo where dim2='a'),\n"
|
||||
|
@ -5597,7 +5566,6 @@ public class CalciteArraysQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testUnnestWithFiltersOutside()
|
||||
{
|
||||
cannotVectorize();
|
||||
testQuery(
|
||||
"SELECT d3 FROM\n"
|
||||
+ " druid.numfoo t,\n"
|
||||
|
@ -5639,7 +5607,6 @@ public class CalciteArraysQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testUnnestWithInFilters()
|
||||
{
|
||||
cannotVectorize();
|
||||
testQuery(
|
||||
"SELECT d3 FROM (select * from druid.numfoo where dim2 IN ('a','b','ab','abc')), UNNEST(MV_TO_ARRAY(dim3)) as unnested (d3)",
|
||||
QUERY_CONTEXT_UNNEST,
|
||||
|
@ -5673,7 +5640,6 @@ public class CalciteArraysQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testUnnestVirtualWithColumns()
|
||||
{
|
||||
cannotVectorize();
|
||||
testQuery(
|
||||
"SELECT strings FROM druid.numfoo, UNNEST(ARRAY[dim4, dim5]) as unnested (strings)",
|
||||
QUERY_CONTEXT_UNNEST,
|
||||
|
@ -5766,7 +5732,6 @@ public class CalciteArraysQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testUnnestWithJoinOnTheLeft()
|
||||
{
|
||||
cannotVectorize();
|
||||
testQuery(
|
||||
"SELECT d3 from (SELECT * from druid.numfoo JOIN (select dim2 as t from druid.numfoo where dim2 IN ('a','b','ab','abc')) ON dim2=t), UNNEST(MV_TO_ARRAY(dim3)) as unnested (d3)",
|
||||
QUERY_CONTEXT_UNNEST,
|
||||
|
@ -5830,7 +5795,6 @@ public class CalciteArraysQueryTest extends BaseCalciteQueryTest
|
|||
// Since there is a constant on the right,
|
||||
// Druid will plan this as a join query
|
||||
// as there is nothing to correlate between left and right
|
||||
cannotVectorize();
|
||||
testQuery(
|
||||
"SELECT longs FROM druid.numfoo, UNNEST(ARRAY[1,2,3]) as unnested (longs)",
|
||||
QUERY_CONTEXT_UNNEST,
|
||||
|
@ -5885,7 +5849,6 @@ public class CalciteArraysQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testUnnestWithSQLFunctionOnUnnestedColumn()
|
||||
{
|
||||
cannotVectorize();
|
||||
testQuery(
|
||||
"SELECT strlen(d3) FROM druid.numfoo, UNNEST(MV_TO_ARRAY(dim3)) as unnested (d3)",
|
||||
QUERY_CONTEXT_UNNEST,
|
||||
|
@ -5931,7 +5894,6 @@ public class CalciteArraysQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testUnnestWithINFiltersWithLeftRewrite()
|
||||
{
|
||||
cannotVectorize();
|
||||
testQuery(
|
||||
"SELECT d3 FROM druid.numfoo, UNNEST(MV_TO_ARRAY(dim3)) as unnested (d3) where d3 IN ('a','b') and m1 < 10",
|
||||
QUERY_CONTEXT_UNNEST,
|
||||
|
@ -5963,7 +5925,6 @@ public class CalciteArraysQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testUnnestWithINFiltersWithNoLeftRewrite()
|
||||
{
|
||||
cannotVectorize();
|
||||
testQuery(
|
||||
"SELECT d45 FROM druid.numfoo, UNNEST(ARRAY[dim4,dim5]) as unnested (d45) where d45 IN ('a','b')",
|
||||
QUERY_CONTEXT_UNNEST,
|
||||
|
@ -5995,7 +5956,6 @@ public class CalciteArraysQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testUnnestWithInvalidINFiltersOnUnnestedColumn()
|
||||
{
|
||||
cannotVectorize();
|
||||
testQuery(
|
||||
"SELECT d3 FROM druid.numfoo, UNNEST(MV_TO_ARRAY(dim3)) as unnested (d3) where d3 IN ('foo','bar')",
|
||||
QUERY_CONTEXT_UNNEST,
|
||||
|
@ -6020,7 +5980,6 @@ public class CalciteArraysQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testUnnestWithNotFiltersOnUnnestedColumn()
|
||||
{
|
||||
cannotVectorize();
|
||||
testQuery(
|
||||
"SELECT d3 FROM druid.numfoo, UNNEST(MV_TO_ARRAY(dim3)) as unnested (d3) where d3!='d' ",
|
||||
QUERY_CONTEXT_UNNEST,
|
||||
|
@ -6061,7 +6020,6 @@ public class CalciteArraysQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testUnnestWithSelectorFiltersOnSelectedColumn()
|
||||
{
|
||||
cannotVectorize();
|
||||
testQuery(
|
||||
"SELECT d3 FROM druid.numfoo, UNNEST(MV_TO_ARRAY(dim3)) as unnested (d3) where d3='b'",
|
||||
QUERY_CONTEXT_UNNEST,
|
||||
|
@ -6089,7 +6047,6 @@ public class CalciteArraysQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testUnnestWithSelectorFiltersOnVirtualColumn()
|
||||
{
|
||||
cannotVectorize();
|
||||
testQuery(
|
||||
"SELECT d12 FROM druid.numfoo, UNNEST(ARRAY[m1,m2]) as unnested (d12) where d12=1",
|
||||
QUERY_CONTEXT_UNNEST,
|
||||
|
@ -6119,7 +6076,6 @@ public class CalciteArraysQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testUnnestWithSelectorFiltersOnVirtualStringColumn()
|
||||
{
|
||||
cannotVectorize();
|
||||
testQuery(
|
||||
"SELECT d45 FROM druid.numfoo, UNNEST(ARRAY[dim4,dim5]) as unnested (d45) where d45 IN ('a','ab')",
|
||||
QUERY_CONTEXT_UNNEST,
|
||||
|
@ -6150,7 +6106,6 @@ public class CalciteArraysQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testUnnestWithMultipleAndFiltersOnSelectedColumns()
|
||||
{
|
||||
cannotVectorize();
|
||||
testQuery(
|
||||
"SELECT d3 FROM druid.numfoo, UNNEST(MV_TO_ARRAY(dim3)) as unnested (d3) where d3='b' and m1 < 10 and m2 < 10",
|
||||
QUERY_CONTEXT_UNNEST,
|
||||
|
@ -6184,7 +6139,6 @@ public class CalciteArraysQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testUnnestWithMultipleOrFiltersOnSelectedColumns()
|
||||
{
|
||||
cannotVectorize();
|
||||
testQuery(
|
||||
"SELECT d3 FROM druid.numfoo, UNNEST(MV_TO_ARRAY(dim3)) as unnested (d3) where d3='b' or m1 < 2 ",
|
||||
QUERY_CONTEXT_UNNEST,
|
||||
|
@ -6219,7 +6173,6 @@ public class CalciteArraysQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testUnnestWithMultipleAndFiltersOnSelectedUnnestedColumns()
|
||||
{
|
||||
cannotVectorize();
|
||||
testQuery(
|
||||
"SELECT d3 FROM druid.numfoo, UNNEST(MV_TO_ARRAY(dim3)) as unnested (d3) where d3 IN ('a','b') and d3 < 'e' ",
|
||||
QUERY_CONTEXT_UNNEST,
|
||||
|
@ -6248,7 +6201,6 @@ public class CalciteArraysQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testUnnestWithMultipleOrFiltersOnUnnestedColumns()
|
||||
{
|
||||
cannotVectorize();
|
||||
testQuery(
|
||||
"SELECT d3 FROM druid.numfoo, UNNEST(MV_TO_ARRAY(dim3)) as unnested (d3) where d3='b' or d3='d' ",
|
||||
QUERY_CONTEXT_UNNEST,
|
||||
|
@ -6277,7 +6229,6 @@ public class CalciteArraysQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testUnnestWithMultipleOrFiltersOnVariationsOfUnnestedColumns()
|
||||
{
|
||||
cannotVectorize();
|
||||
testQuery(
|
||||
"SELECT d3 FROM druid.numfoo, UNNEST(MV_TO_ARRAY(dim3)) as unnested (d3) where strlen(d3) < 2 or d3='d' ",
|
||||
QUERY_CONTEXT_UNNEST,
|
||||
|
@ -6323,7 +6274,6 @@ public class CalciteArraysQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testUnnestWithMultipleOrFiltersOnSelectedNonUnnestedColumns()
|
||||
{
|
||||
cannotVectorize();
|
||||
testQuery(
|
||||
"SELECT d3 FROM druid.numfoo, UNNEST(MV_TO_ARRAY(dim3)) as unnested (d3) where m1 < 2 or m2 < 2 ",
|
||||
QUERY_CONTEXT_UNNEST,
|
||||
|
@ -6357,7 +6307,6 @@ public class CalciteArraysQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testUnnestWithMultipleOrFiltersOnSelectedVirtualColumns()
|
||||
{
|
||||
cannotVectorize();
|
||||
testQuery(
|
||||
"SELECT d45 FROM druid.numfoo, UNNEST(ARRAY[dim4,dim5]) as unnested (d45) where d45 IN ('a','aa') or m1 < 2 ",
|
||||
QUERY_CONTEXT_UNNEST,
|
||||
|
@ -6394,7 +6343,6 @@ public class CalciteArraysQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testUnnestWithMultipleOrFiltersOnUnnestedColumnsAndOnOriginalColumn()
|
||||
{
|
||||
cannotVectorize();
|
||||
testQuery(
|
||||
"SELECT d3 FROM druid.numfoo, UNNEST(MV_TO_ARRAY(dim3)) as unnested (d3) where d3='b' or dim3='d' ",
|
||||
QUERY_CONTEXT_UNNEST,
|
||||
|
@ -6429,7 +6377,6 @@ public class CalciteArraysQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testUnnestWithMultipleOrFiltersOnUnnestedColumnsAndOnOriginalColumnDiffOrdering()
|
||||
{
|
||||
cannotVectorize();
|
||||
testQuery(
|
||||
"SELECT dim3, d3 FROM druid.numfoo, UNNEST(MV_TO_ARRAY(dim3)) as unnested (d3) where dim3='b' or d3='a' ",
|
||||
QUERY_CONTEXT_UNNEST,
|
||||
|
@ -6717,7 +6664,6 @@ public class CalciteArraysQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testUnnestVirtualWithColumnsAndNullIf()
|
||||
{
|
||||
cannotVectorize();
|
||||
testQuery(
|
||||
"select c,m2 from druid.foo, unnest(ARRAY[\"m1\", \"m2\"]) as u(c) where NULLIF(c,m2) IS NULL",
|
||||
QUERY_CONTEXT_UNNEST,
|
||||
|
@ -7361,7 +7307,6 @@ public class CalciteArraysQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testUnnestExtractionFn()
|
||||
{
|
||||
cannotVectorize();
|
||||
testQuery(
|
||||
"SELECT substring(d3,1) FROM druid.numfoo, UNNEST(MV_TO_ARRAY(dim3)) as unnested (d3) WHERE substring(d3,1) <> 'b'",
|
||||
QUERY_CONTEXT_UNNEST,
|
||||
|
@ -7402,7 +7347,6 @@ public class CalciteArraysQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testUnnestExtractionFnNull()
|
||||
{
|
||||
cannotVectorize();
|
||||
testQuery(
|
||||
"SELECT substring(d3,1) FROM druid.numfoo, UNNEST(MV_TO_ARRAY(dim3)) as unnested (d3) WHERE substring(d3,1) is not null",
|
||||
QUERY_CONTEXT_UNNEST,
|
||||
|
|
|
@ -4226,9 +4226,6 @@ public class CalciteJoinQueryTest extends BaseCalciteQueryTest
|
|||
@NotYetSupported(Modes.JOIN_CONDITION_NOT_PUSHED_CONDITION)
|
||||
public void testSemiJoinWithOuterTimeExtractAggregateWithOrderBy()
|
||||
{
|
||||
// Cannot vectorize due to virtual columns.
|
||||
cannotVectorize();
|
||||
|
||||
testQuery(
|
||||
"SELECT COUNT(DISTINCT dim1), EXTRACT(MONTH FROM __time) FROM druid.foo\n"
|
||||
+ " WHERE dim2 IN (\n"
|
||||
|
@ -4699,7 +4696,6 @@ public class CalciteJoinQueryTest extends BaseCalciteQueryTest
|
|||
{
|
||||
// Native JOIN operator cannot handle the condition, so a SQL JOIN with greater-than is translated into a
|
||||
// cross join with a filter.
|
||||
cannotVectorize();
|
||||
|
||||
// We don't handle non-equi join conditions for non-sql compatible mode.
|
||||
assumeFalse(NullHandling.replaceWithDefault());
|
||||
|
@ -4762,7 +4758,6 @@ public class CalciteJoinQueryTest extends BaseCalciteQueryTest
|
|||
{
|
||||
// Native JOIN operator cannot handle the condition, so a SQL JOIN with greater-than is translated into a
|
||||
// cross join with a filter.
|
||||
cannotVectorize();
|
||||
|
||||
// We don't handle non-equi join conditions for non-sql compatible mode.
|
||||
assumeFalse(NullHandling.replaceWithDefault());
|
||||
|
|
|
@ -84,8 +84,6 @@ public class CalciteLookupFunctionQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testFilterEquals()
|
||||
{
|
||||
cannotVectorize();
|
||||
|
||||
testQuery(
|
||||
buildFilterTestSql("LOOKUP(dim1, 'lookyloo') = 'xabc'"),
|
||||
QUERY_CONTEXT,
|
||||
|
@ -129,8 +127,6 @@ public class CalciteLookupFunctionQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testFilterLookupOfConcat()
|
||||
{
|
||||
cannotVectorize();
|
||||
|
||||
testQuery(
|
||||
buildFilterTestSql("LOOKUP(CONCAT(dim1, 'b', dim2), 'lookyloo') = 'xabc'"),
|
||||
QUERY_CONTEXT,
|
||||
|
@ -196,8 +192,6 @@ public class CalciteLookupFunctionQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testFilterConcatOfLookup()
|
||||
{
|
||||
cannotVectorize();
|
||||
|
||||
testQuery(
|
||||
buildFilterTestSql("CONCAT(LOOKUP(dim1, 'lookyloo'), ' (', dim1, ')') = 'xabc (abc)'"),
|
||||
QUERY_CONTEXT,
|
||||
|
@ -235,8 +229,6 @@ public class CalciteLookupFunctionQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testFilterConcatOfLookupOfConcat()
|
||||
{
|
||||
cannotVectorize();
|
||||
|
||||
testQuery(
|
||||
buildFilterTestSql(
|
||||
"CONCAT(LOOKUP(CONCAT(dim1, 'b', dim2), 'lookyloo'), ' (', CONCAT(dim1, 'b', dim2), ')') = 'xabc (abc)'"),
|
||||
|
@ -291,8 +283,6 @@ public class CalciteLookupFunctionQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testFilterConcatOfCoalesceLookupOfConcat()
|
||||
{
|
||||
cannotVectorize();
|
||||
|
||||
testQuery(
|
||||
buildFilterTestSql(
|
||||
"CONCAT(COALESCE(LOOKUP(CONCAT(dim1, 'b', dim2), 'lookyloo'), 'N/A'), ' (', CONCAT(dim1, 'b', dim2), ')') = 'xabc (abc)'"),
|
||||
|
@ -311,8 +301,6 @@ public class CalciteLookupFunctionQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testFilterImpossibleLookupOfConcat()
|
||||
{
|
||||
cannotVectorize();
|
||||
|
||||
// No keys in the lookup table begin with 'key:', so this is always false.
|
||||
testQuery(
|
||||
buildFilterTestSql("LOOKUP('key:' || dim1, 'lookyloo') = 'xabc'"),
|
||||
|
@ -325,8 +313,6 @@ public class CalciteLookupFunctionQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testFilterChainedEquals()
|
||||
{
|
||||
cannotVectorize();
|
||||
|
||||
testQuery(
|
||||
buildFilterTestSql("LOOKUP(LOOKUP(dim1, 'lookyloo'), 'lookyloo-chain') = 'zabc'"),
|
||||
QUERY_CONTEXT,
|
||||
|
@ -338,8 +324,6 @@ public class CalciteLookupFunctionQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testFilterEqualsLiteralFirst()
|
||||
{
|
||||
cannotVectorize();
|
||||
|
||||
testQuery(
|
||||
buildFilterTestSql("'xabc' = LOOKUP(dim1, 'lookyloo')"),
|
||||
QUERY_CONTEXT,
|
||||
|
@ -351,8 +335,6 @@ public class CalciteLookupFunctionQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testFilterEqualsAlwaysFalse()
|
||||
{
|
||||
cannotVectorize();
|
||||
|
||||
testQuery(
|
||||
buildFilterTestSql("LOOKUP(dim1, 'lookyloo') = 'nonexistent'"),
|
||||
QUERY_CONTEXT,
|
||||
|
@ -364,8 +346,6 @@ public class CalciteLookupFunctionQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testFilterIsNotDistinctFrom()
|
||||
{
|
||||
cannotVectorize();
|
||||
|
||||
testQuery(
|
||||
buildFilterTestSql("LOOKUP(dim1, 'lookyloo') IS NOT DISTINCT FROM 'xabc'"),
|
||||
QUERY_CONTEXT,
|
||||
|
@ -552,7 +532,9 @@ public class CalciteLookupFunctionQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testFilterInOrIsNullInjective()
|
||||
{
|
||||
if (NullHandling.sqlCompatible()) {
|
||||
cannotVectorize();
|
||||
}
|
||||
|
||||
testQuery(
|
||||
buildFilterTestSql(
|
||||
|
@ -825,7 +807,10 @@ public class CalciteLookupFunctionQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testFilterMvContainsNullInjective()
|
||||
{
|
||||
if (NullHandling.sqlCompatible()) {
|
||||
// cannotVectorize doesn't trip in default-value mode, due to buildFilterTestExpectedQueryAlwaysFalse.
|
||||
cannotVectorize();
|
||||
}
|
||||
|
||||
testQuery(
|
||||
buildFilterTestSql("MV_CONTAINS(LOOKUP(dim1, 'lookyloo121'), NULL)"),
|
||||
|
@ -1045,7 +1030,10 @@ public class CalciteLookupFunctionQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testFilterIsNullInjective()
|
||||
{
|
||||
if (NullHandling.sqlCompatible()) {
|
||||
// cannotVectorize doesn't trip in default-value mode, due to buildFilterTestExpectedQueryAlwaysFalse.
|
||||
cannotVectorize();
|
||||
}
|
||||
|
||||
testQuery(
|
||||
buildFilterTestSql("LOOKUP(dim1, 'lookyloo121') IS NULL"),
|
||||
|
@ -1314,7 +1302,10 @@ public class CalciteLookupFunctionQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testFilterCoalesceSameLiteralInjective()
|
||||
{
|
||||
if (NullHandling.sqlCompatible()) {
|
||||
// cannotVectorize doesn't trip in default-value mode, due to buildFilterTestExpectedQueryAlwaysFalse.
|
||||
cannotVectorize();
|
||||
}
|
||||
|
||||
testQuery(
|
||||
buildFilterTestSql("COALESCE(LOOKUP(dim1, 'lookyloo121'), 'x6') = 'x6'"),
|
||||
|
@ -1450,8 +1441,6 @@ public class CalciteLookupFunctionQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testFilterCoalesceDifferentLiteral()
|
||||
{
|
||||
cannotVectorize();
|
||||
|
||||
testQuery(
|
||||
buildFilterTestSql("COALESCE(LOOKUP(dim1, 'lookyloo'), 'xyzzy') = 'x6'"),
|
||||
QUERY_CONTEXT,
|
||||
|
@ -1463,8 +1452,6 @@ public class CalciteLookupFunctionQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testFilterCoalesceDifferentLiteralAlwaysFalse()
|
||||
{
|
||||
cannotVectorize();
|
||||
|
||||
testQuery(
|
||||
buildFilterTestSql("COALESCE(LOOKUP(dim1, 'lookyloo'), 'xyzzy') = 'nonexistent'"),
|
||||
QUERY_CONTEXT,
|
||||
|
@ -1476,8 +1463,6 @@ public class CalciteLookupFunctionQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testFilterCoalesceCastVarcharDifferentLiteral()
|
||||
{
|
||||
cannotVectorize();
|
||||
|
||||
testQuery(
|
||||
buildFilterTestSql("COALESCE(CAST(LOOKUP(dim1, 'lookyloo') AS VARCHAR), 'xyzzy') = 'x6'"),
|
||||
QUERY_CONTEXT,
|
||||
|
@ -1953,8 +1938,6 @@ public class CalciteLookupFunctionQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testDontPullUpLookupWhenUsedByAggregation()
|
||||
{
|
||||
cannotVectorize();
|
||||
|
||||
testQuery(
|
||||
"SELECT LOOKUP(dim1, 'lookyloo121'), COUNT(LOOKUP(dim1, 'lookyloo121')) FROM druid.foo GROUP BY 1",
|
||||
ImmutableList.of(
|
||||
|
|
|
@ -1122,9 +1122,6 @@ public class CalciteMultiValueStringQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testSelectAndFilterByStringToMV()
|
||||
{
|
||||
// Cannot vectorize due to usage of expressions.
|
||||
cannotVectorize();
|
||||
|
||||
testBuilder()
|
||||
.sql("SELECT STRING_TO_MV(CONCAT(MV_TO_STRING(dim3, ','), ',d'), ',') FROM druid.numfoo "
|
||||
+ "WHERE MV_CONTAINS(STRING_TO_MV(CONCAT(MV_TO_STRING(dim3, ','), ',d'), ','), 'd')")
|
||||
|
@ -1167,9 +1164,6 @@ public class CalciteMultiValueStringQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testStringToMVOfConstant()
|
||||
{
|
||||
// Cannot vectorize due to usage of expressions.
|
||||
cannotVectorize();
|
||||
|
||||
testBuilder()
|
||||
.sql("SELECT m1, STRING_TO_MV('a,b', ',') AS mv FROM druid.numfoo GROUP BY 1")
|
||||
.expectedQuery(
|
||||
|
@ -1339,9 +1333,6 @@ public class CalciteMultiValueStringQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testMultiValueListFilterNonLiteral()
|
||||
{
|
||||
// Cannot vectorize due to usage of expressions.
|
||||
cannotVectorize();
|
||||
|
||||
testQuery(
|
||||
"SELECT MV_FILTER_ONLY(dim3, ARRAY[dim2]) FROM druid.numfoo",
|
||||
ImmutableList.of(
|
||||
|
@ -1429,9 +1420,6 @@ public class CalciteMultiValueStringQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testMultiValueListFilterDenyNonLiteral()
|
||||
{
|
||||
// Cannot vectorize due to usage of expressions.
|
||||
cannotVectorize();
|
||||
|
||||
testQuery(
|
||||
"SELECT MV_FILTER_NONE(dim3, ARRAY[dim2]) FROM druid.numfoo",
|
||||
ImmutableList.of(
|
||||
|
@ -2170,7 +2158,6 @@ public class CalciteMultiValueStringQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testMultiValueStringOverlapFilterCoalesceNvl()
|
||||
{
|
||||
cannotVectorize();
|
||||
testQuery(
|
||||
"SELECT COALESCE(dim3, 'other') FROM druid.numfoo "
|
||||
+ "WHERE MV_OVERLAP(COALESCE(MV_TO_ARRAY(dim3), ARRAY['other']), ARRAY['a', 'b', 'other']) OR "
|
||||
|
@ -2392,7 +2379,6 @@ public class CalciteMultiValueStringQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testMvContainsSelectColumns()
|
||||
{
|
||||
cannotVectorize();
|
||||
testQuery(
|
||||
"SELECT MV_CONTAINS(dim3, ARRAY['a', 'b']), MV_OVERLAP(dim3, ARRAY['a', 'b']) FROM druid.numfoo LIMIT 5",
|
||||
ImmutableList.of(
|
||||
|
|
|
@ -1314,7 +1314,6 @@ public class CalciteNestedDataQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testUnnestRootSingleTypeArrayStringNulls()
|
||||
{
|
||||
cannotVectorize();
|
||||
testBuilder()
|
||||
.sql("SELECT strings FROM druid.arrays, UNNEST(arrayStringNulls) as u(strings)")
|
||||
.queryContext(QUERY_CONTEXT_NO_STRINGIFY_ARRAY)
|
||||
|
@ -1372,7 +1371,6 @@ public class CalciteNestedDataQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testUnnestRootSingleTypeArrayDoubleNulls()
|
||||
{
|
||||
cannotVectorize();
|
||||
testBuilder()
|
||||
.sql("SELECT doubles FROM druid.arrays, UNNEST(arrayDoubleNulls) as u(doubles)")
|
||||
.queryContext(QUERY_CONTEXT_NO_STRINGIFY_ARRAY)
|
||||
|
@ -2280,7 +2278,6 @@ public class CalciteNestedDataQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testGroupByRootSingleTypeArrayLongElement()
|
||||
{
|
||||
cannotVectorize();
|
||||
testBuilder()
|
||||
.sql(
|
||||
"SELECT "
|
||||
|
@ -2328,7 +2325,6 @@ public class CalciteNestedDataQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testGroupByRootSingleTypeArrayLongElementFiltered()
|
||||
{
|
||||
cannotVectorize();
|
||||
testBuilder()
|
||||
.sql(
|
||||
"SELECT "
|
||||
|
@ -2376,7 +2372,6 @@ public class CalciteNestedDataQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testGroupByRootSingleTypeArrayLongElementDefault()
|
||||
{
|
||||
cannotVectorize();
|
||||
testBuilder()
|
||||
.sql(
|
||||
"SELECT "
|
||||
|
@ -2692,7 +2687,6 @@ public class CalciteNestedDataQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testGroupByPathSelectorFilterCoalesce()
|
||||
{
|
||||
cannotVectorize();
|
||||
testQuery(
|
||||
"SELECT "
|
||||
+ "JSON_VALUE(nest, '$.x'), "
|
||||
|
@ -6821,7 +6815,6 @@ public class CalciteNestedDataQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testJsonQueryDynamicArg()
|
||||
{
|
||||
cannotVectorize();
|
||||
testQuery(
|
||||
"SELECT JSON_PATHS(nester), JSON_QUERY(nester, ARRAY_OFFSET(JSON_PATHS(nester), 0))\n"
|
||||
+ "FROM druid.nested",
|
||||
|
@ -6866,7 +6859,6 @@ public class CalciteNestedDataQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testJsonQueryArrays()
|
||||
{
|
||||
cannotVectorize();
|
||||
testBuilder()
|
||||
.sql("SELECT JSON_QUERY_ARRAY(arrayObject, '$') FROM druid.arrays")
|
||||
.queryContext(QUERY_CONTEXT_DEFAULT)
|
||||
|
@ -6920,7 +6912,6 @@ public class CalciteNestedDataQueryTest extends BaseCalciteQueryTest
|
|||
{
|
||||
// Array complex JSON isn't supported
|
||||
msqIncompatible();
|
||||
cannotVectorize();
|
||||
testBuilder()
|
||||
.sql("SELECT JSON_QUERY_ARRAY(arrayObject, '$.') FROM druid.arrays where arrayObject is null limit 1")
|
||||
.queryContext(QUERY_CONTEXT_DEFAULT)
|
||||
|
@ -6960,7 +6951,6 @@ public class CalciteNestedDataQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testUnnestJsonQueryArrays()
|
||||
{
|
||||
cannotVectorize();
|
||||
testBuilder()
|
||||
.sql("SELECT objects FROM druid.arrays, UNNEST(JSON_QUERY_ARRAY(arrayObject, '$')) as u(objects)")
|
||||
.queryContext(QUERY_CONTEXT_NO_STRINGIFY_ARRAY)
|
||||
|
|
|
@ -652,10 +652,6 @@ public class CalciteParameterQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testWrongTypeParameter()
|
||||
{
|
||||
if (!useDefault) {
|
||||
// cannot vectorize inline datasource
|
||||
cannotVectorize();
|
||||
}
|
||||
testQuery(
|
||||
"SELECT COUNT(*)\n"
|
||||
+ "FROM druid.numfoo\n"
|
||||
|
@ -701,7 +697,6 @@ public class CalciteParameterQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testNullParameter()
|
||||
{
|
||||
cannotVectorize();
|
||||
// contrived example of using null as an sql parameter to at least test the codepath because lots of things dont
|
||||
// actually work as null and things like 'IS NULL' fail to parse in calcite if expressed as 'IS ?'
|
||||
|
||||
|
|
|
@ -581,7 +581,6 @@ public class CalciteQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testSafeDivide()
|
||||
{
|
||||
cannotVectorize();
|
||||
final Map<String, Object> context = new HashMap<>(QUERY_CONTEXT_DEFAULT);
|
||||
|
||||
testQuery(
|
||||
|
@ -605,12 +604,9 @@ public class CalciteQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testDiv()
|
||||
{
|
||||
cannotVectorize();
|
||||
final Map<String, Object> context = new HashMap<>(QUERY_CONTEXT_DEFAULT);
|
||||
|
||||
testQuery(
|
||||
"select cnt, m1, div(m1, 2), div(cnt+2, cnt+1) from foo",
|
||||
context,
|
||||
QUERY_CONTEXT_DEFAULT,
|
||||
ImmutableList.of(
|
||||
newScanQueryBuilder()
|
||||
.dataSource(CalciteTests.DATASOURCE1)
|
||||
|
@ -2635,11 +2631,6 @@ public class CalciteQueryTest extends BaseCalciteQueryTest
|
|||
)
|
||||
);
|
||||
|
||||
if (NullHandling.sqlCompatible()) {
|
||||
// Cannot vectorize due to "istrue" operator.
|
||||
cannotVectorize();
|
||||
}
|
||||
|
||||
testQuery(
|
||||
PLANNER_CONFIG_NO_HLL.withOverrides(
|
||||
ImmutableMap.of(
|
||||
|
@ -3127,9 +3118,6 @@ public class CalciteQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testGroupByCaseWhen()
|
||||
{
|
||||
// Cannot vectorize due to virtual columns.
|
||||
cannotVectorize();
|
||||
|
||||
testQuery(
|
||||
"SELECT\n"
|
||||
+ " CASE EXTRACT(DAY FROM __time)\n"
|
||||
|
@ -3180,9 +3168,6 @@ public class CalciteQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testDecomposeCaseWhenThreeArg()
|
||||
{
|
||||
// Cannot vectorize due to virtual columns.
|
||||
cannotVectorize();
|
||||
|
||||
testQuery(
|
||||
"SELECT\n"
|
||||
+ " dim1, dim2, CASE WHEN dim1 = 'abc' THEN dim1 ELSE dim2 END\n"
|
||||
|
@ -3224,9 +3209,6 @@ public class CalciteQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testDecomposeCaseWhenTwoArg()
|
||||
{
|
||||
// Cannot vectorize due to virtual columns.
|
||||
cannotVectorize();
|
||||
|
||||
testQuery(
|
||||
"SELECT\n"
|
||||
+ " dim1, dim2, CASE WHEN dim1 = 'def' THEN dim2 END\n"
|
||||
|
@ -3259,9 +3241,6 @@ public class CalciteQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testGroupByCaseWhenOfTripleAnd()
|
||||
{
|
||||
// Cannot vectorize due to virtual columns.
|
||||
cannotVectorize();
|
||||
|
||||
testQuery(
|
||||
"SELECT\n"
|
||||
+ " CASE WHEN m1 > 1 AND m1 < 5 AND cnt = 1 THEN 'x' ELSE NULL END,"
|
||||
|
@ -3858,10 +3837,6 @@ public class CalciteQueryTest extends BaseCalciteQueryTest
|
|||
{
|
||||
// Doesn't conform to the SQL standard, but it's how we do it.
|
||||
// This example is used in the sql.md doc.
|
||||
|
||||
// Cannot vectorize due to virtual columns.
|
||||
cannotVectorize();
|
||||
|
||||
testQuery(
|
||||
"SELECT COALESCE(dim2, dim1), COUNT(*) FROM druid.foo GROUP BY COALESCE(dim2, dim1)\n",
|
||||
ImmutableList.of(
|
||||
|
@ -3900,9 +3875,6 @@ public class CalciteQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testCoalesceColumnsFilter()
|
||||
{
|
||||
// Cannot vectorize due to virtual columns.
|
||||
cannotVectorize();
|
||||
|
||||
testQuery(
|
||||
"SELECT COALESCE(dim2, dim1), COUNT(*) FROM druid.foo WHERE COALESCE(dim2, dim1) IN ('a', 'abc') GROUP BY COALESCE(dim2, dim1)\n",
|
||||
ImmutableList.of(
|
||||
|
@ -3936,9 +3908,6 @@ public class CalciteQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testCoalesceColumnsFilterWithEquality()
|
||||
{
|
||||
// Cannot vectorize due to virtual columns.
|
||||
cannotVectorize();
|
||||
|
||||
// we can remove this test if PlannerContext.CTX_SQL_USE_BOUNDS_AND_SELECTORS ever defaults to false all the time
|
||||
// since it otherwise is a duplicate of testCoalesceColumnsFilter
|
||||
|
||||
|
@ -4389,10 +4358,6 @@ public class CalciteQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testCountStarOnCommonTableExpression()
|
||||
{
|
||||
if (NullHandling.sqlCompatible()) {
|
||||
// cannot vectorize due to substring expression
|
||||
cannotVectorize();
|
||||
}
|
||||
Druids.TimeseriesQueryBuilder builder =
|
||||
Druids.newTimeseriesQueryBuilder()
|
||||
.dataSource(CalciteTests.DATASOURCE1)
|
||||
|
@ -4430,11 +4395,6 @@ public class CalciteQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testCountStarOnView()
|
||||
{
|
||||
if (NullHandling.sqlCompatible()) {
|
||||
// cannot vectorize due to substring expression
|
||||
cannotVectorize();
|
||||
}
|
||||
|
||||
Druids.TimeseriesQueryBuilder builder =
|
||||
Druids.newTimeseriesQueryBuilder()
|
||||
.dataSource(CalciteTests.DATASOURCE1)
|
||||
|
@ -4471,10 +4431,6 @@ public class CalciteQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testConfusedView()
|
||||
{
|
||||
if (NullHandling.sqlCompatible()) {
|
||||
// cannot vectorize due to substring expression
|
||||
cannotVectorize();
|
||||
}
|
||||
Druids.TimeseriesQueryBuilder builder =
|
||||
Druids.newTimeseriesQueryBuilder()
|
||||
.dataSource(CalciteTests.DATASOURCE1)
|
||||
|
@ -5065,10 +5021,6 @@ public class CalciteQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testFilteredAggregations()
|
||||
{
|
||||
if (NullHandling.sqlCompatible()) {
|
||||
// cannot vectorize due to expression filter
|
||||
cannotVectorize();
|
||||
}
|
||||
Druids.TimeseriesQueryBuilder builder =
|
||||
Druids.newTimeseriesQueryBuilder()
|
||||
.dataSource(CalciteTests.DATASOURCE1)
|
||||
|
@ -5318,9 +5270,6 @@ public class CalciteQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testExpressionAggregations()
|
||||
{
|
||||
// Cannot vectorize due to expressions.
|
||||
cannotVectorize();
|
||||
|
||||
final ExprMacroTable macroTable = CalciteTests.createExprMacroTable();
|
||||
|
||||
testQuery(
|
||||
|
@ -5785,9 +5734,6 @@ public class CalciteQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testInExpressionBelowThreshold()
|
||||
{
|
||||
// Cannot vectorize scalar_in_array expression.
|
||||
cannotVectorize();
|
||||
|
||||
testQuery(
|
||||
"SELECT dim1 IN ('abc', 'def', 'ghi'), COUNT(*)\n"
|
||||
+ "FROM druid.foo\n"
|
||||
|
@ -6276,9 +6222,6 @@ public class CalciteQueryTest extends BaseCalciteQueryTest
|
|||
public void testCountStarWithNotOfDegenerateFilter()
|
||||
{
|
||||
msqIncompatible();
|
||||
// HashJoinSegmentStorageAdapter is not vectorizable
|
||||
cannotVectorize();
|
||||
|
||||
testQuery(
|
||||
"SELECT COUNT(*) FROM druid.foo WHERE dim2 = 'a' and not (dim1 > 'a' OR dim1 < 'b')",
|
||||
ImmutableList.of(
|
||||
|
@ -7226,9 +7169,6 @@ public class CalciteQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testSumOfExtractionFn()
|
||||
{
|
||||
// Cannot vectorize due to expressions in aggregators.
|
||||
cannotVectorize();
|
||||
|
||||
testQuery(
|
||||
"SELECT SUM(CAST(SUBSTRING(dim1, 1, 10) AS INTEGER)) FROM druid.foo",
|
||||
ImmutableList.of(
|
||||
|
@ -7748,7 +7688,8 @@ public class CalciteQueryTest extends BaseCalciteQueryTest
|
|||
public void testApproxCountDistinct()
|
||||
{
|
||||
msqIncompatible();
|
||||
// Cannot vectorize due to virtual columns.
|
||||
|
||||
// Cannot vectorize due to multi-valued dim2.
|
||||
cannotVectorize();
|
||||
|
||||
testQuery(
|
||||
|
@ -8523,9 +8464,6 @@ public class CalciteQueryTest extends BaseCalciteQueryTest
|
|||
public void testCountDistinctOfTrim()
|
||||
{
|
||||
// Test a couple different syntax variants of TRIM.
|
||||
// Cannot vectorize due to virtual columns.
|
||||
cannotVectorize();
|
||||
|
||||
testQuery(
|
||||
"SELECT COUNT(DISTINCT TRIM(BOTH ' ' FROM dim1)) FROM druid.foo WHERE TRIM(dim1) <> ''",
|
||||
ImmutableList.of(
|
||||
|
@ -8559,10 +8497,6 @@ public class CalciteQueryTest extends BaseCalciteQueryTest
|
|||
public void testSillyQuarters()
|
||||
{
|
||||
// Like FLOOR(__time TO QUARTER) but silly.
|
||||
|
||||
// Cannot vectorize due to virtual columns.
|
||||
cannotVectorize();
|
||||
|
||||
testQuery(
|
||||
"SELECT CAST((EXTRACT(MONTH FROM __time) - 1 ) / 3 + 1 AS INTEGER) AS quarter, COUNT(*)\n"
|
||||
+ "FROM foo\n"
|
||||
|
@ -8672,8 +8606,6 @@ public class CalciteQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testRegexpExtractFilterViaNotNullCheck()
|
||||
{
|
||||
// Cannot vectorize due to extractionFn in dimension spec.
|
||||
cannotVectorize();
|
||||
Druids.TimeseriesQueryBuilder builder =
|
||||
Druids.newTimeseriesQueryBuilder()
|
||||
.dataSource(CalciteTests.DATASOURCE1)
|
||||
|
@ -8849,7 +8781,8 @@ public class CalciteQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testGroupByLimitPushdownExtraction()
|
||||
{
|
||||
cannotVectorize();
|
||||
// Skip vectorization because this can vectorize with decoupled planning, but cannot with regular planning.
|
||||
skipVectorize();
|
||||
|
||||
testQuery(
|
||||
"SELECT dim4, substring(dim5, 1, 1), count(*) FROM druid.numfoo WHERE dim4 = 'a' GROUP BY 1,2 LIMIT 2",
|
||||
|
@ -9107,9 +9040,6 @@ public class CalciteQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testFilterOnTimeExtract()
|
||||
{
|
||||
// Cannot vectorize due to expression filter.
|
||||
cannotVectorize();
|
||||
|
||||
testQuery(
|
||||
"SELECT COUNT(*) FROM druid.foo\n"
|
||||
+ "WHERE EXTRACT(YEAR FROM __time) = 2000\n"
|
||||
|
@ -9142,9 +9072,6 @@ public class CalciteQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testFilterOnTimeExtractWithMultipleDays()
|
||||
{
|
||||
// Cannot vectorize due to expression filters.
|
||||
cannotVectorize();
|
||||
|
||||
testQuery(
|
||||
"SELECT COUNT(*) FROM druid.foo\n"
|
||||
+ "WHERE EXTRACT(YEAR FROM __time) = 2000\n"
|
||||
|
@ -9186,9 +9113,6 @@ public class CalciteQueryTest extends BaseCalciteQueryTest
|
|||
public void testFilterOnTimeExtractWithVariousTimeUnits()
|
||||
{
|
||||
msqIncompatible();
|
||||
// Cannot vectorize due to virtual columns.
|
||||
cannotVectorize();
|
||||
|
||||
testQuery(
|
||||
"SELECT COUNT(*) FROM druid.foo4\n"
|
||||
+ "WHERE EXTRACT(YEAR FROM __time) = 2000\n"
|
||||
|
@ -9298,7 +9222,6 @@ public class CalciteQueryTest extends BaseCalciteQueryTest
|
|||
public void testQueryWithSelectProjectAndIdentityProjectDoesNotRename()
|
||||
{
|
||||
msqIncompatible();
|
||||
cannotVectorize();
|
||||
testQuery(
|
||||
PLANNER_CONFIG_NO_HLL.withOverrides(
|
||||
ImmutableMap.of(
|
||||
|
@ -9580,9 +9503,6 @@ public class CalciteQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testGroupByStringLength()
|
||||
{
|
||||
// Cannot vectorize due to virtual columns.
|
||||
cannotVectorize();
|
||||
|
||||
testQuery(
|
||||
"SELECT CHARACTER_LENGTH(dim1), COUNT(*) FROM druid.foo GROUP BY CHARACTER_LENGTH(dim1)",
|
||||
ImmutableList.of(
|
||||
|
@ -9824,9 +9744,6 @@ public class CalciteQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testTimeseriesUsingTimeFloorWithTimeShift()
|
||||
{
|
||||
// Cannot vectorize due to virtual columns.
|
||||
cannotVectorize();
|
||||
|
||||
testQuery(
|
||||
"SELECT SUM(cnt), gran FROM (\n"
|
||||
+ " SELECT TIME_FLOOR(TIME_SHIFT(__time, 'P1D', -1), 'P1M') AS gran,\n"
|
||||
|
@ -10779,9 +10696,6 @@ public class CalciteQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testGroupByExtractYear()
|
||||
{
|
||||
// Cannot vectorize due to virtual columns.
|
||||
cannotVectorize();
|
||||
|
||||
testQuery(
|
||||
"SELECT\n"
|
||||
+ " EXTRACT(YEAR FROM __time) AS \"year\",\n"
|
||||
|
@ -10830,9 +10744,6 @@ public class CalciteQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testGroupByFormatYearAndMonth()
|
||||
{
|
||||
// Cannot vectorize due to virtual columns.
|
||||
cannotVectorize();
|
||||
|
||||
testQuery(
|
||||
"SELECT\n"
|
||||
+ " TIME_FORMAt(__time, 'yyyy MM') AS \"year\",\n"
|
||||
|
@ -10881,9 +10792,6 @@ public class CalciteQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testGroupByExtractFloorTime()
|
||||
{
|
||||
// Cannot vectorize due to virtual columns.
|
||||
cannotVectorize();
|
||||
|
||||
testQuery(
|
||||
"SELECT\n"
|
||||
+ "EXTRACT(YEAR FROM FLOOR(__time TO YEAR)) AS \"year\", SUM(cnt)\n"
|
||||
|
@ -10916,9 +10824,6 @@ public class CalciteQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testGroupByExtractFloorTimeLosAngeles()
|
||||
{
|
||||
// Cannot vectorize due to virtual columns.
|
||||
cannotVectorize();
|
||||
|
||||
testQuery(
|
||||
PLANNER_CONFIG_DEFAULT,
|
||||
QUERY_CONTEXT_LOS_ANGELES,
|
||||
|
@ -11242,9 +11147,6 @@ public class CalciteQueryTest extends BaseCalciteQueryTest
|
|||
public void testGroupingSets()
|
||||
{
|
||||
msqIncompatible();
|
||||
// Cannot vectorize due to virtual columns.
|
||||
cannotVectorize();
|
||||
|
||||
testQuery(
|
||||
"SELECT dim2, gran, SUM(cnt), GROUPING(dim2, gran)\n"
|
||||
+ "FROM (SELECT FLOOR(__time TO MONTH) AS gran, COALESCE(dim2, '') dim2, cnt FROM druid.foo) AS x\n"
|
||||
|
@ -11462,9 +11364,6 @@ public class CalciteQueryTest extends BaseCalciteQueryTest
|
|||
public void testGroupByRollup()
|
||||
{
|
||||
msqIncompatible();
|
||||
// Cannot vectorize due to virtual columns.
|
||||
cannotVectorize();
|
||||
|
||||
testQuery(
|
||||
"SELECT dim2, gran, SUM(cnt)\n"
|
||||
+ "FROM (SELECT FLOOR(__time TO MONTH) AS gran, COALESCE(dim2, '') dim2, cnt FROM druid.foo) AS x\n"
|
||||
|
@ -11578,9 +11477,6 @@ public class CalciteQueryTest extends BaseCalciteQueryTest
|
|||
public void testGroupByCube()
|
||||
{
|
||||
msqIncompatible();
|
||||
// Cannot vectorize due to virtual columns.
|
||||
cannotVectorize();
|
||||
|
||||
testQuery(
|
||||
"SELECT dim2, gran, SUM(cnt)\n"
|
||||
+ "FROM (SELECT FLOOR(__time TO MONTH) AS gran, COALESCE(dim2, '') dim2, cnt FROM druid.foo) AS x\n"
|
||||
|
@ -11641,9 +11537,6 @@ public class CalciteQueryTest extends BaseCalciteQueryTest
|
|||
public void testGroupingSetsWithDummyDimension()
|
||||
{
|
||||
msqIncompatible();
|
||||
// Cannot vectorize due to virtual columns.
|
||||
cannotVectorize();
|
||||
|
||||
testQuery(
|
||||
"SELECT dim2, gran, SUM(cnt)\n"
|
||||
+ "FROM (SELECT FLOOR(__time TO MONTH) AS gran, COALESCE(dim2, '') dim2, cnt FROM druid.foo) AS x\n"
|
||||
|
@ -11704,9 +11597,6 @@ public class CalciteQueryTest extends BaseCalciteQueryTest
|
|||
public void testGroupingSetsNoSuperset()
|
||||
{
|
||||
msqIncompatible();
|
||||
// Cannot vectorize due to virtual columns.
|
||||
cannotVectorize();
|
||||
|
||||
// Note: the grouping sets are reordered in the output of this query, but this is allowed.
|
||||
testQuery(
|
||||
"SELECT dim2, gran, SUM(cnt)\n"
|
||||
|
@ -11834,9 +11724,6 @@ public class CalciteQueryTest extends BaseCalciteQueryTest
|
|||
public void testGroupingSetsWithOrderByAggregator()
|
||||
{
|
||||
msqIncompatible();
|
||||
// Cannot vectorize due to virtual columns.
|
||||
cannotVectorize();
|
||||
|
||||
testQuery(
|
||||
"SELECT dim2, gran, SUM(cnt)\n"
|
||||
+ "FROM (SELECT FLOOR(__time TO MONTH) AS gran, COALESCE(dim2, '') dim2, cnt FROM druid.foo) AS x\n"
|
||||
|
@ -11904,9 +11791,6 @@ public class CalciteQueryTest extends BaseCalciteQueryTest
|
|||
public void testGroupingSetsWithOrderByAggregatorWithLimit()
|
||||
{
|
||||
msqIncompatible();
|
||||
// Cannot vectorize due to virtual columns.
|
||||
cannotVectorize();
|
||||
|
||||
testQuery(
|
||||
"SELECT dim2, gran, SUM(cnt)\n"
|
||||
+ "FROM (SELECT FLOOR(__time TO MONTH) AS gran, COALESCE(dim2, '') dim2, cnt FROM druid.foo) AS x\n"
|
||||
|
@ -13574,8 +13458,6 @@ public class CalciteQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testTimeStampAddZeroYearPeriod()
|
||||
{
|
||||
cannotVectorize();
|
||||
|
||||
testQuery(
|
||||
"SELECT TIMESTAMPADD(YEAR, 0, \"__time\") FROM druid.foo",
|
||||
|
||||
|
@ -13740,9 +13622,6 @@ public class CalciteQueryTest extends BaseCalciteQueryTest
|
|||
public void testGroupingSetsWithLimitOrderByGran()
|
||||
{
|
||||
msqIncompatible();
|
||||
// Cannot vectorize due to virtual columns.
|
||||
cannotVectorize();
|
||||
|
||||
testQuery(
|
||||
"SELECT dim2, gran, SUM(cnt)\n"
|
||||
+ "FROM (SELECT FLOOR(__time TO MONTH) AS gran, COALESCE(dim2, '') dim2, cnt FROM druid.foo) AS x\n"
|
||||
|
@ -13926,7 +13805,6 @@ public class CalciteQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testExpressionCounts()
|
||||
{
|
||||
cannotVectorize();
|
||||
testQuery(
|
||||
"SELECT\n"
|
||||
+ " COUNT(reverse(dim2)),\n"
|
||||
|
@ -14786,7 +14664,6 @@ public class CalciteQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testReturnEmptyRowWhenGroupByIsConvertedToTimeseriesWithSingleConstantDimension()
|
||||
{
|
||||
cannotVectorize();
|
||||
testQuery(
|
||||
"SELECT 'A' from foo WHERE m1 = 50 AND dim1 = 'wat' GROUP BY 'foobar'",
|
||||
ImmutableList.of(
|
||||
|
@ -14840,7 +14717,6 @@ public class CalciteQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testReturnEmptyRowWhenGroupByIsConvertedToTimeseriesWithMultipleConstantDimensions()
|
||||
{
|
||||
cannotVectorize();
|
||||
testQuery(
|
||||
"SELECT 'A', dim1 from foo WHERE m1 = 50 AND dim1 = 'wat' GROUP BY dim1",
|
||||
ImmutableList.of(
|
||||
|
@ -14951,7 +14827,6 @@ public class CalciteQueryTest extends BaseCalciteQueryTest
|
|||
.build()
|
||||
);
|
||||
} else {
|
||||
cannotVectorize();
|
||||
expectedResult = ImmutableList.of(
|
||||
new Object[]{"", false},
|
||||
new Object[]{"1", true},
|
||||
|
@ -14992,7 +14867,6 @@ public class CalciteQueryTest extends BaseCalciteQueryTest
|
|||
public void testGreatestFunctionForStringWithIsNull()
|
||||
{
|
||||
msqIncompatible();
|
||||
cannotVectorize();
|
||||
|
||||
String query = "SELECT l1, LATEST(GREATEST(dim1, dim2)) IS NULL FROM druid.numfoo GROUP BY l1";
|
||||
|
||||
|
@ -15139,7 +15013,6 @@ public class CalciteQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testComplexDecode()
|
||||
{
|
||||
cannotVectorize();
|
||||
for (String complexDecode : Arrays.asList("COMPLEX_DECODE_BASE64", "DECODE_BASE64_COMPLEX")) {
|
||||
testQuery(
|
||||
StringUtils.format(
|
||||
|
@ -15179,7 +15052,6 @@ public class CalciteQueryTest extends BaseCalciteQueryTest
|
|||
public void testComplexDecodeAgg()
|
||||
{
|
||||
msqIncompatible();
|
||||
cannotVectorize();
|
||||
testQuery(
|
||||
"SELECT APPROX_COUNT_DISTINCT_BUILTIN(COMPLEX_DECODE_BASE64('hyperUnique',PARSE_JSON(TO_JSON_STRING(unique_dim1)))) from druid.foo",
|
||||
ImmutableList.of(
|
||||
|
@ -15213,7 +15085,6 @@ public class CalciteQueryTest extends BaseCalciteQueryTest
|
|||
public void testComplexDecodeAggWithCastedTypeName()
|
||||
{
|
||||
msqIncompatible();
|
||||
cannotVectorize();
|
||||
testQuery(
|
||||
"SELECT "
|
||||
+ "APPROX_COUNT_DISTINCT_BUILTIN(COMPLEX_DECODE_BASE64(CAST('hyperUnique' AS VARCHAR),PARSE_JSON(TO_JSON_STRING(unique_dim1)))) "
|
||||
|
@ -15461,7 +15332,6 @@ public class CalciteQueryTest extends BaseCalciteQueryTest
|
|||
public void testLatestByOnStringColumnWithoutMaxBytesSpecified()
|
||||
{
|
||||
String defaultString = useDefault ? "" : null;
|
||||
cannotVectorize();
|
||||
testQuery(
|
||||
"SELECT dim2,LATEST(dim3),LATEST_BY(dim1, __time),EARLIEST(dim3),EARLIEST_BY(dim1, __time),ANY_VALUE(dim3) FROM druid.foo where dim2='abc' group by 1",
|
||||
ImmutableList.of(
|
||||
|
@ -15584,8 +15454,6 @@ public class CalciteQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testInGroupByLimitOutGroupByOrderBy()
|
||||
{
|
||||
cannotVectorize();
|
||||
|
||||
testBuilder()
|
||||
.sql(
|
||||
"with t AS (SELECT m2, COUNT(m1) as trend_score\n"
|
||||
|
@ -15661,7 +15529,6 @@ public class CalciteQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testInGroupByOrderByLimitOutGroupByOrderByLimit()
|
||||
{
|
||||
cannotVectorize();
|
||||
String sql = "with t AS (SELECT m2 as mo, COUNT(m1) as trend_score\n"
|
||||
+ "FROM \"foo\"\n"
|
||||
+ "GROUP BY 1\n"
|
||||
|
@ -15756,7 +15623,6 @@ public class CalciteQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testWindowingWithScanAndSort()
|
||||
{
|
||||
cannotVectorize();
|
||||
msqIncompatible();
|
||||
String sql = "with t AS (\n"
|
||||
+ "SELECT \n"
|
||||
|
@ -15987,7 +15853,6 @@ public class CalciteQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testCastCharToVarcharInFlattenConcat()
|
||||
{
|
||||
cannotVectorize();
|
||||
testQuery(
|
||||
"select 'A'||cast(col as char)||'B' from (values(1)) as t(col)",
|
||||
ImmutableList.of(
|
||||
|
@ -16229,9 +16094,8 @@ public class CalciteQueryTest extends BaseCalciteQueryTest
|
|||
|
||||
@SqlTestFrameworkConfig.NumMergeBuffers(4)
|
||||
@Test
|
||||
public void testGroupingSetsWithAggrgateCase()
|
||||
public void testGroupingSetsWithAggregateCase()
|
||||
{
|
||||
cannotVectorize();
|
||||
msqIncompatible();
|
||||
final Map<String, Object> queryContext = ImmutableMap.of(
|
||||
PlannerConfig.CTX_KEY_USE_APPROXIMATE_COUNT_DISTINCT, false,
|
||||
|
|
|
@ -480,7 +480,6 @@ public class CalciteSelectQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testSafeDivideWithoutTable()
|
||||
{
|
||||
cannotVectorize();
|
||||
final Map<String, Object> context = new HashMap<>(QUERY_CONTEXT_DEFAULT);
|
||||
|
||||
testQuery(
|
||||
|
@ -657,10 +656,6 @@ public class CalciteSelectQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testSelectDistinctWithCascadeExtractionFilter()
|
||||
{
|
||||
if (NullHandling.sqlCompatible()) {
|
||||
// cannot vectorize due to expression filter
|
||||
cannotVectorize();
|
||||
}
|
||||
testQuery(
|
||||
"SELECT distinct dim1 FROM druid.foo WHERE substring(substring(dim1, 2), 1, 1) = 'e' OR dim2 = 'a'",
|
||||
ImmutableList.of(
|
||||
|
@ -705,9 +700,6 @@ public class CalciteSelectQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testSelectDistinctWithStrlenFilter()
|
||||
{
|
||||
// Cannot vectorize due to usage of expressions.
|
||||
cannotVectorize();
|
||||
|
||||
testQuery(
|
||||
"SELECT distinct dim1 FROM druid.foo "
|
||||
+ "WHERE CHARACTER_LENGTH(dim1) = 3 OR CAST(CHARACTER_LENGTH(dim1) AS varchar) = 3",
|
||||
|
@ -1996,7 +1988,6 @@ public class CalciteSelectQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testCountDistinctNonApproximateEmptySet()
|
||||
{
|
||||
cannotVectorize();
|
||||
testQuery(
|
||||
PLANNER_CONFIG_DEFAULT.withOverrides(
|
||||
ImmutableMap.of(
|
||||
|
@ -2033,7 +2024,6 @@ public class CalciteSelectQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testCountDistinctNonApproximateBasic()
|
||||
{
|
||||
cannotVectorize();
|
||||
testQuery(
|
||||
PLANNER_CONFIG_DEFAULT.withOverrides(
|
||||
ImmutableMap.of(
|
||||
|
@ -2069,8 +2059,6 @@ public class CalciteSelectQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testCountDistinctNonApproximateWithFilter()
|
||||
{
|
||||
cannotVectorize();
|
||||
|
||||
testQuery(
|
||||
PLANNER_CONFIG_DEFAULT.withOverrides(
|
||||
ImmutableMap.of(
|
||||
|
@ -2109,8 +2097,6 @@ public class CalciteSelectQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testCountDistinctNonApproximateWithFilterHaving()
|
||||
{
|
||||
cannotVectorize();
|
||||
|
||||
testQuery(
|
||||
PLANNER_CONFIG_DEFAULT.withOverrides(
|
||||
ImmutableMap.of(
|
||||
|
|
|
@ -103,9 +103,9 @@ public class CalciteSubqueryTest extends BaseCalciteQueryTest
|
|||
@ParameterizedTest(name = "{0}")
|
||||
public void testExactCountDistinctUsingSubqueryWithWhereToOuterFilter(String testName, Map<String, Object> queryContext)
|
||||
{
|
||||
// Cannot vectorize topN operator.
|
||||
if (!queryContext.containsKey(QueryContexts.MAX_SUBQUERY_BYTES_KEY)) {
|
||||
cannotVectorize();
|
||||
|
||||
}
|
||||
testQuery(
|
||||
"SELECT\n"
|
||||
+ " SUM(cnt),\n"
|
||||
|
@ -374,8 +374,9 @@ public class CalciteSubqueryTest extends BaseCalciteQueryTest
|
|||
@ParameterizedTest(name = "{0}")
|
||||
public void testGroupByWithPostAggregatorReferencingTimeFloorColumnOnTimeseries(String testName, Map<String, Object> queryContext)
|
||||
{
|
||||
if (!queryContext.containsKey(QueryContexts.MAX_SUBQUERY_BYTES_KEY)) {
|
||||
cannotVectorize();
|
||||
|
||||
}
|
||||
testQuery(
|
||||
"SELECT TIME_FORMAT(\"date\", 'yyyy-MM'), SUM(x)\n"
|
||||
+ "FROM (\n"
|
||||
|
@ -537,9 +538,9 @@ public class CalciteSubqueryTest extends BaseCalciteQueryTest
|
|||
@ParameterizedTest(name = "{0}")
|
||||
public void testMinMaxAvgDailyCountWithLimit(String testName, Map<String, Object> queryContext)
|
||||
{
|
||||
// Cannot vectorize due to virtual columns.
|
||||
if (!queryContext.containsKey(QueryContexts.MAX_SUBQUERY_BYTES_KEY)) {
|
||||
cannotVectorize();
|
||||
|
||||
}
|
||||
testQuery(
|
||||
"SELECT * FROM ("
|
||||
+ " SELECT max(cnt), min(cnt), avg(cnt), TIME_EXTRACT(max(t), 'EPOCH') last_time, count(1) num_days FROM (\n"
|
||||
|
@ -782,8 +783,9 @@ public class CalciteSubqueryTest extends BaseCalciteQueryTest
|
|||
@ParameterizedTest(name = "{0}")
|
||||
public void testUseTimeFloorInsteadOfGranularityOnJoinResult(String testName, Map<String, Object> queryContext)
|
||||
{
|
||||
if (!queryContext.containsKey(QueryContexts.MAX_SUBQUERY_BYTES_KEY)) {
|
||||
cannotVectorize();
|
||||
|
||||
}
|
||||
testQuery(
|
||||
"WITH main AS (SELECT * FROM foo LIMIT 2)\n"
|
||||
+ "SELECT TIME_FLOOR(__time, 'PT1H') AS \"time\", dim1, COUNT(*)\n"
|
||||
|
@ -902,9 +904,9 @@ public class CalciteSubqueryTest extends BaseCalciteQueryTest
|
|||
@ParameterizedTest(name = "{0}")
|
||||
public void testUsingSubqueryWithLimit(String testName, Map<String, Object> queryContext)
|
||||
{
|
||||
// Cannot vectorize scan query.
|
||||
if (!queryContext.containsKey(QueryContexts.MAX_SUBQUERY_BYTES_KEY)) {
|
||||
cannotVectorize();
|
||||
|
||||
}
|
||||
testQuery(
|
||||
"SELECT COUNT(*) AS cnt FROM ( SELECT * FROM druid.foo LIMIT 10 ) tmpA",
|
||||
queryContext,
|
||||
|
@ -987,8 +989,6 @@ public class CalciteSubqueryTest extends BaseCalciteQueryTest
|
|||
@ParameterizedTest(name = "{0}")
|
||||
public void testJoinWithSubqueries(String testName, Map<String, Object> queryContext)
|
||||
{
|
||||
cannotVectorize();
|
||||
|
||||
List<Object[]> results = new ArrayList<>(ImmutableList.of(
|
||||
new Object[]{"", NullHandling.defaultStringValue()},
|
||||
new Object[]{"10.1", NullHandling.defaultStringValue()},
|
||||
|
|
|
@ -141,9 +141,6 @@ public class CalciteTimeBoundaryQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testMinTimeQueryWithTimeAndExpressionFilters()
|
||||
{
|
||||
// Cannot vectorize due to UPPER expression.
|
||||
cannotVectorize();
|
||||
|
||||
HashMap<String, Object> queryContext = new HashMap<>(QUERY_CONTEXT_DEFAULT);
|
||||
queryContext.put(QueryContexts.TIME_BOUNDARY_PLANNING_KEY, true);
|
||||
testQuery(
|
||||
|
@ -200,9 +197,6 @@ public class CalciteTimeBoundaryQueryTest extends BaseCalciteQueryTest
|
|||
@Test
|
||||
public void testMaxTimeQueryWithJoin()
|
||||
{
|
||||
// Cannot vectorize due to JOIN.
|
||||
cannotVectorize();
|
||||
|
||||
HashMap<String, Object> context = new HashMap<>(QUERY_CONTEXT_DEFAULT);
|
||||
context.put(QueryContexts.TIME_BOUNDARY_PLANNING_KEY, true);
|
||||
|
||||
|
|
|
@ -629,12 +629,15 @@ public class QueryTestRunner
|
|||
// times. Pick the first failure as that emulates the original code flow
|
||||
// where the first exception ended the test.
|
||||
for (QueryResults queryResults : execStep.results()) {
|
||||
if (queryResults.exception == null) {
|
||||
continue;
|
||||
}
|
||||
|
||||
// Delayed exception checking to let other verify steps run before running vectorized checks
|
||||
if (builder.queryCannotVectorize && "force".equals(queryResults.vectorizeOption)) {
|
||||
if (queryResults.exception == null) {
|
||||
Assert.fail(
|
||||
"Expected vectorized execution to fail, but it did not. "
|
||||
+ "Please remove cannotVectorize() from this test case."
|
||||
);
|
||||
}
|
||||
|
||||
MatcherAssert.assertThat(
|
||||
queryResults.exception,
|
||||
CoreMatchers.allOf(
|
||||
|
@ -644,7 +647,7 @@ public class QueryTestRunner
|
|||
)
|
||||
)
|
||||
);
|
||||
} else {
|
||||
} else if (queryResults.exception != null) {
|
||||
throw queryResults.exception;
|
||||
}
|
||||
}
|
||||
|
|
Loading…
Reference in New Issue