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