Class DeclarativeAggregateFunction
- java.lang.Object
-
- org.apache.flink.table.functions.UserDefinedFunction
-
- org.apache.flink.table.functions.DeclarativeAggregateFunction
-
- All Implemented Interfaces:
Serializable
,FunctionDefinition
- Direct Known Subclasses:
AvgAggFunction
,Count1AggFunction
,CountAggFunction
,CumeDistAggFunction
,HiveDeclarativeAggregateFunction
,LeadLagAggFunction
,ListAggFunction
,MaxAggFunction
,MinAggFunction
,NTILEAggFunction
,RankLikeAggFunctionBase
,RowNumberAggFunction
,SingleValueAggFunction
,Sum0AggFunction
,SumAggFunction
,SumWithRetractAggFunction
@PublicEvolving public abstract class DeclarativeAggregateFunction extends UserDefinedFunction
API for aggregation functions that are expressed in terms of expressions.When implementing a new expression-based aggregate function, you should first decide how many operands your function will have by implementing
operandCount()
method. And then you can useoperand(int)
fields to represent your operand, like `operand(0)`, `operand(2)`.Then you should declare all your buffer attributes by implementing
aggBufferAttributes()
. You should declare all buffer attributes asUnresolvedReferenceExpression
, and make sure the name of your attributes are unique within the function and it should not conflict with operandIndex. You can then use these attributes when defininginitialValuesExpressions()
,accumulateExpressions()
,mergeExpressions()
andgetValueExpression()
.Note: Developer of DeclarativeAggregateFunction should guarantee that the inferred type of
getValueExpression()
is the same asgetResultType()
See an full example:
AvgAggFunction
.- See Also:
- Serialized Form
-
-
Constructor Summary
Constructors Constructor Description DeclarativeAggregateFunction()
-
Method Summary
All Methods Instance Methods Abstract Methods Concrete Methods Modifier and Type Method Description abstract Expression[]
accumulateExpressions()
Expressions for accumulating the mutable aggregation buffer based on an input row.abstract UnresolvedReferenceExpression[]
aggBufferAttributes()
All fields of the aggregate buffer.abstract DataType[]
getAggBufferTypes()
All types of the aggregate buffer.FunctionKind
getKind()
Returns the kind of function this definition describes.abstract DataType
getResultType()
The result type of the function.TypeInference
getTypeInference(DataTypeFactory factory)
Returns the logic for performing type inference of a call to this function definition.abstract Expression
getValueExpression()
An expression which returns the final value for this aggregate function.abstract Expression[]
initialValuesExpressions()
Expressions for initializing empty aggregation buffers.abstract Expression[]
mergeExpressions()
A sequence of expressions for merging two aggregation buffers together.UnresolvedReferenceExpression
mergeOperand(UnresolvedReferenceExpression aggBuffer)
Merge input ofmergeExpressions()
, the input are AGG buffer generated by user definition.UnresolvedReferenceExpression[]
mergeOperands()
Merge inputs ofmergeExpressions()
, these inputs are agg buffer generated by user definition.UnresolvedReferenceExpression
operand(int i)
Arg of accumulate and retract, the input value (usually obtained from a new arrived data).abstract int
operandCount()
How many operands your function will deal with.UnresolvedReferenceExpression[]
operands()
Args of accumulate and retract, the input value (usually obtained from a new arrived data).abstract Expression[]
retractExpressions()
Expressions for retracting the mutable aggregation buffer based on an input row.-
Methods inherited from class org.apache.flink.table.functions.UserDefinedFunction
close, functionIdentifier, open, toString
-
Methods inherited from class java.lang.Object
clone, equals, finalize, getClass, hashCode, notify, notifyAll, wait, wait, wait
-
Methods inherited from interface org.apache.flink.table.functions.FunctionDefinition
getRequirements, isDeterministic, supportsConstantFolding
-
-
-
-
Method Detail
-
operandCount
public abstract int operandCount()
How many operands your function will deal with.
-
aggBufferAttributes
public abstract UnresolvedReferenceExpression[] aggBufferAttributes()
All fields of the aggregate buffer.
-
getAggBufferTypes
public abstract DataType[] getAggBufferTypes()
All types of the aggregate buffer.
-
getResultType
public abstract DataType getResultType()
The result type of the function.
-
initialValuesExpressions
public abstract Expression[] initialValuesExpressions()
Expressions for initializing empty aggregation buffers.
-
accumulateExpressions
public abstract Expression[] accumulateExpressions()
Expressions for accumulating the mutable aggregation buffer based on an input row.
-
retractExpressions
public abstract Expression[] retractExpressions()
Expressions for retracting the mutable aggregation buffer based on an input row.
-
mergeExpressions
public abstract Expression[] mergeExpressions()
A sequence of expressions for merging two aggregation buffers together. When defining these expressions, you can use the syntaxattributeName
andmergeOperand(attributeName)
to refer to the attributes corresponding to each of the buffers being merged.
-
getValueExpression
public abstract Expression getValueExpression()
An expression which returns the final value for this aggregate function.
-
operands
public final UnresolvedReferenceExpression[] operands()
Args of accumulate and retract, the input value (usually obtained from a new arrived data).
-
operand
public final UnresolvedReferenceExpression operand(int i)
Arg of accumulate and retract, the input value (usually obtained from a new arrived data).
-
mergeOperand
public final UnresolvedReferenceExpression mergeOperand(UnresolvedReferenceExpression aggBuffer)
Merge input ofmergeExpressions()
, the input are AGG buffer generated by user definition.
-
mergeOperands
public final UnresolvedReferenceExpression[] mergeOperands()
Merge inputs ofmergeExpressions()
, these inputs are agg buffer generated by user definition.
-
getKind
public FunctionKind getKind()
Description copied from interface:FunctionDefinition
Returns the kind of function this definition describes.
-
getTypeInference
public TypeInference getTypeInference(DataTypeFactory factory)
Description copied from class:UserDefinedFunction
Returns the logic for performing type inference of a call to this function definition.The type inference process is responsible for inferring unknown types of input arguments, validating input arguments, and producing result types. The type inference process happens independent of a function body. The output of the type inference is used to search for a corresponding runtime implementation.
Instances of type inference can be created by using
TypeInference.newBuilder()
.See
BuiltInFunctionDefinitions
for concrete usage examples.The type inference for user-defined functions is automatically extracted using reflection. It does this by analyzing implementation methods such as
eval() or accumulate()
and the generic parameters of a function class if present. If the reflective information is not sufficient, it can be supported and enriched withDataTypeHint
andFunctionHint
annotations.Note: Overriding this method is only recommended for advanced users. If a custom type inference is specified, it is the responsibility of the implementer to make sure that the output of the type inference process matches with the implementation method:
The implementation method must comply with each
DataType.getConversionClass()
returned by the type inference. For example, ifDataTypes.TIMESTAMP(3).bridgedTo(java.sql.Timestamp.class)
is an expected argument type, the method must accept a calleval(java.sql.Timestamp)
.Regular Java calling semantics (including type widening and autoboxing) are applied when calling an implementation method which means that the signature can be
eval(java.lang.Object)
.The runtime will take care of converting the data to the data format specified by the
DataType.getConversionClass()
coming from the type inference logic.- Specified by:
getTypeInference
in interfaceFunctionDefinition
- Specified by:
getTypeInference
in classUserDefinedFunction
-
-