risingwave_rpc_client/
hummock_meta_client.rs1use async_trait::async_trait;
16use futures::stream::BoxStream;
17use risingwave_hummock_sdk::version::HummockVersion;
18use risingwave_hummock_sdk::{HummockEpoch, HummockVersionId, ObjectIdRange, SyncResult};
19use risingwave_pb::hummock::{
20 PbHummockVersion, SubscribeCompactionEventRequest, SubscribeCompactionEventResponse,
21};
22use risingwave_pb::iceberg_compaction::{
23 SubscribeIcebergCompactionEventRequest, SubscribeIcebergCompactionEventResponse,
24};
25use risingwave_pb::id::{JobId, TableId};
26use tokio::sync::mpsc::UnboundedSender;
27
28pub type CompactionEventItem = std::result::Result<SubscribeCompactionEventResponse, tonic::Status>;
29pub type IcebergCompactionEventItem =
30 std::result::Result<SubscribeIcebergCompactionEventResponse, tonic::Status>;
31
32use crate::error::Result;
33
34pub type HummockMetaClientChangeLogInfo = Vec<u64>;
35
36#[async_trait]
37pub trait HummockMetaClient: Send + Sync + 'static {
38 async fn unpin_version_before(&self, unpin_version_before: HummockVersionId) -> Result<()>;
39 async fn get_current_version(&self) -> Result<HummockVersion>;
40 async fn get_new_object_ids(&self, number: u32) -> Result<ObjectIdRange>;
41 async fn commit_epoch_with_change_log(
43 &self,
44 epoch: HummockEpoch,
45 sync_result: SyncResult,
46 change_log_info: Option<HummockMetaClientChangeLogInfo>,
47 ) -> Result<()>;
48 async fn commit_epoch(&self, epoch: HummockEpoch, sync_result: SyncResult) -> Result<()> {
49 self.commit_epoch_with_change_log(epoch, sync_result, None)
50 .await
51 }
52 async fn trigger_manual_compaction(
53 &self,
54 compaction_group_id: u64,
55 table_id: JobId,
56 level: u32,
57 sst_ids: Vec<u64>,
58 ) -> Result<()>;
59 async fn trigger_full_gc(
60 &self,
61 sst_retention_time_sec: u64,
62 prefix: Option<String>,
63 ) -> Result<()>;
64
65 async fn subscribe_compaction_event(
66 &self,
67 ) -> Result<(
68 UnboundedSender<SubscribeCompactionEventRequest>,
69 BoxStream<'static, CompactionEventItem>,
70 )>;
71
72 async fn get_version_by_epoch(
73 &self,
74 epoch: HummockEpoch,
75 table_id: TableId,
76 ) -> Result<PbHummockVersion>;
77
78 async fn subscribe_iceberg_compaction_event(
79 &self,
80 ) -> Result<(
81 UnboundedSender<SubscribeIcebergCompactionEventRequest>,
82 BoxStream<'static, IcebergCompactionEventItem>,
83 )>;
84}