risingwave_frontend/handler/
alter_source_with_sr.rs1use std::sync::Arc;
16
17use either::Either;
18use itertools::Itertools;
19use pgwire::pg_response::StatementType;
20use risingwave_common::bail_not_implemented;
21use risingwave_common::catalog::{ColumnCatalog, max_column_id};
22use risingwave_connector::WithPropertiesExt;
23use risingwave_pb::catalog::StreamSourceInfo;
24use risingwave_pb::plan_common::{EncodeType, FormatType};
25use risingwave_sqlparser::ast::{
26 CompatibleFormatEncode, CreateSourceStatement, Encode, Format, FormatEncodeOptions, ObjectName,
27 SqlOption, Statement,
28};
29
30use super::create_source::{
31 generate_stream_graph_for_source, schema_has_schema_registry, validate_compatibility,
32};
33use super::util::SourceSchemaCompatExt;
34use super::{HandlerArgs, RwPgResponse};
35use crate::catalog::root_catalog::SchemaPath;
36use crate::catalog::source_catalog::SourceCatalog;
37use crate::error::{ErrorCode, Result};
38use crate::handler::create_source::{CreateSourceType, bind_columns_from_source};
39use crate::session::SessionImpl;
40use crate::utils::resolve_secret_ref_in_with_options;
41use crate::{Binder, WithOptions};
42
43fn format_type_to_format(from: FormatType) -> Option<Format> {
44 Some(match from {
45 FormatType::Unspecified => return None,
46 FormatType::Native => Format::Native,
47 FormatType::Debezium => Format::Debezium,
48 FormatType::DebeziumMongo => Format::DebeziumMongo,
49 FormatType::Maxwell => Format::Maxwell,
50 FormatType::Canal => Format::Canal,
51 FormatType::Upsert => Format::Upsert,
52 FormatType::Plain => Format::Plain,
53 FormatType::None => Format::None,
54 })
55}
56
57fn encode_type_to_encode(from: EncodeType) -> Option<Encode> {
58 Some(match from {
59 EncodeType::Unspecified => return None,
60 EncodeType::Native => Encode::Native,
61 EncodeType::Avro => Encode::Avro,
62 EncodeType::Csv => Encode::Csv,
63 EncodeType::Protobuf => Encode::Protobuf,
64 EncodeType::Json => Encode::Json,
65 EncodeType::Bytes => Encode::Bytes,
66 EncodeType::Template => Encode::Template,
67 EncodeType::Parquet => Encode::Parquet,
68 EncodeType::None => Encode::None,
69 EncodeType::Text => Encode::Text,
70 })
71}
72
73fn columns_minus(columns_a: &[ColumnCatalog], columns_b: &[ColumnCatalog]) -> Vec<ColumnCatalog> {
85 columns_a
86 .iter()
87 .filter(|col_a| {
88 !col_a.is_hidden()
89 && !col_a.is_connector_additional_column()
90 && !col_a.is_generated()
91 && !columns_b.iter().any(|col_b| {
92 col_a.name() == col_b.name() && col_a.data_type() == col_b.data_type()
93 })
94 })
95 .cloned()
96 .collect()
97}
98
99pub fn fetch_source_catalog_with_db_schema_id(
101 session: &SessionImpl,
102 name: &ObjectName,
103) -> Result<Arc<SourceCatalog>> {
104 let db_name = &session.database();
105 let (schema_name, real_source_name) = Binder::resolve_schema_qualified_name(db_name, name)?;
106 let search_path = session.config().search_path();
107 let user_name = &session.user_name();
108
109 let schema_path = SchemaPath::new(schema_name.as_deref(), &search_path, user_name);
110
111 let reader = session.env().catalog_reader().read_guard();
112 let (source, schema_name) =
113 reader.get_source_by_name(db_name, schema_path, &real_source_name)?;
114
115 session.check_privilege_for_drop_alter(schema_name, &**source)?;
116
117 Ok(Arc::clone(source))
118}
119
120pub fn check_format_encode(
123 original_source: &SourceCatalog,
124 new_format_encode: &FormatEncodeOptions,
125) -> Result<()> {
126 let StreamSourceInfo {
127 format, row_encode, ..
128 } = original_source.info;
129 let (Some(old_format), Some(old_row_encode)) = (
130 format_type_to_format(FormatType::try_from(format).unwrap()),
131 encode_type_to_encode(EncodeType::try_from(row_encode).unwrap()),
132 ) else {
133 return Err(ErrorCode::NotSupported(
134 "altering a legacy source which is not created using `FORMAT .. ENCODE ..` Clause"
135 .to_owned(),
136 "try this feature by creating a fresh source".to_owned(),
137 )
138 .into());
139 };
140
141 if new_format_encode.format != old_format || new_format_encode.row_encode != old_row_encode {
142 bail_not_implemented!(
143 "the original definition is FORMAT {:?} ENCODE {:?}, and altering them is not supported yet",
144 &old_format,
145 &old_row_encode,
146 );
147 }
148
149 Ok(())
150}
151
152pub async fn refresh_sr_and_get_columns_diff(
154 original_source: &SourceCatalog,
155 format_encode: &FormatEncodeOptions,
156 session: &Arc<SessionImpl>,
157) -> Result<(StreamSourceInfo, Vec<ColumnCatalog>, Vec<ColumnCatalog>)> {
158 let mut with_properties = original_source.with_properties.clone();
159 validate_compatibility(format_encode, &mut with_properties)?;
160
161 if with_properties.is_cdc_connector() {
162 bail_not_implemented!("altering a cdc source is not supported");
163 }
164
165 let (Some(columns_from_resolve_source), source_info) = bind_columns_from_source(
166 session,
167 format_encode,
168 Either::Right(&with_properties),
169 CreateSourceType::for_replace(original_source),
170 )
171 .await?
172 else {
173 unreachable!("source without schema registry is rejected")
175 };
176
177 let mut added_columns = columns_minus(&columns_from_resolve_source, &original_source.columns);
178 let mut next_col_id = max_column_id(&original_source.columns).next();
180 for col in &mut added_columns {
181 col.column_desc.column_id = next_col_id;
182 next_col_id = next_col_id.next();
183 }
184 let dropped_columns = columns_minus(&original_source.columns, &columns_from_resolve_source);
185 tracing::debug!(
186 ?added_columns,
187 ?dropped_columns,
188 ?columns_from_resolve_source,
189 original_source = ?original_source.columns
190 );
191
192 Ok((source_info, added_columns, dropped_columns))
193}
194
195fn get_format_encode_from_source(source: &SourceCatalog) -> Result<FormatEncodeOptions> {
196 let stmt = source.create_sql_ast()?;
197 let Statement::CreateSource {
198 stmt: CreateSourceStatement { format_encode, .. },
199 } = stmt
200 else {
201 unreachable!()
202 };
203 Ok(format_encode.into_v2_with_warning())
204}
205
206pub async fn handler_refresh_schema(
207 handler_args: HandlerArgs,
208 name: ObjectName,
209) -> Result<RwPgResponse> {
210 let source = fetch_source_catalog_with_db_schema_id(&handler_args.session, &name)?;
211 let format_encode = get_format_encode_from_source(&source)?;
212 handle_alter_source_with_sr(handler_args, name, format_encode).await
213}
214
215pub async fn handle_alter_source_with_sr(
216 handler_args: HandlerArgs,
217 name: ObjectName,
218 format_encode: FormatEncodeOptions,
219) -> Result<RwPgResponse> {
220 let session = handler_args.session.clone();
221 let source = fetch_source_catalog_with_db_schema_id(&session, &name)?;
222 let mut source = source.as_ref().clone();
223
224 if source.associated_table_id.is_some() {
225 return Err(ErrorCode::NotSupported(
226 "alter table with connector using ALTER SOURCE statement".to_owned(),
227 "try to use ALTER TABLE instead".to_owned(),
228 )
229 .into());
230 };
231
232 check_format_encode(&source, &format_encode)?;
233
234 if !schema_has_schema_registry(&format_encode) {
235 return Err(ErrorCode::NotSupported(
236 "altering a source without schema registry".to_owned(),
237 "try `ALTER SOURCE .. ADD COLUMN ...` instead".to_owned(),
238 )
239 .into());
240 }
241
242 let (source_info, added_columns, dropped_columns) =
243 refresh_sr_and_get_columns_diff(&source, &format_encode, &session).await?;
244
245 if !dropped_columns.is_empty() {
246 bail_not_implemented!(
247 "this altering statement will drop columns, which is not supported yet: {}",
248 dropped_columns
249 .iter()
250 .map(|col| format!("({}: {})", col.name(), col.data_type()))
251 .join(", ")
252 );
253 }
254
255 source.info = source_info;
256 source.columns.extend(added_columns);
257 source.definition = alter_definition_format_encode(
258 source.create_sql_ast_purified()?,
259 format_encode.row_options.clone(),
260 )?;
261
262 let (format_encode_options, format_encode_secret_ref) = resolve_secret_ref_in_with_options(
263 WithOptions::try_from(format_encode.row_options())?,
264 session.as_ref(),
265 )?
266 .into_parts();
267 source
268 .info
269 .format_encode_options
270 .extend(format_encode_options);
271
272 source
273 .info
274 .format_encode_secret_refs
275 .extend(format_encode_secret_ref);
276
277 source.version += 1;
279
280 let pb_source = source.to_prost();
281 let catalog_writer = session.catalog_writer()?;
282 if source.info.is_shared() {
283 let graph = generate_stream_graph_for_source(handler_args, source.clone())?;
284 catalog_writer.replace_source(pb_source, graph).await?
285 } else {
286 catalog_writer.alter_source(pb_source).await?;
287 }
288 Ok(RwPgResponse::empty_result(StatementType::ALTER_SOURCE))
289}
290
291pub fn alter_definition_format_encode(
293 mut stmt: Statement,
294 format_encode_options: Vec<SqlOption>,
295) -> Result<String> {
296 match &mut stmt {
297 Statement::CreateSource {
298 stmt: CreateSourceStatement { format_encode, .. },
299 }
300 | Statement::CreateTable {
301 format_encode: Some(format_encode),
302 ..
303 } => {
304 match format_encode {
305 CompatibleFormatEncode::V2(schema) => {
306 schema.row_options = format_encode_options;
307 }
308 CompatibleFormatEncode::RowFormat(_schema) => unreachable!(),
311 }
312 }
313 _ => unreachable!(),
314 }
315
316 Ok(stmt.to_string())
317}
318
319#[cfg(test)]
320pub mod tests {
321 use risingwave_common::catalog::{DEFAULT_DATABASE_NAME, DEFAULT_SCHEMA_NAME};
322 use risingwave_common::types::DataType;
323
324 use crate::catalog::root_catalog::SchemaPath;
325 use crate::test_utils::{LocalFrontend, PROTO_FILE_DATA, create_proto_file};
326
327 #[tokio::test]
328 async fn test_alter_source_with_sr_handler() {
329 let proto_file = create_proto_file(PROTO_FILE_DATA);
330 let sql = format!(
331 r#"CREATE SOURCE src
332 WITH (
333 connector = 'kafka',
334 topic = 'test-topic',
335 properties.bootstrap.server = 'localhost:29092'
336 )
337 FORMAT PLAIN ENCODE PROTOBUF (
338 message = '.test.TestRecord',
339 schema.location = 'file://{}'
340 )"#,
341 proto_file.path().to_str().unwrap()
342 );
343 let frontend = LocalFrontend::new(Default::default()).await;
344 let session = frontend.session_ref();
345 let schema_path = SchemaPath::Name(DEFAULT_SCHEMA_NAME);
346
347 frontend
348 .run_sql_with_session(session.clone(), "SET streaming_use_shared_source TO false;")
349 .await
350 .unwrap();
351 frontend
352 .run_sql_with_session(session.clone(), sql)
353 .await
354 .unwrap();
355
356 let get_source = || {
357 let catalog_reader = session.env().catalog_reader().read_guard();
358 catalog_reader
359 .get_source_by_name(DEFAULT_DATABASE_NAME, schema_path, "src")
360 .unwrap()
361 .0
362 .clone()
363 };
364
365 let source = get_source();
366 expect_test::expect!["CREATE SOURCE src (id INT, country STRUCT<address CHARACTER VARYING, city STRUCT<address CHARACTER VARYING, zipcode CHARACTER VARYING>, zipcode CHARACTER VARYING>, zipcode BIGINT, rate REAL) WITH (connector = 'kafka', topic = 'test-topic', properties.bootstrap.server = 'localhost:29092') FORMAT PLAIN ENCODE PROTOBUF (message = '.test.TestRecord', schema.location = 'file://')"].assert_eq(&source.create_sql_purified().replace(proto_file.path().to_str().unwrap(), ""));
367
368 let sql = format!(
369 r#"ALTER SOURCE src FORMAT UPSERT ENCODE PROTOBUF (
370 message = '.test.TestRecord',
371 schema.location = 'file://{}'
372 )"#,
373 proto_file.path().to_str().unwrap()
374 );
375 assert!(
376 frontend
377 .run_sql(sql)
378 .await
379 .unwrap_err()
380 .to_string()
381 .contains("the original definition is FORMAT Plain ENCODE Protobuf")
382 );
383
384 let sql = format!(
385 r#"ALTER SOURCE src FORMAT PLAIN ENCODE PROTOBUF (
386 message = '.test.TestRecordAlterType',
387 schema.location = 'file://{}'
388 )"#,
389 proto_file.path().to_str().unwrap()
390 );
391 let res_str = frontend.run_sql(sql).await.unwrap_err().to_string();
392 assert!(res_str.contains("id: integer"));
393 assert!(res_str.contains("zipcode: bigint"));
394
395 let sql = format!(
396 r#"ALTER SOURCE src FORMAT PLAIN ENCODE PROTOBUF (
397 message = '.test.TestRecordExt',
398 schema.location = 'file://{}'
399 )"#,
400 proto_file.path().to_str().unwrap()
401 );
402 frontend.run_sql(sql).await.unwrap();
403
404 let altered_source = get_source();
405
406 let name_column = altered_source
407 .columns
408 .iter()
409 .find(|col| col.column_desc.name == "name")
410 .unwrap();
411 assert_eq!(name_column.column_desc.data_type, DataType::Varchar);
412
413 expect_test::expect!["CREATE SOURCE src (id INT, country STRUCT<address CHARACTER VARYING, city STRUCT<address CHARACTER VARYING, zipcode CHARACTER VARYING>, zipcode CHARACTER VARYING>, zipcode BIGINT, rate REAL, name CHARACTER VARYING) WITH (connector = 'kafka', topic = 'test-topic', properties.bootstrap.server = 'localhost:29092') FORMAT PLAIN ENCODE PROTOBUF (message = '.test.TestRecordExt', schema.location = 'file://')"].assert_eq(&altered_source.create_sql_purified().replace(proto_file.path().to_str().unwrap(), ""));
414 }
415}