risingwave_frontend/handler/
create_sink.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::{BTreeMap, BTreeSet, HashMap, HashSet};
16use std::sync::{Arc, LazyLock};
17
18use anyhow::Context;
19use either::Either;
20use iceberg::arrow::type_to_arrow_type;
21use iceberg::spec::Transform;
22use itertools::Itertools;
23use maplit::{convert_args, hashmap, hashset};
24use pgwire::pg_response::{PgResponse, StatementType};
25use risingwave_common::array::arrow::IcebergArrowConvert;
26use risingwave_common::array::arrow::arrow_schema_iceberg::DataType as ArrowDataType;
27use risingwave_common::bail;
28use risingwave_common::catalog::{
29    ColumnCatalog, ICEBERG_SINK_PREFIX, ObjectId, RISINGWAVE_ICEBERG_ROW_ID, ROW_ID_COLUMN_NAME,
30    Schema,
31};
32use risingwave_common::license::Feature;
33use risingwave_common::secret::LocalSecretManager;
34use risingwave_common::system_param::reader::SystemParamsRead;
35use risingwave_common::types::DataType;
36use risingwave_connector::sink::catalog::{SinkCatalog, SinkFormatDesc};
37use risingwave_connector::sink::file_sink::s3::SnowflakeSink;
38use risingwave_connector::sink::iceberg::{ICEBERG_SINK, IcebergConfig};
39use risingwave_connector::sink::kafka::KAFKA_SINK;
40use risingwave_connector::sink::snowflake_redshift::redshift::RedshiftSink;
41use risingwave_connector::sink::snowflake_redshift::snowflake::SnowflakeV2Sink;
42use risingwave_connector::sink::{
43    CONNECTOR_TYPE_KEY, SINK_SNAPSHOT_OPTION, SINK_TYPE_OPTION, SINK_USER_FORCE_APPEND_ONLY_OPTION,
44    SINK_USER_IGNORE_DELETE_OPTION, Sink, enforce_secret_sink,
45};
46use risingwave_connector::{
47    AUTO_SCHEMA_CHANGE_KEY, SINK_CREATE_TABLE_IF_NOT_EXISTS_KEY, SINK_INTERMEDIATE_TABLE_NAME,
48    SINK_TARGET_TABLE_NAME, WithPropertiesExt,
49};
50use risingwave_pb::catalog::connection_params::PbConnectionType;
51use risingwave_pb::telemetry::TelemetryDatabaseObject;
52use risingwave_sqlparser::ast::{
53    CreateSink, CreateSinkStatement, EmitMode, Encode, ExplainOptions, Format, FormatEncodeOptions,
54    ObjectName, Query, Statement,
55};
56use risingwave_sqlparser::parser::Parser;
57
58use super::RwPgResponse;
59use super::create_mv::get_column_names;
60use super::create_source::UPSTREAM_SOURCE_KEY;
61use super::util::gen_query_from_table_name;
62use crate::binder::{Binder, Relation};
63use crate::catalog::table_catalog::TableType;
64use crate::error::{ErrorCode, Result, RwError};
65use crate::expr::{ExprImpl, InputRef, rewrite_now_to_proctime};
66use crate::handler::HandlerArgs;
67use crate::handler::alter_table_column::fetch_table_catalog_for_alter;
68use crate::handler::create_mv::parse_column_names;
69use crate::handler::util::{
70    LongRunningNotificationAction, check_connector_match_connection_type,
71    ensure_connection_type_allowed, execute_with_long_running_notification,
72    get_table_catalog_by_table_name,
73};
74use crate::optimizer::backfill_order_strategy::plan_backfill_order;
75use crate::optimizer::plan_node::{
76    IcebergPartitionInfo, LogicalSource, PartitionComputeInfo, StreamPlanRef as PlanRef,
77    StreamProject, generic,
78};
79use crate::optimizer::{OptimizerContext, RelationCollectorVisitor};
80use crate::scheduler::streaming_manager::CreatingStreamingJobInfo;
81use crate::session::SessionImpl;
82use crate::session::current::notice_to_user;
83use crate::stream_fragmenter::{GraphJobType, build_graph_with_strategy};
84use crate::utils::{resolve_connection_ref_and_secret_ref, resolve_privatelink_in_with_option};
85use crate::{Explain, Planner, TableCatalog, WithOptions, WithOptionsSecResolved};
86
87static SINK_ALLOWED_CONNECTION_CONNECTOR: LazyLock<HashSet<PbConnectionType>> =
88    LazyLock::new(|| {
89        hashset! {
90            PbConnectionType::Unspecified,
91            PbConnectionType::Kafka,
92            PbConnectionType::Iceberg,
93            PbConnectionType::Elasticsearch,
94        }
95    });
96
97static SINK_ALLOWED_CONNECTION_SCHEMA_REGISTRY: LazyLock<HashSet<PbConnectionType>> =
98    LazyLock::new(|| {
99        hashset! {
100            PbConnectionType::Unspecified,
101            PbConnectionType::SchemaRegistry,
102        }
103    });
104
105// used to store result of `gen_sink_plan`
106pub struct SinkPlanContext {
107    pub query: Box<Query>,
108    pub sink_plan: PlanRef,
109    pub sink_catalog: SinkCatalog,
110    pub target_table_catalog: Option<Arc<TableCatalog>>,
111    pub dependencies: HashSet<ObjectId>,
112}
113
114pub async fn gen_sink_plan(
115    handler_args: HandlerArgs,
116    stmt: CreateSinkStatement,
117    explain_options: Option<ExplainOptions>,
118    is_iceberg_engine_internal: bool,
119) -> Result<SinkPlanContext> {
120    let session = handler_args.session.clone();
121    let session = session.as_ref();
122    let user_specified_columns = !stmt.columns.is_empty();
123    let db_name = &session.database();
124    let (sink_schema_name, sink_table_name) =
125        Binder::resolve_schema_qualified_name(db_name, &stmt.sink_name)?;
126
127    let mut with_options = handler_args.with_options.clone();
128
129    if session
130        .env()
131        .system_params_manager()
132        .get_params()
133        .load()
134        .enforce_secret()
135        && Feature::SecretManagement.check_available().is_ok()
136    {
137        enforce_secret_sink(&with_options)?;
138    }
139
140    resolve_privatelink_in_with_option(&mut with_options)?;
141    let (mut resolved_with_options, connection_type, connector_conn_ref) =
142        resolve_connection_ref_and_secret_ref(
143            with_options,
144            session,
145            Some(TelemetryDatabaseObject::Sink),
146        )?;
147    ensure_connection_type_allowed(connection_type, &SINK_ALLOWED_CONNECTION_CONNECTOR)?;
148
149    // if not using connection, we don't need to check connector match connection type
150    if !matches!(connection_type, PbConnectionType::Unspecified) {
151        let Some(connector) = resolved_with_options.get_connector() else {
152            return Err(RwError::from(ErrorCode::ProtocolError(format!(
153                "missing field '{}' in WITH clause",
154                CONNECTOR_TYPE_KEY
155            ))));
156        };
157        check_connector_match_connection_type(connector.as_str(), &connection_type)?;
158    }
159
160    let partition_info = get_partition_compute_info(&resolved_with_options).await?;
161
162    let context = if let Some(explain_options) = explain_options {
163        OptimizerContext::new(handler_args.clone(), explain_options)
164    } else {
165        OptimizerContext::from_handler_args(handler_args.clone())
166    };
167
168    let is_auto_schema_change = resolved_with_options
169        .remove(AUTO_SCHEMA_CHANGE_KEY)
170        .map(|value| {
171            value.parse::<bool>().map_err(|_| {
172                ErrorCode::InvalidInputSyntax(format!(
173                    "invalid value {} of '{}' option, expect",
174                    value, AUTO_SCHEMA_CHANGE_KEY
175                ))
176            })
177        })
178        .transpose()?
179        .unwrap_or(false);
180
181    if is_auto_schema_change {
182        Feature::SinkAutoSchemaChange.check_available()?;
183    }
184
185    let sink_into_table_name = stmt.into_table_name.as_ref().map(|name| name.real_value());
186    if sink_into_table_name.is_some() {
187        let prev = resolved_with_options.insert(CONNECTOR_TYPE_KEY.to_owned(), "table".to_owned());
188
189        if prev.is_some() {
190            return Err(RwError::from(ErrorCode::BindError(
191                "In the case of sinking into table, the 'connector' parameter should not be provided.".to_owned(),
192            )));
193        }
194    }
195    let connector = resolved_with_options
196        .get(CONNECTOR_TYPE_KEY)
197        .cloned()
198        .ok_or_else(|| ErrorCode::BindError(format!("missing field '{CONNECTOR_TYPE_KEY}'")))?;
199
200    // Used for debezium's table name
201    let sink_from_table_name;
202    // `true` means that sink statement has the form: `CREATE SINK s1 FROM ...`
203    // `false` means that sink statement has the form: `CREATE SINK s1 AS <query>`
204    let direct_sink_from_name: Option<(ObjectName, bool)>;
205    let mut query = match stmt.sink_from {
206        CreateSink::From(from_name) => {
207            sink_from_table_name = from_name.0.last().unwrap().real_value();
208            direct_sink_from_name = Some((from_name.clone(), is_auto_schema_change));
209            if is_auto_schema_change && sink_into_table_name.is_some() {
210                return Err(RwError::from(ErrorCode::InvalidInputSyntax(
211                    "auto schema change not supported for sink-into-table".to_owned(),
212                )));
213            }
214            if resolved_with_options
215                .value_eq_ignore_case(SINK_CREATE_TABLE_IF_NOT_EXISTS_KEY, "true")
216                && connector == RedshiftSink::SINK_NAME
217                || connector == SnowflakeV2Sink::SINK_NAME
218            {
219                if let Some(table_name) = resolved_with_options.get(SINK_TARGET_TABLE_NAME) {
220                    // auto fill intermediate table name if target table name is specified
221                    if resolved_with_options
222                        .get(SINK_INTERMEDIATE_TABLE_NAME)
223                        .is_none()
224                    {
225                        // generate the intermediate table name with random value appended to the target table name
226                        let intermediate_table_name = format!(
227                            "rw_{}_{}_{}",
228                            sink_table_name,
229                            table_name,
230                            uuid::Uuid::new_v4()
231                        );
232                        resolved_with_options.insert(
233                            SINK_INTERMEDIATE_TABLE_NAME.to_owned(),
234                            intermediate_table_name,
235                        );
236                    }
237                } else {
238                    return Err(RwError::from(ErrorCode::BindError(
239                        "'table.name' option must be specified.".to_owned(),
240                    )));
241                }
242            }
243            Box::new(gen_query_from_table_name(from_name))
244        }
245        CreateSink::AsQuery(query) => {
246            if is_auto_schema_change {
247                return Err(RwError::from(ErrorCode::InvalidInputSyntax(
248                    "auto schema change not supported for CREATE SINK AS QUERY".to_owned(),
249                )));
250            }
251            sink_from_table_name = sink_table_name.clone();
252            direct_sink_from_name = None;
253            query
254        }
255    };
256
257    if is_iceberg_engine_internal && let Some((from_name, _)) = &direct_sink_from_name {
258        let (table, _) = get_table_catalog_by_table_name(session, from_name)?;
259        let pk_names = table.pk_column_names();
260        if pk_names.len() == 1 && pk_names[0].eq(ROW_ID_COLUMN_NAME) {
261            let [stmt]: [_; 1] = Parser::parse_sql(&format!(
262                "select {} as {}, * from {}",
263                ROW_ID_COLUMN_NAME, RISINGWAVE_ICEBERG_ROW_ID, from_name
264            ))
265            .context("unable to parse query")?
266            .try_into()
267            .unwrap();
268            let Statement::Query(parsed_query) = stmt else {
269                panic!("unexpected statement: {:?}", stmt);
270            };
271            query = parsed_query;
272        }
273    }
274
275    let (sink_database_id, sink_schema_id) =
276        session.get_database_and_schema_id_for_create(sink_schema_name.clone())?;
277
278    let (dependent_relations, dependent_udfs, bound, auto_refresh_schema_from_table) = {
279        let mut binder = Binder::new_for_stream(session);
280        let auto_refresh_schema_from_table = if let Some((from_name, true)) = &direct_sink_from_name
281        {
282            let from_relation = binder.bind_relation_by_name(from_name, None, None, true)?;
283            if let Relation::BaseTable(table) = from_relation {
284                if table.table_catalog.table_type != TableType::Table {
285                    return Err(ErrorCode::InvalidInputSyntax(format!(
286                        "auto schema change only support on TABLE, but got {:?}",
287                        table.table_catalog.table_type
288                    ))
289                    .into());
290                }
291                if table.table_catalog.database_id != sink_database_id {
292                    return Err(ErrorCode::InvalidInputSyntax(
293                        "auto schema change sink does not support created from cross database table".to_owned()
294                    )
295                        .into());
296                }
297                for col in &table.table_catalog.columns {
298                    if !col.is_hidden() && (col.is_generated() || col.is_rw_sys_column()) {
299                        return Err(ErrorCode::InvalidInputSyntax(format!("auto schema change not supported for table with non-hidden generated column or sys column, but got {}", col.name())).into());
300                    }
301                }
302                Some(table.table_catalog)
303            } else {
304                return Err(RwError::from(ErrorCode::NotSupported(
305                    "auto schema change only supported for TABLE".to_owned(),
306                    "try recreating the sink from table".to_owned(),
307                )));
308            }
309        } else {
310            None
311        };
312
313        let bound = binder.bind_query(&query)?;
314
315        (
316            binder.included_relations().clone(),
317            binder.included_udfs().clone(),
318            bound,
319            auto_refresh_schema_from_table,
320        )
321    };
322
323    let col_names = if sink_into_table_name.is_some() {
324        parse_column_names(&stmt.columns)
325    } else {
326        // If column names not specified, use the name in the bound query, which is equal with the plan root's original field name.
327        get_column_names(&bound, stmt.columns)?
328    };
329
330    let emit_on_window_close = stmt.emit_mode == Some(EmitMode::OnWindowClose);
331    if emit_on_window_close {
332        context.warn_to_user("EMIT ON WINDOW CLOSE is currently an experimental feature. Please use it with caution.");
333    }
334
335    let format_desc = match stmt.sink_schema {
336        // Case A: new syntax `format ... encode ...`
337        Some(f) => {
338            validate_compatibility(&connector, &f)?;
339            Some(bind_sink_format_desc(session,f)?)
340        }
341        None => match resolved_with_options.get(SINK_TYPE_OPTION) {
342            // Case B: old syntax `type = '...'`
343            Some(t) => SinkFormatDesc::from_legacy_type(&connector, t)?.map(|mut f| {
344                session.notice_to_user("Consider using the newer syntax `FORMAT ... ENCODE ...` instead of `type = '...'`.");
345                if let Some(v) = resolved_with_options.get(SINK_USER_FORCE_APPEND_ONLY_OPTION) {
346                    f.options.insert(SINK_USER_FORCE_APPEND_ONLY_OPTION.into(), v.into());
347                }
348                if let Some(v) = resolved_with_options.get(SINK_USER_IGNORE_DELETE_OPTION) {
349                    f.options.insert(SINK_USER_IGNORE_DELETE_OPTION.into(), v.into());
350                }
351                f
352            }),
353            // Case C: no format + encode required
354            None => None,
355        },
356    };
357
358    let definition = context.normalized_sql().to_owned();
359    let mut plan_root = if is_iceberg_engine_internal {
360        Planner::new_for_iceberg_table_engine_sink(context.into()).plan_query(bound)?
361    } else {
362        Planner::new_for_stream(context.into()).plan_query(bound)?
363    };
364    if let Some(col_names) = &col_names {
365        plan_root.set_out_names(col_names.clone())?;
366    };
367
368    let without_backfill = match resolved_with_options.remove(SINK_SNAPSHOT_OPTION) {
369        Some(flag) if flag.eq_ignore_ascii_case("false") => {
370            if direct_sink_from_name.is_some() || is_iceberg_engine_internal {
371                true
372            } else {
373                return Err(ErrorCode::BindError(
374                    "`snapshot = false` only support `CREATE SINK FROM MV or TABLE`".to_owned(),
375                )
376                .into());
377            }
378        }
379        _ => false,
380    };
381
382    let target_table_catalog = stmt
383        .into_table_name
384        .as_ref()
385        .map(|table_name| fetch_table_catalog_for_alter(session, table_name).map(|t| t.0))
386        .transpose()?;
387
388    if let Some(target_table_catalog) = &target_table_catalog {
389        if let Some(col_names) = col_names {
390            let target_table_columns = target_table_catalog
391                .columns()
392                .iter()
393                .map(|c| c.name())
394                .collect::<BTreeSet<_>>();
395            for c in col_names {
396                if !target_table_columns.contains(c.as_str()) {
397                    return Err(RwError::from(ErrorCode::BindError(format!(
398                        "Column {} not found in table {}",
399                        c,
400                        target_table_catalog.name()
401                    ))));
402                }
403            }
404        }
405        if target_table_catalog
406            .columns()
407            .iter()
408            .any(|col| !col.nullable())
409        {
410            notice_to_user(format!(
411                "The target table `{}` contains columns with NOT NULL constraints. Any sinked rows violating the constraints will be ignored silently.",
412                target_table_catalog.name(),
413            ));
414        }
415    }
416
417    let allow_snapshot_backfill = target_table_catalog.is_none() && !is_iceberg_engine_internal;
418
419    let sink_plan = plan_root.gen_sink_plan(
420        sink_table_name,
421        definition,
422        resolved_with_options,
423        emit_on_window_close,
424        db_name.to_owned(),
425        sink_from_table_name,
426        format_desc,
427        without_backfill,
428        target_table_catalog.clone(),
429        partition_info,
430        user_specified_columns,
431        auto_refresh_schema_from_table,
432        allow_snapshot_backfill,
433    )?;
434
435    let sink_desc = sink_plan.sink_desc().clone();
436
437    let mut sink_plan: PlanRef = sink_plan.into();
438
439    let ctx = sink_plan.ctx();
440    let explain_trace = ctx.is_explain_trace();
441    if explain_trace {
442        ctx.trace("Create Sink:");
443        ctx.trace(sink_plan.explain_to_string());
444    }
445    tracing::trace!("sink_plan: {:?}", sink_plan.explain_to_string());
446
447    // TODO(rc): To be consistent with UDF dependency check, we should collect relation dependencies
448    // during binding instead of visiting the optimized plan.
449    let dependencies =
450        RelationCollectorVisitor::collect_with(dependent_relations, sink_plan.clone())
451            .into_iter()
452            .chain(dependent_udfs.iter().copied().map_into())
453            .collect();
454
455    let sink_catalog = sink_desc.into_catalog(
456        sink_schema_id,
457        sink_database_id,
458        session.user_id(),
459        connector_conn_ref,
460    );
461
462    if let Some(table_catalog) = &target_table_catalog {
463        for column in sink_catalog.full_columns() {
464            if !column.can_dml() {
465                unreachable!(
466                    "can not derive generated columns and system column `_rw_timestamp` in a sink's catalog, but meet one"
467                );
468            }
469        }
470
471        let table_columns_without_rw_timestamp = table_catalog.columns_without_rw_timestamp();
472        let exprs = derive_default_column_project_for_sink(
473            &sink_catalog,
474            sink_plan.schema(),
475            &table_columns_without_rw_timestamp,
476            user_specified_columns,
477        )?;
478
479        let logical_project = generic::Project::new(exprs, sink_plan);
480
481        sink_plan = StreamProject::new(logical_project).into();
482
483        let exprs = LogicalSource::derive_output_exprs_from_generated_columns(
484            &table_columns_without_rw_timestamp,
485        )?;
486
487        if let Some(exprs) = exprs {
488            let logical_project = generic::Project::new(exprs, sink_plan);
489            sink_plan = StreamProject::new(logical_project).into();
490        }
491    };
492
493    Ok(SinkPlanContext {
494        query,
495        sink_plan,
496        sink_catalog,
497        target_table_catalog,
498        dependencies,
499    })
500}
501
502// This function is used to return partition compute info for a sink. More details refer in `PartitionComputeInfo`.
503// Return:
504// `Some(PartitionComputeInfo)` if the sink need to compute partition.
505// `None` if the sink does not need to compute partition.
506pub async fn get_partition_compute_info(
507    with_options: &WithOptionsSecResolved,
508) -> Result<Option<PartitionComputeInfo>> {
509    let (options, secret_refs) = with_options.clone().into_parts();
510    let Some(connector) = options.get(UPSTREAM_SOURCE_KEY).cloned() else {
511        return Ok(None);
512    };
513    let properties = LocalSecretManager::global().fill_secrets(options, secret_refs)?;
514    match connector.as_str() {
515        ICEBERG_SINK => {
516            let iceberg_config = IcebergConfig::from_btreemap(properties)?;
517            get_partition_compute_info_for_iceberg(&iceberg_config).await
518        }
519        _ => Ok(None),
520    }
521}
522
523#[allow(clippy::unused_async)]
524async fn get_partition_compute_info_for_iceberg(
525    _iceberg_config: &IcebergConfig,
526) -> Result<Option<PartitionComputeInfo>> {
527    // TODO: check table if exists
528    if _iceberg_config.create_table_if_not_exists {
529        return Ok(None);
530    }
531    let table = _iceberg_config.load_table().await?;
532    let partition_spec = table.metadata().default_partition_spec();
533    if partition_spec.is_unpartitioned() {
534        return Ok(None);
535    }
536
537    // Separate the partition spec into two parts: sparse partition and range partition.
538    // Sparse partition means that the data distribution is more sparse at a given time.
539    // Range partition means that the data distribution is likely same at a given time.
540    // Only compute the partition and shuffle by them for the sparse partition.
541    let has_sparse_partition = partition_spec.fields().iter().any(|f| match f.transform {
542        // Sparse partition
543        Transform::Identity | Transform::Truncate(_) | Transform::Bucket(_) => true,
544        // Range partition
545        Transform::Year
546        | Transform::Month
547        | Transform::Day
548        | Transform::Hour
549        | Transform::Void
550        | Transform::Unknown => false,
551    });
552    if !has_sparse_partition {
553        return Ok(None);
554    }
555
556    let arrow_type = type_to_arrow_type(&iceberg::spec::Type::Struct(
557        table.metadata().default_partition_type().clone(),
558    ))
559    .map_err(|_| {
560        RwError::from(ErrorCode::SinkError(
561            "Fail to convert iceberg partition type to arrow type".into(),
562        ))
563    })?;
564    let ArrowDataType::Struct(struct_fields) = arrow_type else {
565        return Err(RwError::from(ErrorCode::SinkError(
566            "Partition type of iceberg should be a struct type".into(),
567        )));
568    };
569
570    let schema = table.metadata().current_schema();
571    let partition_fields = partition_spec
572        .fields()
573        .iter()
574        .map(|f| {
575            let source_f =
576                schema
577                    .field_by_id(f.source_id)
578                    .ok_or(RwError::from(ErrorCode::SinkError(
579                        "Fail to look up iceberg partition field".into(),
580                    )))?;
581            Ok((source_f.name.clone(), f.transform))
582        })
583        .collect::<Result<Vec<_>>>()?;
584
585    Ok(Some(PartitionComputeInfo::Iceberg(IcebergPartitionInfo {
586        partition_type: IcebergArrowConvert.struct_from_fields(&struct_fields)?,
587        partition_fields,
588    })))
589}
590
591pub async fn handle_create_sink(
592    handle_args: HandlerArgs,
593    stmt: CreateSinkStatement,
594    is_iceberg_engine_internal: bool,
595) -> Result<RwPgResponse> {
596    let session = handle_args.session.clone();
597
598    session.check_cluster_limits().await?;
599
600    let if_not_exists = stmt.if_not_exists;
601    if let Either::Right(resp) = session.check_relation_name_duplicated(
602        stmt.sink_name.clone(),
603        StatementType::CREATE_SINK,
604        if_not_exists,
605    )? {
606        return Ok(resp);
607    }
608
609    if stmt.sink_name.base_name().starts_with(ICEBERG_SINK_PREFIX) {
610        return Err(RwError::from(ErrorCode::InvalidInputSyntax(format!(
611            "Sink name cannot start with reserved prefix '{}'",
612            ICEBERG_SINK_PREFIX
613        ))));
614    }
615
616    let (mut sink, graph, target_table_catalog, dependencies) = {
617        let backfill_order_strategy = handle_args.with_options.backfill_order_strategy();
618
619        let SinkPlanContext {
620            query,
621            sink_plan: plan,
622            sink_catalog: sink,
623            target_table_catalog,
624            dependencies,
625        } = gen_sink_plan(handle_args, stmt, None, is_iceberg_engine_internal).await?;
626
627        let has_order_by = !query.order_by.is_empty();
628        if has_order_by {
629            plan.ctx().warn_to_user(
630                r#"The ORDER BY clause in the CREATE SINK statement has no effect at all."#
631                    .to_owned(),
632            );
633        }
634
635        let backfill_order =
636            plan_backfill_order(session.as_ref(), backfill_order_strategy, plan.clone())?;
637
638        let graph =
639            build_graph_with_strategy(plan, Some(GraphJobType::Sink), Some(backfill_order))?;
640
641        (sink, graph, target_table_catalog, dependencies)
642    };
643
644    if let Some(table_catalog) = target_table_catalog {
645        sink.original_target_columns = table_catalog.columns_without_rw_timestamp();
646    }
647
648    let _job_guard =
649        session
650            .env()
651            .creating_streaming_job_tracker()
652            .guard(CreatingStreamingJobInfo::new(
653                session.session_id(),
654                sink.database_id,
655                sink.schema_id,
656                sink.name.clone(),
657            ));
658
659    let catalog_writer = session.catalog_writer()?;
660    execute_with_long_running_notification(
661        catalog_writer.create_sink(sink.to_proto(), graph, dependencies, if_not_exists),
662        &session,
663        "CREATE SINK",
664        LongRunningNotificationAction::MonitorBackfillJob,
665    )
666    .await?;
667
668    Ok(PgResponse::empty_result(StatementType::CREATE_SINK))
669}
670
671pub fn fetch_incoming_sinks(
672    session: &Arc<SessionImpl>,
673    table: &TableCatalog,
674) -> Result<Vec<Arc<SinkCatalog>>> {
675    let reader = session.env().catalog_reader().read_guard();
676    let schema = reader.get_schema_by_id(table.database_id, table.schema_id)?;
677    let Some(incoming_sinks) = schema.table_incoming_sinks(table.id) else {
678        return Ok(vec![]);
679    };
680    let mut sinks = vec![];
681    for sink_id in incoming_sinks {
682        sinks.push(
683            schema
684                .get_sink_by_id(*sink_id)
685                .expect("should exist")
686                .clone(),
687        );
688    }
689    Ok(sinks)
690}
691
692fn derive_sink_to_table_expr(
693    sink_schema: &Schema,
694    idx: usize,
695    target_type: &DataType,
696) -> Result<ExprImpl> {
697    let input_type = &sink_schema.fields()[idx].data_type;
698
699    if !target_type.equals_datatype(input_type) {
700        bail!(
701            "column type mismatch: {:?} vs {:?}, column name: {:?}",
702            target_type,
703            input_type,
704            sink_schema.fields()[idx].name
705        );
706    } else {
707        Ok(ExprImpl::InputRef(Box::new(InputRef::new(
708            idx,
709            input_type.clone(),
710        ))))
711    }
712}
713
714pub(crate) fn derive_default_column_project_for_sink(
715    sink: &SinkCatalog,
716    sink_schema: &Schema,
717    columns: &[ColumnCatalog],
718    user_specified_columns: bool,
719) -> Result<Vec<ExprImpl>> {
720    assert_eq!(sink.full_schema().len(), sink_schema.len());
721
722    let default_column_exprs = TableCatalog::default_column_exprs(columns);
723
724    let mut exprs = vec![];
725
726    let sink_visible_col_idxes = sink
727        .full_columns()
728        .iter()
729        .positions(|c| !c.is_hidden())
730        .collect_vec();
731    let sink_visible_col_idxes_by_name = sink
732        .full_columns()
733        .iter()
734        .enumerate()
735        .filter(|(_, c)| !c.is_hidden())
736        .map(|(i, c)| (c.name(), i))
737        .collect::<BTreeMap<_, _>>();
738
739    for (idx, column) in columns.iter().enumerate() {
740        if !column.can_dml() {
741            continue;
742        }
743
744        let default_col_expr =
745            || -> ExprImpl { rewrite_now_to_proctime(default_column_exprs[idx].clone()) };
746
747        let sink_col_expr = |sink_col_idx: usize| -> Result<ExprImpl> {
748            derive_sink_to_table_expr(sink_schema, sink_col_idx, column.data_type())
749        };
750
751        // If users specified the columns to be inserted e.g. `CREATE SINK s INTO t(a, b)`, the expressions of `Project` will be generated accordingly.
752        // The missing columns will be filled with default value (`null` if not explicitly defined).
753        // Otherwise, e.g. `CREATE SINK s INTO t`, the columns will be matched by their order in `select` query and the target table.
754        #[allow(clippy::collapsible_else_if)]
755        if user_specified_columns {
756            if let Some(idx) = sink_visible_col_idxes_by_name.get(column.name()) {
757                exprs.push(sink_col_expr(*idx)?);
758            } else {
759                exprs.push(default_col_expr());
760            }
761        } else {
762            if idx < sink_visible_col_idxes.len() {
763                exprs.push(sink_col_expr(sink_visible_col_idxes[idx])?);
764            } else {
765                exprs.push(default_col_expr());
766            };
767        }
768    }
769    Ok(exprs)
770}
771
772/// Transforms the (format, encode, options) from sqlparser AST into an internal struct `SinkFormatDesc`.
773/// This is an analogy to (part of) [`crate::handler::create_source::bind_columns_from_source`]
774/// which transforms sqlparser AST `SourceSchemaV2` into `StreamSourceInfo`.
775fn bind_sink_format_desc(
776    session: &SessionImpl,
777    value: FormatEncodeOptions,
778) -> Result<SinkFormatDesc> {
779    use risingwave_connector::sink::catalog::{SinkEncode, SinkFormat};
780    use risingwave_connector::sink::encoder::TimestamptzHandlingMode;
781    use risingwave_sqlparser::ast::{Encode as E, Format as F};
782
783    let format = match value.format {
784        F::Plain => SinkFormat::AppendOnly,
785        F::Upsert => SinkFormat::Upsert,
786        F::Debezium => SinkFormat::Debezium,
787        f @ (F::Native | F::DebeziumMongo | F::Maxwell | F::Canal | F::None) => {
788            return Err(ErrorCode::BindError(format!("sink format unsupported: {f}")).into());
789        }
790    };
791    let encode = match value.row_encode {
792        E::Json => SinkEncode::Json,
793        E::Protobuf => SinkEncode::Protobuf,
794        E::Avro => SinkEncode::Avro,
795        E::Template => SinkEncode::Template,
796        E::Parquet => SinkEncode::Parquet,
797        E::Bytes => SinkEncode::Bytes,
798        e @ (E::Native | E::Csv | E::None | E::Text) => {
799            return Err(ErrorCode::BindError(format!("sink encode unsupported: {e}")).into());
800        }
801    };
802
803    let mut key_encode = None;
804    if let Some(encode) = value.key_encode {
805        match encode {
806            E::Text => key_encode = Some(SinkEncode::Text),
807            E::Bytes => key_encode = Some(SinkEncode::Bytes),
808            _ => {
809                return Err(ErrorCode::BindError(format!(
810                    "sink key encode unsupported: {encode}, only TEXT and BYTES supported"
811                ))
812                .into());
813            }
814        }
815    }
816
817    let (props, connection_type_flag, schema_registry_conn_ref) =
818        resolve_connection_ref_and_secret_ref(
819            WithOptions::try_from(value.row_options.as_slice())?,
820            session,
821            Some(TelemetryDatabaseObject::Sink),
822        )?;
823    ensure_connection_type_allowed(
824        connection_type_flag,
825        &SINK_ALLOWED_CONNECTION_SCHEMA_REGISTRY,
826    )?;
827    let (mut options, secret_refs) = props.into_parts();
828
829    options
830        .entry(TimestamptzHandlingMode::OPTION_KEY.to_owned())
831        .or_insert(TimestamptzHandlingMode::FRONTEND_DEFAULT.to_owned());
832
833    Ok(SinkFormatDesc {
834        format,
835        encode,
836        options,
837        secret_refs,
838        key_encode,
839        connection_id: schema_registry_conn_ref,
840    })
841}
842
843static CONNECTORS_COMPATIBLE_FORMATS: LazyLock<HashMap<String, HashMap<Format, Vec<Encode>>>> =
844    LazyLock::new(|| {
845        use risingwave_connector::sink::Sink as _;
846        use risingwave_connector::sink::file_sink::azblob::AzblobSink;
847        use risingwave_connector::sink::file_sink::fs::FsSink;
848        use risingwave_connector::sink::file_sink::gcs::GcsSink;
849        use risingwave_connector::sink::file_sink::opendal_sink::FileSink;
850        use risingwave_connector::sink::file_sink::s3::S3Sink;
851        use risingwave_connector::sink::file_sink::webhdfs::WebhdfsSink;
852        use risingwave_connector::sink::google_pubsub::GooglePubSubSink;
853        use risingwave_connector::sink::kafka::KafkaSink;
854        use risingwave_connector::sink::kinesis::KinesisSink;
855        use risingwave_connector::sink::mqtt::MqttSink;
856        use risingwave_connector::sink::pulsar::PulsarSink;
857        use risingwave_connector::sink::redis::RedisSink;
858
859        convert_args!(hashmap!(
860                GooglePubSubSink::SINK_NAME => hashmap!(
861                    Format::Plain => vec![Encode::Json],
862                ),
863                KafkaSink::SINK_NAME => hashmap!(
864                    Format::Plain => vec![Encode::Json, Encode::Avro, Encode::Protobuf, Encode::Bytes],
865                    Format::Upsert => vec![Encode::Json, Encode::Avro, Encode::Protobuf],
866                    Format::Debezium => vec![Encode::Json],
867                ),
868                FileSink::<S3Sink>::SINK_NAME => hashmap!(
869                    Format::Plain => vec![Encode::Parquet, Encode::Json],
870                ),
871                FileSink::<SnowflakeSink>::SINK_NAME => hashmap!(
872                    Format::Plain => vec![Encode::Parquet, Encode::Json],
873                ),
874                FileSink::<GcsSink>::SINK_NAME => hashmap!(
875                    Format::Plain => vec![Encode::Parquet, Encode::Json],
876                ),
877                FileSink::<AzblobSink>::SINK_NAME => hashmap!(
878                    Format::Plain => vec![Encode::Parquet, Encode::Json],
879                ),
880                FileSink::<WebhdfsSink>::SINK_NAME => hashmap!(
881                    Format::Plain => vec![Encode::Parquet, Encode::Json],
882                ),
883                FileSink::<FsSink>::SINK_NAME => hashmap!(
884                    Format::Plain => vec![Encode::Parquet, Encode::Json],
885                ),
886                KinesisSink::SINK_NAME => hashmap!(
887                    Format::Plain => vec![Encode::Json],
888                    Format::Upsert => vec![Encode::Json],
889                    Format::Debezium => vec![Encode::Json],
890                ),
891                MqttSink::SINK_NAME => hashmap!(
892                    Format::Plain => vec![Encode::Json, Encode::Protobuf],
893                ),
894                PulsarSink::SINK_NAME => hashmap!(
895                    Format::Plain => vec![Encode::Json],
896                    Format::Upsert => vec![Encode::Json],
897                    Format::Debezium => vec![Encode::Json],
898                ),
899                RedisSink::SINK_NAME => hashmap!(
900                    Format::Plain => vec![Encode::Json, Encode::Template],
901                    Format::Upsert => vec![Encode::Json, Encode::Template],
902                ),
903        ))
904    });
905
906pub fn validate_compatibility(connector: &str, format_desc: &FormatEncodeOptions) -> Result<()> {
907    let compatible_formats = CONNECTORS_COMPATIBLE_FORMATS
908        .get(connector)
909        .ok_or_else(|| {
910            ErrorCode::BindError(format!(
911                "connector {} is not supported by FORMAT ... ENCODE ... syntax",
912                connector
913            ))
914        })?;
915    let compatible_encodes = compatible_formats.get(&format_desc.format).ok_or_else(|| {
916        ErrorCode::BindError(format!(
917            "connector {} does not support format {:?}",
918            connector, format_desc.format
919        ))
920    })?;
921    if !compatible_encodes.contains(&format_desc.row_encode) {
922        return Err(ErrorCode::BindError(format!(
923            "connector {} does not support format {:?} with encode {:?}",
924            connector, format_desc.format, format_desc.row_encode
925        ))
926        .into());
927    }
928
929    // only allow Kafka connector work with `bytes` as key encode
930    if let Some(encode) = &format_desc.key_encode
931        && connector != KAFKA_SINK
932        && matches!(encode, Encode::Bytes)
933    {
934        return Err(ErrorCode::BindError(format!(
935            "key encode bytes only works with kafka connector, but found {}",
936            connector
937        ))
938        .into());
939    }
940
941    Ok(())
942}
943
944#[cfg(test)]
945pub mod tests {
946    use risingwave_common::catalog::{DEFAULT_DATABASE_NAME, DEFAULT_SCHEMA_NAME};
947
948    use crate::catalog::root_catalog::SchemaPath;
949    use crate::test_utils::{LocalFrontend, PROTO_FILE_DATA, create_proto_file};
950
951    #[tokio::test]
952    async fn test_create_sink_handler() {
953        let proto_file = create_proto_file(PROTO_FILE_DATA);
954        let sql = format!(
955            r#"CREATE SOURCE t1
956    WITH (connector = 'kafka', kafka.topic = 'abc', kafka.brokers = 'localhost:1001')
957    FORMAT PLAIN ENCODE PROTOBUF (message = '.test.TestRecord', schema.location = 'file://{}')"#,
958            proto_file.path().to_str().unwrap()
959        );
960        let frontend = LocalFrontend::new(Default::default()).await;
961        frontend.run_sql(sql).await.unwrap();
962
963        let sql = "create materialized view mv1 as select t1.country from t1;";
964        frontend.run_sql(sql).await.unwrap();
965
966        let sql = r#"CREATE SINK snk1 FROM mv1
967                    WITH (connector = 'jdbc', mysql.endpoint = '127.0.0.1:3306', mysql.table =
968                        '<table_name>', mysql.database = '<database_name>', mysql.user = '<user_name>',
969                        mysql.password = '<password>', type = 'append-only', force_append_only = 'true');"#.to_owned();
970        frontend.run_sql(sql).await.unwrap();
971
972        let session = frontend.session_ref();
973        let catalog_reader = session.env().catalog_reader().read_guard();
974        let schema_path = SchemaPath::Name(DEFAULT_SCHEMA_NAME);
975
976        // Check source exists.
977        let (source, _) = catalog_reader
978            .get_source_by_name(DEFAULT_DATABASE_NAME, schema_path, "t1")
979            .unwrap();
980        assert_eq!(source.name, "t1");
981
982        // Check table exists.
983        let (table, schema_name) = catalog_reader
984            .get_created_table_by_name(DEFAULT_DATABASE_NAME, schema_path, "mv1")
985            .unwrap();
986        assert_eq!(table.name(), "mv1");
987
988        // Check sink exists.
989        let (sink, _) = catalog_reader
990            .get_created_sink_by_name(DEFAULT_DATABASE_NAME, SchemaPath::Name(schema_name), "snk1")
991            .unwrap();
992        assert_eq!(sink.name, "snk1");
993    }
994}