From f5492053b9ea3caa2e2c16003a6d41fcf8da5065 Mon Sep 17 00:00:00 2001 From: penghuo Date: Wed, 5 Aug 2020 17:43:36 -0700 Subject: [PATCH 1/2] Support NULL and MISSING value in response --- .../sql/analysis/Analyzer.java | 54 ++++-- .../sql/analysis/ExpressionAnalyzer.java | 7 - .../analysis/SelectExpressionAnalyzer.java | 79 +++++++++ .../sql/analysis/TypeEnvironment.java | 23 +++ .../sql/analysis/symbol/SymbolTable.java | 29 ++++ .../sql/ast/AbstractNodeVisitor.java | 5 + .../sql/ast/dsl/AstDSL.java | 1 + .../sql/ast/expression/AllFields.java | 43 +++++ .../sql/ast/tree/Project.java | 11 ++ .../sql/data/model/ExprMissingValue.java | 2 +- .../sql/executor/ExecutionEngine.java | 14 ++ .../sql/planner/physical/PhysicalPlan.java | 6 + .../sql/planner/physical/ProjectOperator.java | 14 +- .../sql/analysis/AnalyzerTest.java | 8 + .../sql/analysis/AnalyzerTestBase.java | 64 ++++++- .../sql/analysis/ExpressionAnalyzerTest.java | 27 +-- .../sql/analysis/SelectAnalyzeTest.java | 161 ++++++++++++++++++ .../SelectExpressionAnalyzerTest.java | 69 ++++++++ .../sql/analysis/symbol/SymbolTableTest.java | 37 +++- .../sql/config/TestConfig.java | 2 +- .../sql/data/model/ExprMissingValueTest.java | 5 +- .../sql/data/model/ExprTupleValueTest.java | 10 ++ .../sql/expression/NamedExpressionTest.java | 9 + .../planner/logical/LogicalDedupeTest.java | 8 + .../sql/planner/logical/LogicalEvalTest.java | 7 + .../sql/planner/logical/LogicalSortTest.java | 8 + .../planner/physical/ProjectOperatorTest.java | 23 ++- .../planner/physical/RemoveOperatorTest.java | 13 ++ docs/category.json | 1 + docs/experiment/ppl/cmd/search.rst | 28 +-- docs/user/general/identifiers.rst | 32 ++-- docs/user/general/values.rst | 91 ++++++++++ doctest/test_data/accounts.json | 2 +- .../ElasticsearchExecutionEngine.java | 2 +- .../ElasticsearchExecutionEngineTest.java | 8 + integ-test/build.gradle | 11 +- .../sql/ppl/DedupCommandIT.java | 6 +- .../sql/ppl/OperatorIT.java | 4 +- .../sql/ppl/StandaloneIT.java | 2 +- .../sql/util/MatcherUtils.java | 2 + .../correctness/expressions/literals.txt | 1 + .../sql/legacy/plugin/RestSQLQueryAction.java | 3 +- .../sql/plugin/rest/RestPPLQueryAction.java | 3 +- ppl/build.gradle | 1 + .../sql/ppl/PPLService.java | 4 +- .../sql/ppl/utils/UnresolvedPlanHelper.java | 42 +++++ .../sql/ppl/PPLServiceTest.java | 20 ++- .../ppl/utils/UnresolvedPlanHelperTest.java | 68 ++++++++ .../sql/protocol/response/QueryResult.java | 33 +--- .../protocol/response/QueryResultTest.java | 61 ++++--- .../SimpleJsonResponseFormatterTest.java | 21 ++- .../jdbc/types/ElasticsearchType.java | 1 + .../sql/sql/parser/AstBuilder.java | 25 ++- .../sql/sql/SQLServiceTest.java | 7 +- .../sql/sql/parser/AstBuilderTest.java | 19 ++- 55 files changed, 1069 insertions(+), 168 deletions(-) create mode 100644 core/src/main/java/com/amazon/opendistroforelasticsearch/sql/analysis/SelectExpressionAnalyzer.java create mode 100644 core/src/main/java/com/amazon/opendistroforelasticsearch/sql/ast/expression/AllFields.java create mode 100644 core/src/test/java/com/amazon/opendistroforelasticsearch/sql/analysis/SelectAnalyzeTest.java create mode 100644 core/src/test/java/com/amazon/opendistroforelasticsearch/sql/analysis/SelectExpressionAnalyzerTest.java create mode 100644 docs/user/general/values.rst create mode 100644 ppl/src/main/java/com/amazon/opendistroforelasticsearch/sql/ppl/utils/UnresolvedPlanHelper.java create mode 100644 ppl/src/test/java/com/amazon/opendistroforelasticsearch/sql/ppl/utils/UnresolvedPlanHelperTest.java diff --git a/core/src/main/java/com/amazon/opendistroforelasticsearch/sql/analysis/Analyzer.java b/core/src/main/java/com/amazon/opendistroforelasticsearch/sql/analysis/Analyzer.java index 628720efec..bc40def04f 100644 --- a/core/src/main/java/com/amazon/opendistroforelasticsearch/sql/analysis/Analyzer.java +++ b/core/src/main/java/com/amazon/opendistroforelasticsearch/sql/analysis/Analyzer.java @@ -15,8 +15,6 @@ package com.amazon.opendistroforelasticsearch.sql.analysis; -import static com.amazon.opendistroforelasticsearch.sql.expression.DSL.named; - import com.amazon.opendistroforelasticsearch.sql.analysis.symbol.Namespace; import com.amazon.opendistroforelasticsearch.sql.analysis.symbol.Symbol; import com.amazon.opendistroforelasticsearch.sql.ast.AbstractNodeVisitor; @@ -65,7 +63,6 @@ import java.util.ArrayList; import java.util.List; import java.util.stream.Collectors; -import lombok.RequiredArgsConstructor; import org.apache.commons.lang3.tuple.ImmutablePair; import org.apache.commons.lang3.tuple.Pair; @@ -73,11 +70,25 @@ * Analyze the {@link UnresolvedPlan} in the {@link AnalysisContext} to construct the {@link * LogicalPlan}. */ -@RequiredArgsConstructor public class Analyzer extends AbstractNodeVisitor { + private final ExpressionAnalyzer expressionAnalyzer; + + private final SelectExpressionAnalyzer selectExpressionAnalyzer; + private final StorageEngine storageEngine; + /** + * Constructor. + */ + public Analyzer( + ExpressionAnalyzer expressionAnalyzer, + StorageEngine storageEngine) { + this.expressionAnalyzer = expressionAnalyzer; + this.storageEngine = storageEngine; + this.selectExpressionAnalyzer = new SelectExpressionAnalyzer(expressionAnalyzer); + } + public LogicalPlan analyze(UnresolvedPlan unresolved, AnalysisContext context) { return unresolved.accept(this, context); } @@ -113,8 +124,11 @@ public LogicalPlan visitRename(Rename node, AnalysisContext context) { ReferenceExpression target = new ReferenceExpression(((Field) renameMap.getTarget()).getField().toString(), origin.type()); - context.peek().define(target); - renameMapBuilder.put(DSL.ref(origin.toString(), origin.type()), target); + ReferenceExpression originExpr = DSL.ref(origin.toString(), origin.type()); + TypeEnvironment curEnv = context.peek(); + curEnv.remove(originExpr); + curEnv.define(target); + renameMapBuilder.put(originExpr, target); } else { throw new SemanticCheckException( String.format("the target expected to be field, but is %s", renameMap.getTarget())); @@ -129,17 +143,27 @@ public LogicalPlan visitRename(Rename node, AnalysisContext context) { */ @Override public LogicalPlan visitAggregation(Aggregation node, AnalysisContext context) { - LogicalPlan child = node.getChild().get(0).accept(this, context); + final LogicalPlan child = node.getChild().get(0).accept(this, context); ImmutableList.Builder aggregatorBuilder = new ImmutableList.Builder<>(); for (UnresolvedExpression expr : node.getAggExprList()) { aggregatorBuilder.add((Aggregator) expressionAnalyzer.analyze(expr, context)); } + ImmutableList aggregators = aggregatorBuilder.build(); ImmutableList.Builder groupbyBuilder = new ImmutableList.Builder<>(); for (UnresolvedExpression expr : node.getGroupExprList()) { groupbyBuilder.add(expressionAnalyzer.analyze(expr, context)); } - return new LogicalAggregation(child, aggregatorBuilder.build(), groupbyBuilder.build()); + ImmutableList groupBys = groupbyBuilder.build(); + + // new context + context.push(); + TypeEnvironment newEnv = context.peek(); + aggregators.forEach(aggregator -> newEnv.define(new Symbol(Namespace.FIELD_NAME, + aggregator.toString()), aggregator.type())); + groupBys.forEach(group -> newEnv.define(new Symbol(Namespace.FIELD_NAME, + group.toString()), group.type())); + return new LogicalAggregation(child, aggregators, groupBys); } /** @@ -161,18 +185,24 @@ public LogicalPlan visitProject(Project node, AnalysisContext context) { Argument argument = node.getArgExprList().get(0); Boolean exclude = (Boolean) argument.getValue().getValue(); if (exclude) { + TypeEnvironment curEnv = context.peek(); List referenceExpressions = node.getProjectList().stream() .map(expr -> (ReferenceExpression) expressionAnalyzer.analyze(expr, context)) .collect(Collectors.toList()); + referenceExpressions.forEach(ref -> curEnv.remove(ref)); return new LogicalRemove(child, ImmutableSet.copyOf(referenceExpressions)); } } - List expressions = node.getProjectList().stream() - .map(expr -> named(expressionAnalyzer.analyze(expr, context))) - .collect(Collectors.toList()); - return new LogicalProject(child, expressions); + List namedExpressions = + selectExpressionAnalyzer.analyze(node.getProjectList(), context); + // new context + context.push(); + TypeEnvironment newEnv = context.peek(); + namedExpressions.forEach(expr -> newEnv.define(new Symbol(Namespace.FIELD_NAME, + expr.getName()), expr.type())); + return new LogicalProject(child, namedExpressions); } /** diff --git a/core/src/main/java/com/amazon/opendistroforelasticsearch/sql/analysis/ExpressionAnalyzer.java b/core/src/main/java/com/amazon/opendistroforelasticsearch/sql/analysis/ExpressionAnalyzer.java index db9a782d94..30936150aa 100644 --- a/core/src/main/java/com/amazon/opendistroforelasticsearch/sql/analysis/ExpressionAnalyzer.java +++ b/core/src/main/java/com/amazon/opendistroforelasticsearch/sql/analysis/ExpressionAnalyzer.java @@ -164,13 +164,6 @@ public Expression visitQualifiedName(QualifiedName node, AnalysisContext context return visitIdentifier(node.toString(), context); } - @Override - public Expression visitAlias(Alias node, AnalysisContext context) { - return DSL.named(node.getName(), - node.getDelegated().accept(this, context), - node.getAlias()); - } - private Expression visitIdentifier(String ident, AnalysisContext context) { TypeEnvironment typeEnv = context.peek(); ReferenceExpression ref = DSL.ref(ident, diff --git a/core/src/main/java/com/amazon/opendistroforelasticsearch/sql/analysis/SelectExpressionAnalyzer.java b/core/src/main/java/com/amazon/opendistroforelasticsearch/sql/analysis/SelectExpressionAnalyzer.java new file mode 100644 index 0000000000..bfb9216020 --- /dev/null +++ b/core/src/main/java/com/amazon/opendistroforelasticsearch/sql/analysis/SelectExpressionAnalyzer.java @@ -0,0 +1,79 @@ +/* + * + * Copyright 2020 Amazon.com, Inc. or its affiliates. All Rights Reserved. + * + * Licensed under the Apache License, Version 2.0 (the "License"). + * You may not use this file except in compliance with the License. + * A copy of the License is located at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * or in the "license" file accompanying this file. This file 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 com.amazon.opendistroforelasticsearch.sql.analysis; + +import com.amazon.opendistroforelasticsearch.sql.analysis.symbol.Namespace; +import com.amazon.opendistroforelasticsearch.sql.ast.AbstractNodeVisitor; +import com.amazon.opendistroforelasticsearch.sql.ast.expression.Alias; +import com.amazon.opendistroforelasticsearch.sql.ast.expression.AllFields; +import com.amazon.opendistroforelasticsearch.sql.ast.expression.Field; +import com.amazon.opendistroforelasticsearch.sql.ast.expression.UnresolvedExpression; +import com.amazon.opendistroforelasticsearch.sql.data.type.ExprType; +import com.amazon.opendistroforelasticsearch.sql.expression.DSL; +import com.amazon.opendistroforelasticsearch.sql.expression.NamedExpression; +import com.amazon.opendistroforelasticsearch.sql.expression.ReferenceExpression; +import com.google.common.collect.ImmutableList; +import java.util.Collections; +import java.util.List; +import java.util.Map; +import java.util.stream.Collectors; +import lombok.RequiredArgsConstructor; + +/** + * Analyze the select list in the {@link AnalysisContext} to construct the list of + * {@link NamedExpression}. + */ +@RequiredArgsConstructor +public class SelectExpressionAnalyzer + extends + AbstractNodeVisitor, AnalysisContext> { + private final ExpressionAnalyzer expressionAnalyzer; + + /** + * Analyze Select fields. + */ + public List analyze(List selectList, + AnalysisContext analysisContext) { + ImmutableList.Builder builder = new ImmutableList.Builder<>(); + for (UnresolvedExpression unresolvedExpression : selectList) { + builder.addAll(unresolvedExpression.accept(this, analysisContext)); + } + return builder.build(); + } + + @Override + public List visitField(Field node, AnalysisContext context) { + return Collections.singletonList(DSL.named(node.accept(expressionAnalyzer, context))); + } + + @Override + public List visitAlias(Alias node, AnalysisContext context) { + return Collections.singletonList(DSL.named(node.getName(), + node.getDelegated().accept(expressionAnalyzer, context), + node.getAlias())); + } + + @Override + public List visitAllFields(AllFields node, + AnalysisContext context) { + TypeEnvironment environment = context.peek(); + Map lookupAllFields = environment.lookupAllFields(Namespace.FIELD_NAME); + return lookupAllFields.entrySet().stream().map(entry -> DSL.named(entry.getKey(), + new ReferenceExpression(entry.getKey(), entry.getValue()))).collect(Collectors.toList()); + } +} diff --git a/core/src/main/java/com/amazon/opendistroforelasticsearch/sql/analysis/TypeEnvironment.java b/core/src/main/java/com/amazon/opendistroforelasticsearch/sql/analysis/TypeEnvironment.java index 2262b05598..07849f92d9 100644 --- a/core/src/main/java/com/amazon/opendistroforelasticsearch/sql/analysis/TypeEnvironment.java +++ b/core/src/main/java/com/amazon/opendistroforelasticsearch/sql/analysis/TypeEnvironment.java @@ -23,6 +23,8 @@ import com.amazon.opendistroforelasticsearch.sql.expression.Expression; import com.amazon.opendistroforelasticsearch.sql.expression.ReferenceExpression; import com.amazon.opendistroforelasticsearch.sql.expression.env.Environment; +import java.util.HashMap; +import java.util.Map; import java.util.Optional; import lombok.Getter; @@ -62,6 +64,17 @@ public ExprType resolve(Symbol symbol) { String.format("can't resolve %s in type env", symbol)); } + /** + * Resolve all fields in the current environment. + * @param namespace a namespace + * @return all symbols in the namespace + */ + public Map lookupAllFields(Namespace namespace) { + Map result = new HashMap<>(); + symbolTable.lookupAllFields(namespace).forEach(result::putIfAbsent); + return result; + } + /** * Define symbol with the type. * @@ -81,4 +94,14 @@ public void define(ReferenceExpression ref) { define(new Symbol(Namespace.FIELD_NAME, ref.getAttr()), ref.type()); } + public void remove(Symbol symbol) { + symbolTable.remove(symbol); + } + + /** + * Remove ref. + */ + public void remove(ReferenceExpression ref) { + remove(new Symbol(Namespace.FIELD_NAME, ref.getAttr())); + } } diff --git a/core/src/main/java/com/amazon/opendistroforelasticsearch/sql/analysis/symbol/SymbolTable.java b/core/src/main/java/com/amazon/opendistroforelasticsearch/sql/analysis/symbol/SymbolTable.java index 35acd06163..18daa49b10 100644 --- a/core/src/main/java/com/amazon/opendistroforelasticsearch/sql/analysis/symbol/SymbolTable.java +++ b/core/src/main/java/com/amazon/opendistroforelasticsearch/sql/analysis/symbol/SymbolTable.java @@ -24,6 +24,7 @@ import java.util.NavigableMap; import java.util.Optional; import java.util.TreeMap; +import java.util.stream.Collectors; /** * Symbol table for symbol definition and resolution. @@ -49,6 +50,19 @@ public void store(Symbol symbol, ExprType type) { ).put(symbol.getName(), type); } + /** + * Remove a symbol from SymbolTable. + */ + public void remove(Symbol symbol) { + tableByNamespace.computeIfPresent( + symbol.getNamespace(), + (k, v) -> { + v.remove(symbol.getName()); + return v; + } + ); + } + /** * Look up symbol in the namespace map. * @@ -78,6 +92,21 @@ public Map lookupByPrefix(Symbol prefix) { return emptyMap(); } + /** + * Look up all top level symbols in the namespace. + * this function is mainly used by SELECT * use case to get the top level fields + * Todo. currently, the top level fields is the field which doesn't include "." in the name. + * + * @param namespace a namespace + * @return all symbols in the namespace map + */ + public Map lookupAllFields(Namespace namespace) { + return tableByNamespace.getOrDefault(namespace, emptyNavigableMap()) + .entrySet().stream() + .filter(entry -> !entry.getKey().contains(".")) + .collect(Collectors.toMap(Map.Entry::getKey, Map.Entry::getValue)); + } + /** * Check if namespace map in empty (none definition). * diff --git a/core/src/main/java/com/amazon/opendistroforelasticsearch/sql/ast/AbstractNodeVisitor.java b/core/src/main/java/com/amazon/opendistroforelasticsearch/sql/ast/AbstractNodeVisitor.java index f91212023a..02806fbcfd 100644 --- a/core/src/main/java/com/amazon/opendistroforelasticsearch/sql/ast/AbstractNodeVisitor.java +++ b/core/src/main/java/com/amazon/opendistroforelasticsearch/sql/ast/AbstractNodeVisitor.java @@ -17,6 +17,7 @@ import com.amazon.opendistroforelasticsearch.sql.ast.expression.AggregateFunction; import com.amazon.opendistroforelasticsearch.sql.ast.expression.Alias; +import com.amazon.opendistroforelasticsearch.sql.ast.expression.AllFields; import com.amazon.opendistroforelasticsearch.sql.ast.expression.And; import com.amazon.opendistroforelasticsearch.sql.ast.expression.Argument; import com.amazon.opendistroforelasticsearch.sql.ast.expression.AttributeList; @@ -183,4 +184,8 @@ public T visitValues(Values node, C context) { public T visitAlias(Alias node, C context) { return visitChildren(node, context); } + + public T visitAllFields(AllFields node, C context) { + return visitChildren(node, context); + } } diff --git a/core/src/main/java/com/amazon/opendistroforelasticsearch/sql/ast/dsl/AstDSL.java b/core/src/main/java/com/amazon/opendistroforelasticsearch/sql/ast/dsl/AstDSL.java index cb9578a6bf..4b85c9b2bf 100644 --- a/core/src/main/java/com/amazon/opendistroforelasticsearch/sql/ast/dsl/AstDSL.java +++ b/core/src/main/java/com/amazon/opendistroforelasticsearch/sql/ast/dsl/AstDSL.java @@ -17,6 +17,7 @@ import com.amazon.opendistroforelasticsearch.sql.ast.expression.AggregateFunction; import com.amazon.opendistroforelasticsearch.sql.ast.expression.Alias; +import com.amazon.opendistroforelasticsearch.sql.ast.expression.AllFields; import com.amazon.opendistroforelasticsearch.sql.ast.expression.And; import com.amazon.opendistroforelasticsearch.sql.ast.expression.Argument; import com.amazon.opendistroforelasticsearch.sql.ast.expression.Compare; diff --git a/core/src/main/java/com/amazon/opendistroforelasticsearch/sql/ast/expression/AllFields.java b/core/src/main/java/com/amazon/opendistroforelasticsearch/sql/ast/expression/AllFields.java new file mode 100644 index 0000000000..13d487a3f8 --- /dev/null +++ b/core/src/main/java/com/amazon/opendistroforelasticsearch/sql/ast/expression/AllFields.java @@ -0,0 +1,43 @@ +/* + * + * Copyright 2020 Amazon.com, Inc. or its affiliates. All Rights Reserved. + * + * Licensed under the Apache License, Version 2.0 (the "License"). + * You may not use this file except in compliance with the License. + * A copy of the License is located at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * or in the "license" file accompanying this file. This file 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 com.amazon.opendistroforelasticsearch.sql.ast.expression; + +import com.amazon.opendistroforelasticsearch.sql.ast.AbstractNodeVisitor; +import lombok.EqualsAndHashCode; +import lombok.ToString; + +/** + * Represent the All fields which is been used in SELECT *. + */ +@ToString +@EqualsAndHashCode(callSuper = false) +public class AllFields extends UnresolvedExpression { + public static final AllFields INSTANCE = new AllFields(); + + private AllFields() { + } + + public static AllFields of() { + return INSTANCE; + } + + @Override + public R accept(AbstractNodeVisitor nodeVisitor, C context) { + return nodeVisitor.visitAllFields(this, context); + } +} diff --git a/core/src/main/java/com/amazon/opendistroforelasticsearch/sql/ast/tree/Project.java b/core/src/main/java/com/amazon/opendistroforelasticsearch/sql/ast/tree/Project.java index c11018192f..e7f58ddd1e 100644 --- a/core/src/main/java/com/amazon/opendistroforelasticsearch/sql/ast/tree/Project.java +++ b/core/src/main/java/com/amazon/opendistroforelasticsearch/sql/ast/tree/Project.java @@ -52,6 +52,17 @@ public boolean hasArgument() { return !argExprList.isEmpty(); } + /** + * The Project could been used to exclude fields from the source. + */ + public boolean isExcluded() { + if (hasArgument()) { + Argument argument = argExprList.get(0); + return (Boolean) argument.getValue().getValue(); + } + return false; + } + @Override public Project attach(UnresolvedPlan child) { this.child = child; diff --git a/core/src/main/java/com/amazon/opendistroforelasticsearch/sql/data/model/ExprMissingValue.java b/core/src/main/java/com/amazon/opendistroforelasticsearch/sql/data/model/ExprMissingValue.java index 2d822c47f1..635d8f2fd4 100644 --- a/core/src/main/java/com/amazon/opendistroforelasticsearch/sql/data/model/ExprMissingValue.java +++ b/core/src/main/java/com/amazon/opendistroforelasticsearch/sql/data/model/ExprMissingValue.java @@ -35,7 +35,7 @@ public static ExprMissingValue of() { @Override public Object value() { - throw new ExpressionEvaluationException("invalid to call value operation on missing value"); + return null; } @Override diff --git a/core/src/main/java/com/amazon/opendistroforelasticsearch/sql/executor/ExecutionEngine.java b/core/src/main/java/com/amazon/opendistroforelasticsearch/sql/executor/ExecutionEngine.java index ec6e5bfbf2..d75379ee1e 100644 --- a/core/src/main/java/com/amazon/opendistroforelasticsearch/sql/executor/ExecutionEngine.java +++ b/core/src/main/java/com/amazon/opendistroforelasticsearch/sql/executor/ExecutionEngine.java @@ -18,6 +18,7 @@ import com.amazon.opendistroforelasticsearch.sql.common.response.ResponseListener; import com.amazon.opendistroforelasticsearch.sql.data.model.ExprValue; +import com.amazon.opendistroforelasticsearch.sql.data.type.ExprType; import com.amazon.opendistroforelasticsearch.sql.planner.physical.PhysicalPlan; import java.util.List; import lombok.Data; @@ -40,7 +41,20 @@ public interface ExecutionEngine { */ @Data class QueryResponse { + private final Schema schema; private final List results; } + @Data + class Schema { + private final List columns; + + @Data + public static class Column { + private final String name; + private final String alias; + private final ExprType exprType; + } + } + } diff --git a/core/src/main/java/com/amazon/opendistroforelasticsearch/sql/planner/physical/PhysicalPlan.java b/core/src/main/java/com/amazon/opendistroforelasticsearch/sql/planner/physical/PhysicalPlan.java index e7f64c84ec..56d6fc6e83 100644 --- a/core/src/main/java/com/amazon/opendistroforelasticsearch/sql/planner/physical/PhysicalPlan.java +++ b/core/src/main/java/com/amazon/opendistroforelasticsearch/sql/planner/physical/PhysicalPlan.java @@ -16,6 +16,7 @@ package com.amazon.opendistroforelasticsearch.sql.planner.physical; import com.amazon.opendistroforelasticsearch.sql.data.model.ExprValue; +import com.amazon.opendistroforelasticsearch.sql.executor.ExecutionEngine; import com.amazon.opendistroforelasticsearch.sql.planner.PlanNode; import java.util.Iterator; @@ -43,4 +44,9 @@ public void open() { public void close() { getChild().forEach(PhysicalPlan::close); } + + public ExecutionEngine.Schema schema() { + throw new IllegalStateException(String.format("[BUG] schema can been only applied to " + + "ProjectOperator, instead of %s", toString())); + } } diff --git a/core/src/main/java/com/amazon/opendistroforelasticsearch/sql/planner/physical/ProjectOperator.java b/core/src/main/java/com/amazon/opendistroforelasticsearch/sql/planner/physical/ProjectOperator.java index 28c1de18dc..4576c80946 100644 --- a/core/src/main/java/com/amazon/opendistroforelasticsearch/sql/planner/physical/ProjectOperator.java +++ b/core/src/main/java/com/amazon/opendistroforelasticsearch/sql/planner/physical/ProjectOperator.java @@ -17,11 +17,13 @@ import com.amazon.opendistroforelasticsearch.sql.data.model.ExprTupleValue; import com.amazon.opendistroforelasticsearch.sql.data.model.ExprValue; +import com.amazon.opendistroforelasticsearch.sql.executor.ExecutionEngine; import com.amazon.opendistroforelasticsearch.sql.expression.NamedExpression; import com.google.common.collect.ImmutableMap; import com.google.common.collect.ImmutableMap.Builder; import java.util.Collections; import java.util.List; +import java.util.stream.Collectors; import lombok.EqualsAndHashCode; import lombok.Getter; import lombok.RequiredArgsConstructor; @@ -60,11 +62,15 @@ public ExprValue next() { ImmutableMap.Builder mapBuilder = new Builder<>(); for (NamedExpression expr : projectList) { ExprValue exprValue = expr.valueOf(inputValue.bindingTuples()); - // missing value is ignored. - if (!exprValue.isMissing()) { - mapBuilder.put(expr.getName(), exprValue); - } + mapBuilder.put(expr.getName(), exprValue); } return ExprTupleValue.fromExprValueMap(mapBuilder.build()); } + + @Override + public ExecutionEngine.Schema schema() { + return new ExecutionEngine.Schema(getProjectList().stream() + .map(expr -> new ExecutionEngine.Schema.Column(expr.getName(), + expr.getName(), expr.type())).collect(Collectors.toList())); + } } diff --git a/core/src/test/java/com/amazon/opendistroforelasticsearch/sql/analysis/AnalyzerTest.java b/core/src/test/java/com/amazon/opendistroforelasticsearch/sql/analysis/AnalyzerTest.java index d705854661..df804cd262 100644 --- a/core/src/test/java/com/amazon/opendistroforelasticsearch/sql/analysis/AnalyzerTest.java +++ b/core/src/test/java/com/amazon/opendistroforelasticsearch/sql/analysis/AnalyzerTest.java @@ -32,13 +32,21 @@ import com.amazon.opendistroforelasticsearch.sql.ast.dsl.AstDSL; import com.amazon.opendistroforelasticsearch.sql.exception.SemanticCheckException; import com.amazon.opendistroforelasticsearch.sql.expression.DSL; +import com.amazon.opendistroforelasticsearch.sql.expression.config.ExpressionConfig; import com.amazon.opendistroforelasticsearch.sql.planner.logical.LogicalPlanDSL; import com.google.common.collect.ImmutableList; import com.google.common.collect.ImmutableMap; import java.util.Collections; import org.junit.jupiter.api.Disabled; import org.junit.jupiter.api.Test; +import org.junit.jupiter.api.extension.ExtendWith; +import org.springframework.context.annotation.Configuration; +import org.springframework.test.context.ContextConfiguration; +import org.springframework.test.context.junit.jupiter.SpringExtension; +@Configuration +@ExtendWith(SpringExtension.class) +@ContextConfiguration(classes = {ExpressionConfig.class, AnalyzerTest.class}) class AnalyzerTest extends AnalyzerTestBase { @Test public void filter_relation() { diff --git a/core/src/test/java/com/amazon/opendistroforelasticsearch/sql/analysis/AnalyzerTestBase.java b/core/src/test/java/com/amazon/opendistroforelasticsearch/sql/analysis/AnalyzerTestBase.java index 18e23cf1ba..1a7173fca5 100644 --- a/core/src/test/java/com/amazon/opendistroforelasticsearch/sql/analysis/AnalyzerTestBase.java +++ b/core/src/test/java/com/amazon/opendistroforelasticsearch/sql/analysis/AnalyzerTestBase.java @@ -17,29 +17,77 @@ import static org.junit.jupiter.api.Assertions.assertEquals; +import com.amazon.opendistroforelasticsearch.sql.analysis.symbol.Namespace; +import com.amazon.opendistroforelasticsearch.sql.analysis.symbol.Symbol; import com.amazon.opendistroforelasticsearch.sql.analysis.symbol.SymbolTable; import com.amazon.opendistroforelasticsearch.sql.ast.tree.UnresolvedPlan; import com.amazon.opendistroforelasticsearch.sql.config.TestConfig; import com.amazon.opendistroforelasticsearch.sql.data.type.ExprType; +import com.amazon.opendistroforelasticsearch.sql.exception.ExpressionEvaluationException; import com.amazon.opendistroforelasticsearch.sql.expression.DSL; import com.amazon.opendistroforelasticsearch.sql.expression.Expression; -import com.amazon.opendistroforelasticsearch.sql.expression.config.ExpressionConfig; +import com.amazon.opendistroforelasticsearch.sql.expression.ReferenceExpression; import com.amazon.opendistroforelasticsearch.sql.expression.env.Environment; import com.amazon.opendistroforelasticsearch.sql.expression.function.BuiltinFunctionRepository; import com.amazon.opendistroforelasticsearch.sql.planner.logical.LogicalPlan; +import com.amazon.opendistroforelasticsearch.sql.planner.physical.PhysicalPlan; import com.amazon.opendistroforelasticsearch.sql.storage.StorageEngine; -import org.junit.jupiter.api.extension.ExtendWith; +import com.amazon.opendistroforelasticsearch.sql.storage.Table; +import java.util.Map; import org.springframework.beans.factory.annotation.Autowired; import org.springframework.context.annotation.Bean; -import org.springframework.context.annotation.Configuration; -import org.springframework.test.context.ContextConfiguration; -import org.springframework.test.context.junit.jupiter.SpringExtension; -@Configuration -@ExtendWith(SpringExtension.class) -@ContextConfiguration(classes = {ExpressionConfig.class, AnalyzerTestBase.class, TestConfig.class}) + public class AnalyzerTestBase { + protected Map typeMapping() { + return TestConfig.typeMapping; + } + + @Bean + protected StorageEngine storageEngine() { + return new StorageEngine() { + @Override + public Table getTable(String name) { + return new Table() { + @Override + public Map getFieldTypes() { + return typeMapping(); + } + + @Override + public PhysicalPlan implement(LogicalPlan plan) { + throw new UnsupportedOperationException(); + } + }; + } + }; + } + + + @Bean + protected SymbolTable symbolTable() { + SymbolTable symbolTable = new SymbolTable(); + typeMapping().entrySet() + .forEach( + entry -> symbolTable + .store(new Symbol(Namespace.FIELD_NAME, entry.getKey()), entry.getValue())); + return symbolTable; + } + + @Bean + protected Environment typeEnv() { + return var -> { + if (var instanceof ReferenceExpression) { + ReferenceExpression refExpr = (ReferenceExpression) var; + if (typeMapping().containsKey(refExpr.getAttr())) { + return typeMapping().get(refExpr.getAttr()); + } + } + throw new ExpressionEvaluationException("type resolved failed"); + }; + } + @Autowired protected DSL dsl; diff --git a/core/src/test/java/com/amazon/opendistroforelasticsearch/sql/analysis/ExpressionAnalyzerTest.java b/core/src/test/java/com/amazon/opendistroforelasticsearch/sql/analysis/ExpressionAnalyzerTest.java index 9d7535253c..68b74e5f38 100644 --- a/core/src/test/java/com/amazon/opendistroforelasticsearch/sql/analysis/ExpressionAnalyzerTest.java +++ b/core/src/test/java/com/amazon/opendistroforelasticsearch/sql/analysis/ExpressionAnalyzerTest.java @@ -29,9 +29,16 @@ import com.amazon.opendistroforelasticsearch.sql.exception.SemanticCheckException; import com.amazon.opendistroforelasticsearch.sql.expression.DSL; import com.amazon.opendistroforelasticsearch.sql.expression.Expression; +import com.amazon.opendistroforelasticsearch.sql.expression.config.ExpressionConfig; import org.junit.jupiter.api.Test; - - +import org.junit.jupiter.api.extension.ExtendWith; +import org.springframework.context.annotation.Configuration; +import org.springframework.test.context.ContextConfiguration; +import org.springframework.test.context.junit.jupiter.SpringExtension; + +@Configuration +@ExtendWith(SpringExtension.class) +@ContextConfiguration(classes = {ExpressionConfig.class, AnalyzerTestBase.class}) class ExpressionAnalyzerTest extends AnalyzerTestBase { @Test @@ -82,22 +89,6 @@ public void qualified_name() { ); } - @Test - public void named_expression() { - assertAnalyzeEqual( - DSL.named("int", DSL.ref("integer_value", INTEGER)), - AstDSL.alias("int", AstDSL.qualifiedName("integer_value")) - ); - } - - @Test - public void named_expression_with_alias() { - assertAnalyzeEqual( - DSL.named("integer", DSL.ref("integer_value", INTEGER), "int"), - AstDSL.alias("integer", AstDSL.qualifiedName("integer_value"), "int") - ); - } - @Test public void skip_identifier_with_qualifier() { SyntaxCheckException exception = diff --git a/core/src/test/java/com/amazon/opendistroforelasticsearch/sql/analysis/SelectAnalyzeTest.java b/core/src/test/java/com/amazon/opendistroforelasticsearch/sql/analysis/SelectAnalyzeTest.java new file mode 100644 index 0000000000..376e7538c9 --- /dev/null +++ b/core/src/test/java/com/amazon/opendistroforelasticsearch/sql/analysis/SelectAnalyzeTest.java @@ -0,0 +1,161 @@ +/* + * + * Copyright 2020 Amazon.com, Inc. or its affiliates. All Rights Reserved. + * + * Licensed under the Apache License, Version 2.0 (the "License"). + * You may not use this file except in compliance with the License. + * A copy of the License is located at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * or in the "license" file accompanying this file. This file 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 com.amazon.opendistroforelasticsearch.sql.analysis; + +import static com.amazon.opendistroforelasticsearch.sql.ast.dsl.AstDSL.argument; +import static com.amazon.opendistroforelasticsearch.sql.ast.dsl.AstDSL.booleanLiteral; +import static com.amazon.opendistroforelasticsearch.sql.ast.dsl.AstDSL.field; +import static com.amazon.opendistroforelasticsearch.sql.data.type.ExprCoreType.DOUBLE; +import static com.amazon.opendistroforelasticsearch.sql.data.type.ExprCoreType.INTEGER; +import static com.amazon.opendistroforelasticsearch.sql.data.type.ExprCoreType.STRING; + +import com.amazon.opendistroforelasticsearch.sql.ast.dsl.AstDSL; +import com.amazon.opendistroforelasticsearch.sql.ast.expression.AllFields; +import com.amazon.opendistroforelasticsearch.sql.data.type.ExprCoreType; +import com.amazon.opendistroforelasticsearch.sql.data.type.ExprType; +import com.amazon.opendistroforelasticsearch.sql.expression.DSL; +import com.amazon.opendistroforelasticsearch.sql.expression.config.ExpressionConfig; +import com.amazon.opendistroforelasticsearch.sql.planner.logical.LogicalPlanDSL; +import com.google.common.collect.ImmutableList; +import com.google.common.collect.ImmutableMap; +import java.util.Map; +import org.junit.jupiter.api.Test; +import org.junit.jupiter.api.extension.ExtendWith; +import org.springframework.context.annotation.Configuration; +import org.springframework.test.context.ContextConfiguration; +import org.springframework.test.context.junit.jupiter.SpringExtension; + +@Configuration +@ExtendWith(SpringExtension.class) +@ContextConfiguration(classes = {ExpressionConfig.class, SelectAnalyzeTest.class}) +public class SelectAnalyzeTest extends AnalyzerTestBase { + + @Override + protected Map typeMapping() { + return new ImmutableMap.Builder() + .put("integer_value", ExprCoreType.INTEGER) + .put("double_value", ExprCoreType.DOUBLE) + .put("string_value", ExprCoreType.STRING) + .build(); + } + + @Test + public void project_all_from_source() { + assertAnalyzeEqual( + LogicalPlanDSL.project( + LogicalPlanDSL.relation("schema"), + DSL.named("integer_value", DSL.ref("integer_value", INTEGER)), + DSL.named("double_value", DSL.ref("double_value", DOUBLE)), + DSL.named("string_value", DSL.ref("string_value", STRING)), + DSL.named("integer_value", DSL.ref("integer_value", INTEGER)), + DSL.named("double_value", DSL.ref("double_value", DOUBLE)) + ), + AstDSL.projectWithArg( + AstDSL.relation("schema"), + AstDSL.defaultFieldsArgs(), + AstDSL.field("integer_value"), // Field not wrapped by Alias + AstDSL.alias("double_value", AstDSL.field("double_value")), + AllFields.of())); + } + + @Test + public void select_and_project_all() { + assertAnalyzeEqual( + LogicalPlanDSL.project( + LogicalPlanDSL.project( + LogicalPlanDSL.relation("schema"), + DSL.named("integer_value", DSL.ref("integer_value", INTEGER)), + DSL.named("double_value", DSL.ref("double_value", DOUBLE)) + ), + DSL.named("integer_value", DSL.ref("integer_value", INTEGER)), + DSL.named("double_value", DSL.ref("double_value", DOUBLE)) + ), + AstDSL.projectWithArg( + AstDSL.projectWithArg( + AstDSL.relation("schema"), + AstDSL.defaultFieldsArgs(), + AstDSL.field("integer_value"), + AstDSL.field("double_value")), + AstDSL.defaultFieldsArgs(), + AllFields.of() + )); + } + + @Test + public void remove_and_project_all() { + assertAnalyzeEqual( + LogicalPlanDSL.project( + LogicalPlanDSL.remove( + LogicalPlanDSL.relation("schema"), + DSL.ref("integer_value", INTEGER), + DSL.ref("double_value", DOUBLE) + ), + DSL.named("string_value", DSL.ref("string_value", STRING)) + ), + AstDSL.projectWithArg( + AstDSL.projectWithArg( + AstDSL.relation("schema"), + AstDSL.exprList(argument("exclude", booleanLiteral(true))), + AstDSL.field("integer_value"), + AstDSL.field("double_value")), + AstDSL.defaultFieldsArgs(), + AllFields.of() + )); + } + + @Test + public void stats_and_project_all() { + assertAnalyzeEqual( + LogicalPlanDSL.project( + LogicalPlanDSL.aggregation( + LogicalPlanDSL.relation("schema"), + ImmutableList.of(dsl.avg(DSL.ref("integer_value", INTEGER))), + ImmutableList.of(DSL.ref("string_value", STRING))), + DSL.named("string_value", DSL.ref("string_value", STRING)), + DSL.named("avg(integer_value)", DSL.ref("avg(integer_value)", DOUBLE)) + ), + AstDSL.projectWithArg( + AstDSL.agg( + AstDSL.relation("schema"), + AstDSL.exprList(AstDSL.aggregate("avg", field("integer_value"))), + null, + ImmutableList.of(field("string_value")), + AstDSL.defaultStatsArgs()), AstDSL.defaultFieldsArgs(), + AllFields.of())); + } + + @Test + public void rename_and_project_all() { + assertAnalyzeEqual( + LogicalPlanDSL.project( + LogicalPlanDSL.rename( + LogicalPlanDSL.relation("schema"), + ImmutableMap.of(DSL.ref("integer_value", INTEGER), DSL.ref("ivalue", INTEGER))), + DSL.named("ivalue", DSL.ref("ivalue", INTEGER)), + DSL.named("string_value", DSL.ref("string_value", STRING)), + DSL.named("double_value", DSL.ref("double_value", DOUBLE)) + ), + AstDSL.projectWithArg( + AstDSL.rename( + AstDSL.relation("schema"), + AstDSL.map(AstDSL.field("integer_value"), AstDSL.field("ivalue"))), + AstDSL.defaultFieldsArgs(), + AllFields.of() + )); + } +} diff --git a/core/src/test/java/com/amazon/opendistroforelasticsearch/sql/analysis/SelectExpressionAnalyzerTest.java b/core/src/test/java/com/amazon/opendistroforelasticsearch/sql/analysis/SelectExpressionAnalyzerTest.java new file mode 100644 index 0000000000..42aa1d06f5 --- /dev/null +++ b/core/src/test/java/com/amazon/opendistroforelasticsearch/sql/analysis/SelectExpressionAnalyzerTest.java @@ -0,0 +1,69 @@ +/* + * + * Copyright 2020 Amazon.com, Inc. or its affiliates. All Rights Reserved. + * + * Licensed under the Apache License, Version 2.0 (the "License"). + * You may not use this file except in compliance with the License. + * A copy of the License is located at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * or in the "license" file accompanying this file. This file 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 com.amazon.opendistroforelasticsearch.sql.analysis; + +import static com.amazon.opendistroforelasticsearch.sql.data.type.ExprCoreType.INTEGER; +import static org.junit.jupiter.api.Assertions.assertEquals; + +import com.amazon.opendistroforelasticsearch.sql.ast.dsl.AstDSL; +import com.amazon.opendistroforelasticsearch.sql.ast.expression.UnresolvedExpression; +import com.amazon.opendistroforelasticsearch.sql.expression.DSL; +import com.amazon.opendistroforelasticsearch.sql.expression.NamedExpression; +import com.amazon.opendistroforelasticsearch.sql.expression.config.ExpressionConfig; +import java.util.Arrays; +import java.util.List; +import org.junit.jupiter.api.Test; +import org.junit.jupiter.api.extension.ExtendWith; +import org.springframework.context.annotation.Configuration; +import org.springframework.test.context.ContextConfiguration; +import org.springframework.test.context.junit.jupiter.SpringExtension; + +@Configuration +@ExtendWith(SpringExtension.class) +@ContextConfiguration(classes = {ExpressionConfig.class, SelectExpressionAnalyzerTest.class}) +public class SelectExpressionAnalyzerTest extends AnalyzerTestBase { + + + @Test + public void named_expression() { + assertAnalyzeEqual( + DSL.named("int", DSL.ref("integer_value", INTEGER)), + AstDSL.alias("int", AstDSL.qualifiedName("integer_value")) + ); + } + + @Test + public void named_expression_with_alias() { + assertAnalyzeEqual( + DSL.named("integer", DSL.ref("integer_value", INTEGER), "int"), + AstDSL.alias("integer", AstDSL.qualifiedName("integer_value"), "int") + ); + } + + protected List analyze(UnresolvedExpression unresolvedExpression) { + + return new SelectExpressionAnalyzer(expressionAnalyzer) + .analyze(Arrays.asList(unresolvedExpression), + analysisContext); + } + + protected void assertAnalyzeEqual(NamedExpression expected, + UnresolvedExpression unresolvedExpression) { + assertEquals(Arrays.asList(expected), analyze(unresolvedExpression)); + } +} diff --git a/core/src/test/java/com/amazon/opendistroforelasticsearch/sql/analysis/symbol/SymbolTableTest.java b/core/src/test/java/com/amazon/opendistroforelasticsearch/sql/analysis/symbol/SymbolTableTest.java index 93cc481667..f2557edbf3 100644 --- a/core/src/test/java/com/amazon/opendistroforelasticsearch/sql/analysis/symbol/SymbolTableTest.java +++ b/core/src/test/java/com/amazon/opendistroforelasticsearch/sql/analysis/symbol/SymbolTableTest.java @@ -30,18 +30,35 @@ import com.amazon.opendistroforelasticsearch.sql.data.type.ExprType; import java.util.Map; import java.util.Optional; +import org.junit.jupiter.api.BeforeEach; import org.junit.jupiter.api.Test; public class SymbolTableTest { - private final SymbolTable symbolTable = new SymbolTable(); + private SymbolTable symbolTable; + + @BeforeEach + public void setup() { + symbolTable = new SymbolTable(); + } @Test public void defineFieldSymbolShouldBeAbleToResolve() { defineSymbolShouldBeAbleToResolve(new Symbol(Namespace.FIELD_NAME, "age"), INTEGER); } + @Test + public void removeSymbolCannotBeResolve() { + symbolTable.store(new Symbol(Namespace.FIELD_NAME, "age"), INTEGER); + + Optional age = symbolTable.lookup(new Symbol(Namespace.FIELD_NAME, "age")); + assertTrue(age.isPresent()); + + symbolTable.remove(new Symbol(Namespace.FIELD_NAME, "age")); + age = symbolTable.lookup(new Symbol(Namespace.FIELD_NAME, "age")); + assertFalse(age.isPresent()); + } @Test public void defineFieldSymbolShouldBeAbleToResolveByPrefix() { @@ -61,6 +78,24 @@ public void defineFieldSymbolShouldBeAbleToResolveByPrefix() { ); } + @Test + public void lookupAllFieldsReturnFieldsWithoutDot() { + symbolTable.store(new Symbol(Namespace.FIELD_NAME, "active"), BOOLEAN); + symbolTable.store(new Symbol(Namespace.FIELD_NAME, "s.address"), STRING); + symbolTable.store(new Symbol(Namespace.FIELD_NAME, "s.manager.name"), STRING); + + Map typeByName = + symbolTable.lookupAllFields(Namespace.FIELD_NAME); + + assertThat( + typeByName, + allOf( + aMapWithSize(1), + hasEntry("active", BOOLEAN) + ) + ); + } + @Test public void failedToResolveSymbolNoNamespaceMatched() { symbolTable.store(new Symbol(Namespace.FUNCTION_NAME, "customFunction"), BOOLEAN); diff --git a/core/src/test/java/com/amazon/opendistroforelasticsearch/sql/config/TestConfig.java b/core/src/test/java/com/amazon/opendistroforelasticsearch/sql/config/TestConfig.java index e3eb0b043c..59a13fb8e2 100644 --- a/core/src/test/java/com/amazon/opendistroforelasticsearch/sql/config/TestConfig.java +++ b/core/src/test/java/com/amazon/opendistroforelasticsearch/sql/config/TestConfig.java @@ -47,7 +47,7 @@ public class TestConfig { public static final String STRING_TYPE_NULL_VALUE_FILED = "string_null_value"; public static final String STRING_TYPE_MISSING_VALUE_FILED = "string_missing_value"; - private static Map typeMapping = new ImmutableMap.Builder() + public static Map typeMapping = new ImmutableMap.Builder() .put("integer_value", ExprCoreType.INTEGER) .put(INT_TYPE_NULL_VALUE_FIELD, ExprCoreType.INTEGER) .put(INT_TYPE_MISSING_VALUE_FIELD, ExprCoreType.INTEGER) diff --git a/core/src/test/java/com/amazon/opendistroforelasticsearch/sql/data/model/ExprMissingValueTest.java b/core/src/test/java/com/amazon/opendistroforelasticsearch/sql/data/model/ExprMissingValueTest.java index 337eef847b..26f2ea3699 100644 --- a/core/src/test/java/com/amazon/opendistroforelasticsearch/sql/data/model/ExprMissingValueTest.java +++ b/core/src/test/java/com/amazon/opendistroforelasticsearch/sql/data/model/ExprMissingValueTest.java @@ -20,6 +20,7 @@ import static com.amazon.opendistroforelasticsearch.sql.utils.ComparisonUtil.compare; import static org.junit.jupiter.api.Assertions.assertEquals; import static org.junit.jupiter.api.Assertions.assertFalse; +import static org.junit.jupiter.api.Assertions.assertNull; import static org.junit.jupiter.api.Assertions.assertThrows; import static org.junit.jupiter.api.Assertions.assertTrue; @@ -37,9 +38,7 @@ public void test_is_missing() { @Test public void getValue() { - ExpressionEvaluationException exception = assertThrows(ExpressionEvaluationException.class, - () -> LITERAL_MISSING.value()); - assertEquals("invalid to call value operation on missing value", exception.getMessage()); + assertNull(LITERAL_MISSING.value()); } @Test diff --git a/core/src/test/java/com/amazon/opendistroforelasticsearch/sql/data/model/ExprTupleValueTest.java b/core/src/test/java/com/amazon/opendistroforelasticsearch/sql/data/model/ExprTupleValueTest.java index cd79cc13d5..e78c830058 100644 --- a/core/src/test/java/com/amazon/opendistroforelasticsearch/sql/data/model/ExprTupleValueTest.java +++ b/core/src/test/java/com/amazon/opendistroforelasticsearch/sql/data/model/ExprTupleValueTest.java @@ -18,6 +18,7 @@ import static com.amazon.opendistroforelasticsearch.sql.utils.ComparisonUtil.compare; import static org.junit.jupiter.api.Assertions.assertEquals; import static org.junit.jupiter.api.Assertions.assertFalse; +import static org.junit.jupiter.api.Assertions.assertNotEquals; import static org.junit.jupiter.api.Assertions.assertThrows; import static org.junit.jupiter.api.Assertions.assertTrue; @@ -39,6 +40,15 @@ public void tuple_compare_int() { assertFalse(tupleValue.equals(intValue)); } + @Test + public void compare_tuple_with_different_key() { + ExprValue tupleValue1 = ExprValueUtils.tupleValue(ImmutableMap.of("value", 2)); + ExprValue tupleValue2 = + ExprValueUtils.tupleValue(ImmutableMap.of("integer_value", 2, "float_value", 1f)); + assertNotEquals(tupleValue1, tupleValue2); + assertNotEquals(tupleValue2, tupleValue1); + } + @Test public void compare_tuple_with_different_size() { ExprValue tupleValue1 = ExprValueUtils.tupleValue(ImmutableMap.of("integer_value", 2)); diff --git a/core/src/test/java/com/amazon/opendistroforelasticsearch/sql/expression/NamedExpressionTest.java b/core/src/test/java/com/amazon/opendistroforelasticsearch/sql/expression/NamedExpressionTest.java index 77196099ff..c8330d2fba 100644 --- a/core/src/test/java/com/amazon/opendistroforelasticsearch/sql/expression/NamedExpressionTest.java +++ b/core/src/test/java/com/amazon/opendistroforelasticsearch/sql/expression/NamedExpressionTest.java @@ -42,4 +42,13 @@ void name_an_expression_with_alias() { assertEquals("ten", namedExpression.getName()); } + @Test + void name_an_named_expression() { + LiteralExpression delegated = DSL.literal(10); + Expression expression = DSL.named("10", delegated, "ten"); + + NamedExpression namedExpression = DSL.named(expression); + assertEquals("ten", namedExpression.getName()); + } + } \ No newline at end of file diff --git a/core/src/test/java/com/amazon/opendistroforelasticsearch/sql/planner/logical/LogicalDedupeTest.java b/core/src/test/java/com/amazon/opendistroforelasticsearch/sql/planner/logical/LogicalDedupeTest.java index ffb8becb04..ce856d3cbe 100644 --- a/core/src/test/java/com/amazon/opendistroforelasticsearch/sql/planner/logical/LogicalDedupeTest.java +++ b/core/src/test/java/com/amazon/opendistroforelasticsearch/sql/planner/logical/LogicalDedupeTest.java @@ -28,8 +28,16 @@ import com.amazon.opendistroforelasticsearch.sql.analysis.AnalyzerTestBase; import com.amazon.opendistroforelasticsearch.sql.expression.DSL; +import com.amazon.opendistroforelasticsearch.sql.expression.config.ExpressionConfig; import org.junit.jupiter.api.Test; +import org.junit.jupiter.api.extension.ExtendWith; +import org.springframework.context.annotation.Configuration; +import org.springframework.test.context.ContextConfiguration; +import org.springframework.test.context.junit.jupiter.SpringExtension; +@Configuration +@ExtendWith(SpringExtension.class) +@ContextConfiguration(classes = {ExpressionConfig.class, AnalyzerTestBase.class}) class LogicalDedupeTest extends AnalyzerTestBase { @Test public void analyze_dedup_with_two_field_with_default_option() { diff --git a/core/src/test/java/com/amazon/opendistroforelasticsearch/sql/planner/logical/LogicalEvalTest.java b/core/src/test/java/com/amazon/opendistroforelasticsearch/sql/planner/logical/LogicalEvalTest.java index cb985f8d6b..df0c7aa0cf 100644 --- a/core/src/test/java/com/amazon/opendistroforelasticsearch/sql/planner/logical/LogicalEvalTest.java +++ b/core/src/test/java/com/amazon/opendistroforelasticsearch/sql/planner/logical/LogicalEvalTest.java @@ -21,11 +21,18 @@ import com.amazon.opendistroforelasticsearch.sql.analysis.AnalyzerTestBase; import com.amazon.opendistroforelasticsearch.sql.ast.dsl.AstDSL; import com.amazon.opendistroforelasticsearch.sql.expression.DSL; +import com.amazon.opendistroforelasticsearch.sql.expression.config.ExpressionConfig; import org.apache.commons.lang3.tuple.ImmutablePair; import org.junit.jupiter.api.Test; import org.junit.jupiter.api.extension.ExtendWith; import org.mockito.junit.jupiter.MockitoExtension; +import org.springframework.context.annotation.Configuration; +import org.springframework.test.context.ContextConfiguration; +import org.springframework.test.context.junit.jupiter.SpringExtension; +@Configuration +@ExtendWith(SpringExtension.class) +@ContextConfiguration(classes = {ExpressionConfig.class, AnalyzerTestBase.class}) @ExtendWith(MockitoExtension.class) public class LogicalEvalTest extends AnalyzerTestBase { diff --git a/core/src/test/java/com/amazon/opendistroforelasticsearch/sql/planner/logical/LogicalSortTest.java b/core/src/test/java/com/amazon/opendistroforelasticsearch/sql/planner/logical/LogicalSortTest.java index a194fa62e6..15bb012c6f 100644 --- a/core/src/test/java/com/amazon/opendistroforelasticsearch/sql/planner/logical/LogicalSortTest.java +++ b/core/src/test/java/com/amazon/opendistroforelasticsearch/sql/planner/logical/LogicalSortTest.java @@ -31,9 +31,17 @@ import com.amazon.opendistroforelasticsearch.sql.analysis.AnalyzerTestBase; import com.amazon.opendistroforelasticsearch.sql.ast.tree.Sort.SortOption; import com.amazon.opendistroforelasticsearch.sql.expression.DSL; +import com.amazon.opendistroforelasticsearch.sql.expression.config.ExpressionConfig; import org.apache.commons.lang3.tuple.ImmutablePair; import org.junit.jupiter.api.Test; +import org.junit.jupiter.api.extension.ExtendWith; +import org.springframework.context.annotation.Configuration; +import org.springframework.test.context.ContextConfiguration; +import org.springframework.test.context.junit.jupiter.SpringExtension; +@Configuration +@ExtendWith(SpringExtension.class) +@ContextConfiguration(classes = {ExpressionConfig.class, AnalyzerTestBase.class}) class LogicalSortTest extends AnalyzerTestBase { @Test public void analyze_sort_with_two_field_with_default_option() { diff --git a/core/src/test/java/com/amazon/opendistroforelasticsearch/sql/planner/physical/ProjectOperatorTest.java b/core/src/test/java/com/amazon/opendistroforelasticsearch/sql/planner/physical/ProjectOperatorTest.java index ec6af198d7..02e4e0734a 100644 --- a/core/src/test/java/com/amazon/opendistroforelasticsearch/sql/planner/physical/ProjectOperatorTest.java +++ b/core/src/test/java/com/amazon/opendistroforelasticsearch/sql/planner/physical/ProjectOperatorTest.java @@ -15,17 +15,22 @@ package com.amazon.opendistroforelasticsearch.sql.planner.physical; +import static com.amazon.opendistroforelasticsearch.sql.data.model.ExprValueUtils.LITERAL_MISSING; +import static com.amazon.opendistroforelasticsearch.sql.data.model.ExprValueUtils.stringValue; import static com.amazon.opendistroforelasticsearch.sql.data.type.ExprCoreType.INTEGER; import static com.amazon.opendistroforelasticsearch.sql.data.type.ExprCoreType.STRING; import static com.amazon.opendistroforelasticsearch.sql.planner.physical.PhysicalPlanDSL.project; import static org.hamcrest.MatcherAssert.assertThat; import static org.hamcrest.Matchers.allOf; +import static org.hamcrest.Matchers.contains; import static org.hamcrest.Matchers.hasItems; import static org.hamcrest.Matchers.iterableWithSize; import static org.mockito.Mockito.when; +import com.amazon.opendistroforelasticsearch.sql.data.model.ExprTupleValue; import com.amazon.opendistroforelasticsearch.sql.data.model.ExprValue; import com.amazon.opendistroforelasticsearch.sql.data.model.ExprValueUtils; +import com.amazon.opendistroforelasticsearch.sql.executor.ExecutionEngine; import com.amazon.opendistroforelasticsearch.sql.expression.DSL; import com.google.common.collect.ImmutableMap; import java.util.List; @@ -74,7 +79,7 @@ public void project_two_field_follow_the_project_order() { } @Test - public void project_ignore_missing_value() { + public void project_keep_missing_value() { when(inputPlan.hasNext()).thenReturn(true, true, false); when(inputPlan.next()) .thenReturn(ExprValueUtils.tupleValue(ImmutableMap.of("action", "GET", "response", 200))) @@ -90,6 +95,20 @@ public void project_ignore_missing_value() { iterableWithSize(2), hasItems( ExprValueUtils.tupleValue(ImmutableMap.of("response", 200, "action", "GET")), - ExprValueUtils.tupleValue(ImmutableMap.of("action", "POST"))))); + ExprTupleValue.fromExprValueMap(ImmutableMap.of("response", + LITERAL_MISSING, + "action", stringValue("POST")))))); + } + + @Test + public void project_schema() { + PhysicalPlan project = project(inputPlan, + DSL.named("response", DSL.ref("response", INTEGER)), + DSL.named("action", DSL.ref("action", STRING))); + + assertThat(project.schema().getColumns(), contains( + new ExecutionEngine.Schema.Column("response", "response", INTEGER), + new ExecutionEngine.Schema.Column("action", "action", STRING) + )); } } diff --git a/core/src/test/java/com/amazon/opendistroforelasticsearch/sql/planner/physical/RemoveOperatorTest.java b/core/src/test/java/com/amazon/opendistroforelasticsearch/sql/planner/physical/RemoveOperatorTest.java index 9d6c9680d3..4962b4c320 100644 --- a/core/src/test/java/com/amazon/opendistroforelasticsearch/sql/planner/physical/RemoveOperatorTest.java +++ b/core/src/test/java/com/amazon/opendistroforelasticsearch/sql/planner/physical/RemoveOperatorTest.java @@ -22,6 +22,8 @@ import static org.hamcrest.Matchers.allOf; import static org.hamcrest.Matchers.hasItems; import static org.hamcrest.Matchers.iterableWithSize; +import static org.junit.jupiter.api.Assertions.assertEquals; +import static org.junit.jupiter.api.Assertions.assertThrows; import static org.mockito.Mockito.when; import com.amazon.opendistroforelasticsearch.sql.data.model.ExprValue; @@ -117,4 +119,15 @@ public void remove_nothing_with_none_tuple_value() { assertThat(result, allOf(iterableWithSize(1), hasItems(ExprValueUtils.integerValue(1)))); } + + @Test + public void invalid_to_retrieve_schema_from_remove() { + PhysicalPlan plan = remove(inputPlan, DSL.ref("response", STRING), DSL.ref("referer", STRING)); + IllegalStateException exception = + assertThrows(IllegalStateException.class, () -> plan.schema()); + assertEquals( + "[BUG] schema can been only applied to ProjectOperator, " + + "instead of RemoveOperator(input=inputPlan, removeList=[response, referer])", + exception.getMessage()); + } } diff --git a/docs/category.json b/docs/category.json index 595121a7d4..684ff35873 100644 --- a/docs/category.json +++ b/docs/category.json @@ -17,6 +17,7 @@ "sql_cli": [ "user/dql/expressions.rst", "user/general/identifiers.rst", + "user/general/values.rst", "user/dql/functions.rst", "user/beyond/partiql.rst" ] diff --git a/docs/experiment/ppl/cmd/search.rst b/docs/experiment/ppl/cmd/search.rst index eacc6eb966..d1a943f24f 100644 --- a/docs/experiment/ppl/cmd/search.rst +++ b/docs/experiment/ppl/cmd/search.rst @@ -32,14 +32,14 @@ PPL query:: od> source=accounts; fetched rows / total rows = 4/4 - +------------------+-----------+-------------+------------+-------+----------+----------------------+------------+-----------------------+--------+---------+ - | account_number | balance | firstname | lastname | age | gender | address | employer | email | city | state | - |------------------+-----------+-------------+------------+-------+----------+----------------------+------------+-----------------------+--------+---------| - | 1 | 39225 | Amber | Duke | 32 | M | 880 Holmes Lane | Pyrami | amberduke@pyrami.com | Brogan | IL | - | 6 | 5686 | Hattie | Bond | 36 | M | 671 Bristol Street | Netagy | hattiebond@netagy.com | Dante | TN | - | 13 | 32838 | Nanette | Bates | 28 | F | 789 Madison Street | Quility | null | Nogal | VA | - | 18 | 4180 | Dale | Adams | 33 | M | 467 Hutchinson Court | null | daleadams@boink.com | Orick | MD | - +------------------+-----------+-------------+------------+-------+----------+----------------------+------------+-----------------------+--------+---------+ + +------------------+-------------+----------------------+-----------+----------+--------+------------+---------+-------+-----------------------+------------+ + | account_number | firstname | address | balance | gender | city | employer | state | age | email | lastname | + |------------------+-------------+----------------------+-----------+----------+--------+------------+---------+-------+-----------------------+------------| + | 1 | Amber | 880 Holmes Lane | 39225 | M | Brogan | Pyrami | IL | 32 | amberduke@pyrami.com | Duke | + | 6 | Hattie | 671 Bristol Street | 5686 | M | Dante | Netagy | TN | 36 | hattiebond@netagy.com | Bond | + | 13 | Nanette | 789 Madison Street | 32838 | F | Nogal | Quility | VA | 28 | null | Bates | + | 18 | Dale | 467 Hutchinson Court | 4180 | M | Orick | null | MD | 33 | daleadams@boink.com | Adams | + +------------------+-------------+----------------------+-----------+----------+--------+------------+---------+-------+-----------------------+------------+ Example 2: Fetch data with condition ==================================== @@ -50,10 +50,10 @@ PPL query:: od> source=accounts account_number=1 or gender="F"; fetched rows / total rows = 2/2 - +------------------+-----------+-------------+------------+-------+----------+--------------------+------------+----------------------+--------+---------+ - | account_number | balance | firstname | lastname | age | gender | address | employer | email | city | state | - |------------------+-----------+-------------+------------+-------+----------+--------------------+------------+----------------------+--------+---------| - | 1 | 39225 | Amber | Duke | 32 | M | 880 Holmes Lane | Pyrami | amberduke@pyrami.com | Brogan | IL | - | 13 | 32838 | Nanette | Bates | 28 | F | 789 Madison Street | Quility | null | Nogal | VA | - +------------------+-----------+-------------+------------+-------+----------+--------------------+------------+----------------------+--------+---------+ + +------------------+-------------+--------------------+-----------+----------+--------+------------+---------+-------+----------------------+------------+ + | account_number | firstname | address | balance | gender | city | employer | state | age | email | lastname | + |------------------+-------------+--------------------+-----------+----------+--------+------------+---------+-------+----------------------+------------| + | 1 | Amber | 880 Holmes Lane | 39225 | M | Brogan | Pyrami | IL | 32 | amberduke@pyrami.com | Duke | + | 13 | Nanette | 789 Madison Street | 32838 | F | Nogal | Quility | VA | 28 | null | Bates | + +------------------+-------------+--------------------+-----------+----------+--------+------------+---------+-------+----------------------+------------+ diff --git a/docs/user/general/identifiers.rst b/docs/user/general/identifiers.rst index 5dbee26131..16b6a105c7 100644 --- a/docs/user/general/identifiers.rst +++ b/docs/user/general/identifiers.rst @@ -40,14 +40,14 @@ Here are examples for using index pattern directly without quotes:: od> SELECT * FROM *cc*nt*; fetched rows / total rows = 4/4 - +------------------+-----------+-------------+------------+-------+----------+----------------------+------------+-----------------------+--------+---------+ - | account_number | balance | firstname | lastname | age | gender | address | employer | email | city | state | - |------------------+-----------+-------------+------------+-------+----------+----------------------+------------+-----------------------+--------+---------| - | 1 | 39225 | Amber | Duke | 32 | M | 880 Holmes Lane | Pyrami | amberduke@pyrami.com | Brogan | IL | - | 6 | 5686 | Hattie | Bond | 36 | M | 671 Bristol Street | Netagy | hattiebond@netagy.com | Dante | TN | - | 13 | 32838 | Nanette | Bates | 28 | F | 789 Madison Street | Quility | null | Nogal | VA | - | 18 | 4180 | Dale | Adams | 33 | M | 467 Hutchinson Court | null | daleadams@boink.com | Orick | MD | - +------------------+-----------+-------------+------------+-------+----------+----------------------+------------+-----------------------+--------+---------+ + +------------------+-------------+----------------------+-----------+----------+--------+------------+---------+-------+-----------------------+------------+ + | account_number | firstname | address | balance | gender | city | employer | state | age | email | lastname | + |------------------+-------------+----------------------+-----------+----------+--------+------------+---------+-------+-----------------------+------------| + | 1 | Amber | 880 Holmes Lane | 39225 | M | Brogan | Pyrami | IL | 32 | amberduke@pyrami.com | Duke | + | 6 | Hattie | 671 Bristol Street | 5686 | M | Dante | Netagy | TN | 36 | hattiebond@netagy.com | Bond | + | 13 | Nanette | 789 Madison Street | 32838 | F | Nogal | Quility | VA | 28 | null | Bates | + | 18 | Dale | 467 Hutchinson Court | 4180 | M | Orick | null | MD | 33 | daleadams@boink.com | Adams | + +------------------+-------------+----------------------+-----------+----------+--------+------------+---------+-------+-----------------------+------------+ Delimited Identifiers @@ -76,14 +76,14 @@ Here are examples for quoting an index name by back ticks:: od> SELECT * FROM `accounts`; fetched rows / total rows = 4/4 - +------------------+-----------+-------------+------------+-------+----------+----------------------+------------+-----------------------+--------+---------+ - | account_number | balance | firstname | lastname | age | gender | address | employer | email | city | state | - |------------------+-----------+-------------+------------+-------+----------+----------------------+------------+-----------------------+--------+---------| - | 1 | 39225 | Amber | Duke | 32 | M | 880 Holmes Lane | Pyrami | amberduke@pyrami.com | Brogan | IL | - | 6 | 5686 | Hattie | Bond | 36 | M | 671 Bristol Street | Netagy | hattiebond@netagy.com | Dante | TN | - | 13 | 32838 | Nanette | Bates | 28 | F | 789 Madison Street | Quility | null | Nogal | VA | - | 18 | 4180 | Dale | Adams | 33 | M | 467 Hutchinson Court | null | daleadams@boink.com | Orick | MD | - +------------------+-----------+-------------+------------+-------+----------+----------------------+------------+-----------------------+--------+---------+ + +------------------+-------------+----------------------+-----------+----------+--------+------------+---------+-------+-----------------------+------------+ + | account_number | firstname | address | balance | gender | city | employer | state | age | email | lastname | + |------------------+-------------+----------------------+-----------+----------+--------+------------+---------+-------+-----------------------+------------| + | 1 | Amber | 880 Holmes Lane | 39225 | M | Brogan | Pyrami | IL | 32 | amberduke@pyrami.com | Duke | + | 6 | Hattie | 671 Bristol Street | 5686 | M | Dante | Netagy | TN | 36 | hattiebond@netagy.com | Bond | + | 13 | Nanette | 789 Madison Street | 32838 | F | Nogal | Quility | VA | 28 | null | Bates | + | 18 | Dale | 467 Hutchinson Court | 4180 | M | Orick | null | MD | 33 | daleadams@boink.com | Adams | + +------------------+-------------+----------------------+-----------+----------+--------+------------+---------+-------+-----------------------+------------+ Case Sensitivity diff --git a/docs/user/general/values.rst b/docs/user/general/values.rst new file mode 100644 index 0000000000..fead02a136 --- /dev/null +++ b/docs/user/general/values.rst @@ -0,0 +1,91 @@ +========== +Data Types +========== + +.. rubric:: Table of contents + +.. contents:: + :local: + :depth: 2 + + +NULL and MISSING Values +======================= +ODFE SQL has two ways to represent missing information. (1) The presence of the field with a NULL for its value. and (2) the absence of the filed. + +Please note, when response is in table format, the MISSING value is translate to NULL value. + +Here is an example, Nanette doesn't have email field and Dail has employer filed with NULL value:: + + od> SELECT firstname, employer, email FROM accounts; + fetched rows / total rows = 4/4 + +-------------+------------+-----------------------+ + | firstname | employer | email | + |-------------+------------+-----------------------| + | Amber | Pyrami | amberduke@pyrami.com | + | Hattie | Netagy | hattiebond@netagy.com | + | Nanette | Quility | null | + | Dale | null | daleadams@boink.com | + +-------------+------------+-----------------------+ + + +General NULL and MISSING Values Handling +---------------------------------------- +In general, if any operand evaluates to a MISSING value, the enclosing operator will return MISSING; if none of operands evaluates to a MISSING value but there is an operand evaluates to a NULL value, the enclosing operator will return NULL. + +Here is an example:: + + od> SELECT firstname, employer LIKE 'Quility', email LIKE '%com' FROM accounts; + fetched rows / total rows = 4/4 + +-------------+---------------------------+---------------------+ + | firstname | employer LIKE 'Quility' | email LIKE '%com' | + |-------------+---------------------------+---------------------| + | Amber | False | True | + | Hattie | False | True | + | Nanette | True | null | + | Dale | null | True | + +-------------+---------------------------+---------------------+ + +Special NULL and MISSING Values Handling +---------------------------------------- +THe AND, OR and NOT have special logic to handling NULL and MISSING value. + +The following table is the truth table for AND and OR. + ++---------+---------+---------+---------+ +| A | B | A AND B | A OR B | ++---------+---------+---------+---------+ +| TRUE | TRUE | TRUE | TRUE | ++---------+---------+---------+---------+ +| TRUE | FALSE | FALSE | TRUE | ++---------+---------+---------+---------+ +| TRUE | NULL | NULL | TRUE | ++---------+---------+---------+---------+ +| TRUE | MISSING | MISSING | TRUE | ++---------+---------+---------+---------+ +| FALSE | FALSE | FALSE | FALSE | ++---------+---------+---------+---------+ +| FALSE | NULL | FALSE | NULL | ++---------+---------+---------+---------+ +| FALSE | MISSING | FALSE | MISSING | ++---------+---------+---------+---------+ +| NULL | NULL | NULL | NULL | ++---------+---------+---------+---------+ +| NULL | MISSING | MISSING | NULL | ++---------+---------+---------+---------+ +| MISSING | MISSING | MISSING | MISSING | ++---------+---------+---------+---------+ + +The following table is the truth table for NOT. + ++---------+---------+ +| A | NOT A | ++---------+---------+ +| TRUE | FALSE | ++---------+---------+ +| FALSE | TRUE | ++---------+---------+ +| NULL | NULL | ++---------+---------+ +| MISSING | MISSING | ++---------+---------+ \ No newline at end of file diff --git a/doctest/test_data/accounts.json b/doctest/test_data/accounts.json index 2ea2b602fd..3ff2b7b3c7 100644 --- a/doctest/test_data/accounts.json +++ b/doctest/test_data/accounts.json @@ -1,4 +1,4 @@ {"account_number":1,"balance":39225,"firstname":"Amber","lastname":"Duke","age":32,"gender":"M","address":"880 Holmes Lane","employer":"Pyrami","email":"amberduke@pyrami.com","city":"Brogan","state":"IL"} {"account_number":6,"balance":5686,"firstname":"Hattie","lastname":"Bond","age":36,"gender":"M","address":"671 Bristol Street","employer":"Netagy","email":"hattiebond@netagy.com","city":"Dante","state":"TN"} -{"account_number":13,"balance":32838,"firstname":"Nanette","lastname":"Bates","age":28,"gender":"F","address":"789 Madison Street","employer":"Quility","email":null,"city":"Nogal","state":"VA"} +{"account_number":13,"balance":32838,"firstname":"Nanette","lastname":"Bates","age":28,"gender":"F","address":"789 Madison Street","employer":"Quility","city":"Nogal","state":"VA"} {"account_number":18,"balance":4180,"firstname":"Dale","lastname":"Adams","age":33,"gender":"M","address":"467 Hutchinson Court","employer":null,"email":"daleadams@boink.com","city":"Orick","state":"MD"} diff --git a/elasticsearch/src/main/java/com/amazon/opendistroforelasticsearch/sql/elasticsearch/executor/ElasticsearchExecutionEngine.java b/elasticsearch/src/main/java/com/amazon/opendistroforelasticsearch/sql/elasticsearch/executor/ElasticsearchExecutionEngine.java index b3caa33b83..97163f7e5b 100644 --- a/elasticsearch/src/main/java/com/amazon/opendistroforelasticsearch/sql/elasticsearch/executor/ElasticsearchExecutionEngine.java +++ b/elasticsearch/src/main/java/com/amazon/opendistroforelasticsearch/sql/elasticsearch/executor/ElasticsearchExecutionEngine.java @@ -47,7 +47,7 @@ public void execute(PhysicalPlan physicalPlan, ResponseListener l result.add(plan.next()); } - QueryResponse response = new QueryResponse(result); + QueryResponse response = new QueryResponse(physicalPlan.schema(), result); listener.onResponse(response); } catch (Exception e) { listener.onFailure(e); diff --git a/elasticsearch/src/test/java/com/amazon/opendistroforelasticsearch/sql/elasticsearch/executor/ElasticsearchExecutionEngineTest.java b/elasticsearch/src/test/java/com/amazon/opendistroforelasticsearch/sql/elasticsearch/executor/ElasticsearchExecutionEngineTest.java index f1ca7ac1f8..c91ecd062d 100644 --- a/elasticsearch/src/test/java/com/amazon/opendistroforelasticsearch/sql/elasticsearch/executor/ElasticsearchExecutionEngineTest.java +++ b/elasticsearch/src/test/java/com/amazon/opendistroforelasticsearch/sql/elasticsearch/executor/ElasticsearchExecutionEngineTest.java @@ -32,6 +32,7 @@ import com.amazon.opendistroforelasticsearch.sql.data.model.ExprValue; import com.amazon.opendistroforelasticsearch.sql.elasticsearch.client.ElasticsearchClient; import com.amazon.opendistroforelasticsearch.sql.elasticsearch.executor.protector.ElasticsearchExecutionProtector; +import com.amazon.opendistroforelasticsearch.sql.executor.ExecutionEngine; import com.amazon.opendistroforelasticsearch.sql.planner.physical.PhysicalPlan; import com.amazon.opendistroforelasticsearch.sql.storage.TableScanOperator; import java.util.ArrayList; @@ -53,6 +54,8 @@ class ElasticsearchExecutionEngineTest { @Mock private ElasticsearchExecutionProtector protector; + @Mock private static ExecutionEngine.Schema schema; + @BeforeEach void setUp() { doAnswer( @@ -148,5 +151,10 @@ public boolean hasNext() { public ExprValue next() { return it.next(); } + + @Override + public ExecutionEngine.Schema schema() { + return schema; + } } } diff --git a/integ-test/build.gradle b/integ-test/build.gradle index 472eda4b82..974973fe49 100644 --- a/integ-test/build.gradle +++ b/integ-test/build.gradle @@ -39,7 +39,10 @@ dependencies { testRuntimeOnly('org.junit.jupiter:junit-jupiter-engine:5.6.2') // JDBC drivers for comparison test. Somehow Apache Derby throws security permission exception. - testCompile group: 'com.amazon.opendistroforelasticsearch.client', name: 'opendistro-sql-jdbc', version: '1.8.0.0' + testCompile fileTree('../sql-jdbc/build/libs') { + include '*.jar' + builtBy 'compileJdbc' + } testCompile group: 'com.h2database', name: 'h2', version: '1.4.200' testCompile group: 'org.xerial', name: 'sqlite-jdbc', version: '3.28.0' //testCompile group: 'org.apache.derby', name: 'derby', version: '10.15.1.3' @@ -183,3 +186,9 @@ testClusters.comparisonTest { plugin file(tasks.getByPath(':plugin:bundlePlugin').archiveFile) } +task compileJdbc(type:Exec) { + workingDir '../sql-jdbc/' + + commandLine './gradlew', 'build' + commandLine './gradlew', 'shadowJar' +} diff --git a/integ-test/src/test/java/com/amazon/opendistroforelasticsearch/sql/ppl/DedupCommandIT.java b/integ-test/src/test/java/com/amazon/opendistroforelasticsearch/sql/ppl/DedupCommandIT.java index 39e6d128e7..9a1d031046 100644 --- a/integ-test/src/test/java/com/amazon/opendistroforelasticsearch/sql/ppl/DedupCommandIT.java +++ b/integ-test/src/test/java/com/amazon/opendistroforelasticsearch/sql/ppl/DedupCommandIT.java @@ -66,11 +66,11 @@ public void testKeepEmptyDedup() throws IOException { verifyDataRows( result, rows("Amber JOHnny", 39225), - rows("Hattie"), + rows("Hattie", null), rows("Nanette", 32838), rows("Dale", 4180), - rows("Elinor"), - rows("Virginia"), + rows("Elinor", null), + rows("Virginia", null), rows("Dillard", 48086)); } } diff --git a/integ-test/src/test/java/com/amazon/opendistroforelasticsearch/sql/ppl/OperatorIT.java b/integ-test/src/test/java/com/amazon/opendistroforelasticsearch/sql/ppl/OperatorIT.java index 2dec2f8311..d8a74e1025 100644 --- a/integ-test/src/test/java/com/amazon/opendistroforelasticsearch/sql/ppl/OperatorIT.java +++ b/integ-test/src/test/java/com/amazon/opendistroforelasticsearch/sql/ppl/OperatorIT.java @@ -22,7 +22,6 @@ import java.io.IOException; import org.elasticsearch.client.ResponseException; -import org.hamcrest.Matchers; import org.json.JSONObject; import org.junit.jupiter.api.Test; @@ -118,7 +117,8 @@ public void testArithmeticOperatorWithMissingValue() throws IOException { String.format( "source=%s | eval f = balance * 1 | fields f", TEST_INDEX_BANK_WITH_NULL_VALUES)); verifyDataRows( - result, rows(39225), rows(32838), rows(4180), rows(48086), rows(), rows(), rows()); + result, rows(39225), rows(32838), rows(4180), rows(48086), rows(JSONObject.NULL), + rows(JSONObject.NULL), rows(JSONObject.NULL)); } @Test diff --git a/integ-test/src/test/java/com/amazon/opendistroforelasticsearch/sql/ppl/StandaloneIT.java b/integ-test/src/test/java/com/amazon/opendistroforelasticsearch/sql/ppl/StandaloneIT.java index b017da6f63..3757fbdf2e 100644 --- a/integ-test/src/test/java/com/amazon/opendistroforelasticsearch/sql/ppl/StandaloneIT.java +++ b/integ-test/src/test/java/com/amazon/opendistroforelasticsearch/sql/ppl/StandaloneIT.java @@ -110,7 +110,7 @@ private String executeByStandaloneQueryEngine(String query) { @Override public void onResponse(QueryResponse response) { - QueryResult result = new QueryResult(response.getResults()); + QueryResult result = new QueryResult(response.getSchema(), response.getResults()); String json = new SimpleJsonResponseFormatter(PRETTY).format(result); actual.set(json); } diff --git a/integ-test/src/test/java/com/amazon/opendistroforelasticsearch/sql/util/MatcherUtils.java b/integ-test/src/test/java/com/amazon/opendistroforelasticsearch/sql/util/MatcherUtils.java index 50d803e020..ff3a2a6327 100644 --- a/integ-test/src/test/java/com/amazon/opendistroforelasticsearch/sql/util/MatcherUtils.java +++ b/integ-test/src/test/java/com/amazon/opendistroforelasticsearch/sql/util/MatcherUtils.java @@ -244,6 +244,8 @@ protected boolean matchesSafely(JSONArray array) { isEqual = ((JSONObject) expected).similar(array.get(i)); } else if (expected instanceof JSONArray) { isEqual = ((JSONArray) expected).similar(array.get(i)); + } else if (null == expected) { + isEqual = JSONObject.NULL == array.get(i); } else { isEqual = expected.equals(array.get(i)); } diff --git a/integ-test/src/test/resources/correctness/expressions/literals.txt b/integ-test/src/test/resources/correctness/expressions/literals.txt index 74710d917b..c56aa9ef05 100644 --- a/integ-test/src/test/resources/correctness/expressions/literals.txt +++ b/integ-test/src/test/resources/correctness/expressions/literals.txt @@ -2,3 +2,4 @@ true -4.567 -123,false +'ODFE' diff --git a/legacy/src/main/java/com/amazon/opendistroforelasticsearch/sql/legacy/plugin/RestSQLQueryAction.java b/legacy/src/main/java/com/amazon/opendistroforelasticsearch/sql/legacy/plugin/RestSQLQueryAction.java index 70eb40cb8c..bb089129c6 100644 --- a/legacy/src/main/java/com/amazon/opendistroforelasticsearch/sql/legacy/plugin/RestSQLQueryAction.java +++ b/legacy/src/main/java/com/amazon/opendistroforelasticsearch/sql/legacy/plugin/RestSQLQueryAction.java @@ -121,7 +121,8 @@ private ResponseListener createListener(RestChannel channel) { return new ResponseListener() { @Override public void onResponse(QueryResponse response) { - sendResponse(OK, formatter.format(new QueryResult(response.getResults()))); + sendResponse(OK, formatter.format(new QueryResult(response.getSchema(), + response.getResults()))); } @Override diff --git a/plugin/src/main/java/com/amazon/opendistroforelasticsearch/sql/plugin/rest/RestPPLQueryAction.java b/plugin/src/main/java/com/amazon/opendistroforelasticsearch/sql/plugin/rest/RestPPLQueryAction.java index 7062b31ccb..40aea2e575 100644 --- a/plugin/src/main/java/com/amazon/opendistroforelasticsearch/sql/plugin/rest/RestPPLQueryAction.java +++ b/plugin/src/main/java/com/amazon/opendistroforelasticsearch/sql/plugin/rest/RestPPLQueryAction.java @@ -119,7 +119,8 @@ private ResponseListener createListener(RestChannel channel) { return new ResponseListener() { @Override public void onResponse(QueryResponse response) { - sendResponse(OK, formatter.format(new QueryResult(response.getResults()))); + sendResponse(OK, formatter.format(new QueryResult(response.getSchema(), + response.getResults()))); } @Override diff --git a/ppl/build.gradle b/ppl/build.gradle index 6f55868cc2..062ddfd606 100644 --- a/ppl/build.gradle +++ b/ppl/build.gradle @@ -36,6 +36,7 @@ dependencies { compile project(':protocol') testCompile group: 'junit', name: 'junit', version: '4.12' + testCompile group: 'org.hamcrest', name: 'hamcrest-library', version: '2.1' testCompile group: 'org.mockito', name: 'mockito-core', version: '3.3.3' } diff --git a/ppl/src/main/java/com/amazon/opendistroforelasticsearch/sql/ppl/PPLService.java b/ppl/src/main/java/com/amazon/opendistroforelasticsearch/sql/ppl/PPLService.java index 0c3e437c68..08ca8f4383 100644 --- a/ppl/src/main/java/com/amazon/opendistroforelasticsearch/sql/ppl/PPLService.java +++ b/ppl/src/main/java/com/amazon/opendistroforelasticsearch/sql/ppl/PPLService.java @@ -29,6 +29,7 @@ import com.amazon.opendistroforelasticsearch.sql.ppl.domain.PPLQueryRequest; import com.amazon.opendistroforelasticsearch.sql.ppl.parser.AstBuilder; import com.amazon.opendistroforelasticsearch.sql.ppl.parser.AstExpressionBuilder; +import com.amazon.opendistroforelasticsearch.sql.ppl.utils.UnresolvedPlanHelper; import com.amazon.opendistroforelasticsearch.sql.storage.StorageEngine; import lombok.RequiredArgsConstructor; import org.antlr.v4.runtime.tree.ParseTree; @@ -55,7 +56,8 @@ public void execute(PPLQueryRequest request, ResponseListener lis UnresolvedPlan ast = cst.accept(new AstBuilder(new AstExpressionBuilder())); // 2.Analyze abstract syntax to generate logical plan - LogicalPlan logicalPlan = analyzer.analyze(ast, new AnalysisContext()); + LogicalPlan logicalPlan = analyzer.analyze(UnresolvedPlanHelper.addSelectAll(ast), + new AnalysisContext()); // 3.Generate optimal physical plan from logical plan PhysicalPlan physicalPlan = new Planner(storageEngine).plan(logicalPlan); diff --git a/ppl/src/main/java/com/amazon/opendistroforelasticsearch/sql/ppl/utils/UnresolvedPlanHelper.java b/ppl/src/main/java/com/amazon/opendistroforelasticsearch/sql/ppl/utils/UnresolvedPlanHelper.java new file mode 100644 index 0000000000..6c2f97671f --- /dev/null +++ b/ppl/src/main/java/com/amazon/opendistroforelasticsearch/sql/ppl/utils/UnresolvedPlanHelper.java @@ -0,0 +1,42 @@ +/* + * + * Copyright 2020 Amazon.com, Inc. or its affiliates. All Rights Reserved. + * + * Licensed under the Apache License, Version 2.0 (the "License"). + * You may not use this file except in compliance with the License. + * A copy of the License is located at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * or in the "license" file accompanying this file. This file 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 com.amazon.opendistroforelasticsearch.sql.ppl.utils; + +import com.amazon.opendistroforelasticsearch.sql.ast.expression.AllFields; +import com.amazon.opendistroforelasticsearch.sql.ast.tree.Project; +import com.amazon.opendistroforelasticsearch.sql.ast.tree.UnresolvedPlan; +import com.google.common.collect.ImmutableList; +import lombok.experimental.UtilityClass; + +/** + * The helper to add select to {@link UnresolvedPlan} if needed. + */ +@UtilityClass +public class UnresolvedPlanHelper { + + /** + * Attach Select All to PPL commands if required. + */ + public UnresolvedPlan addSelectAll(UnresolvedPlan plan) { + if ((plan instanceof Project) && !((Project) plan).isExcluded()) { + return plan; + } else { + return new Project(ImmutableList.of(AllFields.of())).attach(plan); + } + } +} diff --git a/ppl/src/test/java/com/amazon/opendistroforelasticsearch/sql/ppl/PPLServiceTest.java b/ppl/src/test/java/com/amazon/opendistroforelasticsearch/sql/ppl/PPLServiceTest.java index 18df192acc..ed8449bd0c 100644 --- a/ppl/src/test/java/com/amazon/opendistroforelasticsearch/sql/ppl/PPLServiceTest.java +++ b/ppl/src/test/java/com/amazon/opendistroforelasticsearch/sql/ppl/PPLServiceTest.java @@ -56,6 +56,9 @@ public class PPLServiceTest { @Mock private PhysicalPlan plan; + @Mock + private ExecutionEngine.Schema schema; + /** * Setup the test context. */ @@ -76,7 +79,7 @@ public void setUp() { public void testExecuteShouldPass() { doAnswer(invocation -> { ResponseListener listener = invocation.getArgument(1); - listener.onResponse(new QueryResponse(Collections.emptyList())); + listener.onResponse(new QueryResponse(schema, Collections.emptyList())); return null; }).when(executionEngine).execute(any(), any()); @@ -108,4 +111,19 @@ public void onFailure(Exception e) { } }); } + + @Test + public void test() { + pplService.execute(new PPLQueryRequest("search", null), new ResponseListener() { + @Override + public void onResponse(QueryResponse pplQueryResponse) { + Assert.fail(); + } + + @Override + public void onFailure(Exception e) { + + } + }); + } } \ No newline at end of file diff --git a/ppl/src/test/java/com/amazon/opendistroforelasticsearch/sql/ppl/utils/UnresolvedPlanHelperTest.java b/ppl/src/test/java/com/amazon/opendistroforelasticsearch/sql/ppl/utils/UnresolvedPlanHelperTest.java new file mode 100644 index 0000000000..3e4d5297b4 --- /dev/null +++ b/ppl/src/test/java/com/amazon/opendistroforelasticsearch/sql/ppl/utils/UnresolvedPlanHelperTest.java @@ -0,0 +1,68 @@ +/* + * + * Copyright 2020 Amazon.com, Inc. or its affiliates. All Rights Reserved. + * + * Licensed under the Apache License, Version 2.0 (the "License"). + * You may not use this file except in compliance with the License. + * A copy of the License is located at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * or in the "license" file accompanying this file. This file 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 com.amazon.opendistroforelasticsearch.sql.ppl.utils; + +import static org.junit.Assert.assertThat; +import static org.mockito.Mockito.when; + +import com.amazon.opendistroforelasticsearch.sql.ast.expression.AllFields; +import com.amazon.opendistroforelasticsearch.sql.ast.expression.UnresolvedExpression; +import com.amazon.opendistroforelasticsearch.sql.ast.tree.Project; +import com.amazon.opendistroforelasticsearch.sql.ast.tree.Rename; +import com.amazon.opendistroforelasticsearch.sql.ast.tree.UnresolvedPlan; +import java.util.Arrays; +import junit.framework.TestCase; +import org.hamcrest.Matchers; +import org.junit.Test; +import org.junit.runner.RunWith; +import org.mockito.Mockito; +import org.mockito.junit.MockitoJUnitRunner; + +@RunWith(MockitoJUnitRunner.class) +public class UnresolvedPlanHelperTest extends TestCase { + + @Test + public void addProjectForRenameOperator() { + Rename rename = Mockito.mock(Rename.class); + + UnresolvedPlan plan = UnresolvedPlanHelper.addSelectAll(rename); + assertTrue(plan instanceof Project); + } + + @Test + public void addProjectForProjectExcludeOperator() { + Project project = Mockito.mock(Project.class); + when(project.isExcluded()).thenReturn(true); + + UnresolvedPlan plan = UnresolvedPlanHelper.addSelectAll(project); + assertTrue(plan instanceof Project); + assertThat(((Project) plan).getProjectList(), Matchers.contains(AllFields.of())); + } + + @Test + public void dontAddProjectForProjectOperator() { + Project project = Mockito.mock(Project.class); + UnresolvedExpression expression = Mockito.mock(UnresolvedExpression.class); + when(project.isExcluded()).thenReturn(false); + when(project.getProjectList()).thenReturn(Arrays.asList(expression)); + + UnresolvedPlan plan = UnresolvedPlanHelper.addSelectAll(project); + assertTrue(plan instanceof Project); + assertThat(((Project) plan).getProjectList(), Matchers.contains(expression)); + } +} \ No newline at end of file diff --git a/protocol/src/main/java/com/amazon/opendistroforelasticsearch/sql/protocol/response/QueryResult.java b/protocol/src/main/java/com/amazon/opendistroforelasticsearch/sql/protocol/response/QueryResult.java index 389b259861..cc8b4d73bd 100644 --- a/protocol/src/main/java/com/amazon/opendistroforelasticsearch/sql/protocol/response/QueryResult.java +++ b/protocol/src/main/java/com/amazon/opendistroforelasticsearch/sql/protocol/response/QueryResult.java @@ -18,8 +18,8 @@ import com.amazon.opendistroforelasticsearch.sql.data.model.ExprValue; import com.amazon.opendistroforelasticsearch.sql.data.model.ExprValueUtils; +import com.amazon.opendistroforelasticsearch.sql.executor.ExecutionEngine; import java.util.Collection; -import java.util.Collections; import java.util.Iterator; import java.util.LinkedHashMap; import java.util.Map; @@ -31,6 +31,7 @@ */ @RequiredArgsConstructor public class QueryResult implements Iterable { + private final ExecutionEngine.Schema schema; /** * Results which are collection of expression. @@ -52,13 +53,10 @@ public int size() { * @return mapping from column names to its expression type */ public Map columnNameTypes() { - if (exprValues.isEmpty()) { - return Collections.emptyMap(); - } - - // TODO: Need other way to extract header than inferring from data implicitly - Map tupleValue = getFirstTupleValue(); - return populateColumnNameAndTypes(tupleValue); + Map colNameTypes = new LinkedHashMap<>(); + schema.getColumns().forEach(column -> colNameTypes.put(column.getName(), + column.getExprType().typeName().toLowerCase())); + return colNameTypes; } @Override @@ -71,29 +69,10 @@ public Iterator iterator() { .iterator(); } - private Map getFirstTupleValue() { - // Assume expression is always tuple on first level - // and columns (keys) of all tuple values are exactly same - ExprValue firstValue = exprValues.iterator().next(); - return ExprValueUtils.getTupleValue(firstValue); - } - - private Map populateColumnNameAndTypes(Map tupleValue) { - // Use linked hashmap to maintain original order in tuple expression - Map colNameTypes = new LinkedHashMap<>(); - tupleValue.forEach((name, expr) -> colNameTypes.put(name, getTypeString(expr))); - return colNameTypes; - } - private Object[] convertExprValuesToValues(Collection exprValues) { return exprValues .stream() .map(ExprValue::value) .toArray(Object[]::new); } - - private String getTypeString(ExprValue exprValue) { - return exprValue.type().typeName().toLowerCase(); - } - } diff --git a/protocol/src/test/java/com/amazon/opendistroforelasticsearch/sql/protocol/response/QueryResultTest.java b/protocol/src/test/java/com/amazon/opendistroforelasticsearch/sql/protocol/response/QueryResultTest.java index 52cf905d9a..785b8949af 100644 --- a/protocol/src/test/java/com/amazon/opendistroforelasticsearch/sql/protocol/response/QueryResultTest.java +++ b/protocol/src/test/java/com/amazon/opendistroforelasticsearch/sql/protocol/response/QueryResultTest.java @@ -17,34 +17,45 @@ package com.amazon.opendistroforelasticsearch.sql.protocol.response; import static com.amazon.opendistroforelasticsearch.sql.data.model.ExprValueUtils.tupleValue; +import static com.amazon.opendistroforelasticsearch.sql.data.type.ExprCoreType.INTEGER; +import static com.amazon.opendistroforelasticsearch.sql.data.type.ExprCoreType.STRING; import static org.junit.jupiter.api.Assertions.assertArrayEquals; import static org.junit.jupiter.api.Assertions.assertEquals; -import static org.junit.jupiter.api.Assertions.assertTrue; import static org.junit.jupiter.api.Assertions.fail; +import com.amazon.opendistroforelasticsearch.sql.executor.ExecutionEngine; +import com.google.common.collect.ImmutableList; import com.google.common.collect.ImmutableMap; import java.util.Arrays; import java.util.Collections; -import org.junit.jupiter.api.Disabled; import org.junit.jupiter.api.Test; class QueryResultTest { + private ExecutionEngine.Schema schema = new ExecutionEngine.Schema(ImmutableList.of( + new ExecutionEngine.Schema.Column("name", "name", STRING), + new ExecutionEngine.Schema.Column("age", "age", INTEGER))); + + @Test void size() { - QueryResult response = new QueryResult(Arrays.asList( - tupleValue(ImmutableMap.of("name", "John", "age", 20)), - tupleValue(ImmutableMap.of("name", "Allen", "age", 30)), - tupleValue(ImmutableMap.of("name", "Smith", "age", 40)) - )); + QueryResult response = new QueryResult( + schema, + Arrays.asList( + tupleValue(ImmutableMap.of("name", "John", "age", 20)), + tupleValue(ImmutableMap.of("name", "Allen", "age", 30)), + tupleValue(ImmutableMap.of("name", "Smith", "age", 40)) + )); assertEquals(3, response.size()); } @Test void columnNameTypes() { - QueryResult response = new QueryResult(Collections.singletonList( - tupleValue(ImmutableMap.of("name", "John", "age", 20)) - )); + QueryResult response = new QueryResult( + schema, + Collections.singletonList( + tupleValue(ImmutableMap.of("name", "John", "age", 20)) + )); assertEquals( ImmutableMap.of("name", "string", "age", "integer"), @@ -54,17 +65,23 @@ void columnNameTypes() { @Test void columnNameTypesFromEmptyExprValues() { - QueryResult response = new QueryResult(Collections.emptyList()); - assertTrue(response.columnNameTypes().isEmpty()); + QueryResult response = new QueryResult( + schema, + Collections.emptyList()); + assertEquals( + ImmutableMap.of("name", "string", "age", "integer"), + response.columnNameTypes() + ); } - @Disabled("Need to figure out column headers in other way than inferring from data implicitly") @Test void columnNameTypesFromExprValuesWithMissing() { - QueryResult response = new QueryResult(Arrays.asList( - tupleValue(ImmutableMap.of("name", "John")), - tupleValue(ImmutableMap.of("name", "John", "age", 20)) - )); + QueryResult response = new QueryResult( + schema, + Arrays.asList( + tupleValue(ImmutableMap.of("name", "John")), + tupleValue(ImmutableMap.of("name", "John", "age", 20)) + )); assertEquals( ImmutableMap.of("name", "string", "age", "integer"), @@ -74,10 +91,12 @@ void columnNameTypesFromExprValuesWithMissing() { @Test void iterate() { - QueryResult response = new QueryResult(Arrays.asList( - tupleValue(ImmutableMap.of("name", "John", "age", 20)), - tupleValue(ImmutableMap.of("name", "Allen", "age", 30)) - )); + QueryResult response = new QueryResult( + schema, + Arrays.asList( + tupleValue(ImmutableMap.of("name", "John", "age", 20)), + tupleValue(ImmutableMap.of("name", "Allen", "age", 30)) + )); int i = 0; for (Object[] objects : response) { diff --git a/protocol/src/test/java/com/amazon/opendistroforelasticsearch/sql/protocol/response/format/SimpleJsonResponseFormatterTest.java b/protocol/src/test/java/com/amazon/opendistroforelasticsearch/sql/protocol/response/format/SimpleJsonResponseFormatterTest.java index bd9dc9a817..4090e0addd 100644 --- a/protocol/src/test/java/com/amazon/opendistroforelasticsearch/sql/protocol/response/format/SimpleJsonResponseFormatterTest.java +++ b/protocol/src/test/java/com/amazon/opendistroforelasticsearch/sql/protocol/response/format/SimpleJsonResponseFormatterTest.java @@ -16,23 +16,34 @@ package com.amazon.opendistroforelasticsearch.sql.protocol.response.format; +import static com.amazon.opendistroforelasticsearch.sql.data.model.ExprValueUtils.LITERAL_MISSING; +import static com.amazon.opendistroforelasticsearch.sql.data.model.ExprValueUtils.stringValue; import static com.amazon.opendistroforelasticsearch.sql.data.model.ExprValueUtils.tupleValue; +import static com.amazon.opendistroforelasticsearch.sql.data.type.ExprCoreType.INTEGER; +import static com.amazon.opendistroforelasticsearch.sql.data.type.ExprCoreType.STRING; import static com.amazon.opendistroforelasticsearch.sql.protocol.response.format.JsonResponseFormatter.Style.COMPACT; import static com.amazon.opendistroforelasticsearch.sql.protocol.response.format.JsonResponseFormatter.Style.PRETTY; import static org.junit.jupiter.api.Assertions.assertEquals; +import com.amazon.opendistroforelasticsearch.sql.data.model.ExprTupleValue; +import com.amazon.opendistroforelasticsearch.sql.executor.ExecutionEngine; import com.amazon.opendistroforelasticsearch.sql.protocol.response.QueryResult; +import com.google.common.collect.ImmutableList; import com.google.common.collect.ImmutableMap; import java.util.Arrays; -import org.junit.jupiter.api.Disabled; import org.junit.jupiter.api.Test; class SimpleJsonResponseFormatterTest { + private final ExecutionEngine.Schema schema = new ExecutionEngine.Schema(ImmutableList.of( + new ExecutionEngine.Schema.Column("firstname", "name", STRING), + new ExecutionEngine.Schema.Column("age", "age", INTEGER))); + @Test void formatResponse() { QueryResult response = new QueryResult( + schema, Arrays.asList( tupleValue(ImmutableMap.of("firstname", "John", "age", 20)), tupleValue(ImmutableMap.of("firstname", "Smith", "age", 30)))); @@ -48,6 +59,7 @@ void formatResponse() { void formatResponsePretty() { QueryResult response = new QueryResult( + schema, Arrays.asList( tupleValue(ImmutableMap.of("firstname", "John", "age", 20)), tupleValue(ImmutableMap.of("firstname", "Smith", "age", 30)))); @@ -80,19 +92,20 @@ void formatResponsePretty() { formatter.format(response)); } - @Disabled("Need to figure out column headers in other way than inferring from data implicitly") @Test void formatResponseWithMissingValue() { QueryResult response = new QueryResult( + schema, Arrays.asList( - tupleValue(ImmutableMap.of("firstname", "John")), + ExprTupleValue.fromExprValueMap( + ImmutableMap.of("firstname", stringValue("John"), "age", LITERAL_MISSING)), tupleValue(ImmutableMap.of("firstname", "Smith", "age", 30)))); SimpleJsonResponseFormatter formatter = new SimpleJsonResponseFormatter(COMPACT); assertEquals( "{\"schema\":[{\"name\":\"firstname\",\"type\":\"string\"}," + "{\"name\":\"age\",\"type\":\"integer\"}],\"total\":2," - + "\"datarows\":[{\"row\":[\"John\",null]},{\"row\":[\"Smith\",30]}],\"size\":2}", + + "\"datarows\":[[\"John\",null],[\"Smith\",30]],\"size\":2}", formatter.format(response)); } diff --git a/sql-jdbc/src/main/java/com/amazon/opendistroforelasticsearch/jdbc/types/ElasticsearchType.java b/sql-jdbc/src/main/java/com/amazon/opendistroforelasticsearch/jdbc/types/ElasticsearchType.java index eebaa1914e..6f754ca094 100644 --- a/sql-jdbc/src/main/java/com/amazon/opendistroforelasticsearch/jdbc/types/ElasticsearchType.java +++ b/sql-jdbc/src/main/java/com/amazon/opendistroforelasticsearch/jdbc/types/ElasticsearchType.java @@ -66,6 +66,7 @@ public enum ElasticsearchType { SCALED_FLOAT(JDBCType.DOUBLE, Double.class, 15, 25, true), KEYWORD(JDBCType.VARCHAR, String.class, 256, 0, false), TEXT(JDBCType.VARCHAR, String.class, Integer.MAX_VALUE, 0, false), + STRING(JDBCType.VARCHAR, String.class, Integer.MAX_VALUE, 0, false), IP(JDBCType.VARCHAR, String.class, 15, 0, false), NESTED(JDBCType.STRUCT, null, 0, 0, false), OBJECT(JDBCType.STRUCT, null, 0, 0, false), diff --git a/sql/src/main/java/com/amazon/opendistroforelasticsearch/sql/sql/parser/AstBuilder.java b/sql/src/main/java/com/amazon/opendistroforelasticsearch/sql/sql/parser/AstBuilder.java index dd3da3f91f..2d00909967 100644 --- a/sql/src/main/java/com/amazon/opendistroforelasticsearch/sql/sql/parser/AstBuilder.java +++ b/sql/src/main/java/com/amazon/opendistroforelasticsearch/sql/sql/parser/AstBuilder.java @@ -22,6 +22,7 @@ import static com.amazon.opendistroforelasticsearch.sql.sql.antlr.parser.OpenDistroSQLParser.SimpleSelectContext; import com.amazon.opendistroforelasticsearch.sql.ast.expression.Alias; +import com.amazon.opendistroforelasticsearch.sql.ast.expression.AllFields; import com.amazon.opendistroforelasticsearch.sql.ast.expression.UnresolvedExpression; import com.amazon.opendistroforelasticsearch.sql.ast.tree.Project; import com.amazon.opendistroforelasticsearch.sql.ast.tree.Relation; @@ -33,8 +34,7 @@ import com.amazon.opendistroforelasticsearch.sql.sql.antlr.parser.OpenDistroSQLParserBaseVisitor; import com.google.common.collect.ImmutableList; import java.util.Collections; -import java.util.List; -import java.util.stream.Collectors; +import java.util.Optional; import lombok.RequiredArgsConstructor; import org.antlr.v4.runtime.ParserRuleContext; import org.antlr.v4.runtime.Token; @@ -46,8 +46,6 @@ @RequiredArgsConstructor public class AstBuilder extends OpenDistroSQLParserBaseVisitor { - private static final Project SELECT_ALL = null; - private final AstExpressionBuilder expressionBuilder = new AstExpressionBuilder(); /** @@ -62,10 +60,12 @@ public UnresolvedPlan visitSimpleSelect(SimpleSelectContext ctx) { UnresolvedPlan project = visit(query.selectClause()); if (query.fromClause() == null) { - if (project == SELECT_ALL) { + Optional allFields = + ((Project) project).getProjectList().stream().filter(node -> node instanceof AllFields) + .findFirst(); + if (allFields.isPresent()) { throw new SyntaxCheckException("No FROM clause found for select all"); } - // Attach an Values operator with only a empty row inside so that // Project operator can have a chance to evaluate its expression // though the evaluation doesn't have any dependency on what's in Values. @@ -74,19 +74,18 @@ public UnresolvedPlan visitSimpleSelect(SimpleSelectContext ctx) { } UnresolvedPlan relation = visit(query.fromClause()); - return (project == SELECT_ALL) ? relation : project.attach(relation); + return project.attach(relation); } @Override public UnresolvedPlan visitSelectClause(SelectClauseContext ctx) { + ImmutableList.Builder builder = + new ImmutableList.Builder<>(); if (ctx.selectElements().star != null) { //TODO: project operator should be required? - return SELECT_ALL; + builder.add(AllFields.of()); } - - List selectElements = ctx.selectElements().selectElement(); - return new Project(selectElements.stream() - .map(this::visitSelectItem) - .collect(Collectors.toList())); + ctx.selectElements().selectElement().forEach(field -> builder.add(visitSelectItem(field))); + return new Project(builder.build()); } @Override diff --git a/sql/src/test/java/com/amazon/opendistroforelasticsearch/sql/sql/SQLServiceTest.java b/sql/src/test/java/com/amazon/opendistroforelasticsearch/sql/sql/SQLServiceTest.java index 017278454f..2f262a261f 100644 --- a/sql/src/test/java/com/amazon/opendistroforelasticsearch/sql/sql/SQLServiceTest.java +++ b/sql/src/test/java/com/amazon/opendistroforelasticsearch/sql/sql/SQLServiceTest.java @@ -52,6 +52,9 @@ class SQLServiceTest { @Mock private ExecutionEngine executionEngine; + @Mock + private ExecutionEngine.Schema schema; + @BeforeEach public void setUp() { context.registerBean(StorageEngine.class, () -> storageEngine); @@ -65,7 +68,7 @@ public void setUp() { public void canExecuteSqlQuery() { doAnswer(invocation -> { ResponseListener listener = invocation.getArgument(1); - listener.onResponse(new QueryResponse(Collections.emptyList())); + listener.onResponse(new QueryResponse(schema, Collections.emptyList())); return null; }).when(executionEngine).execute(any(), any()); @@ -88,7 +91,7 @@ public void onFailure(Exception e) { public void canExecuteFromPhysicalPlan() { doAnswer(invocation -> { ResponseListener listener = invocation.getArgument(1); - listener.onResponse(new QueryResponse(Collections.emptyList())); + listener.onResponse(new QueryResponse(schema, Collections.emptyList())); return null; }).when(executionEngine).execute(any(), any()); diff --git a/sql/src/test/java/com/amazon/opendistroforelasticsearch/sql/sql/parser/AstBuilderTest.java b/sql/src/test/java/com/amazon/opendistroforelasticsearch/sql/sql/parser/AstBuilderTest.java index 4a80334b94..3de6b9920b 100644 --- a/sql/src/test/java/com/amazon/opendistroforelasticsearch/sql/sql/parser/AstBuilderTest.java +++ b/sql/src/test/java/com/amazon/opendistroforelasticsearch/sql/sql/parser/AstBuilderTest.java @@ -30,6 +30,7 @@ import static org.junit.jupiter.api.Assertions.assertEquals; import static org.junit.jupiter.api.Assertions.assertThrows; +import com.amazon.opendistroforelasticsearch.sql.ast.expression.AllFields; import com.amazon.opendistroforelasticsearch.sql.ast.tree.UnresolvedPlan; import com.amazon.opendistroforelasticsearch.sql.common.antlr.SyntaxCheckException; import com.amazon.opendistroforelasticsearch.sql.sql.antlr.SQLSyntaxParser; @@ -75,13 +76,29 @@ public void can_build_select_function_call_with_alias() { @Test public void can_build_select_all_from_index() { assertEquals( - relation("test"), + project( + relation("test"), + AllFields.of() + ), buildAST("SELECT * FROM test") ); assertThrows(SyntaxCheckException.class, () -> buildAST("SELECT *")); } + @Test + public void can_build_select_all_and_fields_from_index() { + assertEquals( + project( + relation("test"), + AllFields.of(), + alias("age", qualifiedName("age")), + alias("age", qualifiedName("age"), "a") + ), + buildAST("SELECT *, age, age as a FROM test") + ); + } + @Test public void can_build_select_fields_from_index() { assertEquals( From 6974a902ec12c604c0262cfff19752496a5c61f8 Mon Sep 17 00:00:00 2001 From: penghuo Date: Mon, 10 Aug 2020 09:53:45 -0700 Subject: [PATCH 2/2] address comments --- .../sql/expression/NamedExpression.java | 2 ++ .../sql/planner/physical/ProjectOperator.java | 2 +- .../sql/planner/physical/ProjectOperatorTest.java | 4 ++-- 3 files changed, 5 insertions(+), 3 deletions(-) diff --git a/core/src/main/java/com/amazon/opendistroforelasticsearch/sql/expression/NamedExpression.java b/core/src/main/java/com/amazon/opendistroforelasticsearch/sql/expression/NamedExpression.java index 6f3889fa6c..d586cb9263 100644 --- a/core/src/main/java/com/amazon/opendistroforelasticsearch/sql/expression/NamedExpression.java +++ b/core/src/main/java/com/amazon/opendistroforelasticsearch/sql/expression/NamedExpression.java @@ -22,6 +22,7 @@ import com.google.common.base.Strings; import lombok.AllArgsConstructor; import lombok.EqualsAndHashCode; +import lombok.Getter; import lombok.RequiredArgsConstructor; import lombok.ToString; @@ -49,6 +50,7 @@ public class NamedExpression implements Expression { /** * Optional alias. */ + @Getter private String alias; @Override diff --git a/core/src/main/java/com/amazon/opendistroforelasticsearch/sql/planner/physical/ProjectOperator.java b/core/src/main/java/com/amazon/opendistroforelasticsearch/sql/planner/physical/ProjectOperator.java index 4576c80946..32a6906298 100644 --- a/core/src/main/java/com/amazon/opendistroforelasticsearch/sql/planner/physical/ProjectOperator.java +++ b/core/src/main/java/com/amazon/opendistroforelasticsearch/sql/planner/physical/ProjectOperator.java @@ -71,6 +71,6 @@ public ExprValue next() { public ExecutionEngine.Schema schema() { return new ExecutionEngine.Schema(getProjectList().stream() .map(expr -> new ExecutionEngine.Schema.Column(expr.getName(), - expr.getName(), expr.type())).collect(Collectors.toList())); + expr.getAlias(), expr.type())).collect(Collectors.toList())); } } diff --git a/core/src/test/java/com/amazon/opendistroforelasticsearch/sql/planner/physical/ProjectOperatorTest.java b/core/src/test/java/com/amazon/opendistroforelasticsearch/sql/planner/physical/ProjectOperatorTest.java index 02e4e0734a..5baf541f62 100644 --- a/core/src/test/java/com/amazon/opendistroforelasticsearch/sql/planner/physical/ProjectOperatorTest.java +++ b/core/src/test/java/com/amazon/opendistroforelasticsearch/sql/planner/physical/ProjectOperatorTest.java @@ -107,8 +107,8 @@ public void project_schema() { DSL.named("action", DSL.ref("action", STRING))); assertThat(project.schema().getColumns(), contains( - new ExecutionEngine.Schema.Column("response", "response", INTEGER), - new ExecutionEngine.Schema.Column("action", "action", STRING) + new ExecutionEngine.Schema.Column("response", null, INTEGER), + new ExecutionEngine.Schema.Column("action", null, STRING) )); } }