risingwave_connector/
with_options.rs1use std::collections::{BTreeMap, HashMap};
16use std::marker::PhantomData;
17use std::time::Duration;
18
19use risingwave_pb::secret::PbSecretRef;
20
21use crate::sink::catalog::SinkFormatDesc;
22use crate::source::cdc::MYSQL_CDC_CONNECTOR;
23use crate::source::cdc::external::CdcTableType;
24use crate::source::iceberg::ICEBERG_CONNECTOR;
25use crate::source::{
26 AZBLOB_CONNECTOR, GCS_CONNECTOR, KAFKA_CONNECTOR, LEGACY_S3_CONNECTOR, OPENDAL_S3_CONNECTOR,
27 POSIX_FS_CONNECTOR, UPSTREAM_SOURCE_KEY,
28};
29
30pub trait WithOptions {
37 #[doc(hidden)]
38 #[inline(always)]
39 fn assert_receiver_is_with_options(&self) {}
40}
41
42impl<T: crate::source::cdc::CdcSourceTypeTrait> WithOptions
49 for crate::source::cdc::CdcProperties<T>
50{
51}
52
53impl<T: WithOptions> WithOptions for Option<T> {}
56impl WithOptions for Vec<String> {}
57impl WithOptions for Vec<u64> {}
58impl WithOptions for HashMap<String, String> {}
59impl WithOptions for BTreeMap<String, String> {}
60
61impl WithOptions for String {}
62impl WithOptions for bool {}
63impl WithOptions for usize {}
64impl WithOptions for u8 {}
65impl WithOptions for u16 {}
66impl WithOptions for u32 {}
67impl WithOptions for u64 {}
68impl WithOptions for i32 {}
69impl WithOptions for i64 {}
70impl WithOptions for f64 {}
71impl WithOptions for std::time::Duration {}
72impl WithOptions for crate::connector_common::MqttQualityOfService {}
73impl WithOptions for crate::sink::file_sink::opendal_sink::PathPartitionPrefix {}
74impl WithOptions for crate::sink::kafka::CompressionCodec {}
75impl WithOptions for crate::source::filesystem::file_common::CompressionFormat {}
76impl WithOptions for nexmark::config::RateShape {}
77impl WithOptions for nexmark::event::EventType {}
78impl<T> WithOptions for PhantomData<T> {}
79
80pub trait Get {
81 fn get(&self, key: &str) -> Option<&String>;
82}
83
84impl Get for HashMap<String, String> {
85 fn get(&self, key: &str) -> Option<&String> {
86 self.get(key)
87 }
88}
89
90impl Get for BTreeMap<String, String> {
91 fn get(&self, key: &str) -> Option<&String> {
92 self.get(key)
93 }
94}
95
96pub trait WithPropertiesExt: Get + Sized {
98 #[inline(always)]
99 fn get_connector(&self) -> Option<String> {
100 self.get(UPSTREAM_SOURCE_KEY).map(|s| s.to_lowercase())
101 }
102
103 #[inline(always)]
104 fn is_kafka_connector(&self) -> bool {
105 let Some(connector) = self.get_connector() else {
106 return false;
107 };
108 connector == KAFKA_CONNECTOR
109 }
110
111 #[inline(always)]
112 fn is_mysql_cdc_connector(&self) -> bool {
113 let Some(connector) = self.get_connector() else {
114 return false;
115 };
116 connector == MYSQL_CDC_CONNECTOR
117 }
118
119 #[inline(always)]
120 fn get_sync_call_timeout(&self) -> Option<Duration> {
121 const SYNC_CALL_TIMEOUT_KEY: &str = "properties.sync.call.timeout"; self.get(SYNC_CALL_TIMEOUT_KEY)
123 .and_then(|s| duration_str::parse_std(s).ok())
125 }
126
127 #[inline(always)]
128 fn is_cdc_connector(&self) -> bool {
129 let Some(connector) = self.get_connector() else {
130 return false;
131 };
132 connector.contains("-cdc")
133 }
134
135 fn is_shareable_cdc_connector(&self) -> bool {
137 self.is_cdc_connector() && CdcTableType::from_properties(self).can_backfill()
138 }
139
140 fn is_shareable_only_cdc_connector(&self) -> bool {
143 self.is_cdc_connector() && CdcTableType::from_properties(self).shareable_only()
144 }
145
146 fn enable_transaction_metadata(&self) -> bool {
147 CdcTableType::from_properties(self).enable_transaction_metadata()
148 }
149
150 fn is_shareable_non_cdc_connector(&self) -> bool {
151 self.is_kafka_connector()
152 }
153
154 #[inline(always)]
155 fn is_iceberg_connector(&self) -> bool {
156 let Some(connector) = self.get_connector() else {
157 return false;
158 };
159 connector == ICEBERG_CONNECTOR
160 }
161
162 fn connector_need_pk(&self) -> bool {
163 !self.is_iceberg_connector()
169 }
170
171 fn is_legacy_fs_connector(&self) -> bool {
172 self.get(UPSTREAM_SOURCE_KEY)
173 .map(|s| s.eq_ignore_ascii_case(LEGACY_S3_CONNECTOR))
174 .unwrap_or(false)
175 }
176
177 fn is_new_fs_connector(&self) -> bool {
178 self.get(UPSTREAM_SOURCE_KEY)
179 .map(|s| {
180 s.eq_ignore_ascii_case(OPENDAL_S3_CONNECTOR)
181 || s.eq_ignore_ascii_case(POSIX_FS_CONNECTOR)
182 || s.eq_ignore_ascii_case(GCS_CONNECTOR)
183 || s.eq_ignore_ascii_case(AZBLOB_CONNECTOR)
184 })
185 .unwrap_or(false)
186 }
187}
188
189impl<T: Get> WithPropertiesExt for T {}
190
191#[derive(Default, Clone, Debug, PartialEq, Eq, Hash)]
193pub struct WithOptionsSecResolved {
194 inner: BTreeMap<String, String>,
195 secret_ref: BTreeMap<String, PbSecretRef>,
196}
197
198impl std::ops::Deref for WithOptionsSecResolved {
199 type Target = BTreeMap<String, String>;
200
201 fn deref(&self) -> &Self::Target {
202 &self.inner
203 }
204}
205
206impl std::ops::DerefMut for WithOptionsSecResolved {
207 fn deref_mut(&mut self) -> &mut Self::Target {
208 &mut self.inner
209 }
210}
211
212impl WithOptionsSecResolved {
213 pub fn new(inner: BTreeMap<String, String>, secret_ref: BTreeMap<String, PbSecretRef>) -> Self {
215 Self { inner, secret_ref }
216 }
217
218 pub fn without_secrets(inner: BTreeMap<String, String>) -> Self {
220 Self {
221 inner,
222 secret_ref: Default::default(),
223 }
224 }
225
226 pub fn into_parts(self) -> (BTreeMap<String, String>, BTreeMap<String, PbSecretRef>) {
228 (self.inner, self.secret_ref)
229 }
230
231 pub fn value_eq_ignore_case(&self, key: &str, val: &str) -> bool {
232 if let Some(inner_val) = self.inner.get(key) {
233 if inner_val.eq_ignore_ascii_case(val) {
234 return true;
235 }
236 }
237 false
238 }
239}
240
241impl TryFrom<&WithOptionsSecResolved> for Option<SinkFormatDesc> {
243 type Error = crate::sink::SinkError;
244
245 fn try_from(value: &WithOptionsSecResolved) -> std::result::Result<Self, Self::Error> {
246 let connector = value.get(crate::sink::CONNECTOR_TYPE_KEY);
247 let r#type = value.get(crate::sink::SINK_TYPE_OPTION);
248 match (connector, r#type) {
249 (Some(c), Some(t)) => SinkFormatDesc::from_legacy_type(c, t),
250 _ => Ok(None),
251 }
252 }
253}
254
255impl Get for WithOptionsSecResolved {
256 fn get(&self, key: &str) -> Option<&String> {
257 self.inner.get(key)
258 }
259}