Skip to content

Commit

Permalink
refactor: remove stale AggKind::RowCount (#3774)
Browse files Browse the repository at this point in the history
* refactor: remove stale AggKind::RowCount

* update doc

Co-authored-by: mergify[bot] <37929162+mergify[bot]@users.noreply.github.com>
  • Loading branch information
xiangjinwu and mergify[bot] authored Jul 11, 2022
1 parent 2ee98b5 commit f93d8e3
Show file tree
Hide file tree
Showing 10 changed files with 10 additions and 40 deletions.
5 changes: 0 additions & 5 deletions src/expr/src/expr/agg.rs
Original file line number Diff line number Diff line change
Expand Up @@ -24,7 +24,6 @@ pub enum AggKind {
Max,
Sum,
Count,
RowCount,
Avg,
StringAgg,
SingleValue,
Expand All @@ -38,7 +37,6 @@ impl std::fmt::Display for AggKind {
AggKind::Max => write!(f, "max"),
AggKind::Sum => write!(f, "sum"),
AggKind::Count => write!(f, "count"),
AggKind::RowCount => write!(f, "row_count"),
AggKind::Avg => write!(f, "avg"),
AggKind::StringAgg => write!(f, "string_agg"),
AggKind::SingleValue => write!(f, "single_value"),
Expand Down Expand Up @@ -75,9 +73,6 @@ impl AggKind {
Self::Count => Type::Count,
Self::StringAgg => Type::StringAgg,
Self::SingleValue => Type::SingleValue,
Self::RowCount => {
panic!("cannot convert RowCount to prost, TODO: remove RowCount from AggKind")
}
Self::ApproxCountDistinct => Type::ApproxCountDistinct,
}
}
Expand Down
10 changes: 0 additions & 10 deletions src/frontend/src/expr/agg_call.rs
Original file line number Diff line number Diff line change
Expand Up @@ -52,13 +52,6 @@ impl AggCall {
/// Infer the return type for the given agg call.
/// Returns error if not supported or the arguments are invalid.
pub fn infer_return_type(agg_kind: &AggKind, inputs: &[DataType]) -> Result<DataType> {
let unsupported = || {
let args = inputs.iter().map(|t| format!("{:?}", t)).join(", ");
Err(RwError::from(ErrorCode::NotImplemented(
format!("Unsupported aggregation: {}({})", agg_kind, args),
112.into(),
)))
};
let invalid = || {
let args = inputs.iter().map(|t| format!("{:?}", t)).join(", ");
Err(RwError::from(ErrorCode::InvalidInputSyntax(format!(
Expand Down Expand Up @@ -107,9 +100,6 @@ impl AggCall {
// SingleValue
(AggKind::SingleValue, [input]) => input.clone(),
(AggKind::SingleValue, _) => return invalid(),

// Others
_ => return unsupported(),
};

Ok(return_type)
Expand Down
5 changes: 1 addition & 4 deletions src/frontend/src/optimizer/plan_node/logical_agg.rs
Original file line number Diff line number Diff line change
Expand Up @@ -103,9 +103,7 @@ impl PlanAggCall {
| AggKind::StringAgg
| AggKind::SingleValue => self.agg_kind.clone(),

AggKind::Count | AggKind::RowCount | AggKind::Sum | AggKind::ApproxCountDistinct => {
AggKind::Sum
}
AggKind::Count | AggKind::Sum | AggKind::ApproxCountDistinct => AggKind::Sum,
};
PlanAggCall {
agg_kind: total_agg_kind,
Expand Down Expand Up @@ -184,7 +182,6 @@ impl LogicalAgg {
}
AggKind::Sum
| AggKind::Count
| AggKind::RowCount
| AggKind::Avg
| AggKind::SingleValue
| AggKind::ApproxCountDistinct => {
Expand Down
2 changes: 1 addition & 1 deletion src/stream/src/executor/aggregation/agg_call.rs
Original file line number Diff line number Diff line change
Expand Up @@ -20,7 +20,7 @@ use risingwave_expr::expr::AggKind;
/// An aggregation function may accept 0, 1 or 2 arguments.
#[derive(Clone, Debug)]
pub enum AggArgs {
/// `None` is used for aggregation function accepts 0 arguments, such as [`AggKind::RowCount`].
/// `None` is used for aggregation function accepts 0 arguments, such as `count(*)`.
None,
/// `Unary` is used for aggregation function accepts 1 argument, such as [`AggKind::Sum`].
Unary(DataType, usize),
Expand Down
6 changes: 0 additions & 6 deletions src/stream/src/executor/aggregation/mod.rs
Original file line number Diff line number Diff line change
Expand Up @@ -269,12 +269,6 @@ pub fn create_streaming_agg_state(
}
[] => {
match (agg_type, return_type, datum) {
// `AggKind::Count` for partial/local Count(*) == RowCount while `AggKind::Sum` for
// final/global Count(*)
(AggKind::RowCount, DataType::Int64, Some(datum)) => {
Box::new(StreamingRowCountAgg::with_row_cnt(datum))
}
(AggKind::RowCount, DataType::Int64, None) => Box::new(StreamingRowCountAgg::new()),
// According to the function header comments and the link, Count(*) == RowCount
// `StreamingCountAgg` does not count `NULL`, so we use `StreamingRowCountAgg` here.
(AggKind::Count, DataType::Int64, Some(datum)) => {
Expand Down
2 changes: 1 addition & 1 deletion src/stream/src/executor/global_simple_agg.rs
Original file line number Diff line number Diff line change
Expand Up @@ -321,7 +321,7 @@ mod tests {
let append_only = false;
let agg_calls = vec![
AggCall {
kind: AggKind::RowCount,
kind: AggKind::Count,
args: AggArgs::None,
return_type: DataType::Int64,
append_only,
Expand Down
8 changes: 4 additions & 4 deletions src/stream/src/executor/hash_agg.rs
Original file line number Diff line number Diff line change
Expand Up @@ -573,7 +573,7 @@ mod tests {
let append_only = false;
let agg_calls = vec![
AggCall {
kind: AggKind::RowCount,
kind: AggKind::Count,
args: AggArgs::None,
return_type: DataType::Int64,
append_only,
Expand Down Expand Up @@ -660,7 +660,7 @@ mod tests {
let append_only = false;
let agg_calls = vec![
AggCall {
kind: AggKind::RowCount,
kind: AggKind::Count,
args: AggArgs::None,
return_type: DataType::Int64,
append_only,
Expand Down Expand Up @@ -755,7 +755,7 @@ mod tests {
let keys = vec![0];
let agg_calls = vec![
AggCall {
kind: AggKind::RowCount,
kind: AggKind::Count,
args: AggArgs::None,
return_type: DataType::Int64,
append_only: false,
Expand Down Expand Up @@ -844,7 +844,7 @@ mod tests {
let append_only = true;
let agg_calls = vec![
AggCall {
kind: AggKind::RowCount,
kind: AggKind::Count,
args: AggArgs::None,
return_type: DataType::Int64,
append_only,
Expand Down
2 changes: 1 addition & 1 deletion src/stream/src/executor/integration_tests.rs
Original file line number Diff line number Diff line change
Expand Up @@ -60,7 +60,7 @@ async fn test_merger_sum_aggr() {
input.boxed(),
vec![
AggCall {
kind: AggKind::RowCount,
kind: AggKind::Count,
args: AggArgs::None,
return_type: DataType::Int64,
append_only,
Expand Down
4 changes: 2 additions & 2 deletions src/stream/src/executor/local_simple_agg.rs
Original file line number Diff line number Diff line change
Expand Up @@ -183,7 +183,7 @@ mod tests {
tx.push_barrier(3, false);

let agg_calls = vec![AggCall {
kind: AggKind::RowCount,
kind: AggKind::Count,
args: AggArgs::None,
return_type: DataType::Int64,
append_only: false,
Expand Down Expand Up @@ -237,7 +237,7 @@ mod tests {
// This is local simple aggregation, so we add another row count state
let agg_calls = vec![
AggCall {
kind: AggKind::RowCount,
kind: AggKind::Count,
args: AggArgs::None,
return_type: DataType::Int64,
append_only: false,
Expand Down
6 changes: 0 additions & 6 deletions src/stream/src/executor/managed_state/aggregation/mod.rs
Original file line number Diff line number Diff line change
Expand Up @@ -164,12 +164,6 @@ impl<S: StateStore> ManagedStateImpl<S> {
ManagedValueState::new(agg_call, row_count, pk, state_table).await?,
))
}
AggKind::RowCount => {
assert!(is_row_count);
Ok(Self::Value(
ManagedValueState::new(agg_call, row_count, pk, state_table).await?,
))
}
AggKind::SingleValue => Ok(Self::Value(
ManagedValueState::new(agg_call, row_count, pk, state_table).await?,
)),
Expand Down

0 comments on commit f93d8e3

Please sign in to comment.