This is an automated email from the ASF dual-hosted git repository.
mayanks pushed a commit to branch master
in repository https://gitbox.apache.org/repos/asf/incubator-pinot.git
The following commit(s) were added to refs/heads/master by this push:
new bd79861 Issue 5261: Support AggregationFunctions with multiple
arguments (contd). (#5275)
bd79861 is described below
commit bd79861e7bca87406fc184ec70ce504667f76daf
Author: Mayank Shrivastava <[email protected]>
AuthorDate: Sun Apr 19 16:51:21 2020 -0700
Issue 5261: Support AggregationFunctions with multiple arguments (contd).
(#5275)
This PR is a continuation of
https://github.com/apache/incubator-pinot/pull/5259
to address the issue https://github.com/apache/incubator-pinot/issues/5261.
1. Added new field in request.thrift `aggregationFunctionArgs` as a list of
String
arguments for the aggregation funciton.
- Could not use the existing `aggregationParams` as it is a Map, and
functions with
variable arguments may not provide a name for the arg (to be used as
key in Map).
- Maintain backward compatibility by first check for the new field, and
fall back to
the existing one if it does not exist.
2. Ensure that all calls to the old AggregationInfo.getAggregationParams()
is replaced
with backward compatible AgguregationFunctionUtils.getAggregationArgs().
3. Since most aggregation functions today have just one argument, added a
separate api
AggregationFuncitonContext.getFirstArgument() as an optimization.
4. Cleaned up getColumnName() and getResultColumnName() api's in
AggregationFunctionContext
class to not require the column name argument, as this is already stored
in the
AggregationFunction.
5. Modified all tests to use aggregationFunctionArgs instead of
aggregationParams.
TODO:
Remove the AggregationFunctionContext class as AggregationFunctions now
store their arguments,
and this class no longer provides any additional value.
---
.../requesthandler/BaseBrokerRequestHandler.java | 29 ++-
.../request/PqlAndCalciteSqlCompatibilityTest.java | 5 -
.../pinot/common/request/AggregationInfo.java | 215 ++++++++++++++++++---
.../common/utils/request/HavingQueryTree.java | 2 +-
.../utils/BrokerRequestComparisonUtils.java | 25 +--
.../parsers/PinotQuery2BrokerRequestConverter.java | 38 ++--
.../pql/parsers/pql2/ast/FunctionCallAstNode.java | 33 ++--
.../apache/pinot/pql/parsers/Pql2CompilerTest.java | 12 +-
.../pinot/sql/parsers/CalciteSqlCompilerTest.java | 170 +++++++++++-----
pinot-common/src/thrift/request.thrift | 5 +
.../query/DictionaryBasedAggregationOperator.java | 2 +-
.../plan/DictionaryBasedAggregationPlanNode.java | 2 +-
.../plan/MetadataBasedAggregationPlanNode.java | 2 +-
.../apache/pinot/core/plan/TransformPlanNode.java | 9 +-
.../core/plan/maker/InstancePlanMakerImplV2.java | 2 +-
.../aggregation/AggregationFunctionContext.java | 36 +++-
.../aggregation/DefaultAggregationExecutor.java | 16 +-
.../aggregation/function/AggregationFunction.java | 8 +-
.../function/AggregationFunctionFactory.java | 5 +-
.../function/AggregationFunctionUtils.java | 43 ++++-
.../function/AvgAggregationFunction.java | 10 +
.../function/CountAggregationFunction.java | 4 +-
.../function/CountMVAggregationFunction.java | 8 +-
.../function/DistinctAggregationFunction.java | 13 +-
.../function/DistinctCountAggregationFunction.java | 10 +
.../DistinctCountHLLAggregationFunction.java | 10 +
.../DistinctCountRawHLLAggregationFunction.java | 10 +
.../function/FastHLLAggregationFunction.java | 10 +
.../function/MaxAggregationFunction.java | 10 +
.../function/MinAggregationFunction.java | 9 +
.../function/MinMaxRangeAggregationFunction.java | 10 +
.../function/PercentileAggregationFunction.java | 8 +-
.../function/PercentileEstAggregationFunction.java | 8 +-
.../PercentileEstMVAggregationFunction.java | 8 +-
.../function/PercentileMVAggregationFunction.java | 8 +-
.../PercentileTDigestAggregationFunction.java | 8 +-
.../PercentileTDigestMVAggregationFunction.java | 8 +-
.../function/SumAggregationFunction.java | 10 +
.../groupby/DefaultGroupByExecutor.java | 2 +-
.../query/reduce/DistinctDataTableReducer.java | 12 +-
.../core/query/request/ServerQueryRequest.java | 5 +-
.../apache/pinot/core/startree/StarTreeUtils.java | 5 +-
.../executor/StarTreeAggregationExecutor.java | 2 +-
.../startree/executor/StarTreeGroupByExecutor.java | 2 +-
.../pinot/core/data/table/IndexedTableTest.java | 41 ++--
.../pinot/core/data/table/TableResizerTest.java | 27 +--
.../function/AggregationFunctionFactoryTest.java | 147 +++++++-------
.../DefaultAggregationExecutorTest.java | 6 +-
.../AggregationGroupByTrimmingServiceTest.java | 4 +-
.../apache/pinot/perf/BenchmarkCombineGroupBy.java | 15 +-
.../apache/pinot/perf/BenchmarkIndexedTable.java | 16 +-
.../apache/pinot/tools/scan/query/Aggregation.java | 4 +-
.../tools/scan/query/SegmentQueryProcessor.java | 4 +-
53 files changed, 727 insertions(+), 386 deletions(-)
diff --git
a/pinot-broker/src/main/java/org/apache/pinot/broker/requesthandler/BaseBrokerRequestHandler.java
b/pinot-broker/src/main/java/org/apache/pinot/broker/requesthandler/BaseBrokerRequestHandler.java
index 1784bd9..909e12f 100644
---
a/pinot-broker/src/main/java/org/apache/pinot/broker/requesthandler/BaseBrokerRequestHandler.java
+++
b/pinot-broker/src/main/java/org/apache/pinot/broker/requesthandler/BaseBrokerRequestHandler.java
@@ -24,7 +24,6 @@ import com.google.common.base.Splitter;
import com.google.common.util.concurrent.RateLimiter;
import java.net.InetAddress;
import java.util.ArrayList;
-import java.util.Arrays;
import java.util.Collection;
import java.util.Collections;
import java.util.HashMap;
@@ -66,12 +65,11 @@ import org.apache.pinot.common.response.BrokerResponse;
import org.apache.pinot.common.response.broker.BrokerResponseNative;
import org.apache.pinot.common.utils.CommonConstants;
import org.apache.pinot.common.utils.CommonConstants.Broker;
-import org.apache.pinot.common.utils.StringUtil;
import org.apache.pinot.common.utils.helix.TableCache;
+import
org.apache.pinot.core.query.aggregation.function.AggregationFunctionUtils;
import org.apache.pinot.core.query.reduce.BrokerReduceService;
import org.apache.pinot.core.transport.ServerInstance;
import org.apache.pinot.core.util.QueryOptions;
-import org.apache.pinot.parsers.CompilerConstants;
import org.apache.pinot.spi.config.TableType;
import org.apache.pinot.spi.utils.builder.TableNameBuilder;
import org.slf4j.Logger;
@@ -441,17 +439,16 @@ public abstract class BaseBrokerRequestHandler implements
BrokerRequestHandler {
}
if (brokerRequest.isSetAggregationsInfo()) {
for (AggregationInfo info : brokerRequest.getAggregationsInfo()) {
- if (info.getAggregationParams() != null && !info.getAggregationType()
- .equalsIgnoreCase(AggregationFunctionType.COUNT.getName())) {
- String column =
info.getAggregationParams().get(CompilerConstants.COLUMN_KEY_IN_AGGREGATION_INFO);
- String[] expressions =
column.split(CompilerConstants.AGGREGATION_FUNCTION_ARG_SEPARATOR);
- String[] newExpressions = new String[expressions.length];
- for (int i = 0; i < expressions.length; i++) {
- String expression = expressions[i];
- newExpressions[i] = fixColumnNameCase(tableCache, actualTableName,
expression);
+ if
(info.getAggregationType().equalsIgnoreCase(AggregationFunctionType.COUNT.getName()))
{
+
+ // Always read from backward compatible api in
AggregationFunctionUtils.
+ List<String> expressions =
AggregationFunctionUtils.getAggregationExpressions(info);
+
+ List<String> newExpressions = new ArrayList<>(expressions.size());
+ for (String expression : expressions) {
+ newExpressions.add(fixColumnNameCase(tableCache, actualTableName,
expression));
}
- String newColumns =
StringUtil.join(CompilerConstants.AGGREGATION_FUNCTION_ARG_SEPARATOR,
newExpressions);
-
info.getAggregationParams().put(CompilerConstants.COLUMN_KEY_IN_AGGREGATION_INFO,
newColumns);
+ info.setExpressions(newExpressions);
}
}
if (brokerRequest.isSetGroupBy()) {
@@ -713,10 +710,8 @@ public abstract class BaseBrokerRequestHandler implements
BrokerRequestHandler {
throw new UnsupportedOperationException("DISTINCT with GROUP BY is
currently not supported");
}
if (brokerRequest.isSetOrderBy()) {
- String column =
-
aggregationInfo.getAggregationParams().get(CompilerConstants.COLUMN_KEY_IN_AGGREGATION_INFO);
- String[] columns =
column.split(CompilerConstants.AGGREGATION_FUNCTION_ARG_SEPARATOR);
- Set<String> set = new HashSet<>(Arrays.asList(columns));
+ List<String> columns =
AggregationFunctionUtils.getAggregationExpressions(aggregationInfo);
+ Set<String> set = new HashSet<>(columns);
List<SelectionSort> orderByColumns = brokerRequest.getOrderBy();
for (SelectionSort selectionSort : orderByColumns) {
if (!set.contains(selectionSort.getColumn())) {
diff --git
a/pinot-broker/src/test/java/org/apache/pinot/broker/request/PqlAndCalciteSqlCompatibilityTest.java
b/pinot-broker/src/test/java/org/apache/pinot/broker/request/PqlAndCalciteSqlCompatibilityTest.java
index be6371c..6ddb454 100644
---
a/pinot-broker/src/test/java/org/apache/pinot/broker/request/PqlAndCalciteSqlCompatibilityTest.java
+++
b/pinot-broker/src/test/java/org/apache/pinot/broker/request/PqlAndCalciteSqlCompatibilityTest.java
@@ -22,13 +22,8 @@ import java.io.BufferedReader;
import java.io.InputStreamReader;
import java.util.List;
import org.apache.pinot.broker.requesthandler.BrokerRequestOptimizer;
-import org.apache.pinot.common.request.AggregationInfo;
import org.apache.pinot.common.request.BrokerRequest;
-import org.apache.pinot.common.request.FilterQuery;
-import org.apache.pinot.common.request.FilterQueryMap;
-import org.apache.pinot.common.request.GroupBy;
import org.apache.pinot.common.request.PinotQuery;
-import org.apache.pinot.common.request.Selection;
import org.apache.pinot.common.request.SelectionSort;
import org.apache.pinot.parsers.utils.BrokerRequestComparisonUtils;
import org.apache.pinot.pql.parsers.PinotQuery2BrokerRequestConverter;
diff --git
a/pinot-common/src/main/java/org/apache/pinot/common/request/AggregationInfo.java
b/pinot-common/src/main/java/org/apache/pinot/common/request/AggregationInfo.java
index f3dc1d5..d3eb2f1 100644
---
a/pinot-common/src/main/java/org/apache/pinot/common/request/AggregationInfo.java
+++
b/pinot-common/src/main/java/org/apache/pinot/common/request/AggregationInfo.java
@@ -28,15 +28,16 @@ package org.apache.pinot.common.request;
/**
* AUTO GENERATED: DO NOT EDIT
* Aggregation
- *
+ *
*/
[email protected](value = "Autogenerated by Thrift Compiler
(0.12.0)", date = "2019-07-19")
[email protected](value = "Autogenerated by Thrift Compiler
(0.12.0)", date = "2020-04-19")
public class AggregationInfo implements
org.apache.thrift.TBase<AggregationInfo, AggregationInfo._Fields>,
java.io.Serializable, Cloneable, Comparable<AggregationInfo> {
private static final org.apache.thrift.protocol.TStruct STRUCT_DESC = new
org.apache.thrift.protocol.TStruct("AggregationInfo");
private static final org.apache.thrift.protocol.TField
AGGREGATION_TYPE_FIELD_DESC = new
org.apache.thrift.protocol.TField("aggregationType",
org.apache.thrift.protocol.TType.STRING, (short)1);
private static final org.apache.thrift.protocol.TField
AGGREGATION_PARAMS_FIELD_DESC = new
org.apache.thrift.protocol.TField("aggregationParams",
org.apache.thrift.protocol.TType.MAP, (short)2);
private static final org.apache.thrift.protocol.TField
IS_IN_SELECT_LIST_FIELD_DESC = new
org.apache.thrift.protocol.TField("isInSelectList",
org.apache.thrift.protocol.TType.BOOL, (short)3);
+ private static final org.apache.thrift.protocol.TField
EXPRESSIONS_FIELD_DESC = new org.apache.thrift.protocol.TField("expressions",
org.apache.thrift.protocol.TType.LIST, (short)4);
private static final org.apache.thrift.scheme.SchemeFactory
STANDARD_SCHEME_FACTORY = new AggregationInfoStandardSchemeFactory();
private static final org.apache.thrift.scheme.SchemeFactory
TUPLE_SCHEME_FACTORY = new AggregationInfoTupleSchemeFactory();
@@ -44,12 +45,14 @@ public class AggregationInfo implements
org.apache.thrift.TBase<AggregationInfo,
public @org.apache.thrift.annotation.Nullable java.lang.String
aggregationType; // optional
public @org.apache.thrift.annotation.Nullable
java.util.Map<java.lang.String,java.lang.String> aggregationParams; // optional
public boolean isInSelectList; // optional
+ public @org.apache.thrift.annotation.Nullable
java.util.List<java.lang.String> expressions; // optional
/** The set of fields this struct contains, along with convenience methods
for finding and manipulating them. */
public enum _Fields implements org.apache.thrift.TFieldIdEnum {
AGGREGATION_TYPE((short)1, "aggregationType"),
AGGREGATION_PARAMS((short)2, "aggregationParams"),
- IS_IN_SELECT_LIST((short)3, "isInSelectList");
+ IS_IN_SELECT_LIST((short)3, "isInSelectList"),
+ EXPRESSIONS((short)4, "expressions");
private static final java.util.Map<java.lang.String, _Fields> byName = new
java.util.HashMap<java.lang.String, _Fields>();
@@ -71,6 +74,8 @@ public class AggregationInfo implements
org.apache.thrift.TBase<AggregationInfo,
return AGGREGATION_PARAMS;
case 3: // IS_IN_SELECT_LIST
return IS_IN_SELECT_LIST;
+ case 4: // EXPRESSIONS
+ return EXPRESSIONS;
default:
return null;
}
@@ -114,18 +119,21 @@ public class AggregationInfo implements
org.apache.thrift.TBase<AggregationInfo,
// isset id assignments
private static final int __ISINSELECTLIST_ISSET_ID = 0;
private byte __isset_bitfield = 0;
- private static final _Fields optionals[] =
{_Fields.AGGREGATION_TYPE,_Fields.AGGREGATION_PARAMS,_Fields.IS_IN_SELECT_LIST};
+ private static final _Fields optionals[] =
{_Fields.AGGREGATION_TYPE,_Fields.AGGREGATION_PARAMS,_Fields.IS_IN_SELECT_LIST,_Fields.EXPRESSIONS};
public static final java.util.Map<_Fields,
org.apache.thrift.meta_data.FieldMetaData> metaDataMap;
static {
java.util.Map<_Fields, org.apache.thrift.meta_data.FieldMetaData> tmpMap =
new java.util.EnumMap<_Fields,
org.apache.thrift.meta_data.FieldMetaData>(_Fields.class);
- tmpMap.put(_Fields.AGGREGATION_TYPE, new
org.apache.thrift.meta_data.FieldMetaData("aggregationType",
org.apache.thrift.TFieldRequirementType.OPTIONAL,
+ tmpMap.put(_Fields.AGGREGATION_TYPE, new
org.apache.thrift.meta_data.FieldMetaData("aggregationType",
org.apache.thrift.TFieldRequirementType.OPTIONAL,
new
org.apache.thrift.meta_data.FieldValueMetaData(org.apache.thrift.protocol.TType.STRING)));
- tmpMap.put(_Fields.AGGREGATION_PARAMS, new
org.apache.thrift.meta_data.FieldMetaData("aggregationParams",
org.apache.thrift.TFieldRequirementType.OPTIONAL,
- new
org.apache.thrift.meta_data.MapMetaData(org.apache.thrift.protocol.TType.MAP,
- new
org.apache.thrift.meta_data.FieldValueMetaData(org.apache.thrift.protocol.TType.STRING),
+ tmpMap.put(_Fields.AGGREGATION_PARAMS, new
org.apache.thrift.meta_data.FieldMetaData("aggregationParams",
org.apache.thrift.TFieldRequirementType.OPTIONAL,
+ new
org.apache.thrift.meta_data.MapMetaData(org.apache.thrift.protocol.TType.MAP,
+ new
org.apache.thrift.meta_data.FieldValueMetaData(org.apache.thrift.protocol.TType.STRING),
new
org.apache.thrift.meta_data.FieldValueMetaData(org.apache.thrift.protocol.TType.STRING))));
- tmpMap.put(_Fields.IS_IN_SELECT_LIST, new
org.apache.thrift.meta_data.FieldMetaData("isInSelectList",
org.apache.thrift.TFieldRequirementType.OPTIONAL,
+ tmpMap.put(_Fields.IS_IN_SELECT_LIST, new
org.apache.thrift.meta_data.FieldMetaData("isInSelectList",
org.apache.thrift.TFieldRequirementType.OPTIONAL,
new
org.apache.thrift.meta_data.FieldValueMetaData(org.apache.thrift.protocol.TType.BOOL)));
+ tmpMap.put(_Fields.EXPRESSIONS, new
org.apache.thrift.meta_data.FieldMetaData("expressions",
org.apache.thrift.TFieldRequirementType.OPTIONAL,
+ new
org.apache.thrift.meta_data.ListMetaData(org.apache.thrift.protocol.TType.LIST,
+ new
org.apache.thrift.meta_data.FieldValueMetaData(org.apache.thrift.protocol.TType.STRING))));
metaDataMap = java.util.Collections.unmodifiableMap(tmpMap);
org.apache.thrift.meta_data.FieldMetaData.addStructMetaDataMap(AggregationInfo.class,
metaDataMap);
}
@@ -146,6 +154,10 @@ public class AggregationInfo implements
org.apache.thrift.TBase<AggregationInfo,
this.aggregationParams = __this__aggregationParams;
}
this.isInSelectList = other.isInSelectList;
+ if (other.isSetExpressions()) {
+ java.util.List<java.lang.String> __this__expressions = new
java.util.ArrayList<java.lang.String>(other.expressions);
+ this.expressions = __this__expressions;
+ }
}
public AggregationInfo deepCopy() {
@@ -158,6 +170,7 @@ public class AggregationInfo implements
org.apache.thrift.TBase<AggregationInfo,
this.aggregationParams = null;
setIsInSelectListIsSet(false);
this.isInSelectList = false;
+ this.expressions = null;
}
@org.apache.thrift.annotation.Nullable
@@ -244,6 +257,47 @@ public class AggregationInfo implements
org.apache.thrift.TBase<AggregationInfo,
__isset_bitfield =
org.apache.thrift.EncodingUtils.setBit(__isset_bitfield,
__ISINSELECTLIST_ISSET_ID, value);
}
+ public int getExpressionsSize() {
+ return (this.expressions == null) ? 0 : this.expressions.size();
+ }
+
+ @org.apache.thrift.annotation.Nullable
+ public java.util.Iterator<java.lang.String> getExpressionsIterator() {
+ return (this.expressions == null) ? null : this.expressions.iterator();
+ }
+
+ public void addToExpressions(java.lang.String elem) {
+ if (this.expressions == null) {
+ this.expressions = new java.util.ArrayList<java.lang.String>();
+ }
+ this.expressions.add(elem);
+ }
+
+ @org.apache.thrift.annotation.Nullable
+ public java.util.List<java.lang.String> getExpressions() {
+ return this.expressions;
+ }
+
+ public AggregationInfo setExpressions(@org.apache.thrift.annotation.Nullable
java.util.List<java.lang.String> expressions) {
+ this.expressions = expressions;
+ return this;
+ }
+
+ public void unsetExpressions() {
+ this.expressions = null;
+ }
+
+ /** Returns true if field expressions is set (has been assigned a value) and
false otherwise */
+ public boolean isSetExpressions() {
+ return this.expressions != null;
+ }
+
+ public void setExpressionsIsSet(boolean value) {
+ if (!value) {
+ this.expressions = null;
+ }
+ }
+
public void setFieldValue(_Fields field,
@org.apache.thrift.annotation.Nullable java.lang.Object value) {
switch (field) {
case AGGREGATION_TYPE:
@@ -270,6 +324,14 @@ public class AggregationInfo implements
org.apache.thrift.TBase<AggregationInfo,
}
break;
+ case EXPRESSIONS:
+ if (value == null) {
+ unsetExpressions();
+ } else {
+ setExpressions((java.util.List<java.lang.String>)value);
+ }
+ break;
+
}
}
@@ -285,6 +347,9 @@ public class AggregationInfo implements
org.apache.thrift.TBase<AggregationInfo,
case IS_IN_SELECT_LIST:
return isIsInSelectList();
+ case EXPRESSIONS:
+ return getExpressions();
+
}
throw new java.lang.IllegalStateException();
}
@@ -302,6 +367,8 @@ public class AggregationInfo implements
org.apache.thrift.TBase<AggregationInfo,
return isSetAggregationParams();
case IS_IN_SELECT_LIST:
return isSetIsInSelectList();
+ case EXPRESSIONS:
+ return isSetExpressions();
}
throw new java.lang.IllegalStateException();
}
@@ -348,6 +415,15 @@ public class AggregationInfo implements
org.apache.thrift.TBase<AggregationInfo,
return false;
}
+ boolean this_present_expressions = true && this.isSetExpressions();
+ boolean that_present_expressions = true && that.isSetExpressions();
+ if (this_present_expressions || that_present_expressions) {
+ if (!(this_present_expressions && that_present_expressions))
+ return false;
+ if (!this.expressions.equals(that.expressions))
+ return false;
+ }
+
return true;
}
@@ -367,6 +443,10 @@ public class AggregationInfo implements
org.apache.thrift.TBase<AggregationInfo,
if (isSetIsInSelectList())
hashCode = hashCode * 8191 + ((isInSelectList) ? 131071 : 524287);
+ hashCode = hashCode * 8191 + ((isSetExpressions()) ? 131071 : 524287);
+ if (isSetExpressions())
+ hashCode = hashCode * 8191 + expressions.hashCode();
+
return hashCode;
}
@@ -408,6 +488,16 @@ public class AggregationInfo implements
org.apache.thrift.TBase<AggregationInfo,
return lastComparison;
}
}
+ lastComparison =
java.lang.Boolean.valueOf(isSetExpressions()).compareTo(other.isSetExpressions());
+ if (lastComparison != 0) {
+ return lastComparison;
+ }
+ if (isSetExpressions()) {
+ lastComparison =
org.apache.thrift.TBaseHelper.compareTo(this.expressions, other.expressions);
+ if (lastComparison != 0) {
+ return lastComparison;
+ }
+ }
return 0;
}
@@ -454,6 +544,16 @@ public class AggregationInfo implements
org.apache.thrift.TBase<AggregationInfo,
sb.append(this.isInSelectList);
first = false;
}
+ if (isSetExpressions()) {
+ if (!first) sb.append(", ");
+ sb.append("expressions:");
+ if (this.expressions == null) {
+ sb.append("null");
+ } else {
+ sb.append(this.expressions);
+ }
+ first = false;
+ }
sb.append(")");
return sb.toString();
}
@@ -495,7 +595,7 @@ public class AggregationInfo implements
org.apache.thrift.TBase<AggregationInfo,
while (true)
{
schemeField = iprot.readFieldBegin();
- if (schemeField.type == org.apache.thrift.protocol.TType.STOP) {
+ if (schemeField.type == org.apache.thrift.protocol.TType.STOP) {
break;
}
switch (schemeField.id) {
@@ -503,7 +603,7 @@ public class AggregationInfo implements
org.apache.thrift.TBase<AggregationInfo,
if (schemeField.type == org.apache.thrift.protocol.TType.STRING) {
struct.aggregationType = iprot.readString();
struct.setAggregationTypeIsSet(true);
- } else {
+ } else {
org.apache.thrift.protocol.TProtocolUtil.skip(iprot,
schemeField.type);
}
break;
@@ -523,7 +623,7 @@ public class AggregationInfo implements
org.apache.thrift.TBase<AggregationInfo,
iprot.readMapEnd();
}
struct.setAggregationParamsIsSet(true);
- } else {
+ } else {
org.apache.thrift.protocol.TProtocolUtil.skip(iprot,
schemeField.type);
}
break;
@@ -531,7 +631,25 @@ public class AggregationInfo implements
org.apache.thrift.TBase<AggregationInfo,
if (schemeField.type == org.apache.thrift.protocol.TType.BOOL) {
struct.isInSelectList = iprot.readBool();
struct.setIsInSelectListIsSet(true);
- } else {
+ } else {
+ org.apache.thrift.protocol.TProtocolUtil.skip(iprot,
schemeField.type);
+ }
+ break;
+ case 4: // EXPRESSIONS
+ if (schemeField.type == org.apache.thrift.protocol.TType.LIST) {
+ {
+ org.apache.thrift.protocol.TList _list56 =
iprot.readListBegin();
+ struct.expressions = new
java.util.ArrayList<java.lang.String>(_list56.size);
+ @org.apache.thrift.annotation.Nullable java.lang.String
_elem57;
+ for (int _i58 = 0; _i58 < _list56.size; ++_i58)
+ {
+ _elem57 = iprot.readString();
+ struct.expressions.add(_elem57);
+ }
+ iprot.readListEnd();
+ }
+ struct.setExpressionsIsSet(true);
+ } else {
org.apache.thrift.protocol.TProtocolUtil.skip(iprot,
schemeField.type);
}
break;
@@ -562,10 +680,10 @@ public class AggregationInfo implements
org.apache.thrift.TBase<AggregationInfo,
oprot.writeFieldBegin(AGGREGATION_PARAMS_FIELD_DESC);
{
oprot.writeMapBegin(new
org.apache.thrift.protocol.TMap(org.apache.thrift.protocol.TType.STRING,
org.apache.thrift.protocol.TType.STRING, struct.aggregationParams.size()));
- for (java.util.Map.Entry<java.lang.String, java.lang.String>
_iter56 : struct.aggregationParams.entrySet())
+ for (java.util.Map.Entry<java.lang.String, java.lang.String>
_iter59 : struct.aggregationParams.entrySet())
{
- oprot.writeString(_iter56.getKey());
- oprot.writeString(_iter56.getValue());
+ oprot.writeString(_iter59.getKey());
+ oprot.writeString(_iter59.getValue());
}
oprot.writeMapEnd();
}
@@ -577,6 +695,20 @@ public class AggregationInfo implements
org.apache.thrift.TBase<AggregationInfo,
oprot.writeBool(struct.isInSelectList);
oprot.writeFieldEnd();
}
+ if (struct.expressions != null) {
+ if (struct.isSetExpressions()) {
+ oprot.writeFieldBegin(EXPRESSIONS_FIELD_DESC);
+ {
+ oprot.writeListBegin(new
org.apache.thrift.protocol.TList(org.apache.thrift.protocol.TType.STRING,
struct.expressions.size()));
+ for (java.lang.String _iter60 : struct.expressions)
+ {
+ oprot.writeString(_iter60);
+ }
+ oprot.writeListEnd();
+ }
+ oprot.writeFieldEnd();
+ }
+ }
oprot.writeFieldStop();
oprot.writeStructEnd();
}
@@ -604,44 +736,56 @@ public class AggregationInfo implements
org.apache.thrift.TBase<AggregationInfo,
if (struct.isSetIsInSelectList()) {
optionals.set(2);
}
- oprot.writeBitSet(optionals, 3);
+ if (struct.isSetExpressions()) {
+ optionals.set(3);
+ }
+ oprot.writeBitSet(optionals, 4);
if (struct.isSetAggregationType()) {
oprot.writeString(struct.aggregationType);
}
if (struct.isSetAggregationParams()) {
{
oprot.writeI32(struct.aggregationParams.size());
- for (java.util.Map.Entry<java.lang.String, java.lang.String> _iter57
: struct.aggregationParams.entrySet())
+ for (java.util.Map.Entry<java.lang.String, java.lang.String> _iter61
: struct.aggregationParams.entrySet())
{
- oprot.writeString(_iter57.getKey());
- oprot.writeString(_iter57.getValue());
+ oprot.writeString(_iter61.getKey());
+ oprot.writeString(_iter61.getValue());
}
}
}
if (struct.isSetIsInSelectList()) {
oprot.writeBool(struct.isInSelectList);
}
+ if (struct.isSetExpressions()) {
+ {
+ oprot.writeI32(struct.expressions.size());
+ for (java.lang.String _iter62 : struct.expressions)
+ {
+ oprot.writeString(_iter62);
+ }
+ }
+ }
}
@Override
public void read(org.apache.thrift.protocol.TProtocol prot,
AggregationInfo struct) throws org.apache.thrift.TException {
org.apache.thrift.protocol.TTupleProtocol iprot =
(org.apache.thrift.protocol.TTupleProtocol) prot;
- java.util.BitSet incoming = iprot.readBitSet(3);
+ java.util.BitSet incoming = iprot.readBitSet(4);
if (incoming.get(0)) {
struct.aggregationType = iprot.readString();
struct.setAggregationTypeIsSet(true);
}
if (incoming.get(1)) {
{
- org.apache.thrift.protocol.TMap _map58 = new
org.apache.thrift.protocol.TMap(org.apache.thrift.protocol.TType.STRING,
org.apache.thrift.protocol.TType.STRING, iprot.readI32());
- struct.aggregationParams = new
java.util.HashMap<java.lang.String,java.lang.String>(2*_map58.size);
- @org.apache.thrift.annotation.Nullable java.lang.String _key59;
- @org.apache.thrift.annotation.Nullable java.lang.String _val60;
- for (int _i61 = 0; _i61 < _map58.size; ++_i61)
+ org.apache.thrift.protocol.TMap _map63 = new
org.apache.thrift.protocol.TMap(org.apache.thrift.protocol.TType.STRING,
org.apache.thrift.protocol.TType.STRING, iprot.readI32());
+ struct.aggregationParams = new
java.util.HashMap<java.lang.String,java.lang.String>(2*_map63.size);
+ @org.apache.thrift.annotation.Nullable java.lang.String _key64;
+ @org.apache.thrift.annotation.Nullable java.lang.String _val65;
+ for (int _i66 = 0; _i66 < _map63.size; ++_i66)
{
- _key59 = iprot.readString();
- _val60 = iprot.readString();
- struct.aggregationParams.put(_key59, _val60);
+ _key64 = iprot.readString();
+ _val65 = iprot.readString();
+ struct.aggregationParams.put(_key64, _val65);
}
}
struct.setAggregationParamsIsSet(true);
@@ -650,6 +794,19 @@ public class AggregationInfo implements
org.apache.thrift.TBase<AggregationInfo,
struct.isInSelectList = iprot.readBool();
struct.setIsInSelectListIsSet(true);
}
+ if (incoming.get(3)) {
+ {
+ org.apache.thrift.protocol.TList _list67 = new
org.apache.thrift.protocol.TList(org.apache.thrift.protocol.TType.STRING,
iprot.readI32());
+ struct.expressions = new
java.util.ArrayList<java.lang.String>(_list67.size);
+ @org.apache.thrift.annotation.Nullable java.lang.String _elem68;
+ for (int _i69 = 0; _i69 < _list67.size; ++_i69)
+ {
+ _elem68 = iprot.readString();
+ struct.expressions.add(_elem68);
+ }
+ }
+ struct.setExpressionsIsSet(true);
+ }
}
}
diff --git
a/pinot-common/src/main/java/org/apache/pinot/common/utils/request/HavingQueryTree.java
b/pinot-common/src/main/java/org/apache/pinot/common/utils/request/HavingQueryTree.java
index dca93a0..ea09e5a 100644
---
a/pinot-common/src/main/java/org/apache/pinot/common/utils/request/HavingQueryTree.java
+++
b/pinot-common/src/main/java/org/apache/pinot/common/utils/request/HavingQueryTree.java
@@ -59,7 +59,7 @@ public class HavingQueryTree extends QueryTree {
stringBuffer.append(_operator);
} else {
stringBuffer.append(_aggregationInfo.getAggregationType()).append("(")
-
.append(_aggregationInfo.getAggregationParams().toString()).append(")").append("
").append(_operator)
+
.append(_aggregationInfo.getExpressions().toString()).append(")").append("
").append(_operator)
.append(" ").append(_value);
}
if (_children != null) {
diff --git
a/pinot-common/src/main/java/org/apache/pinot/parsers/utils/BrokerRequestComparisonUtils.java
b/pinot-common/src/main/java/org/apache/pinot/parsers/utils/BrokerRequestComparisonUtils.java
index e19a2d3..c08cb66 100644
---
a/pinot-common/src/main/java/org/apache/pinot/parsers/utils/BrokerRequestComparisonUtils.java
+++
b/pinot-common/src/main/java/org/apache/pinot/parsers/utils/BrokerRequestComparisonUtils.java
@@ -89,8 +89,7 @@ public class BrokerRequestComparisonUtils {
return false;
}
} else if (br2.getGroupBy() != null) {
- LOGGER.error("tGroupBy did not match, br1.getGroupBy() = null,
br2.getGroupBy() = {}",
- br2.getGroupBy());
+ LOGGER.error("GroupBy did not match, br1.getGroupBy() = null,
br2.getGroupBy() = {}", br2.getGroupBy());
return false;
}
if (br1.getAggregationsInfo() != null) {
@@ -114,8 +113,7 @@ public class BrokerRequestComparisonUtils {
return false;
}
} else if (br2.getOrderBy() != null) {
- LOGGER.error("OrderBy did not match, br1.getOrderBy() = null,
br2.getOrderBy() = {}",
- br2.getOrderBy());
+ LOGGER.error("OrderBy did not match, br1.getOrderBy() = null,
br2.getOrderBy() = {}", br2.getOrderBy());
return false;
}
}
@@ -178,22 +176,19 @@ public class BrokerRequestComparisonUtils {
LOGGER.error("Failed to validate AggregationInfo: AggregationType
doesn't match.\n\t{}\n\t{}", agg1, agg2);
return false;
}
- if (agg1.getAggregationParamsSize() != agg2.getAggregationParamsSize()) {
- LOGGER.error("Failed to validate AggregationInfo: AggregationParamsSize
doesn't match.\n\t{}\n\t{}", agg1, agg2);
+ if (agg1.getExpressionsSize() != agg2.getExpressionsSize()) {
+ LOGGER.error("Failed to validate AggregationInfo: Expressions doesn't
match.\n\t{}\n\t{}", agg1, agg2);
return false;
}
- for (int i = 0; i < agg1.getAggregationParamsSize(); i++) {
- for (String key : agg1.getAggregationParams().keySet()) {
- if
(!agg1.getAggregationParams().get(key).equals(agg2.getAggregationParams().get(key)))
{
- LOGGER
- .error("Failed to validate AggregationInfo: AggregationParams at
key {} doesn't match.\n\t{}\n\t{}", key,
- agg1.getAggregationParams().get(key),
agg2.getAggregationParams().get(key));
- return false;
- }
+ for (int i = 0; i < agg1.getExpressionsSize(); i++) {
+ if (!agg1.getExpressions().get(i).equals(agg2.getExpressions().get(i))) {
+ LOGGER.error("Failed to validate AggregationInfo: Expressions
mis-match.\n\t{}\n\t{}",
+ agg1.getExpressions().get(i), agg1.getExpressions().get(i));
+ return false;
}
}
return true;
- }
+}
private static boolean validateGroupBy(GroupBy groupBy1, GroupBy groupBy2) {
if (groupBy1.getTopN() != groupBy2.getTopN()) {
diff --git
a/pinot-common/src/main/java/org/apache/pinot/pql/parsers/PinotQuery2BrokerRequestConverter.java
b/pinot-common/src/main/java/org/apache/pinot/pql/parsers/PinotQuery2BrokerRequestConverter.java
index 7ad26f5..eeda5ef 100644
---
a/pinot-common/src/main/java/org/apache/pinot/pql/parsers/PinotQuery2BrokerRequestConverter.java
+++
b/pinot-common/src/main/java/org/apache/pinot/pql/parsers/PinotQuery2BrokerRequestConverter.java
@@ -19,11 +19,12 @@
package org.apache.pinot.pql.parsers;
import java.util.ArrayList;
+import java.util.Collections;
import java.util.HashMap;
-import java.util.HashSet;
import java.util.List;
import java.util.Map;
import java.util.Set;
+import java.util.TreeSet;
import org.apache.calcite.sql.SqlKind;
import org.apache.pinot.common.function.AggregationFunctionType;
import org.apache.pinot.common.function.FunctionDefinitionRegistry;
@@ -41,7 +42,6 @@ import org.apache.pinot.common.request.PinotQuery;
import org.apache.pinot.common.request.QuerySource;
import org.apache.pinot.common.request.Selection;
import org.apache.pinot.common.request.SelectionSort;
-import org.apache.pinot.parsers.CompilerConstants;
import org.apache.pinot.pql.parsers.pql2.ast.FilterKind;
import org.apache.pinot.pql.parsers.pql2.ast.OrderByAstNode;
@@ -206,8 +206,7 @@ public class PinotQuery2BrokerRequestConverter {
return expression.getIdentifier().getName();
case FUNCTION:
Function functionCall = expression.getFunctionCall();
- String functionString = standardizeFunction(functionCall);
- return functionString;
+ return standardizeFunction(functionCall);
default:
throw new UnsupportedOperationException("Unknown Expression type: " +
expression.getType());
}
@@ -233,32 +232,32 @@ public class PinotQuery2BrokerRequestConverter {
throw new Pql2CompilationException("Aggregation function expects non
null argument");
}
- String argumentString;
+ List<String> args = new ArrayList<>(operands.size());
String functionName = function.getOperator();
if
(functionName.equalsIgnoreCase(AggregationFunctionType.COUNT.getName())) {
- argumentString = "*";
+ args = Collections.singletonList("*");
} else {
- Set<String> expressions = new HashSet<>();
- StringBuilder sb = new StringBuilder();
- int numOperands = operands.size();
- for (int i = 0; i < numOperands; i++) {
- Expression expression = operands.get(i);
- String columnExpression = getColumnExpression(expression);
- if (expressions.add(columnExpression)) {
- // deduplicate the columns
- if (i != 0) {
- sb.append(CompilerConstants.AGGREGATION_FUNCTION_ARG_SEPARATOR);
+ // Need to de-dup columns for distinct.
+ if
(functionName.equalsIgnoreCase(AggregationFunctionType.DISTINCT.getName())) {
+ Set<String> expressionSet = new TreeSet<>();
+
+ for (Expression operand : operands) {
+ String expression = getColumnExpression(operand);
+ if (expressionSet.add(expression)) {
+ args.add(expression);
}
- sb.append(getColumnExpression(expression));
+ }
+ } else {
+ for (Expression operand : operands) {
+ args.add(getColumnExpression(operand));
}
}
- argumentString = sb.toString();
}
AggregationInfo aggregationInfo = new AggregationInfo();
aggregationInfo.setAggregationType(functionName);
-
aggregationInfo.putToAggregationParams(CompilerConstants.COLUMN_KEY_IN_AGGREGATION_INFO,
argumentString);
+ aggregationInfo.setExpressions(args);
aggregationInfo.setIsInSelectList(true);
return aggregationInfo;
}
@@ -284,7 +283,6 @@ public class PinotQuery2BrokerRequestConverter {
List<Integer> childFilterIds = new ArrayList<>();
switch (filterExpression.getType()) {
case LITERAL:
- break;
case IDENTIFIER:
break;
case FUNCTION:
diff --git
a/pinot-common/src/main/java/org/apache/pinot/pql/parsers/pql2/ast/FunctionCallAstNode.java
b/pinot-common/src/main/java/org/apache/pinot/pql/parsers/pql2/ast/FunctionCallAstNode.java
index 4af5f81..390be8c 100644
---
a/pinot-common/src/main/java/org/apache/pinot/pql/parsers/pql2/ast/FunctionCallAstNode.java
+++
b/pinot-common/src/main/java/org/apache/pinot/pql/parsers/pql2/ast/FunctionCallAstNode.java
@@ -18,13 +18,13 @@
*/
package org.apache.pinot.pql.parsers.pql2.ast;
-import java.util.HashSet;
+import java.util.ArrayList;
import java.util.List;
import java.util.Set;
+import java.util.TreeSet;
import org.apache.pinot.common.function.AggregationFunctionType;
import org.apache.pinot.common.request.AggregationInfo;
import org.apache.pinot.common.request.transform.TransformExpressionTree;
-import org.apache.pinot.parsers.CompilerConstants;
import org.apache.pinot.pql.parsers.Pql2CompilationException;
import org.apache.pinot.spi.utils.EqualityUtils;
@@ -80,10 +80,10 @@ public class FunctionCallAstNode extends BaseAstNode {
}
AggregationInfo buildAggregationInfo() {
- String expression;
+ List<String> functionArgs = new ArrayList<>();
// COUNT aggregation function always works on '*'
if (_name.equalsIgnoreCase(AggregationFunctionType.COUNT.getName())) {
- expression = "*";
+ functionArgs.add("*");
} else {
List<? extends AstNode> children = getChildren();
if (children == null || children.isEmpty()) {
@@ -98,25 +98,26 @@ public class FunctionCallAstNode extends BaseAstNode {
"Syntax error: Pinot currently does not support DISTINCT with *.
Please specify each column name as argument to DISTINCT function");
}
- Set<String> expressions = new HashSet<>();
- StringBuilder distinctColumnExpr = new StringBuilder();
- int numChildren = children.size();
- for (int i = 0; i < numChildren; ++i) {
- expression =
TransformExpressionTree.getStandardExpression(children.get(i));
- if (expressions.add(expression)) {
- // deduplicate the columns
- if (i != 0) {
-
distinctColumnExpr.append(CompilerConstants.AGGREGATION_FUNCTION_ARG_SEPARATOR);
+ // Need to de-dup the args for Distinct.
+ if (_name.equalsIgnoreCase(AggregationFunctionType.DISTINCT.getName())) {
+ Set<String> expressionsSet = new TreeSet<>();
+ for (AstNode child : children) {
+ String expression =
TransformExpressionTree.getStandardExpression(child);
+
+ if (expressionsSet.add(expression)) {
+ functionArgs.add(expression);
}
- distinctColumnExpr.append(expression);
+ }
+ } else {
+ for (AstNode child : children) {
+
functionArgs.add(TransformExpressionTree.getStandardExpression(child));
}
}
- expression = distinctColumnExpr.toString();
}
AggregationInfo aggregationInfo = new AggregationInfo();
aggregationInfo.setAggregationType(_name);
-
aggregationInfo.putToAggregationParams(CompilerConstants.COLUMN_KEY_IN_AGGREGATION_INFO,
expression);
+ aggregationInfo.setExpressions(functionArgs);
aggregationInfo.setIsInSelectList(_isInSelectList);
return aggregationInfo;
diff --git
a/pinot-common/src/test/java/org/apache/pinot/pql/parsers/Pql2CompilerTest.java
b/pinot-common/src/test/java/org/apache/pinot/pql/parsers/Pql2CompilerTest.java
index 5ce5eff..1ec95bb 100644
---
a/pinot-common/src/test/java/org/apache/pinot/pql/parsers/Pql2CompilerTest.java
+++
b/pinot-common/src/test/java/org/apache/pinot/pql/parsers/Pql2CompilerTest.java
@@ -261,7 +261,7 @@ public class Pql2CompilerTest {
.compileToBrokerRequest("select avg(age) as avg_age from person group
by address_city having avg(age)=20");
Assert.assertEquals(brokerRequest.getHavingFilterQuery().getOperator(),
FilterOperator.EQUALITY);
Assert.assertEquals(brokerRequest.getHavingFilterQuery().getAggregationInfo().getAggregationType(),
"avg");
-
Assert.assertEquals(brokerRequest.getHavingFilterQuery().getAggregationInfo().getAggregationParams().get("column"),
+
Assert.assertEquals(brokerRequest.getHavingFilterQuery().getAggregationInfo().getExpressions().get(0),
"age");
Assert.assertEquals(brokerRequest.getHavingFilterQuery().getValue().get(0),
"20");
brokerRequest = COMPILER.compileToBrokerRequest(
@@ -344,11 +344,11 @@ public class Pql2CompilerTest {
BrokerRequest brokerRequest = COMPILER.compileToBrokerRequest(
"select avg(`attributes.age`) as `avg_age` from `person` group by
`attributes.address_city` having avg(`attributes.age`)=20");
-
Assert.assertEquals(brokerRequest.getAggregationsInfo().get(0).getAggregationParams().get("column"),
+
Assert.assertEquals(brokerRequest.getAggregationsInfo().get(0).getExpressions().get(0),
"attributes.age");
Assert.assertEquals(brokerRequest.getGroupBy().getExpressions(),
Collections.singletonList("attributes.address_city"));
-
Assert.assertEquals(brokerRequest.getHavingFilterQuery().getAggregationInfo().getAggregationParams().get("column"),
+
Assert.assertEquals(brokerRequest.getHavingFilterQuery().getAggregationInfo().getExpressions().get(0),
"attributes.age");
// Test PinotQuery
@@ -370,8 +370,8 @@ public class Pql2CompilerTest {
COMPILER.compileToBrokerRequest("SELECT SUM('foo'), MAX(\"bar\") FROM
table GROUP BY 'foo', \"bar\"");
List<AggregationInfo> aggregationInfos =
brokerRequest.getAggregationsInfo();
Assert.assertEquals(aggregationInfos.size(), 2);
-
Assert.assertEquals(aggregationInfos.get(0).getAggregationParams().get("column"),
"foo");
-
Assert.assertEquals(aggregationInfos.get(1).getAggregationParams().get("column"),
"bar");
+ Assert.assertEquals(aggregationInfos.get(0).getExpressions().get(0),
"foo");
+ Assert.assertEquals(aggregationInfos.get(1).getExpressions().get(0),
"bar");
List<String> expressions = brokerRequest.getGroupBy().getExpressions();
Assert.assertEquals(expressions.size(), 2);
Assert.assertEquals(expressions.get(0), "foo");
@@ -394,7 +394,7 @@ public class Pql2CompilerTest {
COMPILER.compileToBrokerRequest("SELECT SUM(ADD(foo, 'bar')) FROM
table GROUP BY SUB(\"foo\", bar)");
aggregationInfos = brokerRequest.getAggregationsInfo();
Assert.assertEquals(aggregationInfos.size(), 1);
-
Assert.assertEquals(aggregationInfos.get(0).getAggregationParams().get("column"),
"add(foo,'bar')");
+ Assert.assertEquals(aggregationInfos.get(0).getExpressions().get(0),
"add(foo,'bar')");
expressions = brokerRequest.getGroupBy().getExpressions();
Assert.assertEquals(expressions.size(), 1);
Assert.assertEquals(expressions.get(0), "sub('foo',bar)");
diff --git
a/pinot-common/src/test/java/org/apache/pinot/sql/parsers/CalciteSqlCompilerTest.java
b/pinot-common/src/test/java/org/apache/pinot/sql/parsers/CalciteSqlCompilerTest.java
index d3c0e50..1633b66 100644
---
a/pinot-common/src/test/java/org/apache/pinot/sql/parsers/CalciteSqlCompilerTest.java
+++
b/pinot-common/src/test/java/org/apache/pinot/sql/parsers/CalciteSqlCompilerTest.java
@@ -21,6 +21,7 @@ package org.apache.pinot.sql.parsers;
import java.io.BufferedReader;
import java.io.IOException;
import java.io.InputStreamReader;
+import java.util.Arrays;
import java.util.List;
import org.apache.calcite.sql.SqlKind;
import org.apache.calcite.sql.parser.SqlParseException;
@@ -33,7 +34,6 @@ import org.apache.pinot.common.request.FilterOperator;
import org.apache.pinot.common.request.Function;
import org.apache.pinot.common.request.Identifier;
import org.apache.pinot.common.request.PinotQuery;
-import org.apache.pinot.parsers.CompilerConstants;
import org.apache.pinot.pql.parsers.PinotQuery2BrokerRequestConverter;
import org.apache.pinot.pql.parsers.Pql2Compiler;
import org.testng.Assert;
@@ -124,67 +124,133 @@ public class CalciteSqlCompilerTest {
Function func = pinotQuery.getFilterExpression().getFunctionCall();
Assert.assertEquals(func.getOperator(), SqlKind.GREATER_THAN.name());
Assert.assertEquals(func.getOperands().get(0).getFunctionCall().getOperator(),
"MINUS");
-
Assert.assertEquals(func.getOperands().get(0).getFunctionCall().getOperands().get(0).getIdentifier().getName(),
"a");
-
Assert.assertEquals(func.getOperands().get(0).getFunctionCall().getOperands().get(1).getIdentifier().getName(),
"b");
+ Assert
+
.assertEquals(func.getOperands().get(0).getFunctionCall().getOperands().get(0).getIdentifier().getName(),
"a");
+ Assert
+
.assertEquals(func.getOperands().get(0).getFunctionCall().getOperands().get(1).getIdentifier().getName(),
"b");
Assert.assertEquals(func.getOperands().get(1).getLiteral().getLongValue(),
0L);
pinotQuery = CalciteSqlParser.compileToPinotQuery("select * from
vegetables where 0 < a-b");
func = pinotQuery.getFilterExpression().getFunctionCall();
Assert.assertEquals(func.getOperator(), SqlKind.GREATER_THAN.name());
Assert.assertEquals(func.getOperands().get(0).getFunctionCall().getOperator(),
"MINUS");
-
Assert.assertEquals(func.getOperands().get(0).getFunctionCall().getOperands().get(0).getIdentifier().getName(),
"a");
-
Assert.assertEquals(func.getOperands().get(0).getFunctionCall().getOperands().get(1).getIdentifier().getName(),
"b");
+ Assert
+
.assertEquals(func.getOperands().get(0).getFunctionCall().getOperands().get(0).getIdentifier().getName(),
"a");
+ Assert
+
.assertEquals(func.getOperands().get(0).getFunctionCall().getOperands().get(1).getIdentifier().getName(),
"b");
Assert.assertEquals(func.getOperands().get(1).getLiteral().getLongValue(),
0L);
-
pinotQuery = CalciteSqlParser.compileToPinotQuery("select * from
vegetables where b < 100 + c");
func = pinotQuery.getFilterExpression().getFunctionCall();
Assert.assertEquals(func.getOperator(), SqlKind.LESS_THAN.name());
Assert.assertEquals(func.getOperands().get(0).getFunctionCall().getOperator(),
"MINUS");
-
Assert.assertEquals(func.getOperands().get(0).getFunctionCall().getOperands().get(0).getIdentifier().getName(),
"b");
-
Assert.assertEquals(func.getOperands().get(0).getFunctionCall().getOperands().get(1).getFunctionCall().getOperator(),
"PLUS");
-
Assert.assertEquals(func.getOperands().get(0).getFunctionCall().getOperands().get(1).getFunctionCall().getOperands().get(0).getLiteral().getLongValue(),
100L);
-
Assert.assertEquals(func.getOperands().get(0).getFunctionCall().getOperands().get(1).getFunctionCall().getOperands().get(1).getIdentifier().getName(),
"c");
+ Assert
+
.assertEquals(func.getOperands().get(0).getFunctionCall().getOperands().get(0).getIdentifier().getName(),
"b");
+ Assert
+
.assertEquals(func.getOperands().get(0).getFunctionCall().getOperands().get(1).getFunctionCall().getOperator(),
+ "PLUS");
+ Assert.assertEquals(
+
func.getOperands().get(0).getFunctionCall().getOperands().get(1).getFunctionCall().getOperands().get(0)
+ .getLiteral().getLongValue(), 100L);
+ Assert.assertEquals(
+
func.getOperands().get(0).getFunctionCall().getOperands().get(1).getFunctionCall().getOperands().get(1)
+ .getIdentifier().getName(), "c");
Assert.assertEquals(func.getOperands().get(1).getLiteral().getLongValue(),
0L);
pinotQuery = CalciteSqlParser.compileToPinotQuery("select * from
vegetables where b -(100+c)< 0");
func = pinotQuery.getFilterExpression().getFunctionCall();
Assert.assertEquals(func.getOperator(), SqlKind.LESS_THAN.name());
Assert.assertEquals(func.getOperands().get(0).getFunctionCall().getOperator(),
"MINUS");
-
Assert.assertEquals(func.getOperands().get(0).getFunctionCall().getOperands().get(0).getIdentifier().getName(),
"b");
-
Assert.assertEquals(func.getOperands().get(0).getFunctionCall().getOperands().get(1).getFunctionCall().getOperator(),
"PLUS");
-
Assert.assertEquals(func.getOperands().get(0).getFunctionCall().getOperands().get(1).getFunctionCall().getOperands().get(0).getLiteral().getLongValue(),
100L);
-
Assert.assertEquals(func.getOperands().get(0).getFunctionCall().getOperands().get(1).getFunctionCall().getOperands().get(1).getIdentifier().getName(),
"c");
+ Assert
+
.assertEquals(func.getOperands().get(0).getFunctionCall().getOperands().get(0).getIdentifier().getName(),
"b");
+ Assert
+
.assertEquals(func.getOperands().get(0).getFunctionCall().getOperands().get(1).getFunctionCall().getOperator(),
+ "PLUS");
+ Assert.assertEquals(
+
func.getOperands().get(0).getFunctionCall().getOperands().get(1).getFunctionCall().getOperands().get(0)
+ .getLiteral().getLongValue(), 100L);
+ Assert.assertEquals(
+
func.getOperands().get(0).getFunctionCall().getOperands().get(1).getFunctionCall().getOperands().get(1)
+ .getIdentifier().getName(), "c");
Assert.assertEquals(func.getOperands().get(1).getLiteral().getLongValue(),
0L);
-
- pinotQuery = CalciteSqlParser.compileToPinotQuery("select * from
vegetables where foo1(bar1(a-b)) <= foo2(bar2(c+d))");
+ pinotQuery =
+ CalciteSqlParser.compileToPinotQuery("select * from vegetables where
foo1(bar1(a-b)) <= foo2(bar2(c+d))");
func = pinotQuery.getFilterExpression().getFunctionCall();
Assert.assertEquals(func.getOperator(), SqlKind.LESS_THAN_OR_EQUAL.name());
Assert.assertEquals(func.getOperands().get(0).getFunctionCall().getOperator(),
"MINUS");
-
Assert.assertEquals(func.getOperands().get(0).getFunctionCall().getOperands().get(0).getFunctionCall().getOperator(),
"foo1");
-
Assert.assertEquals(func.getOperands().get(0).getFunctionCall().getOperands().get(1).getFunctionCall().getOperator(),
"foo2");
-
Assert.assertEquals(func.getOperands().get(0).getFunctionCall().getOperands().get(0).getFunctionCall().getOperands().get(0).getFunctionCall().getOperator(),
"bar1");
-
Assert.assertEquals(func.getOperands().get(0).getFunctionCall().getOperands().get(1).getFunctionCall().getOperands().get(0).getFunctionCall().getOperator(),
"bar2");
-
Assert.assertEquals(func.getOperands().get(0).getFunctionCall().getOperands().get(0).getFunctionCall().getOperands().get(0).getFunctionCall().getOperands().get(0).getFunctionCall().getOperator(),
"MINUS");
-
Assert.assertEquals(func.getOperands().get(0).getFunctionCall().getOperands().get(1).getFunctionCall().getOperands().get(0).getFunctionCall().getOperands().get(0).getFunctionCall().getOperator(),
"PLUS");
-
Assert.assertEquals(func.getOperands().get(0).getFunctionCall().getOperands().get(0).getFunctionCall().getOperands().get(0).getFunctionCall().getOperands().get(0).getFunctionCall().getOperands().get(0).getIdentifier().getName(),
"a");
-
Assert.assertEquals(func.getOperands().get(0).getFunctionCall().getOperands().get(0).getFunctionCall().getOperands().get(0).getFunctionCall().getOperands().get(0).getFunctionCall().getOperands().get(1).getIdentifier().getName(),
"b");
-
Assert.assertEquals(func.getOperands().get(0).getFunctionCall().getOperands().get(1).getFunctionCall().getOperands().get(0).getFunctionCall().getOperands().get(0).getFunctionCall().getOperands().get(0).getIdentifier().getName(),
"c");
-
Assert.assertEquals(func.getOperands().get(0).getFunctionCall().getOperands().get(1).getFunctionCall().getOperands().get(0).getFunctionCall().getOperands().get(0).getFunctionCall().getOperands().get(1).getIdentifier().getName(),
"d");
+ Assert
+
.assertEquals(func.getOperands().get(0).getFunctionCall().getOperands().get(0).getFunctionCall().getOperator(),
+ "foo1");
+ Assert
+
.assertEquals(func.getOperands().get(0).getFunctionCall().getOperands().get(1).getFunctionCall().getOperator(),
+ "foo2");
+ Assert.assertEquals(
+
func.getOperands().get(0).getFunctionCall().getOperands().get(0).getFunctionCall().getOperands().get(0)
+ .getFunctionCall().getOperator(), "bar1");
+ Assert.assertEquals(
+
func.getOperands().get(0).getFunctionCall().getOperands().get(1).getFunctionCall().getOperands().get(0)
+ .getFunctionCall().getOperator(), "bar2");
+ Assert.assertEquals(
+
func.getOperands().get(0).getFunctionCall().getOperands().get(0).getFunctionCall().getOperands().get(0)
+
.getFunctionCall().getOperands().get(0).getFunctionCall().getOperator(),
"MINUS");
+ Assert.assertEquals(
+
func.getOperands().get(0).getFunctionCall().getOperands().get(1).getFunctionCall().getOperands().get(0)
+
.getFunctionCall().getOperands().get(0).getFunctionCall().getOperator(),
"PLUS");
+ Assert.assertEquals(
+
func.getOperands().get(0).getFunctionCall().getOperands().get(0).getFunctionCall().getOperands().get(0)
+
.getFunctionCall().getOperands().get(0).getFunctionCall().getOperands().get(0).getIdentifier().getName(),
+ "a");
+ Assert.assertEquals(
+
func.getOperands().get(0).getFunctionCall().getOperands().get(0).getFunctionCall().getOperands().get(0)
+
.getFunctionCall().getOperands().get(0).getFunctionCall().getOperands().get(1).getIdentifier().getName(),
+ "b");
+ Assert.assertEquals(
+
func.getOperands().get(0).getFunctionCall().getOperands().get(1).getFunctionCall().getOperands().get(0)
+
.getFunctionCall().getOperands().get(0).getFunctionCall().getOperands().get(0).getIdentifier().getName(),
+ "c");
+ Assert.assertEquals(
+
func.getOperands().get(0).getFunctionCall().getOperands().get(1).getFunctionCall().getOperands().get(0)
+
.getFunctionCall().getOperands().get(0).getFunctionCall().getOperands().get(1).getIdentifier().getName(),
+ "d");
Assert.assertEquals(func.getOperands().get(1).getLiteral().getLongValue(),
0L);
- pinotQuery = CalciteSqlParser.compileToPinotQuery("select * from
vegetables where foo1(bar1(a-b)) - foo2(bar2(c+d)) <= 0");
+ pinotQuery =
+ CalciteSqlParser.compileToPinotQuery("select * from vegetables where
foo1(bar1(a-b)) - foo2(bar2(c+d)) <= 0");
func = pinotQuery.getFilterExpression().getFunctionCall();
Assert.assertEquals(func.getOperator(), SqlKind.LESS_THAN_OR_EQUAL.name());
Assert.assertEquals(func.getOperands().get(0).getFunctionCall().getOperator(),
"MINUS");
-
Assert.assertEquals(func.getOperands().get(0).getFunctionCall().getOperands().get(0).getFunctionCall().getOperator(),
"foo1");
-
Assert.assertEquals(func.getOperands().get(0).getFunctionCall().getOperands().get(1).getFunctionCall().getOperator(),
"foo2");
-
Assert.assertEquals(func.getOperands().get(0).getFunctionCall().getOperands().get(0).getFunctionCall().getOperands().get(0).getFunctionCall().getOperator(),
"bar1");
-
Assert.assertEquals(func.getOperands().get(0).getFunctionCall().getOperands().get(1).getFunctionCall().getOperands().get(0).getFunctionCall().getOperator(),
"bar2");
-
Assert.assertEquals(func.getOperands().get(0).getFunctionCall().getOperands().get(0).getFunctionCall().getOperands().get(0).getFunctionCall().getOperands().get(0).getFunctionCall().getOperator(),
"MINUS");
-
Assert.assertEquals(func.getOperands().get(0).getFunctionCall().getOperands().get(1).getFunctionCall().getOperands().get(0).getFunctionCall().getOperands().get(0).getFunctionCall().getOperator(),
"PLUS");
-
Assert.assertEquals(func.getOperands().get(0).getFunctionCall().getOperands().get(0).getFunctionCall().getOperands().get(0).getFunctionCall().getOperands().get(0).getFunctionCall().getOperands().get(0).getIdentifier().getName(),
"a");
-
Assert.assertEquals(func.getOperands().get(0).getFunctionCall().getOperands().get(0).getFunctionCall().getOperands().get(0).getFunctionCall().getOperands().get(0).getFunctionCall().getOperands().get(1).getIdentifier().getName(),
"b");
-
Assert.assertEquals(func.getOperands().get(0).getFunctionCall().getOperands().get(1).getFunctionCall().getOperands().get(0).getFunctionCall().getOperands().get(0).getFunctionCall().getOperands().get(0).getIdentifier().getName(),
"c");
-
Assert.assertEquals(func.getOperands().get(0).getFunctionCall().getOperands().get(1).getFunctionCall().getOperands().get(0).getFunctionCall().getOperands().get(0).getFunctionCall().getOperands().get(1).getIdentifier().getName(),
"d");
+ Assert
+
.assertEquals(func.getOperands().get(0).getFunctionCall().getOperands().get(0).getFunctionCall().getOperator(),
+ "foo1");
+ Assert
+
.assertEquals(func.getOperands().get(0).getFunctionCall().getOperands().get(1).getFunctionCall().getOperator(),
+ "foo2");
+ Assert.assertEquals(
+
func.getOperands().get(0).getFunctionCall().getOperands().get(0).getFunctionCall().getOperands().get(0)
+ .getFunctionCall().getOperator(), "bar1");
+ Assert.assertEquals(
+
func.getOperands().get(0).getFunctionCall().getOperands().get(1).getFunctionCall().getOperands().get(0)
+ .getFunctionCall().getOperator(), "bar2");
+ Assert.assertEquals(
+
func.getOperands().get(0).getFunctionCall().getOperands().get(0).getFunctionCall().getOperands().get(0)
+
.getFunctionCall().getOperands().get(0).getFunctionCall().getOperator(),
"MINUS");
+ Assert.assertEquals(
+
func.getOperands().get(0).getFunctionCall().getOperands().get(1).getFunctionCall().getOperands().get(0)
+
.getFunctionCall().getOperands().get(0).getFunctionCall().getOperator(),
"PLUS");
+ Assert.assertEquals(
+
func.getOperands().get(0).getFunctionCall().getOperands().get(0).getFunctionCall().getOperands().get(0)
+
.getFunctionCall().getOperands().get(0).getFunctionCall().getOperands().get(0).getIdentifier().getName(),
+ "a");
+ Assert.assertEquals(
+
func.getOperands().get(0).getFunctionCall().getOperands().get(0).getFunctionCall().getOperands().get(0)
+
.getFunctionCall().getOperands().get(0).getFunctionCall().getOperands().get(1).getIdentifier().getName(),
+ "b");
+ Assert.assertEquals(
+
func.getOperands().get(0).getFunctionCall().getOperands().get(1).getFunctionCall().getOperands().get(0)
+
.getFunctionCall().getOperands().get(0).getFunctionCall().getOperands().get(0).getIdentifier().getName(),
+ "c");
+ Assert.assertEquals(
+
func.getOperands().get(0).getFunctionCall().getOperands().get(1).getFunctionCall().getOperands().get(0)
+
.getFunctionCall().getOperands().get(0).getFunctionCall().getOperands().get(1).getIdentifier().getName(),
+ "d");
Assert.assertEquals(func.getOperands().get(1).getLiteral().getLongValue(),
0L);
pinotQuery = CalciteSqlParser.compileToPinotQuery("select * from
vegetables where c >= 10");
@@ -622,8 +688,7 @@ public class CalciteSqlCompilerTest {
Assert.assertEquals(aggregationInfos.size(), 1);
AggregationInfo aggregationInfo = aggregationInfos.get(0);
Assert.assertEquals(aggregationInfo.getAggregationType(),
AggregationFunctionType.DISTINCT.getName());
-
Assert.assertEquals(aggregationInfo.getAggregationParams().get(CompilerConstants.COLUMN_KEY_IN_AGGREGATION_INFO),
- "c1");
+ Assert.assertEquals(aggregationInfo.getExpressions().get(0), "c1");
// test multi column DISTINCT
sql = "SELECT DISTINCT c1, c2 FROM foo";
@@ -648,8 +713,7 @@ public class CalciteSqlCompilerTest {
Assert.assertEquals(aggregationInfos.size(), 1);
aggregationInfo = aggregationInfos.get(0);
Assert.assertEquals(aggregationInfo.getAggregationType(),
AggregationFunctionType.DISTINCT.getName());
-
Assert.assertEquals(aggregationInfo.getAggregationParams().get(CompilerConstants.COLUMN_KEY_IN_AGGREGATION_INFO),
- "c1:c2");
+ Assert.assertEquals(aggregationInfo.getExpressions(), Arrays.asList("c1",
"c2"));
// test multi column DISTINCT with filter
sql = "SELECT DISTINCT c1, c2, c3 FROM foo WHERE c3 > 100";
@@ -682,8 +746,7 @@ public class CalciteSqlCompilerTest {
Assert.assertEquals(aggregationInfos.size(), 1);
aggregationInfo = aggregationInfos.get(0);
Assert.assertEquals(aggregationInfo.getAggregationType(),
AggregationFunctionType.DISTINCT.getName());
-
Assert.assertEquals(aggregationInfo.getAggregationParams().get(CompilerConstants.COLUMN_KEY_IN_AGGREGATION_INFO),
- "c1:c2:c3");
+ Assert.assertEquals(aggregationInfo.getExpressions(), Arrays.asList("c1",
"c2", "c3"));
// not supported by Calcite SQL (this is in compliance with SQL standard)
try {
@@ -809,8 +872,7 @@ public class CalciteSqlCompilerTest {
Assert.assertEquals(aggregationInfos.size(), 1);
aggregationInfo = aggregationInfos.get(0);
Assert.assertEquals(aggregationInfo.getAggregationType(),
AggregationFunctionType.DISTINCT.getName());
-
Assert.assertEquals(aggregationInfo.getAggregationParams().get(CompilerConstants.COLUMN_KEY_IN_AGGREGATION_INFO),
- "add(col1,col2)");
+ Assert.assertEquals(aggregationInfo.getExpressions().get(0),
"add(col1,col2)");
// multi-column distinct with multiple transform functions
sql = "SELECT DISTINCT add(div(col1, col2), mul(col3, col4)), sub(col3,
col4) FROM foo";
@@ -863,8 +925,8 @@ public class CalciteSqlCompilerTest {
Assert.assertEquals(aggregationInfos.size(), 1);
aggregationInfo = aggregationInfos.get(0);
Assert.assertEquals(aggregationInfo.getAggregationType(),
AggregationFunctionType.DISTINCT.getName());
-
Assert.assertEquals(aggregationInfo.getAggregationParams().get(CompilerConstants.COLUMN_KEY_IN_AGGREGATION_INFO),
- "add(div(col1,col2),mul(col3,col4)):sub(col3,col4)");
+ Assert.assertEquals(aggregationInfo.getExpressions(),
+ Arrays.asList("add(div(col1,col2),mul(col3,col4))", "sub(col3,col4)"));
// multi-column distinct with multiple transform columns and additional
identifiers
sql = "SELECT DISTINCT add(div(col1, col2), mul(col3, col4)), sub(col3,
col4), col5, col6 FROM foo";
@@ -924,8 +986,8 @@ public class CalciteSqlCompilerTest {
Assert.assertEquals(aggregationInfos.size(), 1);
aggregationInfo = aggregationInfos.get(0);
Assert.assertEquals(aggregationInfo.getAggregationType(),
AggregationFunctionType.DISTINCT.getName());
-
Assert.assertEquals(aggregationInfo.getAggregationParams().get(CompilerConstants.COLUMN_KEY_IN_AGGREGATION_INFO),
- "add(div(col1,col2),mul(col3,col4)):sub(col3,col4):col5:col6");
+ Assert.assertEquals(aggregationInfo.getExpressions(),
+ Arrays.asList("add(div(col1,col2),mul(col3,col4))", "sub(col3,col4)",
"col5", "col6"));
}
@Test
@@ -1346,11 +1408,15 @@ public class CalciteSqlCompilerTest {
Assert.assertEquals("VARCHAR",
pinotQuery.getSelectList().get(0).getFunctionCall().getOperands().get(1).getLiteral().getStringValue());
- pinotQuery = CalciteSqlParser.compileToPinotQuery("SELECT
SUM(CAST(CAST(ArrTime AS STRING) AS LONG)) FROM mytable WHERE DaysSinceEpoch <>
16312 AND Carrier = 'DL'");
+ pinotQuery = CalciteSqlParser.compileToPinotQuery(
+ "SELECT SUM(CAST(CAST(ArrTime AS STRING) AS LONG)) FROM mytable WHERE
DaysSinceEpoch <> 16312 AND Carrier = 'DL'");
Assert.assertEquals(pinotQuery.getSelectListSize(), 1);
Assert.assertEquals("SUM",
pinotQuery.getSelectList().get(0).getFunctionCall().getOperator());
- Assert.assertEquals("CAST",
pinotQuery.getSelectList().get(0).getFunctionCall().getOperands().get(0).getFunctionCall().getOperator());
- Assert.assertEquals("CAST",
pinotQuery.getSelectList().get(0).getFunctionCall().getOperands().get(0).getFunctionCall().getOperands().get(0).getFunctionCall().getOperator());
+ Assert.assertEquals("CAST",
+
pinotQuery.getSelectList().get(0).getFunctionCall().getOperands().get(0).getFunctionCall().getOperator());
+ Assert.assertEquals("CAST",
+
pinotQuery.getSelectList().get(0).getFunctionCall().getOperands().get(0).getFunctionCall().getOperands().get(0)
+ .getFunctionCall().getOperator());
}
@Test
diff --git a/pinot-common/src/thrift/request.thrift
b/pinot-common/src/thrift/request.thrift
index 5409f25..36518a7 100644
--- a/pinot-common/src/thrift/request.thrift
+++ b/pinot-common/src/thrift/request.thrift
@@ -102,6 +102,11 @@ struct AggregationInfo {
2: optional map<string,string> aggregationParams;
3: optional bool isInSelectList;
+ // Backward compatible change to allow aggregation functions to take
multiple arguments.
+ // We could not reuse aggregationParams, as it requires argument name (as
key), which may not be
+ // available for aggregation functions with variable arguments. Each
argument can be an expression.
+ 4: optional list<string> expressions;
+
}
/**
diff --git
a/pinot-core/src/main/java/org/apache/pinot/core/operator/query/DictionaryBasedAggregationOperator.java
b/pinot-core/src/main/java/org/apache/pinot/core/operator/query/DictionaryBasedAggregationOperator.java
index 8750684..5c230da 100644
---
a/pinot-core/src/main/java/org/apache/pinot/core/operator/query/DictionaryBasedAggregationOperator.java
+++
b/pinot-core/src/main/java/org/apache/pinot/core/operator/query/DictionaryBasedAggregationOperator.java
@@ -72,7 +72,7 @@ public class DictionaryBasedAggregationOperator extends
BaseOperator<Intermediat
for (AggregationFunctionContext aggregationFunctionContext :
_aggregationFunctionContexts) {
AggregationFunction function =
aggregationFunctionContext.getAggregationFunction();
AggregationFunctionType functionType = function.getType();
- String column = aggregationFunctionContext.getColumn();
+ String column = aggregationFunctionContext.getColumnName();
Dictionary dictionary = _dictionaryMap.get(column);
AggregationResultHolder resultHolder;
switch (functionType) {
diff --git
a/pinot-core/src/main/java/org/apache/pinot/core/plan/DictionaryBasedAggregationPlanNode.java
b/pinot-core/src/main/java/org/apache/pinot/core/plan/DictionaryBasedAggregationPlanNode.java
index 36d9293..4bf368d 100644
---
a/pinot-core/src/main/java/org/apache/pinot/core/plan/DictionaryBasedAggregationPlanNode.java
+++
b/pinot-core/src/main/java/org/apache/pinot/core/plan/DictionaryBasedAggregationPlanNode.java
@@ -54,7 +54,7 @@ public class DictionaryBasedAggregationPlanNode implements
PlanNode {
_aggregationFunctionContexts =
AggregationFunctionUtils.getAggregationFunctionContexts(brokerRequest);
for (AggregationFunctionContext aggregationFunctionContext :
_aggregationFunctionContexts) {
- String column = aggregationFunctionContext.getColumn();
+ String column = aggregationFunctionContext.getColumnName();
if (!_dictionaryMap.containsKey(column)) {
_dictionaryMap.put(column,
_indexSegment.getDataSource(column).getDictionary());
}
diff --git
a/pinot-core/src/main/java/org/apache/pinot/core/plan/MetadataBasedAggregationPlanNode.java
b/pinot-core/src/main/java/org/apache/pinot/core/plan/MetadataBasedAggregationPlanNode.java
index 428f47e..3ac5e5d 100644
---
a/pinot-core/src/main/java/org/apache/pinot/core/plan/MetadataBasedAggregationPlanNode.java
+++
b/pinot-core/src/main/java/org/apache/pinot/core/plan/MetadataBasedAggregationPlanNode.java
@@ -66,7 +66,7 @@ public class MetadataBasedAggregationPlanNode implements
PlanNode {
Map<String, DataSource> dataSourceMap = new HashMap<>();
for (AggregationFunctionContext aggregationFunctionContext :
aggregationFunctionContexts) {
if (aggregationFunctionContext.getAggregationFunction().getType() !=
AggregationFunctionType.COUNT) {
- String column = aggregationFunctionContext.getColumn();
+ String column = aggregationFunctionContext.getColumnName();
if (!dataSourceMap.containsKey(column)) {
dataSourceMap.put(column, _indexSegment.getDataSource(column));
}
diff --git
a/pinot-core/src/main/java/org/apache/pinot/core/plan/TransformPlanNode.java
b/pinot-core/src/main/java/org/apache/pinot/core/plan/TransformPlanNode.java
index b35c03e..5bd9286 100644
--- a/pinot-core/src/main/java/org/apache/pinot/core/plan/TransformPlanNode.java
+++ b/pinot-core/src/main/java/org/apache/pinot/core/plan/TransformPlanNode.java
@@ -18,7 +18,6 @@
*/
package org.apache.pinot.core.plan;
-import java.util.Arrays;
import java.util.HashSet;
import java.util.List;
import java.util.Set;
@@ -31,7 +30,6 @@ import
org.apache.pinot.common.request.transform.TransformExpressionTree;
import org.apache.pinot.core.indexsegment.IndexSegment;
import org.apache.pinot.core.operator.transform.TransformOperator;
import
org.apache.pinot.core.query.aggregation.function.AggregationFunctionUtils;
-import org.apache.pinot.parsers.CompilerConstants;
import org.apache.pinot.pql.parsers.pql2.ast.IdentifierAstNode;
import org.slf4j.Logger;
import org.slf4j.LoggerFactory;
@@ -66,11 +64,10 @@ public class TransformPlanNode implements PlanNode {
for (AggregationInfo aggregationInfo :
brokerRequest.getAggregationsInfo()) {
if
(aggregationInfo.getAggregationType().equalsIgnoreCase(AggregationFunctionType.DISTINCT.getName()))
{
// 'DISTINCT(col1, col2 ...)' is modeled as one single aggregation
function
- String[] distinctColumns =
AggregationFunctionUtils.getColumn(aggregationInfo)
- .split(CompilerConstants.AGGREGATION_FUNCTION_ARG_SEPARATOR);
- columns.addAll(Arrays.asList(distinctColumns));
+ List<String> distinctColumns =
AggregationFunctionUtils.getAggregationExpressions(aggregationInfo);
+ columns.addAll(distinctColumns);
} else if
(!aggregationInfo.getAggregationType().equalsIgnoreCase(AggregationFunctionType.COUNT.getName()))
{
- columns.add(AggregationFunctionUtils.getColumn(aggregationInfo));
+
columns.addAll(AggregationFunctionUtils.getAggregationExpressions(aggregationInfo));
}
}
// Extract group-by expressions
diff --git
a/pinot-core/src/main/java/org/apache/pinot/core/plan/maker/InstancePlanMakerImplV2.java
b/pinot-core/src/main/java/org/apache/pinot/core/plan/maker/InstancePlanMakerImplV2.java
index 0164b8e..182f79b 100644
---
a/pinot-core/src/main/java/org/apache/pinot/core/plan/maker/InstancePlanMakerImplV2.java
+++
b/pinot-core/src/main/java/org/apache/pinot/core/plan/maker/InstancePlanMakerImplV2.java
@@ -205,7 +205,7 @@ public class InstancePlanMakerImplV2 implements PlanMaker {
AggregationFunctionType.getAggregationFunctionType(aggregationInfo.getAggregationType());
if (functionType
.isOfType(AggregationFunctionType.MIN, AggregationFunctionType.MAX,
AggregationFunctionType.MINMAXRANGE)) {
- String expression = AggregationFunctionUtils.getColumn(aggregationInfo);
+ String expression =
AggregationFunctionUtils.getAggregationExpressions(aggregationInfo).get(0);
if
(TransformExpressionTree.compileToExpressionTree(expression).isColumn()) {
Dictionary dictionary =
indexSegment.getDataSource(expression).getDictionary();
return dictionary != null && dictionary.isSorted();
diff --git
a/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/AggregationFunctionContext.java
b/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/AggregationFunctionContext.java
index 5fa06a3..9d1027f 100644
---
a/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/AggregationFunctionContext.java
+++
b/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/AggregationFunctionContext.java
@@ -18,19 +18,28 @@
*/
package org.apache.pinot.core.query.aggregation;
+import com.google.common.base.Preconditions;
+import java.util.List;
import org.apache.pinot.core.query.aggregation.function.AggregationFunction;
+import
org.apache.pinot.core.query.aggregation.function.AggregationFunctionUtils;
/**
* This class caches miscellaneous data to perform efficient aggregation.
+ *
+ * TODO: Remove this class, as it no longer provides any value after
aggregation functions now store
+ * their arguments.
*/
public class AggregationFunctionContext {
private final AggregationFunction _aggregationFunction;
- private final String _column;
+ private final List<String> _expressions;
+ private final String columnName;
- public AggregationFunctionContext(AggregationFunction aggregationFunction,
String column) {
+ public AggregationFunctionContext(AggregationFunction aggregationFunction,
List<String> expressions) {
+ Preconditions.checkArgument(expressions.size() >= 1, "Aggregation
functions require at least one argument.");
_aggregationFunction = aggregationFunction;
- _column = column;
+ _expressions = expressions;
+ columnName = AggregationFunctionUtils.concatArgs(expressions);
}
/**
@@ -41,10 +50,21 @@ public class AggregationFunctionContext {
}
/**
- * Returns the aggregation column (could be column name or UDF expression).
+ * Returns the arguments for the aggregation function.
+ *
+ * @return List of Strings containing the arguments for the aggregation
function.
+ */
+ public List<String> getExpressions() {
+ return _expressions;
+ }
+
+ /**
+ * Returns the column for aggregation function.
+ *
+ * @return Aggregation Column (could be column name or UDF expression).
*/
- public String getColumn() {
- return _column;
+ public String getColumnName() {
+ return columnName;
}
/**
@@ -52,7 +72,7 @@ public class AggregationFunctionContext {
* <p>E.g. AVG(foo) -> avg_foo
*/
public String getAggregationColumnName() {
- return _aggregationFunction.getColumnName(_column);
+ return _aggregationFunction.getColumnName();
}
/**
@@ -60,6 +80,6 @@ public class AggregationFunctionContext {
* <p>E.g. AVGMV(foo) -> avgMV(foo)
*/
public String getResultColumnName() {
- return _aggregationFunction.getResultColumnName(_column);
+ return _aggregationFunction.getResultColumnName();
}
}
diff --git
a/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/DefaultAggregationExecutor.java
b/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/DefaultAggregationExecutor.java
index 020b709..852292d 100644
---
a/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/DefaultAggregationExecutor.java
+++
b/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/DefaultAggregationExecutor.java
@@ -29,7 +29,6 @@ import org.apache.pinot.core.common.BlockValSet;
import org.apache.pinot.core.operator.blocks.TransformBlock;
import org.apache.pinot.core.query.aggregation.function.AggregationFunction;
import
org.apache.pinot.core.query.aggregation.function.AggregationFunctionUtils;
-import org.apache.pinot.parsers.CompilerConstants;
public class DefaultAggregationExecutor implements AggregationExecutor {
@@ -48,12 +47,13 @@ public class DefaultAggregationExecutor implements
AggregationExecutor {
// so we need to build expression tree for each column
_functions[0] = functionContexts[0].getAggregationFunction();
_resultHolders[0] = _functions[0].createAggregationResultHolder();
- String multiColumnExpression = functionContexts[0].getColumn();
- String[] distinctColumnExpressions =
-
multiColumnExpression.split(CompilerConstants.AGGREGATION_FUNCTION_ARG_SEPARATOR);
- _expressions = new
TransformExpressionTree[distinctColumnExpressions.length];
- for (int i = 0; i < distinctColumnExpressions.length; i++) {
- _expressions[i] =
TransformExpressionTree.compileToExpressionTree(distinctColumnExpressions[i]);
+
+ List<String> expressions = functionContexts[0].getExpressions();
+ _expressions = new TransformExpressionTree[expressions.size()];
+
+ for (int i = 0; i < _expressions.length; i++) {
+ _expressions[i] =
TransformExpressionTree.compileToExpressionTree(expressions.get(i));
+
}
} else {
_expressions = new TransformExpressionTree[_numFunctions];
@@ -64,7 +64,7 @@ public class DefaultAggregationExecutor implements
AggregationExecutor {
if (function.getType() != AggregationFunctionType.COUNT) {
// count(*) does not have a column so handle rest of the aggregate
// functions -- sum, min, max etc
- _expressions[i] =
TransformExpressionTree.compileToExpressionTree(functionContexts[i].getColumn());
+ _expressions[i] =
TransformExpressionTree.compileToExpressionTree(functionContexts[i].getColumnName());
}
}
}
diff --git
a/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/AggregationFunction.java
b/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/AggregationFunction.java
index 99b72f0..349065e 100644
---
a/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/AggregationFunction.java
+++
b/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/AggregationFunction.java
@@ -42,16 +42,12 @@ public interface AggregationFunction<IntermediateResult,
FinalResult extends Com
/**
* Returns the result column name for the given aggregation column, e.g.
'SUM(foo)' -> 'sum_foo'.
*/
- default String getColumnName(String column) {
- return getType().getName() + "_" + column;
- }
+ String getColumnName();
/**
* Returns the column name to be used in the data schema of results. e.g.
'MINMAXRANGEMV( foo)' -> 'minmaxrangemv(foo)', 'PERCENTILE75(bar)' ->
'percentile75(bar)'
*/
- default String getResultColumnName(String column) {
- return getType().getName().toLowerCase() + "(" + column + ")";
- }
+ String getResultColumnName();
/**
* Accepts an aggregation function visitor to visit.
diff --git
a/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/AggregationFunctionFactory.java
b/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/AggregationFunctionFactory.java
index ce59357..b433328 100644
---
a/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/AggregationFunctionFactory.java
+++
b/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/AggregationFunctionFactory.java
@@ -20,14 +20,12 @@ package org.apache.pinot.core.query.aggregation.function;
import com.google.common.base.Preconditions;
import java.util.ArrayList;
-import java.util.Arrays;
import java.util.List;
import javax.annotation.Nullable;
import org.apache.pinot.common.function.AggregationFunctionType;
import org.apache.pinot.common.request.AggregationInfo;
import org.apache.pinot.common.request.BrokerRequest;
import org.apache.pinot.core.query.exception.BadQueryRequestException;
-import org.apache.pinot.parsers.CompilerConstants;
/**
@@ -47,8 +45,7 @@ public class AggregationFunctionFactory {
public static AggregationFunction getAggregationFunction(AggregationInfo
aggregationInfo,
@Nullable BrokerRequest brokerRequest) {
String functionName = aggregationInfo.getAggregationType();
- String argumentsString =
AggregationFunctionUtils.getColumn(aggregationInfo);
- List<String> arguments =
Arrays.asList(argumentsString.split(CompilerConstants.AGGREGATION_FUNCTION_ARG_SEPARATOR));
+ List<String> arguments =
AggregationFunctionUtils.getAggregationExpressions(aggregationInfo);
try {
String upperCaseFunctionName = functionName.toUpperCase();
diff --git
a/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/AggregationFunctionUtils.java
b/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/AggregationFunctionUtils.java
index e86b014..b4c8e21 100644
---
a/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/AggregationFunctionUtils.java
+++
b/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/AggregationFunctionUtils.java
@@ -21,6 +21,7 @@ package org.apache.pinot.core.query.aggregation.function;
import com.google.common.base.Preconditions;
import com.google.common.math.DoubleMath;
import java.io.Serializable;
+import java.util.Arrays;
import java.util.List;
import java.util.Locale;
import javax.annotation.Nullable;
@@ -41,18 +42,39 @@ public class AggregationFunctionUtils {
/**
* Extracts the aggregation column (could be column name or UDF expression)
from the {@link AggregationInfo}.
+ *
+ *
+ */
+ /**
+ * Returns the arguments for {@link AggregationFunction} as List of Strings.
+ * For backward compatibility, it uses the new Thrift field `expressions` if
found, or else,
+ * falls back to the previous aggregationParams based approach.
+ *
+ * @param aggregationInfo Aggregation Info
+ * @return List of aggregation function arguments
*/
- public static String getColumn(AggregationInfo aggregationInfo) {
- return
aggregationInfo.getAggregationParams().get(CompilerConstants.COLUMN_KEY_IN_AGGREGATION_INFO);
+ public static List<String> getAggregationExpressions(AggregationInfo
aggregationInfo) {
+ List<String> expressions = aggregationInfo.getExpressions();
+ if (expressions != null) {
+ return expressions;
+ }
+
+ String params =
aggregationInfo.getAggregationParams().get(CompilerConstants.COLUMN_KEY_IN_AGGREGATION_INFO);
+ return
Arrays.asList(params.split(CompilerConstants.AGGREGATION_FUNCTION_ARG_SEPARATOR));
}
/**
* Creates an {@link AggregationFunctionColumnPair} from the {@link
AggregationInfo}.
+ * Asserts that the function only expects one argument.
*/
public static AggregationFunctionColumnPair
getFunctionColumnPair(AggregationInfo aggregationInfo) {
+ List<String> aggregationExpressions =
getAggregationExpressions(aggregationInfo);
+ int numExpressions = aggregationExpressions.size();
AggregationFunctionType functionType =
AggregationFunctionType.getAggregationFunctionType(aggregationInfo.getAggregationType());
- return new AggregationFunctionColumnPair(functionType,
getColumn(aggregationInfo));
+ Preconditions
+ .checkState(numExpressions == 1, "Expected one argument for '" +
functionType + "', got: " + numExpressions);
+ return new AggregationFunctionColumnPair(functionType,
aggregationExpressions.get(0));
}
public static boolean isDistinct(AggregationFunctionContext[]
functionContexts) {
@@ -75,10 +97,10 @@ public class AggregationFunctionUtils {
*/
public static AggregationFunctionContext
getAggregationFunctionContext(AggregationInfo aggregationInfo,
@Nullable BrokerRequest brokerRequest) {
- String column = getColumn(aggregationInfo);
+ List<String> aggregationExpressions =
getAggregationExpressions(aggregationInfo);
AggregationFunction aggregationFunction =
AggregationFunctionFactory.getAggregationFunction(aggregationInfo,
brokerRequest);
- return new AggregationFunctionContext(aggregationFunction, column);
+ return new AggregationFunctionContext(aggregationFunction,
aggregationExpressions);
}
public static AggregationFunctionContext[]
getAggregationFunctionContexts(BrokerRequest brokerRequest) {
@@ -149,4 +171,15 @@ public class AggregationFunctionUtils {
Preconditions.checkState(percentile >= 0 && percentile <= 100);
return percentile;
}
+
+ /**
+ * Helper function to concatenate arguments using separator.
+ *
+ * @param arguments Arguments to concatenate
+ * @return Concatenated String of arguments
+ */
+ public static String concatArgs(List<String> arguments) {
+ return (arguments.size() > 1) ?
String.join(CompilerConstants.AGGREGATION_FUNCTION_ARG_SEPARATOR, arguments)
+ : arguments.get(0);
+ }
}
diff --git
a/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/AvgAggregationFunction.java
b/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/AvgAggregationFunction.java
index 5198518..4060c81 100644
---
a/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/AvgAggregationFunction.java
+++
b/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/AvgAggregationFunction.java
@@ -50,6 +50,16 @@ public class AvgAggregationFunction implements
AggregationFunction<AvgPair, Doub
}
@Override
+ public String getColumnName() {
+ return getType().getName() + "_" + _column;
+ }
+
+ @Override
+ public String getResultColumnName() {
+ return getType().getName().toLowerCase() + "(" + _column + ")";
+ }
+
+ @Override
public void accept(AggregationFunctionVisitorBase visitor) {
visitor.visit(this);
}
diff --git
a/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/CountAggregationFunction.java
b/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/CountAggregationFunction.java
index b130767..a00523e 100644
---
a/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/CountAggregationFunction.java
+++
b/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/CountAggregationFunction.java
@@ -48,12 +48,12 @@ public class CountAggregationFunction implements
AggregationFunction<Long, Long>
}
@Override
- public String getColumnName(String column) {
+ public String getColumnName() {
return COLUMN_NAME;
}
@Override
- public String getResultColumnName(String column) {
+ public String getResultColumnName() {
return AggregationFunctionType.COUNT.getName().toLowerCase() + "(*)";
}
diff --git
a/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/CountMVAggregationFunction.java
b/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/CountMVAggregationFunction.java
index 48341c3..3322565 100644
---
a/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/CountMVAggregationFunction.java
+++
b/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/CountMVAggregationFunction.java
@@ -41,13 +41,13 @@ public class CountMVAggregationFunction extends
CountAggregationFunction {
}
@Override
- public String getColumnName(String column) {
- return AggregationFunctionType.COUNTMV.getName() + "_" + column;
+ public String getColumnName() {
+ return getType().getName() + "_" + _column;
}
@Override
- public String getResultColumnName(String column) {
- return AggregationFunctionType.COUNTMV.getName().toLowerCase() + "(" +
column + ")";
+ public String getResultColumnName() {
+ return getType().getName().toLowerCase() + "(" + _column + ")";
}
@Override
diff --git
a/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/DistinctAggregationFunction.java
b/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/DistinctAggregationFunction.java
index b91bdbb..57fde12 100644
---
a/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/DistinctAggregationFunction.java
+++
b/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/DistinctAggregationFunction.java
@@ -68,6 +68,16 @@ public class DistinctAggregationFunction implements
AggregationFunction<Distinct
}
@Override
+ public String getColumnName() {
+ return getType().getName() + "_" +
AggregationFunctionUtils.concatArgs(Arrays.asList(_columns));
+ }
+
+ @Override
+ public String getResultColumnName() {
+ return getType().getName().toLowerCase() + "(" +
AggregationFunctionUtils.concatArgs(Arrays.asList(_columns)) + ")";
+ }
+
+ @Override
public void accept(AggregationFunctionVisitorBase visitor) {
visitor.visit(this);
}
@@ -78,7 +88,8 @@ public class DistinctAggregationFunction implements
AggregationFunction<Distinct
}
@Override
- public void aggregate(int length, AggregationResultHolder
aggregationResultHolder, Map<String, BlockValSet> blockValSetMap) {
+ public void aggregate(int length, AggregationResultHolder
aggregationResultHolder,
+ Map<String, BlockValSet> blockValSetMap) {
int numColumns = _columns.length;
int numBlockValSets = blockValSetMap.size();
Preconditions.checkState(numBlockValSets == numColumns, "Size mismatch:
numBlockValSets = %s, numColumns = %s",
diff --git
a/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/DistinctCountAggregationFunction.java
b/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/DistinctCountAggregationFunction.java
index 358ce0b..904391c 100644
---
a/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/DistinctCountAggregationFunction.java
+++
b/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/DistinctCountAggregationFunction.java
@@ -48,6 +48,16 @@ public class DistinctCountAggregationFunction implements
AggregationFunction<Int
}
@Override
+ public String getColumnName() {
+ return getType().getName() + "_" + _column;
+ }
+
+ @Override
+ public String getResultColumnName() {
+ return getType().getName().toLowerCase() + "(" + _column + ")";
+ }
+
+ @Override
public void accept(AggregationFunctionVisitorBase visitor) {
visitor.visit(this);
}
diff --git
a/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/DistinctCountHLLAggregationFunction.java
b/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/DistinctCountHLLAggregationFunction.java
index 853b2aa..c5a2eef 100644
---
a/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/DistinctCountHLLAggregationFunction.java
+++
b/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/DistinctCountHLLAggregationFunction.java
@@ -51,6 +51,16 @@ public class DistinctCountHLLAggregationFunction implements
AggregationFunction<
}
@Override
+ public String getColumnName() {
+ return getType().getName() + "_" + _column;
+ }
+
+ @Override
+ public String getResultColumnName() {
+ return getType().getName().toLowerCase() + "(" + _column + ")";
+ }
+
+ @Override
public void accept(AggregationFunctionVisitorBase visitor) {
visitor.visit(this);
}
diff --git
a/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/DistinctCountRawHLLAggregationFunction.java
b/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/DistinctCountRawHLLAggregationFunction.java
index d3bd48b..c1a5f76 100644
---
a/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/DistinctCountRawHLLAggregationFunction.java
+++
b/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/DistinctCountRawHLLAggregationFunction.java
@@ -53,6 +53,16 @@ public class DistinctCountRawHLLAggregationFunction
implements AggregationFuncti
}
@Override
+ public String getColumnName() {
+ return getType().getName() + "_" + _column;
+ }
+
+ @Override
+ public String getResultColumnName() {
+ return getType().getName().toLowerCase() + "(" + _column + ")";
+ }
+
+ @Override
public void accept(AggregationFunctionVisitorBase visitor) {
_distinctCountHLLAggregationFunction.accept(visitor);
}
diff --git
a/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/FastHLLAggregationFunction.java
b/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/FastHLLAggregationFunction.java
index 1ea7234..b6f53a4 100644
---
a/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/FastHLLAggregationFunction.java
+++
b/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/FastHLLAggregationFunction.java
@@ -55,6 +55,16 @@ public class FastHLLAggregationFunction implements
AggregationFunction<HyperLogL
}
@Override
+ public String getColumnName() {
+ return getType().getName() + "_" + _column;
+ }
+
+ @Override
+ public String getResultColumnName() {
+ return getType().getName().toLowerCase() + "(" + _column + ")";
+ }
+
+ @Override
public void accept(AggregationFunctionVisitorBase visitor) {
visitor.visit(this);
}
diff --git
a/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/MaxAggregationFunction.java
b/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/MaxAggregationFunction.java
index 26b27cf..530a42d 100644
---
a/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/MaxAggregationFunction.java
+++
b/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/MaxAggregationFunction.java
@@ -47,6 +47,16 @@ public class MaxAggregationFunction implements
AggregationFunction<Double, Doubl
}
@Override
+ public String getColumnName() {
+ return getType().getName() + "_" + _column;
+ }
+
+ @Override
+ public String getResultColumnName() {
+ return getType().getName().toLowerCase() + "(" + _column + ")";
+ }
+
+ @Override
public void accept(AggregationFunctionVisitorBase visitor) {
visitor.visit(this);
}
diff --git
a/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/MinAggregationFunction.java
b/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/MinAggregationFunction.java
index 31ffe8b..88bd84b 100644
---
a/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/MinAggregationFunction.java
+++
b/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/MinAggregationFunction.java
@@ -47,6 +47,15 @@ public class MinAggregationFunction implements
AggregationFunction<Double, Doubl
}
@Override
+ public String getColumnName() {
+ return getType().getName() + "_" + _column;
+ }
+
+ @Override
+ public String getResultColumnName() {
+ return getType().getName().toLowerCase() + "(" + _column + ")";
+ }
+ @Override
public void accept(AggregationFunctionVisitorBase visitor) {
visitor.visit(this);
}
diff --git
a/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/MinMaxRangeAggregationFunction.java
b/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/MinMaxRangeAggregationFunction.java
index 9279e54..cdb562d 100644
---
a/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/MinMaxRangeAggregationFunction.java
+++
b/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/MinMaxRangeAggregationFunction.java
@@ -49,6 +49,16 @@ public class MinMaxRangeAggregationFunction implements
AggregationFunction<MinMa
}
@Override
+ public String getColumnName() {
+ return getType().getName() + "_" + _column;
+ }
+
+ @Override
+ public String getResultColumnName() {
+ return getType().getName().toLowerCase() + "(" + _column + ")";
+ }
+
+ @Override
public void accept(AggregationFunctionVisitorBase visitor) {
visitor.visit(this);
}
diff --git
a/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/PercentileAggregationFunction.java
b/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/PercentileAggregationFunction.java
index db6e5d3..ae45a5e 100644
---
a/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/PercentileAggregationFunction.java
+++
b/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/PercentileAggregationFunction.java
@@ -61,13 +61,13 @@ public class PercentileAggregationFunction implements
AggregationFunction<Double
}
@Override
- public String getColumnName(String column) {
- return AggregationFunctionType.PERCENTILE.getName() + _percentile + "_" +
column;
+ public String getColumnName() {
+ return getType().getName() + _percentile + "_" + _column;
}
@Override
- public String getResultColumnName(String column) {
- return AggregationFunctionType.PERCENTILE.getName().toLowerCase() +
_percentile + "(" + column + ")";
+ public String getResultColumnName() {
+ return getType().getName().toLowerCase() + _percentile + "(" + _column +
")";
}
@Override
diff --git
a/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/PercentileEstAggregationFunction.java
b/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/PercentileEstAggregationFunction.java
index 483691e..a2c3fb1 100644
---
a/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/PercentileEstAggregationFunction.java
+++
b/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/PercentileEstAggregationFunction.java
@@ -62,13 +62,13 @@ public class PercentileEstAggregationFunction implements
AggregationFunction<Qua
}
@Override
- public String getColumnName(String column) {
- return AggregationFunctionType.PERCENTILEEST.getName() + _percentile + "_"
+ column;
+ public String getColumnName() {
+ return AggregationFunctionType.PERCENTILEEST.getName() + _percentile + "_"
+ _column;
}
@Override
- public String getResultColumnName(String column) {
- return AggregationFunctionType.PERCENTILEEST.getName().toLowerCase() +
_percentile + "(" + column + ")";
+ public String getResultColumnName() {
+ return AggregationFunctionType.PERCENTILEEST.getName().toLowerCase() +
_percentile + "(" + _column + ")";
}
@Override
diff --git
a/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/PercentileEstMVAggregationFunction.java
b/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/PercentileEstMVAggregationFunction.java
index 8622a61..8809d42 100644
---
a/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/PercentileEstMVAggregationFunction.java
+++
b/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/PercentileEstMVAggregationFunction.java
@@ -48,13 +48,13 @@ public class PercentileEstMVAggregationFunction extends
PercentileEstAggregation
}
@Override
- public String getColumnName(String column) {
- return AggregationFunctionType.PERCENTILEEST.getName() + _percentile +
"MV_" + column;
+ public String getColumnName() {
+ return AggregationFunctionType.PERCENTILEEST.getName() + _percentile +
"MV_" + _column;
}
@Override
- public String getResultColumnName(String column) {
- return AggregationFunctionType.PERCENTILEEST.getName().toLowerCase() +
_percentile + "mv(" + column + ")";
+ public String getResultColumnName() {
+ return AggregationFunctionType.PERCENTILEEST.getName().toLowerCase() +
_percentile + "mv(" + _column + ")";
}
@Override
diff --git
a/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/PercentileMVAggregationFunction.java
b/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/PercentileMVAggregationFunction.java
index 5b5e7d6..fba5f02 100644
---
a/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/PercentileMVAggregationFunction.java
+++
b/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/PercentileMVAggregationFunction.java
@@ -48,13 +48,13 @@ public class PercentileMVAggregationFunction extends
PercentileAggregationFuncti
}
@Override
- public String getColumnName(String column) {
- return AggregationFunctionType.PERCENTILE.getName() + _percentile + "MV_"
+ column;
+ public String getColumnName() {
+ return AggregationFunctionType.PERCENTILE.getName() + _percentile + "MV_"
+ _column;
}
@Override
- public String getResultColumnName(String column) {
- return AggregationFunctionType.PERCENTILE.getName().toLowerCase() +
_percentile + "mv(" + column + ")";
+ public String getResultColumnName() {
+ return AggregationFunctionType.PERCENTILE.getName().toLowerCase() +
_percentile + "mv(" + _column + ")";
}
@Override
diff --git
a/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/PercentileTDigestAggregationFunction.java
b/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/PercentileTDigestAggregationFunction.java
index 4e99e68..ba64586 100644
---
a/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/PercentileTDigestAggregationFunction.java
+++
b/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/PercentileTDigestAggregationFunction.java
@@ -64,13 +64,13 @@ public class PercentileTDigestAggregationFunction
implements AggregationFunction
}
@Override
- public String getColumnName(String column) {
- return AggregationFunctionType.PERCENTILETDIGEST.getName() + _percentile +
"_" + column;
+ public String getColumnName() {
+ return AggregationFunctionType.PERCENTILETDIGEST.getName() + _percentile +
"_" + _column;
}
@Override
- public String getResultColumnName(String column) {
- return AggregationFunctionType.PERCENTILETDIGEST.getName().toLowerCase() +
_percentile + "(" + column + ")";
+ public String getResultColumnName() {
+ return AggregationFunctionType.PERCENTILETDIGEST.getName().toLowerCase() +
_percentile + "(" + _column + ")";
}
@Override
diff --git
a/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/PercentileTDigestMVAggregationFunction.java
b/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/PercentileTDigestMVAggregationFunction.java
index 17a2289..d71a7fe 100644
---
a/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/PercentileTDigestMVAggregationFunction.java
+++
b/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/PercentileTDigestMVAggregationFunction.java
@@ -48,13 +48,13 @@ public class PercentileTDigestMVAggregationFunction extends
PercentileTDigestAgg
}
@Override
- public String getColumnName(String column) {
- return AggregationFunctionType.PERCENTILETDIGEST.getName() + _percentile +
"MV_" + column;
+ public String getColumnName() {
+ return AggregationFunctionType.PERCENTILETDIGEST.getName() + _percentile +
"MV_" + _column;
}
@Override
- public String getResultColumnName(String column) {
- return AggregationFunctionType.PERCENTILETDIGEST.getName().toLowerCase() +
_percentile + "mv(" + column + ")";
+ public String getResultColumnName() {
+ return AggregationFunctionType.PERCENTILETDIGEST.getName().toLowerCase() +
_percentile + "mv(" + _column + ")";
}
@Override
diff --git
a/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/SumAggregationFunction.java
b/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/SumAggregationFunction.java
index c149473..af3e04e 100644
---
a/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/SumAggregationFunction.java
+++
b/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/function/SumAggregationFunction.java
@@ -46,6 +46,16 @@ public class SumAggregationFunction implements
AggregationFunction<Double, Doubl
}
@Override
+ public String getColumnName() {
+ return getType().getName() + "_" + _column;
+ }
+
+ @Override
+ public String getResultColumnName() {
+ return getType().getName().toLowerCase() + "(" + _column + ")";
+ }
+
+ @Override
public void accept(AggregationFunctionVisitorBase visitor) {
visitor.visit(this);
}
diff --git
a/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/groupby/DefaultGroupByExecutor.java
b/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/groupby/DefaultGroupByExecutor.java
index 8407a9b..f9bd100 100644
---
a/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/groupby/DefaultGroupByExecutor.java
+++
b/pinot-core/src/main/java/org/apache/pinot/core/query/aggregation/groupby/DefaultGroupByExecutor.java
@@ -79,7 +79,7 @@ public class DefaultGroupByExecutor implements
GroupByExecutor {
AggregationFunction function =
functionContexts[i].getAggregationFunction();
_functions[i] = function;
if (function.getType() != AggregationFunctionType.COUNT) {
- _aggregationExpressions[i] =
TransformExpressionTree.compileToExpressionTree(functionContexts[i].getColumn());
+ _aggregationExpressions[i] =
TransformExpressionTree.compileToExpressionTree(functionContexts[i].getColumnName());
}
}
diff --git
a/pinot-core/src/main/java/org/apache/pinot/core/query/reduce/DistinctDataTableReducer.java
b/pinot-core/src/main/java/org/apache/pinot/core/query/reduce/DistinctDataTableReducer.java
index be23e67..de5db1c 100644
---
a/pinot-core/src/main/java/org/apache/pinot/core/query/reduce/DistinctDataTableReducer.java
+++
b/pinot-core/src/main/java/org/apache/pinot/core/query/reduce/DistinctDataTableReducer.java
@@ -37,10 +37,10 @@ import org.apache.pinot.common.utils.DataTable;
import org.apache.pinot.core.data.table.Record;
import org.apache.pinot.core.query.aggregation.DistinctTable;
import org.apache.pinot.core.query.aggregation.function.AggregationFunction;
+import
org.apache.pinot.core.query.aggregation.function.AggregationFunctionUtils;
import org.apache.pinot.core.query.selection.SelectionOperatorUtils;
import org.apache.pinot.core.transport.ServerRoutingInstance;
import org.apache.pinot.core.util.QueryOptions;
-import org.apache.pinot.parsers.CompilerConstants;
/**
@@ -77,7 +77,7 @@ public class DistinctDataTableReducer implements
DataTableReducer {
brokerResponseNative.setResultTable(new ResultTable(finalDataSchema,
Collections.emptyList()));
} else {
brokerResponseNative
- .setSelectionResults(new
SelectionResults(Arrays.asList(getDistinctColumns()), Collections.emptyList()));
+ .setSelectionResults(new SelectionResults(getDistinctColumns(),
Collections.emptyList()));
}
return;
}
@@ -162,14 +162,12 @@ public class DistinctDataTableReducer implements
DataTableReducer {
return new ResultTable(dataSchema, rows);
}
- private String[] getDistinctColumns() {
- return _brokerRequest.getAggregationsInfo().get(0).getAggregationParams()
- .get(CompilerConstants.COLUMN_KEY_IN_AGGREGATION_INFO)
- .split(CompilerConstants.AGGREGATION_FUNCTION_ARG_SEPARATOR);
+ private List<String> getDistinctColumns() {
+ return
AggregationFunctionUtils.getAggregationExpressions(_brokerRequest.getAggregationsInfo().get(0));
}
private DataSchema getEmptyResultTableDataSchema() {
- String[] columns = getDistinctColumns();
+ String[] columns = getDistinctColumns().toArray(new String[0]);
DataSchema.ColumnDataType[] columnDataTypes = new
DataSchema.ColumnDataType[columns.length];
Arrays.fill(columnDataTypes, DataSchema.ColumnDataType.STRING);
return new DataSchema(columns, columnDataTypes);
diff --git
a/pinot-core/src/main/java/org/apache/pinot/core/query/request/ServerQueryRequest.java
b/pinot-core/src/main/java/org/apache/pinot/core/query/request/ServerQueryRequest.java
index bafabcc..3865f2e 100644
---
a/pinot-core/src/main/java/org/apache/pinot/core/query/request/ServerQueryRequest.java
+++
b/pinot-core/src/main/java/org/apache/pinot/core/query/request/ServerQueryRequest.java
@@ -95,8 +95,9 @@ public class ServerQueryRequest {
_aggregationExpressions = new HashSet<>();
for (AggregationInfo aggregationInfo : aggregationsInfo) {
if
(!aggregationInfo.getAggregationType().equalsIgnoreCase(AggregationFunctionType.COUNT.getName()))
{
- _aggregationExpressions.add(
-
TransformExpressionTree.compileToExpressionTree(AggregationFunctionUtils.getColumn(aggregationInfo)));
+ for (String expressions :
AggregationFunctionUtils.getAggregationExpressions(aggregationInfo)) {
+
_aggregationExpressions.add(TransformExpressionTree.compileToExpressionTree(expressions));
+ }
}
}
_aggregationColumns =
RequestUtils.extractColumnsFromExpressions(_aggregationExpressions);
diff --git
a/pinot-core/src/main/java/org/apache/pinot/core/startree/StarTreeUtils.java
b/pinot-core/src/main/java/org/apache/pinot/core/startree/StarTreeUtils.java
index ebdef14..f8e9bd1 100644
--- a/pinot-core/src/main/java/org/apache/pinot/core/startree/StarTreeUtils.java
+++ b/pinot-core/src/main/java/org/apache/pinot/core/startree/StarTreeUtils.java
@@ -18,6 +18,7 @@
*/
package org.apache.pinot.core.startree;
+import java.util.Arrays;
import java.util.HashSet;
import java.util.Map;
import java.util.Set;
@@ -109,8 +110,8 @@ public class StarTreeUtils {
public static AggregationFunctionContext
createStarTreeFunctionContext(AggregationFunctionContext functionContext) {
AggregationFunction function = functionContext.getAggregationFunction();
AggregationFunctionColumnPair functionColumnPair =
- new AggregationFunctionColumnPair(function.getType(),
functionContext.getColumn());
- return new AggregationFunctionContext(function,
functionColumnPair.toColumnName());
+ new AggregationFunctionColumnPair(function.getType(),
functionContext.getColumnName());
+ return new AggregationFunctionContext(function,
Arrays.asList(functionColumnPair.toColumnName()));
}
/**
diff --git
a/pinot-core/src/main/java/org/apache/pinot/core/startree/executor/StarTreeAggregationExecutor.java
b/pinot-core/src/main/java/org/apache/pinot/core/startree/executor/StarTreeAggregationExecutor.java
index faca98c..e83a7f8 100644
---
a/pinot-core/src/main/java/org/apache/pinot/core/startree/executor/StarTreeAggregationExecutor.java
+++
b/pinot-core/src/main/java/org/apache/pinot/core/startree/executor/StarTreeAggregationExecutor.java
@@ -47,7 +47,7 @@ public class StarTreeAggregationExecutor extends
DefaultAggregationExecutor {
_functionArgs = new String[functionContexts.length];
for (int i = 0; i < functionContexts.length; i++) {
- _functionArgs[i] = functionContexts[i].getColumn();
+ _functionArgs[i] = functionContexts[i].getColumnName();
}
}
diff --git
a/pinot-core/src/main/java/org/apache/pinot/core/startree/executor/StarTreeGroupByExecutor.java
b/pinot-core/src/main/java/org/apache/pinot/core/startree/executor/StarTreeGroupByExecutor.java
index a681e20..86fd53b 100644
---
a/pinot-core/src/main/java/org/apache/pinot/core/startree/executor/StarTreeGroupByExecutor.java
+++
b/pinot-core/src/main/java/org/apache/pinot/core/startree/executor/StarTreeGroupByExecutor.java
@@ -55,7 +55,7 @@ public class StarTreeGroupByExecutor extends
DefaultGroupByExecutor {
_functionArgs = new String[functionContexts.length];
for (int i = 0; i < functionContexts.length; i++) {
- _functionArgs[i] = functionContexts[i].getColumn();
+ _functionArgs[i] = functionContexts[i].getColumnName();
}
}
diff --git
a/pinot-core/src/test/java/org/apache/pinot/core/data/table/IndexedTableTest.java
b/pinot-core/src/test/java/org/apache/pinot/core/data/table/IndexedTableTest.java
index 8bb91dc..6c90739 100644
---
a/pinot-core/src/test/java/org/apache/pinot/core/data/table/IndexedTableTest.java
+++
b/pinot-core/src/test/java/org/apache/pinot/core/data/table/IndexedTableTest.java
@@ -20,10 +20,8 @@ package org.apache.pinot.core.data.table;
import com.google.common.collect.Lists;
import java.util.ArrayList;
-import java.util.HashMap;
import java.util.Iterator;
import java.util.List;
-import java.util.Map;
import java.util.concurrent.Callable;
import java.util.concurrent.ExecutionException;
import java.util.concurrent.ExecutorService;
@@ -53,14 +51,15 @@ public class IndexedTableTest {
ColumnDataType.DOUBLE});
AggregationInfo agg1 = new AggregationInfo();
- Map<String, String> params1 = new HashMap<>();
- params1.put("column", "m1");
- agg1.setAggregationParams(params1);
+ List<String> args1 = new ArrayList<>(1);
+ args1.add("m1");
+ agg1.setExpressions(args1);
agg1.setAggregationType("sum");
+
AggregationInfo agg2 = new AggregationInfo();
- Map<String, String> params2 = new HashMap<>();
- params2.put("column", "m2");
- agg2.setAggregationParams(params2);
+ List<String> args2 = new ArrayList<>(1);
+ args2.add("m2");
+ agg2.setExpressions(args2);
agg2.setAggregationType("max");
List<AggregationInfo> aggregationInfos = Lists.newArrayList(agg1, agg2);
@@ -136,14 +135,15 @@ public class IndexedTableTest {
new ColumnDataType[]{ColumnDataType.STRING, ColumnDataType.INT,
ColumnDataType.DOUBLE, ColumnDataType.INT, ColumnDataType.DOUBLE,
ColumnDataType.DOUBLE});
AggregationInfo agg1 = new AggregationInfo();
- Map<String, String> params1 = new HashMap<>();
- params1.put("column", "m1");
- agg1.setAggregationParams(params1);
+ List<String> args1 = new ArrayList<>(1);
+ args1.add("m1");
+ agg1.setExpressions(args1);
agg1.setAggregationType("sum");
+
AggregationInfo agg2 = new AggregationInfo();
- Map<String, String> params2 = new HashMap<>();
- params2.put("column", "m2");
- agg2.setAggregationParams(params2);
+ List<String> args2 = new ArrayList<>(1);
+ args2.add("m2");
+ agg2.setExpressions(args2);
agg2.setAggregationType("max");
List<AggregationInfo> aggregationInfos = Lists.newArrayList(agg1, agg2);
@@ -311,14 +311,15 @@ public class IndexedTableTest {
ColumnDataType.DOUBLE});
AggregationInfo agg1 = new AggregationInfo();
- Map<String, String> params1 = new HashMap<>();
- params1.put("column", "m1");
- agg1.setAggregationParams(params1);
+ List<String> args1 = new ArrayList<>(1);
+ args1.add("m1");
+ agg1.setExpressions(args1);
agg1.setAggregationType("sum");
+
AggregationInfo agg2 = new AggregationInfo();
- Map<String, String> params2 = new HashMap<>();
- params2.put("column", "m2");
- agg2.setAggregationParams(params2);
+ List<String> args2 = new ArrayList<>(1);
+ args2.add("m2");
+ agg2.setExpressions(args2);
agg2.setAggregationType("max");
List<AggregationInfo> aggregationInfos = Lists.newArrayList(agg1, agg2);
diff --git
a/pinot-core/src/test/java/org/apache/pinot/core/data/table/TableResizerTest.java
b/pinot-core/src/test/java/org/apache/pinot/core/data/table/TableResizerTest.java
index 7e1cf4e..15c1e3f 100644
---
a/pinot-core/src/test/java/org/apache/pinot/core/data/table/TableResizerTest.java
+++
b/pinot-core/src/test/java/org/apache/pinot/core/data/table/TableResizerTest.java
@@ -61,24 +61,27 @@ public class TableResizerTest {
dataSchema = new DataSchema(new String[]{"d1", "d2", "d3", "sum(m1)",
"max(m2)", "distinctcount(m3)", "avg(m4)"},
new DataSchema.ColumnDataType[]{DataSchema.ColumnDataType.STRING,
DataSchema.ColumnDataType.INT, DataSchema.ColumnDataType.DOUBLE,
DataSchema.ColumnDataType.DOUBLE, DataSchema.ColumnDataType.DOUBLE,
DataSchema.ColumnDataType.OBJECT, DataSchema.ColumnDataType.OBJECT});
AggregationInfo agg1 = new AggregationInfo();
- Map<String, String> params1 = new HashMap<>(1);
- params1.put("column", "m1");
- agg1.setAggregationParams(params1);
+ List<String> args1 = new ArrayList<>(1);
+ args1.add("m1");
+ agg1.setExpressions(args1);
agg1.setAggregationType("sum");
+
AggregationInfo agg2 = new AggregationInfo();
- Map<String, String> params2 = new HashMap<>(1);
- params2.put("column", "m2");
- agg2.setAggregationParams(params2);
+ List<String> args2 = new ArrayList<>(1);
+ args2.add("m2");
+ agg2.setExpressions(args2);
agg2.setAggregationType("max");
+
AggregationInfo agg3 = new AggregationInfo();
- Map<String, String> params3 = new HashMap<>(1);
- params3.put("column", "m3");
- agg3.setAggregationParams(params3);
+ List<String> args3 = new ArrayList<>(1);
+ args3.add("m3");
+ agg3.setExpressions(args3);
agg3.setAggregationType("distinctcount");
+
AggregationInfo agg4 = new AggregationInfo();
- Map<String, String> params4 = new HashMap<>(1);
- params4.put("column", "m4");
- agg4.setAggregationParams(params4);
+ List<String> args4 = new ArrayList<>(1);
+ args4.add("m4");
+ agg4.setExpressions(args4);
agg4.setAggregationType("avg");
aggregationInfos = Lists.newArrayList(agg1, agg2, agg3, agg4);
diff --git
a/pinot-core/src/test/java/org/apache/pinot/core/query/aggregation/function/AggregationFunctionFactoryTest.java
b/pinot-core/src/test/java/org/apache/pinot/core/query/aggregation/function/AggregationFunctionFactoryTest.java
index e5d6998..442d86b 100644
---
a/pinot-core/src/test/java/org/apache/pinot/core/query/aggregation/function/AggregationFunctionFactoryTest.java
+++
b/pinot-core/src/test/java/org/apache/pinot/core/query/aggregation/function/AggregationFunctionFactoryTest.java
@@ -18,11 +18,12 @@
*/
package org.apache.pinot.core.query.aggregation.function;
+import java.util.Arrays;
import java.util.Collections;
+import java.util.List;
import org.apache.pinot.common.function.AggregationFunctionType;
import org.apache.pinot.common.request.AggregationInfo;
import org.apache.pinot.common.request.BrokerRequest;
-import org.apache.pinot.parsers.CompilerConstants;
import org.testng.Assert;
import org.testng.annotations.Test;
@@ -39,252 +40,244 @@ public class AggregationFunctionFactoryTest {
AggregationInfo aggregationInfo = new AggregationInfo();
aggregationInfo.setAggregationType("CoUnT");
- aggregationInfo
-
.setAggregationParams(Collections.singletonMap(CompilerConstants.COLUMN_KEY_IN_AGGREGATION_INFO,
column));
+ aggregationInfo.setExpressions(Collections.singletonList(COLUMN));
aggregationFunction =
AggregationFunctionFactory.getAggregationFunction(aggregationInfo,
brokerRequest);
Assert.assertTrue(aggregationFunction instanceof CountAggregationFunction);
Assert.assertEquals(aggregationFunction.getType(),
AggregationFunctionType.COUNT);
- Assert.assertEquals(aggregationFunction.getColumnName(COLUMN),
"count_star");
+ Assert.assertEquals(aggregationFunction.getColumnName(), "count_star");
aggregationInfo = new AggregationInfo();
aggregationInfo.setAggregationType("MiN");
column = "min_column";
- aggregationInfo
-
.setAggregationParams(Collections.singletonMap(CompilerConstants.COLUMN_KEY_IN_AGGREGATION_INFO,
column));
+ aggregationInfo.setExpressions(Collections.singletonList(COLUMN));
aggregationFunction =
AggregationFunctionFactory.getAggregationFunction(aggregationInfo,
brokerRequest);
Assert.assertTrue(aggregationFunction instanceof MinAggregationFunction);
Assert.assertEquals(aggregationFunction.getType(),
AggregationFunctionType.MIN);
- Assert.assertEquals(aggregationFunction.getColumnName(COLUMN), column);
+ Assert.assertEquals(aggregationFunction.getColumnName(), column);
aggregationInfo = new AggregationInfo();
aggregationInfo.setAggregationType("MaX");
column = "max_column";
- aggregationInfo
-
.setAggregationParams(Collections.singletonMap(CompilerConstants.COLUMN_KEY_IN_AGGREGATION_INFO,
column));
+ aggregationInfo.setExpressions(Collections.singletonList(COLUMN));
aggregationFunction =
AggregationFunctionFactory.getAggregationFunction(aggregationInfo,
brokerRequest);
Assert.assertTrue(aggregationFunction instanceof MaxAggregationFunction);
Assert.assertEquals(aggregationFunction.getType(),
AggregationFunctionType.MAX);
- Assert.assertEquals(aggregationFunction.getColumnName(COLUMN), column);
+ Assert.assertEquals(aggregationFunction.getColumnName(), column);
aggregationInfo = new AggregationInfo();
aggregationInfo.setAggregationType("SuM");
column = "sum_column";
- aggregationInfo
-
.setAggregationParams(Collections.singletonMap(CompilerConstants.COLUMN_KEY_IN_AGGREGATION_INFO,
column));
+ aggregationInfo.setExpressions(Collections.singletonList(COLUMN));
aggregationFunction =
AggregationFunctionFactory.getAggregationFunction(aggregationInfo,
brokerRequest);
Assert.assertTrue(aggregationFunction instanceof SumAggregationFunction);
Assert.assertEquals(aggregationFunction.getType(),
AggregationFunctionType.SUM);
- Assert.assertEquals(aggregationFunction.getColumnName(COLUMN), column);
+ Assert.assertEquals(aggregationFunction.getColumnName(), column);
aggregationInfo = new AggregationInfo();
aggregationInfo.setAggregationType("AvG");
column = "avg_column";
- aggregationInfo
-
.setAggregationParams(Collections.singletonMap(CompilerConstants.COLUMN_KEY_IN_AGGREGATION_INFO,
column));
+ aggregationInfo.setExpressions(Collections.singletonList(COLUMN));
aggregationFunction =
AggregationFunctionFactory.getAggregationFunction(aggregationInfo,
brokerRequest);
Assert.assertTrue(aggregationFunction instanceof AvgAggregationFunction);
Assert.assertEquals(aggregationFunction.getType(),
AggregationFunctionType.AVG);
- Assert.assertEquals(aggregationFunction.getColumnName(COLUMN), column);
+ Assert.assertEquals(aggregationFunction.getColumnName(), column);
aggregationInfo = new AggregationInfo();
aggregationInfo.setAggregationType("MiNmAxRaNgE");
column = "minMaxRange_column";
- aggregationInfo.setAggregationParams(
-
Collections.singletonMap(CompilerConstants.COLUMN_KEY_IN_AGGREGATION_INFO,
column));
+ aggregationInfo.setExpressions(Collections.singletonList(COLUMN));
aggregationFunction =
AggregationFunctionFactory.getAggregationFunction(aggregationInfo,
brokerRequest);
Assert.assertTrue(aggregationFunction instanceof
MinMaxRangeAggregationFunction);
Assert.assertEquals(aggregationFunction.getType(),
AggregationFunctionType.MINMAXRANGE);
- Assert.assertEquals(aggregationFunction.getColumnName(COLUMN), column);
+ Assert.assertEquals(aggregationFunction.getColumnName(), column);
aggregationInfo = new AggregationInfo();
aggregationInfo.setAggregationType("DiStInCtCoUnT");
column = "distinctCount_column";
- aggregationInfo.setAggregationParams(
-
Collections.singletonMap(CompilerConstants.COLUMN_KEY_IN_AGGREGATION_INFO,
column));
+ aggregationInfo.setExpressions(Collections.singletonList(COLUMN));
aggregationFunction =
AggregationFunctionFactory.getAggregationFunction(aggregationInfo,
brokerRequest);
Assert.assertTrue(aggregationFunction instanceof
DistinctCountAggregationFunction);
Assert.assertEquals(aggregationFunction.getType(),
AggregationFunctionType.DISTINCTCOUNT);
- Assert.assertEquals(aggregationFunction.getColumnName(COLUMN), column);
+ Assert.assertEquals(aggregationFunction.getColumnName(), column);
aggregationInfo = new AggregationInfo();
aggregationInfo.setAggregationType("DiStInCtCoUnThLl");
column = "distinctCountHLL_column";
- aggregationInfo
-
.setAggregationParams(Collections.singletonMap(CompilerConstants.COLUMN_KEY_IN_AGGREGATION_INFO,
column));
+ aggregationInfo.setExpressions(Collections.singletonList(COLUMN));
aggregationFunction =
AggregationFunctionFactory.getAggregationFunction(aggregationInfo,
brokerRequest);
Assert.assertTrue(aggregationFunction instanceof
DistinctCountHLLAggregationFunction);
Assert.assertEquals(aggregationFunction.getType(),
AggregationFunctionType.DISTINCTCOUNTHLL);
- Assert.assertEquals(aggregationFunction.getColumnName(COLUMN), column);
+ Assert.assertEquals(aggregationFunction.getColumnName(), column);
aggregationInfo = new AggregationInfo();
aggregationInfo.setAggregationType("DiStInCtCoUnTrAwHlL");
column = "distinctCountRawHLL_column";
- aggregationInfo
-
.setAggregationParams(Collections.singletonMap(CompilerConstants.COLUMN_KEY_IN_AGGREGATION_INFO,
column));
+ aggregationInfo.setExpressions(Collections.singletonList(COLUMN));
aggregationFunction =
AggregationFunctionFactory.getAggregationFunction(aggregationInfo,
brokerRequest);
Assert.assertTrue(aggregationFunction instanceof
DistinctCountRawHLLAggregationFunction);
Assert.assertEquals(aggregationFunction.getType(),
AggregationFunctionType.DISTINCTCOUNTRAWHLL);
- Assert.assertEquals(aggregationFunction.getColumnName(COLUMN), column);
- ;
+ Assert.assertEquals(aggregationFunction.getColumnName(), column);
aggregationInfo = new AggregationInfo();
aggregationInfo.setAggregationType("FaStHlL");
column = "fastHLL_column";
- aggregationInfo
-
.setAggregationParams(Collections.singletonMap(CompilerConstants.COLUMN_KEY_IN_AGGREGATION_INFO,
column));
+ aggregationInfo.setExpressions(Collections.singletonList(COLUMN));
aggregationFunction =
AggregationFunctionFactory.getAggregationFunction(aggregationInfo,
brokerRequest);
Assert.assertTrue(aggregationFunction instanceof
FastHLLAggregationFunction);
Assert.assertEquals(aggregationFunction.getType(),
AggregationFunctionType.FASTHLL);
- Assert.assertEquals(aggregationFunction.getColumnName(COLUMN), column);
+ Assert.assertEquals(aggregationFunction.getColumnName(), column);
aggregationInfo = new AggregationInfo();
aggregationInfo.setAggregationType("PeRcEnTiLe5");
column = "percentile5_column";
- aggregationInfo
-
.setAggregationParams(Collections.singletonMap(CompilerConstants.COLUMN_KEY_IN_AGGREGATION_INFO,
column));
+ aggregationInfo.setExpressions(Collections.singletonList(COLUMN));
aggregationFunction =
AggregationFunctionFactory.getAggregationFunction(aggregationInfo,
brokerRequest);
Assert.assertTrue(aggregationFunction instanceof
PercentileAggregationFunction);
Assert.assertEquals(aggregationFunction.getType(),
AggregationFunctionType.PERCENTILE);
- Assert.assertEquals(aggregationFunction.getColumnName(COLUMN), column);
+ Assert.assertEquals(aggregationFunction.getColumnName(), column);
aggregationInfo = new AggregationInfo();
aggregationInfo.setAggregationType("PeRcEnTiLeEsT50");
column = "percentileEst50_column";
- aggregationInfo
-
.setAggregationParams(Collections.singletonMap(CompilerConstants.COLUMN_KEY_IN_AGGREGATION_INFO,
column));
+ aggregationInfo.setExpressions(Collections.singletonList(COLUMN));
aggregationFunction =
AggregationFunctionFactory.getAggregationFunction(aggregationInfo,
brokerRequest);
Assert.assertTrue(aggregationFunction instanceof
PercentileEstAggregationFunction);
Assert.assertEquals(aggregationFunction.getType(),
AggregationFunctionType.PERCENTILEEST);
- Assert.assertEquals(aggregationFunction.getColumnName(COLUMN), column);
+ Assert.assertEquals(aggregationFunction.getColumnName(), column);
aggregationInfo = new AggregationInfo();
aggregationInfo.setAggregationType("PeRcEnTiLeTdIgEsT99");
column = "percentileTDigest99_column";
- aggregationInfo
-
.setAggregationParams(Collections.singletonMap(CompilerConstants.COLUMN_KEY_IN_AGGREGATION_INFO,
column));
+ aggregationInfo.setExpressions(Collections.singletonList(COLUMN));
aggregationFunction =
AggregationFunctionFactory.getAggregationFunction(aggregationInfo,
brokerRequest);
Assert.assertTrue(aggregationFunction instanceof
PercentileTDigestAggregationFunction);
Assert.assertEquals(aggregationFunction.getType(),
AggregationFunctionType.PERCENTILETDIGEST);
- Assert.assertEquals(aggregationFunction.getColumnName(COLUMN), column);
+ Assert.assertEquals(aggregationFunction.getColumnName(), column);
aggregationInfo = new AggregationInfo();
aggregationInfo.setAggregationType("CoUnTmV");
column = "countMV_column";
- aggregationInfo
-
.setAggregationParams(Collections.singletonMap(CompilerConstants.COLUMN_KEY_IN_AGGREGATION_INFO,
column));
+ aggregationInfo.setExpressions(Collections.singletonList(COLUMN));
aggregationFunction =
AggregationFunctionFactory.getAggregationFunction(aggregationInfo,
brokerRequest);
Assert.assertTrue(aggregationFunction instanceof
CountMVAggregationFunction);
Assert.assertEquals(aggregationFunction.getType(),
AggregationFunctionType.COUNTMV);
- Assert.assertEquals(aggregationFunction.getColumnName(COLUMN), column);
+ Assert.assertEquals(aggregationFunction.getColumnName(), column);
aggregationInfo = new AggregationInfo();
aggregationInfo.setAggregationType("MiNmV");
column = "minMV_column";
- aggregationInfo
-
.setAggregationParams(Collections.singletonMap(CompilerConstants.COLUMN_KEY_IN_AGGREGATION_INFO,
column));
+ aggregationInfo.setExpressions(Collections.singletonList(COLUMN));
aggregationFunction =
AggregationFunctionFactory.getAggregationFunction(aggregationInfo,
brokerRequest);
Assert.assertTrue(aggregationFunction instanceof MinMVAggregationFunction);
Assert.assertEquals(aggregationFunction.getType(),
AggregationFunctionType.MINMV);
- Assert.assertEquals(aggregationFunction.getColumnName(COLUMN), column);
+ Assert.assertEquals(aggregationFunction.getColumnName(), column);
aggregationInfo = new AggregationInfo();
aggregationInfo.setAggregationType("MaXmV");
column = "maxMV_column";
- aggregationInfo
-
.setAggregationParams(Collections.singletonMap(CompilerConstants.COLUMN_KEY_IN_AGGREGATION_INFO,
column));
+ aggregationInfo.setExpressions(Collections.singletonList(COLUMN));
aggregationFunction =
AggregationFunctionFactory.getAggregationFunction(aggregationInfo,
brokerRequest);
Assert.assertTrue(aggregationFunction instanceof MaxMVAggregationFunction);
Assert.assertEquals(aggregationFunction.getType(),
AggregationFunctionType.MAXMV);
- Assert.assertEquals(aggregationFunction.getColumnName(COLUMN), column);
+ Assert.assertEquals(aggregationFunction.getColumnName(), column);
aggregationInfo = new AggregationInfo();
aggregationInfo.setAggregationType("SuMmV");
column = "sumMV_column";
- aggregationInfo
-
.setAggregationParams(Collections.singletonMap(CompilerConstants.COLUMN_KEY_IN_AGGREGATION_INFO,
column));
+ aggregationInfo.setExpressions(Collections.singletonList(COLUMN));
aggregationFunction =
AggregationFunctionFactory.getAggregationFunction(aggregationInfo,
brokerRequest);
Assert.assertTrue(aggregationFunction instanceof SumMVAggregationFunction);
Assert.assertEquals(aggregationFunction.getType(),
AggregationFunctionType.SUMMV);
- Assert.assertEquals(aggregationFunction.getColumnName(COLUMN), column);
+ Assert.assertEquals(aggregationFunction.getColumnName(), column);
aggregationInfo = new AggregationInfo();
aggregationInfo.setAggregationType("AvGmV");
column = "avgMV_column";
- aggregationInfo
-
.setAggregationParams(Collections.singletonMap(CompilerConstants.COLUMN_KEY_IN_AGGREGATION_INFO,
column));
+ aggregationInfo.setExpressions(Collections.singletonList(COLUMN));
aggregationFunction =
AggregationFunctionFactory.getAggregationFunction(aggregationInfo,
brokerRequest);
Assert.assertTrue(aggregationFunction instanceof AvgMVAggregationFunction);
Assert.assertEquals(aggregationFunction.getType(),
AggregationFunctionType.AVGMV);
- Assert.assertEquals(aggregationFunction.getColumnName(COLUMN), column);
+ Assert.assertEquals(aggregationFunction.getColumnName(), column);
aggregationInfo = new AggregationInfo();
aggregationInfo.setAggregationType("MiNmAxRaNgEmV");
column = "minMaxRangeMV_column";
- aggregationInfo
-
.setAggregationParams(Collections.singletonMap(CompilerConstants.COLUMN_KEY_IN_AGGREGATION_INFO,
column));
+ aggregationInfo.setExpressions(Collections.singletonList(COLUMN));
aggregationFunction =
AggregationFunctionFactory.getAggregationFunction(aggregationInfo,
brokerRequest);
Assert.assertTrue(aggregationFunction instanceof
MinMaxRangeMVAggregationFunction);
Assert.assertEquals(aggregationFunction.getType(),
AggregationFunctionType.MINMAXRANGEMV);
- Assert.assertEquals(aggregationFunction.getColumnName(COLUMN), column);
+ Assert.assertEquals(aggregationFunction.getColumnName(), column);
aggregationInfo = new AggregationInfo();
aggregationInfo.setAggregationType("DiStInCtCoUnTmV");
column = "distinctCountMV_column";
- aggregationInfo
-
.setAggregationParams(Collections.singletonMap(CompilerConstants.COLUMN_KEY_IN_AGGREGATION_INFO,
column));
+ aggregationInfo.setExpressions(Collections.singletonList(COLUMN));
aggregationFunction =
AggregationFunctionFactory.getAggregationFunction(aggregationInfo,
brokerRequest);
Assert.assertTrue(aggregationFunction instanceof
DistinctCountMVAggregationFunction);
Assert.assertEquals(aggregationFunction.getType(),
AggregationFunctionType.DISTINCTCOUNTMV);
- Assert.assertEquals(aggregationFunction.getColumnName(COLUMN), column);
+ Assert.assertEquals(aggregationFunction.getColumnName(), column);
aggregationInfo = new AggregationInfo();
aggregationInfo.setAggregationType("DiStInCtCoUnThLlMv");
column = "distinctCountHLLMV_column";
- aggregationInfo
-
.setAggregationParams(Collections.singletonMap(CompilerConstants.COLUMN_KEY_IN_AGGREGATION_INFO,
column));
+ aggregationInfo.setExpressions(Collections.singletonList(COLUMN));
aggregationFunction =
AggregationFunctionFactory.getAggregationFunction(aggregationInfo,
brokerRequest);
Assert.assertTrue(aggregationFunction instanceof
DistinctCountHLLMVAggregationFunction);
Assert.assertEquals(aggregationFunction.getType(),
AggregationFunctionType.DISTINCTCOUNTHLLMV);
- Assert.assertEquals(aggregationFunction.getColumnName(COLUMN), column);
+ Assert.assertEquals(aggregationFunction.getColumnName(), column);
aggregationInfo = new AggregationInfo();
aggregationInfo.setAggregationType("DiStInCtCoUnTrAwHlLmV");
column = "distinctCountRawHLLMV_column";
- aggregationInfo
-
.setAggregationParams(Collections.singletonMap(CompilerConstants.COLUMN_KEY_IN_AGGREGATION_INFO,
column));
+ aggregationInfo.setExpressions(Collections.singletonList(COLUMN));
aggregationFunction =
AggregationFunctionFactory.getAggregationFunction(aggregationInfo,
brokerRequest);
Assert.assertTrue(aggregationFunction instanceof
DistinctCountRawHLLMVAggregationFunction);
Assert.assertEquals(aggregationFunction.getType(),
AggregationFunctionType.DISTINCTCOUNTRAWHLLMV);
- Assert.assertEquals(aggregationFunction.getColumnName(COLUMN), column);
+ Assert.assertEquals(aggregationFunction.getColumnName(), column);
aggregationInfo = new AggregationInfo();
aggregationInfo.setAggregationType("PeRcEnTiLe10Mv");
column = "percentile10MV_column";
- aggregationInfo
-
.setAggregationParams(Collections.singletonMap(CompilerConstants.COLUMN_KEY_IN_AGGREGATION_INFO,
column));
+ aggregationInfo.setExpressions(Collections.singletonList(COLUMN));
aggregationFunction =
AggregationFunctionFactory.getAggregationFunction(aggregationInfo,
brokerRequest);
Assert.assertTrue(aggregationFunction instanceof
PercentileMVAggregationFunction);
Assert.assertEquals(aggregationFunction.getType(),
AggregationFunctionType.PERCENTILEMV);
- Assert.assertEquals(aggregationFunction.getColumnName(COLUMN), column);
+ Assert.assertEquals(aggregationFunction.getColumnName(), column);
aggregationInfo = new AggregationInfo();
aggregationInfo.setAggregationType("PeRcEnTiLeEsT90mV");
column = "percentileEst90MV_column";
- aggregationInfo
-
.setAggregationParams(Collections.singletonMap(CompilerConstants.COLUMN_KEY_IN_AGGREGATION_INFO,
column));
+ aggregationInfo.setExpressions(Collections.singletonList(COLUMN));
aggregationFunction =
AggregationFunctionFactory.getAggregationFunction(aggregationInfo,
brokerRequest);
Assert.assertTrue(aggregationFunction instanceof
PercentileEstMVAggregationFunction);
Assert.assertEquals(aggregationFunction.getType(),
AggregationFunctionType.PERCENTILEESTMV);
- Assert.assertEquals(aggregationFunction.getColumnName(COLUMN), column);
+ Assert.assertEquals(aggregationFunction.getColumnName(), column);
aggregationInfo = new AggregationInfo();
aggregationInfo.setAggregationType("PeRcEnTiLeTdIgEsT95mV");
column = "percentileTDigest95MV_column";
- aggregationInfo
-
.setAggregationParams(Collections.singletonMap(CompilerConstants.COLUMN_KEY_IN_AGGREGATION_INFO,
column));
+ aggregationInfo.setExpressions(Collections.singletonList(COLUMN));
aggregationFunction =
AggregationFunctionFactory.getAggregationFunction(aggregationInfo,
brokerRequest);
Assert.assertTrue(aggregationFunction instanceof
PercentileTDigestMVAggregationFunction);
Assert.assertEquals(aggregationFunction.getType(),
AggregationFunctionType.PERCENTILETDIGESTMV);
- Assert.assertEquals(aggregationFunction.getColumnName(COLUMN), column);
+ Assert.assertEquals(aggregationFunction.getColumnName(), column);
+ }
+
+ @Test
+ public void testAggregationFunctionWithMultipleArgs() {
+ // Test using new field `expressions` in AggregationInfo.
+ BrokerRequest brokerRequest = new BrokerRequest();
+ AggregationInfo aggregationInfo = new AggregationInfo();
+
+ aggregationInfo.setAggregationType("distinct");
+ List<String> args = Arrays.asList("column1", "column2", "column3");
+ String expected = "distinct_" + AggregationFunctionUtils.concatArgs(args);
+ aggregationInfo.setExpressions(args);
+
+ AggregationFunction aggregationFunction =
+ AggregationFunctionFactory.getAggregationFunction(aggregationInfo,
brokerRequest);
+ Assert.assertTrue(aggregationFunction instanceof
DistinctAggregationFunction);
+ Assert.assertEquals(aggregationFunction.getType(),
AggregationFunctionType.DISTINCT);
+ Assert.assertEquals(aggregationFunction.getColumnName(), expected);
}
}
diff --git
a/pinot-core/src/test/java/org/apache/pinot/query/aggregation/DefaultAggregationExecutorTest.java
b/pinot-core/src/test/java/org/apache/pinot/query/aggregation/DefaultAggregationExecutorTest.java
index 03abe88..b703a3d 100644
---
a/pinot-core/src/test/java/org/apache/pinot/query/aggregation/DefaultAggregationExecutorTest.java
+++
b/pinot-core/src/test/java/org/apache/pinot/query/aggregation/DefaultAggregationExecutorTest.java
@@ -20,6 +20,7 @@ package org.apache.pinot.query.aggregation;
import java.io.File;
import java.util.ArrayList;
+import java.util.Collections;
import java.util.HashMap;
import java.util.HashSet;
import java.util.List;
@@ -113,10 +114,7 @@ public class DefaultAggregationExecutorTest {
AggregationInfo aggregationInfo = new AggregationInfo();
aggregationInfo.setAggregationType(AGGREGATION_FUNCTIONS[i]);
- Map<String, String> params = new HashMap<>();
- params.put("column", _columns[i]);
-
- aggregationInfo.setAggregationParams(params);
+ aggregationInfo.setExpressions(Collections.singletonList(_columns[i]));
_aggregationInfoList.add(aggregationInfo);
}
}
diff --git
a/pinot-core/src/test/java/org/apache/pinot/query/aggregation/groupby/AggregationGroupByTrimmingServiceTest.java
b/pinot-core/src/test/java/org/apache/pinot/query/aggregation/groupby/AggregationGroupByTrimmingServiceTest.java
index 859fabd..3ef8f60 100644
---
a/pinot-core/src/test/java/org/apache/pinot/query/aggregation/groupby/AggregationGroupByTrimmingServiceTest.java
+++
b/pinot-core/src/test/java/org/apache/pinot/query/aggregation/groupby/AggregationGroupByTrimmingServiceTest.java
@@ -35,7 +35,6 @@ import
org.apache.pinot.core.query.aggregation.function.AggregationFunction;
import
org.apache.pinot.core.query.aggregation.function.AggregationFunctionFactory;
import
org.apache.pinot.core.query.aggregation.groupby.AggregationGroupByTrimmingService;
import org.apache.pinot.core.query.aggregation.groupby.GroupKeyGenerator;
-import org.apache.pinot.parsers.CompilerConstants;
import org.testng.Assert;
import org.testng.annotations.BeforeClass;
import org.testng.annotations.Test;
@@ -147,8 +146,7 @@ public class AggregationGroupByTrimmingServiceTest {
private static AggregationFunction createAggregationFunction(String
aggregationType, String column) {
AggregationInfo aggregationInfo = new AggregationInfo();
aggregationInfo.setAggregationType(aggregationType);
- aggregationInfo
-
.setAggregationParams(Collections.singletonMap(CompilerConstants.COLUMN_KEY_IN_AGGREGATION_INFO,
column));
+ aggregationInfo.setExpressions(Collections.singletonList(column));
return AggregationFunctionFactory.getAggregationFunction(aggregationInfo,
new BrokerRequest());
}
}
diff --git
a/pinot-perf/src/main/java/org/apache/pinot/perf/BenchmarkCombineGroupBy.java
b/pinot-perf/src/main/java/org/apache/pinot/perf/BenchmarkCombineGroupBy.java
index 8be1283..954db47 100644
---
a/pinot-perf/src/main/java/org/apache/pinot/perf/BenchmarkCombineGroupBy.java
+++
b/pinot-perf/src/main/java/org/apache/pinot/perf/BenchmarkCombineGroupBy.java
@@ -21,7 +21,6 @@ package org.apache.pinot.perf;
import com.google.common.base.Joiner;
import com.google.common.collect.Lists;
import java.util.ArrayList;
-import java.util.HashMap;
import java.util.HashSet;
import java.util.List;
import java.util.Map;
@@ -44,7 +43,6 @@ import org.apache.pinot.common.request.SelectionSort;
import org.apache.pinot.common.utils.DataSchema;
import org.apache.pinot.core.data.table.ConcurrentIndexedTable;
import org.apache.pinot.core.data.table.IndexedTable;
-import org.apache.pinot.core.data.table.Key;
import org.apache.pinot.core.data.table.Record;
import org.apache.pinot.core.query.aggregation.function.AggregationFunction;
import
org.apache.pinot.core.query.aggregation.function.AggregationFunctionFactory;
@@ -111,14 +109,15 @@ public class BenchmarkCombineGroupBy {
DataSchema.ColumnDataType.DOUBLE,
DataSchema.ColumnDataType.DOUBLE});
AggregationInfo agg1 = new AggregationInfo();
- Map<String, String> params1 = new HashMap<>();
- params1.put("column", "m1");
- agg1.setAggregationParams(params1);
+ List<String> args1 = new ArrayList<>(1);
+ args1.add("m1");
+ agg1.setExpressions(args1);
agg1.setAggregationType("sum");
+
AggregationInfo agg2 = new AggregationInfo();
- Map<String, String> params2 = new HashMap<>();
- params2.put("column", "m2");
- agg2.setAggregationParams(params2);
+ List<String> args2 = new ArrayList<>(1);
+ args2.add("m2");
+ agg2.setExpressions(args2);
agg2.setAggregationType("max");
_aggregationInfos = Lists.newArrayList(agg1, agg2);
diff --git
a/pinot-perf/src/main/java/org/apache/pinot/perf/BenchmarkIndexedTable.java
b/pinot-perf/src/main/java/org/apache/pinot/perf/BenchmarkIndexedTable.java
index 0b6ece3..7312b6b 100644
--- a/pinot-perf/src/main/java/org/apache/pinot/perf/BenchmarkIndexedTable.java
+++ b/pinot-perf/src/main/java/org/apache/pinot/perf/BenchmarkIndexedTable.java
@@ -20,10 +20,8 @@ package org.apache.pinot.perf;
import com.google.common.collect.Lists;
import java.util.ArrayList;
-import java.util.HashMap;
import java.util.HashSet;
import java.util.List;
-import java.util.Map;
import java.util.Random;
import java.util.Set;
import java.util.concurrent.Callable;
@@ -40,7 +38,6 @@ import org.apache.pinot.common.request.SelectionSort;
import org.apache.pinot.common.utils.DataSchema;
import org.apache.pinot.core.data.table.ConcurrentIndexedTable;
import org.apache.pinot.core.data.table.IndexedTable;
-import org.apache.pinot.core.data.table.Key;
import org.apache.pinot.core.data.table.Record;
import org.apache.pinot.core.data.table.SimpleIndexedTable;
import org.apache.pinot.core.util.trace.TraceRunnable;
@@ -97,14 +94,15 @@ public class BenchmarkIndexedTable {
DataSchema.ColumnDataType.DOUBLE,
DataSchema.ColumnDataType.DOUBLE});
AggregationInfo agg1 = new AggregationInfo();
- Map<String, String> params1 = new HashMap<>();
- params1.put("column", "m1");
- agg1.setAggregationParams(params1);
+ List<String> args1 = new ArrayList<>();
+ args1.add("m1");
+ agg1.setExpressions(args1);
agg1.setAggregationType("sum");
+
AggregationInfo agg2 = new AggregationInfo();
- Map<String, String> params2 = new HashMap<>();
- params2.put("column", "m2");
- agg2.setAggregationParams(params2);
+ List<String> args2 = new ArrayList<>();
+ args2.add("m2");
+ agg2.setExpressions(args2);
agg2.setAggregationType("max");
_aggregationInfos = Lists.newArrayList(agg1, agg2);
diff --git
a/pinot-tools/src/main/java/org/apache/pinot/tools/scan/query/Aggregation.java
b/pinot-tools/src/main/java/org/apache/pinot/tools/scan/query/Aggregation.java
index 67225a0..21eefe8 100644
---
a/pinot-tools/src/main/java/org/apache/pinot/tools/scan/query/Aggregation.java
+++
b/pinot-tools/src/main/java/org/apache/pinot/tools/scan/query/Aggregation.java
@@ -49,9 +49,7 @@ public class Aggregation {
_addCountStar = false;
for (AggregationInfo aggregationInfo : _aggregationsInfo) {
- Map<String, String> aggregationParams =
aggregationInfo.getAggregationParams();
- for (Map.Entry<String, String> entry : aggregationParams.entrySet()) {
- String column = entry.getValue();
+ for (String column : aggregationInfo.getExpressions()) {
// Apparently in case of multiple group by's '*' is replaced by
empty/null in brokerRequest.
if (column == null || column.isEmpty() || column.equals("*")) {
_addCountStar = true;
diff --git
a/pinot-tools/src/main/java/org/apache/pinot/tools/scan/query/SegmentQueryProcessor.java
b/pinot-tools/src/main/java/org/apache/pinot/tools/scan/query/SegmentQueryProcessor.java
index 9b692a6..2d03d91 100644
---
a/pinot-tools/src/main/java/org/apache/pinot/tools/scan/query/SegmentQueryProcessor.java
+++
b/pinot-tools/src/main/java/org/apache/pinot/tools/scan/query/SegmentQueryProcessor.java
@@ -141,9 +141,7 @@ class SegmentQueryProcessor {
Set<String> allColumns = _metadata.getAllColumns();
if (brokerRequest.isSetAggregationsInfo()) {
for (AggregationInfo aggregationInfo :
brokerRequest.getAggregationsInfo()) {
- Map<String, String> aggregationParams =
aggregationInfo.getAggregationParams();
-
- for (String column : aggregationParams.values()) {
+ for (String column : aggregationInfo.getExpressions()) {
if (column != null && !column.isEmpty() && !column.equals("*") &&
!allColumns.contains(column)) {
LOGGER.debug("Skipping segment '{}', as it does not have column
'{}'", _metadata.getName(), column);
return true;
---------------------------------------------------------------------
To unsubscribe, e-mail: [email protected]
For additional commands, e-mail: [email protected]