risingwave_meta_service/
lib.rs

1// Copyright 2025 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
15#![feature(let_chains)]
16#![feature(impl_trait_in_assoc_type)]
17#![cfg_attr(coverage, feature(coverage_attribute))]
18
19use risingwave_meta::*;
20
21pub mod backup_service;
22pub mod cloud_service;
23pub mod cluster_limit_service;
24pub mod cluster_service;
25pub mod ddl_service;
26pub mod event_log_service;
27pub mod health_service;
28pub mod heartbeat_service;
29pub mod hummock_service;
30pub mod meta_member_service;
31pub mod notification_service;
32pub mod scale_service;
33pub mod serving_service;
34pub mod session_config;
35pub mod sink_coordination_service;
36pub mod stream_service;
37pub mod system_params_service;
38pub mod telemetry_service;
39pub mod user_service;
40
41use std::pin::Pin;
42use std::task::{Context, Poll};
43
44use futures::Stream;
45use tokio::sync::mpsc::UnboundedReceiver;
46
47use crate::MetaError;
48
49/// `RwReceiverStream` is a wrapper around `tokio::sync::mpsc::UnboundedReceiver` that implements
50/// Stream. `RwReceiverStream` is similar to `tokio_stream::wrappers::ReceiverStream`, but it
51/// maps Result<S, `MetaError`> to Result<S, `tonic::Status`>.
52pub struct RwReceiverStream<S> {
53    inner: UnboundedReceiver<Result<S, MetaError>>,
54}
55
56impl<S> RwReceiverStream<S> {
57    pub fn new(inner: UnboundedReceiver<Result<S, MetaError>>) -> Self {
58        Self { inner }
59    }
60}
61
62impl<S> Stream for RwReceiverStream<S> {
63    type Item = Result<S, tonic::Status>;
64
65    fn poll_next(mut self: Pin<&mut Self>, cx: &mut Context<'_>) -> Poll<Option<Self::Item>> {
66        self.inner
67            .poll_recv(cx)
68            .map(|opt| opt.map(|res| res.map_err(Into::into)))
69    }
70}
71
72use std::net::SocketAddr;
73
74#[derive(Clone)]
75pub struct AddressInfo {
76    pub advertise_addr: String,
77    pub listen_addr: SocketAddr,
78    pub prometheus_addr: Option<SocketAddr>,
79    pub dashboard_addr: Option<SocketAddr>,
80}
81impl Default for AddressInfo {
82    fn default() -> Self {
83        Self {
84            advertise_addr: "".to_owned(),
85            listen_addr: SocketAddr::V4("127.0.0.1:0000".parse().unwrap()),
86            prometheus_addr: None,
87            dashboard_addr: None,
88        }
89    }
90}