From 3dd1393e0e88d1794879e0154564515d6aa4c95d Mon Sep 17 00:00:00 2001 From: William Wen <44139337+wenym1@users.noreply.github.com> Date: Fri, 16 Jun 2023 17:34:34 +0800 Subject: [PATCH] feat(sink): enable delta lake sink (#10374) --- src/connector/src/sink/remote.rs | 2 +- src/frontend/src/optimizer/plan_node/stream_sink.rs | 2 +- 2 files changed, 2 insertions(+), 2 deletions(-) diff --git a/src/connector/src/sink/remote.rs b/src/connector/src/sink/remote.rs index e91fccac23a19..542180f42c9ba 100644 --- a/src/connector/src/sink/remote.rs +++ b/src/connector/src/sink/remote.rs @@ -44,7 +44,7 @@ use super::catalog::SinkCatalog; use crate::sink::{record_to_json, Result, Sink, SinkError, TimestampHandlingMode}; use crate::ConnectorParams; -pub const VALID_REMOTE_SINKS: [&str; 3] = ["jdbc", "file", "iceberg"]; +pub const VALID_REMOTE_SINKS: [&str; 3] = ["jdbc", "iceberg", "deltalake"]; pub fn is_valid_remote_sink(connector_type: &str) -> bool { VALID_REMOTE_SINKS.contains(&connector_type) diff --git a/src/frontend/src/optimizer/plan_node/stream_sink.rs b/src/frontend/src/optimizer/plan_node/stream_sink.rs index d159abf709887..0e3a40e8ed09d 100644 --- a/src/frontend/src/optimizer/plan_node/stream_sink.rs +++ b/src/frontend/src/optimizer/plan_node/stream_sink.rs @@ -75,7 +75,7 @@ impl StreamSink { Distribution::Single => RequiredDist::single(), _ => { match properties.get("connector") { - Some(s) if s == "iceberg" => { + Some(s) if s == "iceberg" || s == "deltalake" => { // iceberg with multiple parallelism will fail easily with concurrent commit // on metadata // TODO: reset iceberg sink to have multiple parallelism