pub trait SinkConnector: Send {
Show 13 methods
// Required methods
fn open<'life0, 'life1, 'async_trait>(
&'life0 mut self,
config: &'life1 ConnectorConfig,
) -> Pin<Box<dyn Future<Output = Result<(), ConnectorError>> + Send + 'async_trait>>
where Self: 'async_trait,
'life0: 'async_trait,
'life1: 'async_trait;
fn write_batch<'life0, 'life1, 'async_trait>(
&'life0 mut self,
batch: &'life1 RecordBatch,
) -> Pin<Box<dyn Future<Output = Result<WriteResult, ConnectorError>> + Send + 'async_trait>>
where Self: 'async_trait,
'life0: 'async_trait,
'life1: 'async_trait;
fn schema(&self) -> SchemaRef;
fn capabilities(&self) -> SinkConnectorCapabilities;
fn close<'life0, 'async_trait>(
&'life0 mut self,
) -> Pin<Box<dyn Future<Output = Result<(), ConnectorError>> + Send + 'async_trait>>
where Self: 'async_trait,
'life0: 'async_trait;
// Provided methods
fn begin_epoch<'life0, 'async_trait>(
&'life0 mut self,
_epoch: u64,
) -> Pin<Box<dyn Future<Output = Result<(), ConnectorError>> + Send + 'async_trait>>
where Self: 'async_trait,
'life0: 'async_trait { ... }
fn pre_commit<'life0, 'async_trait>(
&'life0 mut self,
_epoch: u64,
) -> Pin<Box<dyn Future<Output = Result<(), ConnectorError>> + Send + 'async_trait>>
where Self: 'async_trait,
'life0: 'async_trait { ... }
fn commit_epoch<'life0, 'async_trait>(
&'life0 mut self,
_epoch: u64,
) -> Pin<Box<dyn Future<Output = Result<(), ConnectorError>> + Send + 'async_trait>>
where Self: 'async_trait,
'life0: 'async_trait { ... }
fn rollback_epoch<'life0, 'async_trait>(
&'life0 mut self,
_epoch: u64,
) -> Pin<Box<dyn Future<Output = Result<(), ConnectorError>> + Send + 'async_trait>>
where Self: 'async_trait,
'life0: 'async_trait { ... }
fn health_check(&self) -> HealthStatus { ... }
fn metrics(&self) -> ConnectorMetrics { ... }
fn flush<'life0, 'async_trait>(
&'life0 mut self,
) -> Pin<Box<dyn Future<Output = Result<(), ConnectorError>> + Send + 'async_trait>>
where Self: 'async_trait,
'life0: 'async_trait { ... }
fn as_schema_registry_aware(&self) -> Option<&dyn SchemaRegistryAware> { ... }
}Expand description
Trait for sink connectors that write data to external systems.
Sink connectors operate in Ring 1, receiving data from Ring 0 and
writing to external systems. Implementations that advertise
exactly_once also implement begin_epoch/pre_commit/commit_epoch/
rollback_epoch; the runtime drives them via the checkpoint coordinator.
Lifecycle: open() → loop over epochs of begin_epoch(),
write_batch()*, pre_commit(), commit_epoch() (or rollback_epoch()
on failure) → close().
Required Methods§
Sourcefn open<'life0, 'life1, 'async_trait>(
&'life0 mut self,
config: &'life1 ConnectorConfig,
) -> Pin<Box<dyn Future<Output = Result<(), ConnectorError>> + Send + 'async_trait>>where
Self: 'async_trait,
'life0: 'async_trait,
'life1: 'async_trait,
fn open<'life0, 'life1, 'async_trait>(
&'life0 mut self,
config: &'life1 ConnectorConfig,
) -> Pin<Box<dyn Future<Output = Result<(), ConnectorError>> + Send + 'async_trait>>where
Self: 'async_trait,
'life0: 'async_trait,
'life1: 'async_trait,
Open the connection and prepare to accept writes.
Sourcefn write_batch<'life0, 'life1, 'async_trait>(
&'life0 mut self,
batch: &'life1 RecordBatch,
) -> Pin<Box<dyn Future<Output = Result<WriteResult, ConnectorError>> + Send + 'async_trait>>where
Self: 'async_trait,
'life0: 'async_trait,
'life1: 'async_trait,
fn write_batch<'life0, 'life1, 'async_trait>(
&'life0 mut self,
batch: &'life1 RecordBatch,
) -> Pin<Box<dyn Future<Output = Result<WriteResult, ConnectorError>> + Send + 'async_trait>>where
Self: 'async_trait,
'life0: 'async_trait,
'life1: 'async_trait,
Must be cancellation-safe: the runtime wraps this in
tokio::time::timeout. Don’t split a &mut self mutation
across an .await. In-flight transactional state may remain open
after cancellation; the caller will rollback_epoch it.
Sourcefn capabilities(&self) -> SinkConnectorCapabilities
fn capabilities(&self) -> SinkConnectorCapabilities
Required (no default) so every implementation declares
suggested_write_timeout.
Provided Methods§
Sourcefn begin_epoch<'life0, 'async_trait>(
&'life0 mut self,
_epoch: u64,
) -> Pin<Box<dyn Future<Output = Result<(), ConnectorError>> + Send + 'async_trait>>where
Self: 'async_trait,
'life0: 'async_trait,
fn begin_epoch<'life0, 'async_trait>(
&'life0 mut self,
_epoch: u64,
) -> Pin<Box<dyn Future<Output = Result<(), ConnectorError>> + Send + 'async_trait>>where
Self: 'async_trait,
'life0: 'async_trait,
Default: no-op (at-least-once semantics).
Sourcefn pre_commit<'life0, 'async_trait>(
&'life0 mut self,
_epoch: u64,
) -> Pin<Box<dyn Future<Output = Result<(), ConnectorError>> + Send + 'async_trait>>where
Self: 'async_trait,
'life0: 'async_trait,
fn pre_commit<'life0, 'async_trait>(
&'life0 mut self,
_epoch: u64,
) -> Pin<Box<dyn Future<Output = Result<(), ConnectorError>> + Send + 'async_trait>>where
Self: 'async_trait,
'life0: 'async_trait,
Phase 1 of 2PC: flush + prepare, do NOT finalize the txn. The
runtime persists the manifest between pre_commit and
commit_epoch; on failure it calls rollback_epoch. Default
delegates to flush().
Sourcefn commit_epoch<'life0, 'async_trait>(
&'life0 mut self,
_epoch: u64,
) -> Pin<Box<dyn Future<Output = Result<(), ConnectorError>> + Send + 'async_trait>>where
Self: 'async_trait,
'life0: 'async_trait,
fn commit_epoch<'life0, 'async_trait>(
&'life0 mut self,
_epoch: u64,
) -> Pin<Box<dyn Future<Output = Result<(), ConnectorError>> + Send + 'async_trait>>where
Self: 'async_trait,
'life0: 'async_trait,
Phase 2 of 2PC: finalize the txn. Called after the manifest is durable. Default: no-op (at-least-once semantics).
Sourcefn rollback_epoch<'life0, 'async_trait>(
&'life0 mut self,
_epoch: u64,
) -> Pin<Box<dyn Future<Output = Result<(), ConnectorError>> + Send + 'async_trait>>where
Self: 'async_trait,
'life0: 'async_trait,
fn rollback_epoch<'life0, 'async_trait>(
&'life0 mut self,
_epoch: u64,
) -> Pin<Box<dyn Future<Output = Result<(), ConnectorError>> + Send + 'async_trait>>where
Self: 'async_trait,
'life0: 'async_trait,
Must be idempotent: the runtime calls this on every exactly-once
sink after a pre_commit failure, including sinks that never
pre_commited.
Sourcefn health_check(&self) -> HealthStatus
fn health_check(&self) -> HealthStatus
Defaults to Unknown; connectors should override.
Sourcefn metrics(&self) -> ConnectorMetrics
fn metrics(&self) -> ConnectorMetrics
Current sink metrics snapshot.
Sourcefn flush<'life0, 'async_trait>(
&'life0 mut self,
) -> Pin<Box<dyn Future<Output = Result<(), ConnectorError>> + Send + 'async_trait>>where
Self: 'async_trait,
'life0: 'async_trait,
fn flush<'life0, 'async_trait>(
&'life0 mut self,
) -> Pin<Box<dyn Future<Output = Result<(), ConnectorError>> + Send + 'async_trait>>where
Self: 'async_trait,
'life0: 'async_trait,
Must be internally bounded — the sink task’s periodic timer
calls this on every tick. Thorough drains belong in pre_commit
/ commit_epoch / close, not here.
Sourcefn as_schema_registry_aware(&self) -> Option<&dyn SchemaRegistryAware>
fn as_schema_registry_aware(&self) -> Option<&dyn SchemaRegistryAware>
Return a SchemaRegistryAware
view, if the sink speaks a schema registry protocol.