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