risingwave_stream/executor/
agg_common.rsuse std::collections::HashMap;
use risingwave_expr::aggregate::AggCall;
use risingwave_pb::stream_plan::PbAggNodeVersion;
use super::aggregation::AggStateStorage;
use crate::executor::prelude::*;
pub struct AggExecutorArgs<S: StateStore, E: AggExecutorExtraArgs> {
pub version: PbAggNodeVersion,
pub input: Executor,
pub actor_ctx: ActorContextRef,
pub info: ExecutorInfo,
pub extreme_cache_size: usize,
pub agg_calls: Vec<AggCall>,
pub row_count_index: usize,
pub storages: Vec<AggStateStorage<S>>,
pub intermediate_state_table: StateTable<S>,
pub distinct_dedup_tables: HashMap<usize, StateTable<S>>,
pub watermark_epoch: AtomicU64Ref,
pub extra: E,
}
pub trait AggExecutorExtraArgs {}
pub struct SimpleAggExecutorExtraArgs {
pub must_output_per_barrier: bool,
}
impl AggExecutorExtraArgs for SimpleAggExecutorExtraArgs {}
pub struct HashAggExecutorExtraArgs {
pub group_key_indices: Vec<usize>,
pub chunk_size: usize,
pub max_dirty_groups_heap_size: usize,
pub emit_on_window_close: bool,
}
impl AggExecutorExtraArgs for HashAggExecutorExtraArgs {}