Trait sea_streamer::Streamer

source ·
pub trait Streamer: Sized {
    type Error: Error;
    type Producer: Producer<Error = Self::Error>;
    type Consumer: Consumer<Error = Self::Error>;
    type ConnectOptions: ConnectOptions;
    type ConsumerOptions: ConsumerOptions;
    type ProducerOptions: ProducerOptions;

    // Required methods
    fn connect(
        streamer: StreamerUri,
        options: Self::ConnectOptions
    ) -> impl Future<Output = Result<Self, StreamErr<Self::Error>>> + Send;
    fn disconnect(
        self
    ) -> impl Future<Output = Result<(), StreamErr<Self::Error>>> + Send;
    fn create_generic_producer(
        &self,
        options: Self::ProducerOptions
    ) -> impl Future<Output = Result<Self::Producer, StreamErr<Self::Error>>> + Send;
    fn create_consumer(
        &self,
        streams: &[StreamKey],
        options: Self::ConsumerOptions
    ) -> impl Future<Output = Result<Self::Consumer, StreamErr<Self::Error>>> + Send;

    // Provided method
    fn create_producer(
        &self,
        stream: StreamKey,
        options: Self::ProducerOptions
    ) -> impl Future<Output = Result<Self::Producer, StreamErr<Self::Error>>> + Send { ... }
}
Expand description

Common interface of streamer clients.

Required Associated Types§

Required Methods§

source

fn connect( streamer: StreamerUri, options: Self::ConnectOptions ) -> impl Future<Output = Result<Self, StreamErr<Self::Error>>> + Send

Establish a connection to the streaming server.

source

fn disconnect( self ) -> impl Future<Output = Result<(), StreamErr<Self::Error>>> + Send

Flush and disconnect from the streaming server.

source

fn create_generic_producer( &self, options: Self::ProducerOptions ) -> impl Future<Output = Result<Self::Producer, StreamErr<Self::Error>>> + Send

Create a producer that can stream to any stream key.

source

fn create_consumer( &self, streams: &[StreamKey], options: Self::ConsumerOptions ) -> impl Future<Output = Result<Self::Consumer, StreamErr<Self::Error>>> + Send

Create a consumer subscribing to the specified streams.

Provided Methods§

source

fn create_producer( &self, stream: StreamKey, options: Self::ProducerOptions ) -> impl Future<Output = Result<Self::Producer, StreamErr<Self::Error>>> + Send

Create a producer that streams to the specified stream.

Object Safety§

This trait is not object safe.

Implementations on Foreign Types§

source§

impl Streamer for KafkaStreamer

source§

async fn disconnect(self) -> Result<(), StreamErr<KafkaError>>

It will flush all producers

source§

async fn create_consumer( &self, streams: &[StreamKey], options: <KafkaStreamer as Streamer>::ConsumerOptions ) -> Result<<KafkaStreamer as Streamer>::Consumer, StreamErr<KafkaError>>

If ConsumerMode is RealTime, auto commit will be disabled and it always stream from latest. group_id must not be set.

If ConsumerMode is Resumable, it will use a group id unique to this host: on a physical machine, it will use the mac address. Inside a docker container, it will use the container id. So when the process restarts, it will resume from last committed offset. group_id must not be set.

If ConsumerMode is LoadBalanced, shards will be divided-and-assigned by the broker to consumers sharing the same group_id. group_id must already be set.

If you need to override the HOST ID, you can set the ENV var HOST_ID.

§

type Error = KafkaError

§

type Producer = KafkaProducer

§

type Consumer = KafkaConsumer

§

type ConnectOptions = KafkaConnectOptions

§

type ConsumerOptions = KafkaConsumerOptions

§

type ProducerOptions = KafkaProducerOptions

source§

async fn connect( uri: StreamerUri, options: <KafkaStreamer as Streamer>::ConnectOptions ) -> Result<KafkaStreamer, StreamErr<KafkaError>>

source§

async fn create_generic_producer( &self, options: <KafkaStreamer as Streamer>::ProducerOptions ) -> Result<<KafkaStreamer as Streamer>::Producer, StreamErr<KafkaError>>

source§

impl Streamer for StdioStreamer

source§

async fn connect( _: StreamerUri, options: <StdioStreamer as Streamer>::ConnectOptions ) -> Result<StdioStreamer, StreamErr<StdioErr>>

Nothing will happen until you create a producer/consumer

source§

async fn disconnect(self) -> Result<(), StreamErr<StdioErr>>

Call this method if you want to exit gracefully. This waits asynchronously until all pending messages are sent.

The side effects is global: all existing consumers and producers will become unusable, until you connect again.

source§

async fn create_consumer( &self, streams: &[StreamKey], options: <StdioStreamer as Streamer>::ConsumerOptions ) -> Result<<StdioStreamer as Streamer>::Consumer, StreamErr<StdioErr>>

A background thread will be spawned to read stdin dedicatedly. It is safe to spawn multiple consumers.

§

type Error = StdioErr

§

type Producer = StdioProducer

§

type Consumer = StdioConsumer

§

type ConnectOptions = StdioConnectOptions

§

type ConsumerOptions = StdioConsumerOptions

§

type ProducerOptions = StdioProducerOptions

source§

async fn create_generic_producer( &self, _: <StdioStreamer as Streamer>::ProducerOptions ) -> Result<<StdioStreamer as Streamer>::Producer, StreamErr<StdioErr>>

Implementors§