risingwave_expr::aggregate

Trait AggregateFunction

source
pub trait AggregateFunction:
    Send
    + Sync
    + 'static {
    // Required methods
    fn return_type(&self) -> DataType;
    fn update<'life0, 'life1, 'life2, 'async_trait>(
        &'life0 self,
        state: &'life1 mut AggregateState,
        input: &'life2 StreamChunk,
    ) -> Pin<Box<dyn Future<Output = Result<()>> + Send + 'async_trait>>
       where Self: 'async_trait,
             'life0: 'async_trait,
             'life1: 'async_trait,
             'life2: 'async_trait;
    fn update_range<'life0, 'life1, 'life2, 'async_trait>(
        &'life0 self,
        state: &'life1 mut AggregateState,
        input: &'life2 StreamChunk,
        range: Range<usize>,
    ) -> Pin<Box<dyn Future<Output = Result<()>> + Send + 'async_trait>>
       where Self: 'async_trait,
             'life0: 'async_trait,
             'life1: 'async_trait,
             'life2: 'async_trait;
    fn get_result<'life0, 'life1, 'async_trait>(
        &'life0 self,
        state: &'life1 AggregateState,
    ) -> Pin<Box<dyn Future<Output = Result<Datum>> + Send + 'async_trait>>
       where Self: 'async_trait,
             'life0: 'async_trait,
             'life1: 'async_trait;

    // Provided methods
    fn create_state(&self) -> Result<AggregateState> { ... }
    fn encode_state(&self, state: &AggregateState) -> Result<Datum> { ... }
    fn decode_state(&self, datum: Datum) -> Result<AggregateState> { ... }
}
Expand description

A trait over all aggregate functions.

Required Methods§

source

fn return_type(&self) -> DataType

Returns the return type of the aggregate function.

source

fn update<'life0, 'life1, 'life2, 'async_trait>( &'life0 self, state: &'life1 mut AggregateState, input: &'life2 StreamChunk, ) -> Pin<Box<dyn Future<Output = Result<()>> + Send + 'async_trait>>
where Self: 'async_trait, 'life0: 'async_trait, 'life1: 'async_trait, 'life2: 'async_trait,

Update the state with multiple rows.

source

fn update_range<'life0, 'life1, 'life2, 'async_trait>( &'life0 self, state: &'life1 mut AggregateState, input: &'life2 StreamChunk, range: Range<usize>, ) -> Pin<Box<dyn Future<Output = Result<()>> + Send + 'async_trait>>
where Self: 'async_trait, 'life0: 'async_trait, 'life1: 'async_trait, 'life2: 'async_trait,

Update the state with a range of rows.

source

fn get_result<'life0, 'life1, 'async_trait>( &'life0 self, state: &'life1 AggregateState, ) -> Pin<Box<dyn Future<Output = Result<Datum>> + Send + 'async_trait>>
where Self: 'async_trait, 'life0: 'async_trait, 'life1: 'async_trait,

Get aggregate result from the state.

Provided Methods§

source

fn create_state(&self) -> Result<AggregateState>

Creates an initial state of the aggregate function.

source

fn encode_state(&self, state: &AggregateState) -> Result<Datum>

Encode the state into a datum that can be stored in state table.

source

fn decode_state(&self, datum: Datum) -> Result<AggregateState>

Decode the state from a datum in state table.

Implementors§