risingwave_rpc_client/
hummock_meta_client.rs

1// Copyright 2022 RisingWave Labs
2//
3// Licensed under the Apache License, Version 2.0 (the "License");
4// you may not use this file except in compliance with the License.
5// You may obtain a copy of the License at
6//
7//     http://www.apache.org/licenses/LICENSE-2.0
8//
9// Unless required by applicable law or agreed to in writing, software
10// distributed under the License is distributed on an "AS IS" BASIS,
11// WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
12// See the License for the specific language governing permissions and
13// limitations under the License.
14
15use std::collections::HashSet;
16
17use async_trait::async_trait;
18use futures::stream::BoxStream;
19use risingwave_hummock_sdk::change_log::TableChangeLogs;
20use risingwave_hummock_sdk::version::HummockVersion;
21use risingwave_hummock_sdk::{
22    CompactionGroupId, HummockEpoch, HummockVersionId, ObjectIdRange, SyncResult,
23};
24use risingwave_pb::hummock::{
25    PbHummockVersion, SubscribeCompactionEventRequest, SubscribeCompactionEventResponse,
26};
27use risingwave_pb::iceberg_compaction::{
28    SubscribeIcebergCompactionEventRequest, SubscribeIcebergCompactionEventResponse,
29};
30use risingwave_pb::id::{HummockSstableId, JobId, TableId};
31use tokio::sync::mpsc::UnboundedSender;
32
33pub type CompactionEventItem = std::result::Result<SubscribeCompactionEventResponse, tonic::Status>;
34pub type IcebergCompactionEventItem =
35    std::result::Result<SubscribeIcebergCompactionEventResponse, tonic::Status>;
36
37use crate::error::Result;
38
39pub type HummockMetaClientChangeLogInfo = Vec<u64>;
40
41#[async_trait]
42pub trait HummockMetaClient: Send + Sync + 'static {
43    async fn unpin_version_before(&self, unpin_version_before: HummockVersionId) -> Result<()>;
44    async fn get_current_version(&self) -> Result<HummockVersion>;
45    async fn get_new_object_ids(&self, number: u32) -> Result<ObjectIdRange>;
46    // We keep `commit_epoch` only for test/benchmark.
47    async fn commit_epoch_with_change_log(
48        &self,
49        epoch: HummockEpoch,
50        sync_result: SyncResult,
51        change_log_info: Option<HummockMetaClientChangeLogInfo>,
52    ) -> Result<()>;
53    async fn commit_epoch(&self, epoch: HummockEpoch, sync_result: SyncResult) -> Result<()> {
54        self.commit_epoch_with_change_log(epoch, sync_result, None)
55            .await
56    }
57    async fn trigger_manual_compaction(
58        &self,
59        compaction_group_id: CompactionGroupId,
60        table_id: JobId,
61        level: u32,
62        sst_ids: Vec<HummockSstableId>,
63        exclusive: bool,
64    ) -> Result<bool>;
65    async fn trigger_full_gc(
66        &self,
67        sst_retention_time_sec: u64,
68        prefix: Option<String>,
69    ) -> Result<()>;
70
71    async fn subscribe_compaction_event(
72        &self,
73    ) -> Result<(
74        UnboundedSender<SubscribeCompactionEventRequest>,
75        BoxStream<'static, CompactionEventItem>,
76    )>;
77
78    async fn get_version_by_epoch(
79        &self,
80        epoch: HummockEpoch,
81        table_id: TableId,
82    ) -> Result<PbHummockVersion>;
83
84    async fn subscribe_iceberg_compaction_event(
85        &self,
86    ) -> Result<(
87        UnboundedSender<SubscribeIcebergCompactionEventRequest>,
88        BoxStream<'static, IcebergCompactionEventItem>,
89    )>;
90
91    async fn get_table_change_logs(
92        &self,
93        epoch_only: bool,
94        start_epoch_inclusive: Option<u64>,
95        end_epoch_inclusive: Option<u64>,
96        table_ids: Option<HashSet<TableId>>,
97        exclude_empty: bool,
98        limit: Option<u32>,
99    ) -> Result<TableChangeLogs>;
100}