-
Notifications
You must be signed in to change notification settings - Fork 5.8k
New issue
Have a question about this project? Sign up for a free GitHub account to open an issue and contact its maintainers and the community.
By clicking “Sign up for GitHub”, you agree to our terms of service and privacy statement. We’ll occasionally send you account related emails.
Already on GitHub? Sign in to your account
executor: introduce a new execution framework for aggregate functions #6852
Changes from 6 commits
182658b
01ec369
faa3f7d
1ebf4d5
038135a
7f1fd3a
e5acefb
e25d273
dca132a
81c812b
cacad7c
d3a1ced
cf34a16
b1335c3
ce821f9
dbbef24
57f57b8
5f17822
aa71782
f3b005f
c7f7144
66801be
0d859db
7f8134c
File filter
Filter by extension
Conversations
Jump to
Diff view
Diff view
There are no files selected for viewing
Original file line number | Diff line number | Diff line change |
---|---|---|
@@ -0,0 +1,61 @@ | ||
// Copyright 2018 PingCAP, Inc. | ||
// | ||
// Licensed under the Apache License, Version 2.0 (the "License"); | ||
// you may not use this file except in compliance with the License. | ||
// You may obtain a copy of the License at | ||
// | ||
// http://www.apache.org/licenses/LICENSE-2.0 | ||
// | ||
// Unless required by applicable law or agreed to in writing, software | ||
// distributed under the License is distributed on an "AS IS" BASIS, | ||
// See the License for the specific language governing permissions and | ||
// limitations under the License. | ||
|
||
package aggfuncs | ||
|
||
import ( | ||
"github.com/pingcap/tidb/expression" | ||
"github.com/pingcap/tidb/sessionctx" | ||
"github.com/pingcap/tidb/util/chunk" | ||
) | ||
|
||
// All the AggFunc implementations are listed here for navigation. | ||
var ( | ||
// All the AggFunc implementations for "COUNT". | ||
// All the AggFunc implementations for "SUM". | ||
// All the AggFunc implementations for "AVG". | ||
_ AggFunc = (*avgDedup4Decimal)(nil) | ||
_ AggFunc = (*avgOriginal4Decimal)(nil) | ||
_ AggFunc = (*avgPartial4Decimal)(nil) | ||
|
||
_ AggFunc = (*avgDedup4Float64)(nil) | ||
_ AggFunc = (*avgOriginal4Float64)(nil) | ||
_ AggFunc = (*avgPartial4Float64)(nil) | ||
|
||
_ AggFunc = (*avgDedup4Float32)(nil) | ||
_ AggFunc = (*avgOriginal4Float32)(nil) | ||
_ AggFunc = (*avgPartial4Float32)(nil) | ||
|
||
// All the AggFunc implementations for "FIRSTROW". | ||
// All the AggFunc implementations for "MAX". | ||
// All the AggFunc implementations for "MIN". | ||
// All the AggFunc implementations for "GROUP_CONCAT". | ||
// All the AggFunc implementations for "BIT_OR". | ||
// All the AggFunc implementations for "BIT_XOR". | ||
// All the AggFunc implementations for "BIT_AND". | ||
) | ||
|
||
// AggFunc is the interface to evaluate aggregate functions. | ||
type AggFunc interface { | ||
AllocPartialResult() []byte | ||
ResetPartialResult(partialBytes []byte) | ||
UpdatePartialResult(sctx sessionctx.Context, rowsInGroup []chunk.Row, partialBytes []byte) error | ||
|
||
AppendPartialResult2Chunk(sctx sessionctx.Context, partialBytes []byte, chk *chunk.Chunk) error | ||
AppendFinalResult2Chunk(sctx sessionctx.Context, partialBytes []byte, chk *chunk.Chunk) error | ||
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. s/ AppendFinalResult2Chunk/ GetFinalResult There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. I prefer the original name, which indicates the result is appended to the output chunk |
||
} | ||
|
||
type baseAggFunc struct { | ||
input []expression.Expression | ||
output []int | ||
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. add comments for these two args. |
||
} |
Original file line number | Diff line number | Diff line change |
---|---|---|
@@ -0,0 +1,141 @@ | ||
// Copyright 2018 PingCAP, Inc. | ||
// | ||
// Licensed under the Apache License, Version 2.0 (the "License"); | ||
// you may not use this file except in compliance with the License. | ||
// You may obtain a copy of the License at | ||
// | ||
// http://www.apache.org/licenses/LICENSE-2.0 | ||
// | ||
// Unless required by applicable law or agreed to in writing, software | ||
// distributed under the License is distributed on an "AS IS" BASIS, | ||
// See the License for the specific language governing permissions and | ||
// limitations under the License. | ||
|
||
package aggfuncs | ||
|
||
import ( | ||
"github.com/pingcap/tidb/ast" | ||
"github.com/pingcap/tidb/expression/aggregation" | ||
"github.com/pingcap/tidb/mysql" | ||
"github.com/pingcap/tidb/types" | ||
) | ||
|
||
// Build is used to build AggFunc according to the aggFuncDesc | ||
func Build(aggFuncDesc *aggregation.AggFuncDesc, output []int) AggFunc { | ||
switch aggFuncDesc.Name { | ||
case ast.AggFuncCount: | ||
return buildCount(aggFuncDesc, output) | ||
case ast.AggFuncSum: | ||
return buildSum(aggFuncDesc, output) | ||
case ast.AggFuncAvg: | ||
return buildAvg(aggFuncDesc, output) | ||
case ast.AggFuncFirstRow: | ||
return buildFirstRow(aggFuncDesc, output) | ||
case ast.AggFuncMax: | ||
return buildMax(aggFuncDesc, output) | ||
case ast.AggFuncMin: | ||
return buildMin(aggFuncDesc, output) | ||
case ast.AggFuncGroupConcat: | ||
return buildGroupConcat(aggFuncDesc, output) | ||
case ast.AggFuncBitOr: | ||
return buildBitOr(aggFuncDesc, output) | ||
case ast.AggFuncBitXor: | ||
return buildBitXor(aggFuncDesc, output) | ||
case ast.AggFuncBitAnd: | ||
return buildBitAnd(aggFuncDesc, output) | ||
} | ||
return nil | ||
} | ||
|
||
func buildCount(aggFuncDesc *aggregation.AggFuncDesc, output []int) AggFunc { | ||
return nil | ||
} | ||
|
||
func buildSum(aggFuncDesc *aggregation.AggFuncDesc, output []int) AggFunc { | ||
return nil | ||
} | ||
|
||
func buildAvg(aggFuncDesc *aggregation.AggFuncDesc, output []int) AggFunc { | ||
base := baseAggFunc{ | ||
input: aggFuncDesc.Args, | ||
output: output, | ||
} | ||
|
||
switch aggFuncDesc.Mode { | ||
// Build avg functions which consume the original data and remove the | ||
// duplicated input of the same group. | ||
case aggregation.DedupMode: | ||
return nil | ||
// TODO: implement UpdatePartialResult for the following structs. | ||
// switch aggFuncDesc.Args[0].GetType().Tp { | ||
// case mysql.TypeDecimal: | ||
// return &avgDedup4Decimal{baseAvgDecimal{base}} | ||
// case mysql.TypeFloat: | ||
// return &avgDedup4Float32{baseAvgFloat32{base}} | ||
// case mysql.TypeDouble: | ||
// return &avgDedup4Float64{baseAvgFloat64{base}} | ||
// } | ||
|
||
// Build avg functions which consume the original data and update their | ||
// partial results. | ||
case aggregation.CompleteMode, aggregation.Partial1Mode: | ||
switch aggFuncDesc.Args[0].GetType().Tp { | ||
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. we should consider all the input types, |
||
case mysql.TypeNewDecimal: | ||
if aggFuncDesc.HasDistinct { | ||
return &avgOriginal4Decimal{baseAvgDecimal{base}, make(map[types.MyDecimal]bool)} | ||
} | ||
return &avgOriginal4Decimal{baseAvgDecimal{base}, nil} | ||
case mysql.TypeFloat: | ||
if aggFuncDesc.HasDistinct { | ||
return &avgOriginal4Float32{baseAvgFloat32{base}, make(map[float32]bool)} | ||
} | ||
return &avgOriginal4Float32{baseAvgFloat32{base}, nil} | ||
case mysql.TypeDouble: | ||
if aggFuncDesc.HasDistinct { | ||
return &avgOriginal4Float64{baseAvgFloat64{base}, make(map[float64]bool)} | ||
} | ||
return &avgOriginal4Float64{baseAvgFloat64{base}, nil} | ||
} | ||
|
||
// Build avg functions which consume the partial result of other agg | ||
// functions and update their partial results. | ||
case aggregation.Partial2Mode, aggregation.FinalMode: | ||
switch aggFuncDesc.Args[1].GetType().Tp { | ||
case mysql.TypeNewDecimal: | ||
return &avgPartial4Decimal{baseAvgDecimal{base}} | ||
case mysql.TypeFloat: | ||
return &avgPartial4Float32{baseAvgFloat32{base}} | ||
case mysql.TypeDouble: | ||
return &avgPartial4Float64{baseAvgFloat64{base}} | ||
} | ||
} | ||
return nil | ||
} | ||
|
||
func buildFirstRow(aggFuncDesc *aggregation.AggFuncDesc, output []int) AggFunc { | ||
return nil | ||
} | ||
|
||
func buildMax(aggFuncDesc *aggregation.AggFuncDesc, output []int) AggFunc { | ||
return nil | ||
} | ||
|
||
func buildMin(aggFuncDesc *aggregation.AggFuncDesc, output []int) AggFunc { | ||
return nil | ||
} | ||
|
||
func buildGroupConcat(aggFuncDesc *aggregation.AggFuncDesc, output []int) AggFunc { | ||
return nil | ||
} | ||
|
||
func buildBitOr(aggFuncDesc *aggregation.AggFuncDesc, output []int) AggFunc { | ||
return nil | ||
} | ||
|
||
func buildBitXor(aggFuncDesc *aggregation.AggFuncDesc, output []int) AggFunc { | ||
return nil | ||
} | ||
|
||
func buildBitAnd(aggFuncDesc *aggregation.AggFuncDesc, output []int) AggFunc { | ||
return nil | ||
} |
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
It seems that we need another struct which contains partialResultBytes to handle the hashagg evaluation and aggfunc with distinct?
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
no need for now, we can just add a
map
field in a specific aggregate function implementation, during the execution ofUpdatePartialResult
we use that map to deduplicate the input, whenResetPartialResult
, we reset that map.