You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@inlong.apache.org by GitBox <gi...@apache.org> on 2022/07/05 14:28:51 UTC

[GitHub] [inlong] Greedyu opened a new pull request, #4882: [INLONG-4814][Sort][Manager] please declare primary key for sink table when query contains update/delete record

Greedyu opened a new pull request, #4882:
URL: https://github.com/apache/inlong/pull/4882

   Handling exceptions: please declare primary key for sink table when query contains update/delete record.
   
   - Fixes #4814 
   
   Briefly describe ideas
   
   1. The problem is that with flink JDBC SQL you have to specify the primary key.
   ![wecom-temp-5cf30d79023ca92a09bfb0eaee00ac84](https://user-images.githubusercontent.com/20356765/177348452-d69faf2a-cc61-4d8e-bbde-655f3309b610.png)
   
   2. When the Sort module sees the processing of primaryKey, it should be a problem with the parameters in the node. Gotta go check it out
   ![wecom-temp-ae9621b69bbce4feb44abe2b79454ee2](https://user-images.githubusercontent.com/20356765/177348105-93502218-fd38-4f0d-8f90-f75caa7929a8.png)
   
   3. Debug test locates that the primaryKey in the file read by sort is indeed empty
   <img width="1581" alt="image" src="https://user-images.githubusercontent.com/20356765/177348864-00113314-63bb-4c8c-a540-10cc42af4d9a.png">
   
   4.  To query the source of assignment of GroupInfo.StreamInfo.nodes, there are only CreateSortConfigListenerV2 and CreateStreamSortConfigListener classes, and they are both created by the createNodesForStream() method.
   ![image](https://user-images.githubusercontent.com/20356765/177351283-b31ecc2e-507e-4fd3-8461-9fb69db6b4bf.png)
   
   5. Test after bug fixes
   <img width="1435" alt="image" src="https://user-images.githubusercontent.com/20356765/177351479-b449221e-0169-401a-8996-51f29a07cc99.png">
   
   
   


-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: commits-unsubscribe@inlong.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [inlong] thesumery commented on pull request #4882: [INLONG-4814][Sort][Manager] Declare the primary key for sink table when the query contains update/delete

Posted by GitBox <gi...@apache.org>.
thesumery commented on PR #4882:
URL: https://github.com/apache/inlong/pull/4882#issuecomment-1175715305

   There are some reasons that clickhouse sink didn't release the ability of update/delete:
   1. clickhouse `update` operations is heavy, it will takes precedence over all others operation to execute, so it has a poor performance. It is more suitable for batch scenes but not streaming scenes where operations are performed frequently.
   


-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: commits-unsubscribe@inlong.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [inlong] Greedyu commented on a diff in pull request #4882: [INLONG-4814][Sort][Manager] Declare the primary key for sink table when the query contains update/delete

Posted by GitBox <gi...@apache.org>.
Greedyu commented on code in PR #4882:
URL: https://github.com/apache/inlong/pull/4882#discussion_r918661474


##########
inlong-sort/sort-common/src/test/java/org/apache/inlong/sort/protocol/node/load/ClickHouseLoadNodeTest.java:
##########
@@ -44,6 +44,11 @@ public Node getTestObject() {
                 "ck_demo",
                 "jdbc:clickhouse://localhost:8023/default",
                 "root",
-                "root");
+                "root",
+                "Log",

Review Comment:
   yes, [https://clickhouse.com/docs/en/engines/table-engines/log-family/log/](https://clickhouse.com/docs/en/engines/table-engines/log-family/log/)



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: commits-unsubscribe@inlong.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [inlong] gong commented on a diff in pull request #4882: [INLONG-4814][Sort][Manager] Declare the primary key for sink table when the query contains update/delete

Posted by GitBox <gi...@apache.org>.
gong commented on code in PR #4882:
URL: https://github.com/apache/inlong/pull/4882#discussion_r934307281


##########
inlong-sort/sort-connectors/jdbc/src/main/java/org/apache/inlong/sort/jdbc/table/JdbcDialects.java:
##########
@@ -44,6 +45,7 @@ public final class JdbcDialects {
         DIALECTS.add(new TDSQLPostgresDialect());
         DIALECTS.add(new SqlServerDialect());
         DIALECTS.add(new OracleDialect());
+        DIALECTS.add(new ClickHouseDialect());

Review Comment:
   It is redundancy. Because clickhouse load node use `dialect-impl` to register dialect.



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: commits-unsubscribe@inlong.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [inlong] EMsnap merged pull request #4882: [INLONG-4814][Sort][Manager] Declare the primary key for sink table when the query contains update/delete

Posted by GitBox <gi...@apache.org>.
EMsnap merged PR #4882:
URL: https://github.com/apache/inlong/pull/4882


-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: commits-unsubscribe@inlong.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [inlong] Greedyu commented on a diff in pull request #4882: [INLONG-4814][Sort][Manager] Declare the primary key for sink table when the query contains update/delete

Posted by GitBox <gi...@apache.org>.
Greedyu commented on code in PR #4882:
URL: https://github.com/apache/inlong/pull/4882#discussion_r934091338


##########
inlong-sort/sort-common/src/main/java/org/apache/inlong/sort/protocol/node/load/ClickHouseLoadNode.java:
##########
@@ -73,12 +85,21 @@ public ClickHouseLoadNode(@JsonProperty("id") String id,
             @Nonnull @JsonProperty("tableName") String tableName,
             @Nonnull @JsonProperty("url") String url,
             @Nonnull @JsonProperty("userName") String userName,
-            @Nonnull @JsonProperty("passWord") String password) {
+            @Nonnull @JsonProperty("passWord") String password,
+            @JsonProperty("engine") String engine,
+            @JsonProperty("partitionBy") String partitionBy,

Review Comment:
   These are important attributes of clickhouse. Even if you don't use them directly now, you can add them first.



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: commits-unsubscribe@inlong.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [inlong] haibo-duan commented on a diff in pull request #4882: [INLONG-4814][Sort][Manager] Declare the primary key for sink table when the query contains update/delete

Posted by GitBox <gi...@apache.org>.
haibo-duan commented on code in PR #4882:
URL: https://github.com/apache/inlong/pull/4882#discussion_r933949933


##########
inlong-sort/sort-end-to-end-tests/src/test/java/org/apache/inlong/sort/tests/ClickHouseCase.java:
##########
@@ -0,0 +1,161 @@
+/*
+ *  Licensed to the Apache Software Foundation (ASF) under one
+ *  or more contributor license agreements.  See the NOTICE file
+ *  distributed with this work for additional information
+ *  regarding copyright ownership.  The ASF licenses this file
+ *  to you under the Apache License, Version 2.0 (the
+ *  "License"); you may not use this file except in compliance
+ *  with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ *  Unless required by applicable law or agreed to in writing, software
+ *  distributed under the License is distributed on an "AS IS" BASIS,
+ *  WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ *  See the License for the specific language governing permissions and
+ *  limitations under the License.
+ */
+
+package org.apache.inlong.sort.tests;
+
+import org.apache.inlong.sort.tests.utils.FlinkContainerTestEnv;
+import org.apache.inlong.sort.tests.utils.JdbcProxy;
+import org.apache.inlong.sort.tests.utils.TestUtils;
+import org.junit.After;
+import org.junit.Before;
+import org.junit.ClassRule;
+import org.junit.Test;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+import org.testcontainers.containers.ClickHouseContainer;
+import org.testcontainers.containers.output.Slf4jLogConsumer;
+
+import java.net.URISyntaxException;
+import java.nio.file.Path;
+import java.nio.file.Paths;
+import java.sql.Connection;
+import java.sql.DriverManager;
+import java.sql.SQLException;
+import java.sql.Statement;
+import java.time.Duration;
+import java.util.Arrays;
+import java.util.List;
+
+/**
+ * End-to-end tests
+ * Test flink sql mysql cdc to clickHouse
+ */
+public class ClickHouseCase extends FlinkContainerTestEnv {
+
+    private static final Logger LOG = LoggerFactory.getLogger(ClickHouseCase.class);
+
+    private static final Path jdbcJar = TestUtils.getResource("sort-connector-jdbc.jar");
+    private static final Path mysqlJar = TestUtils.getResource("sort-connector-mysql-cdc.jar");
+    private static final Path mysqlJdbcJar = TestUtils.getResource("mysql-driver.jar");
+    // Can't use getResource("xxx").getPath(), windows will don't know that path
+    private static final String sqlFile;
+
+    static {
+        try {
+            sqlFile = Paths.get(ClickHouseCase.class.getResource("/flinkSql/clickhouse_test.sql").toURI()).toString();
+        } catch (URISyntaxException e) {
+            throw new RuntimeException(e);
+        }
+    }
+
+    @ClassRule
+    public static final ClickHouseContainer CLICK_HOUSE_CONTAINER = (ClickHouseContainer) new ClickHouseContainer(
+            "yandex/clickhouse-server:20.1.8.41")
+            .withNetwork(NETWORK)
+            .withNetworkAliases("clickhouse")
+            .withLogConsumer(new Slf4jLogConsumer(LOG));
+
+    @Before
+    public void setup() {
+        initializeMysqlTable();
+        initializeClickHouseTable();
+    }
+
+    @After
+    public void teardown() {
+        if (CLICK_HOUSE_CONTAINER != null) {
+            CLICK_HOUSE_CONTAINER.stop();
+        }
+    }
+
+    private void initializeClickHouseTable() {
+        try {
+            Class.forName(CLICK_HOUSE_CONTAINER.getDriverClassName());
+            Connection conn = DriverManager
+                    .getConnection(CLICK_HOUSE_CONTAINER.getJdbcUrl(), CLICK_HOUSE_CONTAINER.getUsername(),

Review Comment:
   please close the JDBC connection At the end of your JDBC program.



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: commits-unsubscribe@inlong.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [inlong] Greedyu commented on a diff in pull request #4882: [INLONG-4814][Sort][Manager] Declare the primary key for sink table when the query contains update/delete

Posted by GitBox <gi...@apache.org>.
Greedyu commented on code in PR #4882:
URL: https://github.com/apache/inlong/pull/4882#discussion_r933994789


##########
inlong-sort/sort-end-to-end-tests/src/test/java/org/apache/inlong/sort/tests/ClickHouseCase.java:
##########
@@ -0,0 +1,161 @@
+/*
+ *  Licensed to the Apache Software Foundation (ASF) under one
+ *  or more contributor license agreements.  See the NOTICE file
+ *  distributed with this work for additional information
+ *  regarding copyright ownership.  The ASF licenses this file
+ *  to you under the Apache License, Version 2.0 (the
+ *  "License"); you may not use this file except in compliance
+ *  with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ *  Unless required by applicable law or agreed to in writing, software
+ *  distributed under the License is distributed on an "AS IS" BASIS,
+ *  WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ *  See the License for the specific language governing permissions and
+ *  limitations under the License.
+ */
+
+package org.apache.inlong.sort.tests;
+
+import org.apache.inlong.sort.tests.utils.FlinkContainerTestEnv;
+import org.apache.inlong.sort.tests.utils.JdbcProxy;
+import org.apache.inlong.sort.tests.utils.TestUtils;
+import org.junit.After;
+import org.junit.Before;
+import org.junit.ClassRule;
+import org.junit.Test;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+import org.testcontainers.containers.ClickHouseContainer;
+import org.testcontainers.containers.output.Slf4jLogConsumer;
+
+import java.net.URISyntaxException;
+import java.nio.file.Path;
+import java.nio.file.Paths;
+import java.sql.Connection;
+import java.sql.DriverManager;
+import java.sql.SQLException;
+import java.sql.Statement;
+import java.time.Duration;
+import java.util.Arrays;
+import java.util.List;
+
+/**
+ * End-to-end tests
+ * Test flink sql mysql cdc to clickHouse
+ */
+public class ClickHouseCase extends FlinkContainerTestEnv {
+
+    private static final Logger LOG = LoggerFactory.getLogger(ClickHouseCase.class);
+
+    private static final Path jdbcJar = TestUtils.getResource("sort-connector-jdbc.jar");
+    private static final Path mysqlJar = TestUtils.getResource("sort-connector-mysql-cdc.jar");
+    private static final Path mysqlJdbcJar = TestUtils.getResource("mysql-driver.jar");
+    // Can't use getResource("xxx").getPath(), windows will don't know that path
+    private static final String sqlFile;
+
+    static {
+        try {
+            sqlFile = Paths.get(ClickHouseCase.class.getResource("/flinkSql/clickhouse_test.sql").toURI()).toString();
+        } catch (URISyntaxException e) {
+            throw new RuntimeException(e);
+        }
+    }
+
+    @ClassRule
+    public static final ClickHouseContainer CLICK_HOUSE_CONTAINER = (ClickHouseContainer) new ClickHouseContainer(
+            "yandex/clickhouse-server:20.1.8.41")
+            .withNetwork(NETWORK)
+            .withNetworkAliases("clickhouse")
+            .withLogConsumer(new Slf4jLogConsumer(LOG));
+
+    @Before
+    public void setup() {
+        initializeMysqlTable();
+        initializeClickHouseTable();
+    }
+
+    @After
+    public void teardown() {
+        if (CLICK_HOUSE_CONTAINER != null) {
+            CLICK_HOUSE_CONTAINER.stop();
+        }
+    }
+
+    private void initializeClickHouseTable() {
+        try {
+            Class.forName(CLICK_HOUSE_CONTAINER.getDriverClassName());
+            Connection conn = DriverManager
+                    .getConnection(CLICK_HOUSE_CONTAINER.getJdbcUrl(), CLICK_HOUSE_CONTAINER.getUsername(),

Review Comment:
   resolve



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: commits-unsubscribe@inlong.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [inlong] thesumery commented on a diff in pull request #4882: [INLONG-4814][Sort][Manager] Declare the primary key for sink table when the query contains update/delete

Posted by GitBox <gi...@apache.org>.
thesumery commented on code in PR #4882:
URL: https://github.com/apache/inlong/pull/4882#discussion_r934097558


##########
inlong-sort/sort-common/src/main/java/org/apache/inlong/sort/protocol/node/load/ClickHouseLoadNode.java:
##########
@@ -100,7 +121,7 @@ public String genTableName() {
 
     @Override
     public String getPrimaryKey() {
-        return super.getPrimaryKey();
+        return primaryKey;
     }
 
     @Override

Review Comment:
   MayBe this is unnecessary.Clickhouse sink currently is implement by jdbc connector.But [jdbc table connector  sink](https://nightlies.apache.org/flink/flink-docs-master/api/java/org/apache/flink/connector/jdbc/table/JdbcDynamicTableSink.html) don't support applyPartition currently. So get partitionFiled is useless.



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: commits-unsubscribe@inlong.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [inlong] thesumery commented on a diff in pull request #4882: [INLONG-4814][Sort][Manager] Declare the primary key for sink table when the query contains update/delete

Posted by GitBox <gi...@apache.org>.
thesumery commented on code in PR #4882:
URL: https://github.com/apache/inlong/pull/4882#discussion_r934319199


##########
inlong-sort/sort-common/src/main/java/org/apache/inlong/sort/protocol/node/load/ClickHouseLoadNode.java:
##########
@@ -73,12 +85,21 @@ public ClickHouseLoadNode(@JsonProperty("id") String id,
             @Nonnull @JsonProperty("tableName") String tableName,
             @Nonnull @JsonProperty("url") String url,
             @Nonnull @JsonProperty("userName") String userName,
-            @Nonnull @JsonProperty("passWord") String password) {
+            @Nonnull @JsonProperty("passWord") String password,
+            @JsonProperty("engine") String engine,
+            @JsonProperty("partitionBy") String partitionBy,

Review Comment:
   > why add `partitionBy`、`orderBy`、`engine` params? Where are they used?
   
   ok, allright , although it is useless temporally.



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: commits-unsubscribe@inlong.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [inlong] Greedyu commented on a diff in pull request #4882: [INLONG-4814][Sort][Manager] Declare the primary key for sink table when the query contains update/delete

Posted by GitBox <gi...@apache.org>.
Greedyu commented on code in PR #4882:
URL: https://github.com/apache/inlong/pull/4882#discussion_r934099236


##########
inlong-sort/sort-common/src/main/java/org/apache/inlong/sort/protocol/node/load/ClickHouseLoadNode.java:
##########
@@ -100,7 +121,7 @@ public String genTableName() {
 
     @Override
     public String getPrimaryKey() {
-        return super.getPrimaryKey();
+        return primaryKey;
     }
 
     @Override

Review Comment:
   partitionFiled is useless, the method for obtaining partitionFiled is not modified here, but the getPrimaryKey() method is used



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: commits-unsubscribe@inlong.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [inlong] healchow commented on a diff in pull request #4882: [INLONG-4814][Sort][Manager] Declare the primary key for sink table when the query contains update/delete

Posted by GitBox <gi...@apache.org>.
healchow commented on code in PR #4882:
URL: https://github.com/apache/inlong/pull/4882#discussion_r917377346


##########
inlong-sort/sort-common/src/test/java/org/apache/inlong/sort/protocol/node/load/ClickHouseLoadNodeTest.java:
##########
@@ -44,6 +44,11 @@ public Node getTestObject() {
                 "ck_demo",
                 "jdbc:clickhouse://localhost:8023/default",
                 "root",
-                "root");
+                "root",
+                "Log",

Review Comment:
   The `Log` was one of the engines of ClickHouse?



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: commits-unsubscribe@inlong.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [inlong] gong commented on a diff in pull request #4882: [INLONG-4814][Sort][Manager] Declare the primary key for sink table when the query contains update/delete

Posted by GitBox <gi...@apache.org>.
gong commented on code in PR #4882:
URL: https://github.com/apache/inlong/pull/4882#discussion_r934347173


##########
inlong-sort/sort-connectors/jdbc/src/main/java/org/apache/inlong/sort/jdbc/table/JdbcDialects.java:
##########
@@ -44,6 +45,7 @@ public final class JdbcDialects {
         DIALECTS.add(new TDSQLPostgresDialect());
         DIALECTS.add(new SqlServerDialect());
         DIALECTS.add(new OracleDialect());
+        DIALECTS.add(new ClickHouseDialect());

Review Comment:
   > If it is not added to the list, the unit test will be abnormal here ![企业微信截图_f539e39a-2dcc-4498-99d7-bb1d593e032f](https://user-images.githubusercontent.com/20356765/182117661-b52daf46-6d1e-4b13-b2fd-f3d07d48f35c.png)
   
   UT error reason is  https://github.com/apache/inlong/pull/4882#discussion_r934339275. 



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: commits-unsubscribe@inlong.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [inlong] gong commented on a diff in pull request #4882: [INLONG-4814][Sort][Manager] Declare the primary key for sink table when the query contains update/delete

Posted by GitBox <gi...@apache.org>.
gong commented on code in PR #4882:
URL: https://github.com/apache/inlong/pull/4882#discussion_r934358441


##########
inlong-sort/sort-end-to-end-tests/src/test/resources/flinkSql/clickhouse_test.sql:
##########
@@ -0,0 +1,33 @@
+CREATE TABLE test_input1 (
+    `id` INT primary key,
+    name STRING,
+    description STRING
+) WITH (
+    'connector' = 'mysql-cdc-inlong',
+    'hostname' = 'mysql',
+    'port' = '3306',
+    'username' = 'inlong',
+    'password' = 'inlong',
+    'database-name' = 'test',
+    'table-name' = 'test_input1',
+    'scan.incremental.snapshot.chunk.size' = '4',
+    'scan.incremental.snapshot.enabled' = 'false'
+);
+
+CREATE TABLE test_output1 (
+    `id` INT primary key,
+    name STRING,
+    description STRING
+) WITH (
+    'connector' = 'jdbc-inlong',
+    'url' = 'jdbc:clickhouse://clickhouse:8123/default',
+    'table-name' = 'test_output1',
+    'username' = 'default',
+    'password' = ''

Review Comment:
   > I understand what you mean, but isn't it better to be able to identify the dialect based on the url? It's not an elegant way to test the need to add dialect configuration.
   
   clickhouse url is ok. posrgres variant(greenplum, gaussdb, tdsqlpostgresql) url will same. So I just suggest keep unify. 



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: commits-unsubscribe@inlong.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [inlong] Greedyu commented on a diff in pull request #4882: [INLONG-4814][Sort][Manager] Declare the primary key for sink table when the query contains update/delete

Posted by GitBox <gi...@apache.org>.
Greedyu commented on code in PR #4882:
URL: https://github.com/apache/inlong/pull/4882#discussion_r934398627


##########
inlong-sort/sort-common/src/main/java/org/apache/inlong/sort/protocol/node/load/ClickHouseLoadNode.java:
##########
@@ -73,12 +85,21 @@ public ClickHouseLoadNode(@JsonProperty("id") String id,
             @Nonnull @JsonProperty("tableName") String tableName,
             @Nonnull @JsonProperty("url") String url,
             @Nonnull @JsonProperty("userName") String userName,
-            @Nonnull @JsonProperty("passWord") String password) {
+            @Nonnull @JsonProperty("passWord") String password,
+            @JsonProperty("engine") String engine,
+            @JsonProperty("partitionBy") String partitionBy,

Review Comment:
   These properties have been removed



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: commits-unsubscribe@inlong.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [inlong] thesumery commented on pull request #4882: [INLONG-4814][Sort][Manager] Declare the primary key for sink table when the query contains update/delete

Posted by GitBox <gi...@apache.org>.
thesumery commented on PR #4882:
URL: https://github.com/apache/inlong/pull/4882#issuecomment-1201931465

   LGTM


-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: commits-unsubscribe@inlong.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [inlong] Greedyu commented on pull request #4882: [INLONG-4814][Sort][Manager] Declare the primary key for sink table when the query contains update/delete

Posted by GitBox <gi...@apache.org>.
Greedyu commented on PR #4882:
URL: https://github.com/apache/inlong/pull/4882#issuecomment-1175795626

   > @yunqingmoswu @Greedyu If user want to use update scenes, how about this two different solution:
   > 
   > 1. Import `update` ability,but should tell users the limitation(all code and docs should comment it). In additional what you have submitte in this pr, you should also change update and delete statement in `ClickHouseDialect`.
   > 2. Enable upstream `append-mode`,so upstream produce record is only append stream. Add `time` and `rowkind` field in clickhouse table, those two filed comes from upstream table metadata fileds.Usersdecides how to deduplicate the downstream olap sql.
   
   I thought this was just a bug before, but I didn't expect this ability to be purposely not supported.
   But there is still a question, why does the normal mode create group also generate Flink cdc sql


-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: commits-unsubscribe@inlong.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [inlong] thesumery commented on a diff in pull request #4882: [INLONG-4814][Sort][Manager] Declare the primary key for sink table when the query contains update/delete

Posted by GitBox <gi...@apache.org>.
thesumery commented on code in PR #4882:
URL: https://github.com/apache/inlong/pull/4882#discussion_r934087199


##########
inlong-sort/sort-common/src/main/java/org/apache/inlong/sort/protocol/node/load/ClickHouseLoadNode.java:
##########
@@ -73,12 +85,21 @@ public ClickHouseLoadNode(@JsonProperty("id") String id,
             @Nonnull @JsonProperty("tableName") String tableName,
             @Nonnull @JsonProperty("url") String url,
             @Nonnull @JsonProperty("userName") String userName,
-            @Nonnull @JsonProperty("passWord") String password) {
+            @Nonnull @JsonProperty("passWord") String password,
+            @JsonProperty("engine") String engine,
+            @JsonProperty("partitionBy") String partitionBy,

Review Comment:
   why add `partitionBy`、`orderBy`、`engine` params?  Where are they used?



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: commits-unsubscribe@inlong.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [inlong] Greedyu commented on a diff in pull request #4882: [INLONG-4814][Sort][Manager] Declare the primary key for sink table when the query contains update/delete

Posted by GitBox <gi...@apache.org>.
Greedyu commented on code in PR #4882:
URL: https://github.com/apache/inlong/pull/4882#discussion_r934352925


##########
inlong-sort/sort-end-to-end-tests/src/test/resources/flinkSql/clickhouse_test.sql:
##########
@@ -0,0 +1,33 @@
+CREATE TABLE test_input1 (
+    `id` INT primary key,
+    name STRING,
+    description STRING
+) WITH (
+    'connector' = 'mysql-cdc-inlong',
+    'hostname' = 'mysql',
+    'port' = '3306',
+    'username' = 'inlong',
+    'password' = 'inlong',
+    'database-name' = 'test',
+    'table-name' = 'test_input1',
+    'scan.incremental.snapshot.chunk.size' = '4',
+    'scan.incremental.snapshot.enabled' = 'false'
+);
+
+CREATE TABLE test_output1 (
+    `id` INT primary key,
+    name STRING,
+    description STRING
+) WITH (
+    'connector' = 'jdbc-inlong',
+    'url' = 'jdbc:clickhouse://clickhouse:8123/default',
+    'table-name' = 'test_output1',
+    'username' = 'default',
+    'password' = ''

Review Comment:
   I understand what you mean, but isn't it better to be able to identify the dialect based on the url? It's not an elegant way to test the need to add dialect configuration.



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: commits-unsubscribe@inlong.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [inlong] Greedyu commented on pull request #4882: [INLONG-4814][Sort][Manager] Declare the primary key for sink table when the query contains update/delete

Posted by GitBox <gi...@apache.org>.
Greedyu commented on PR #4882:
URL: https://github.com/apache/inlong/pull/4882#issuecomment-1200653630

   > added
   
   Add in ClickHouseLoadNode class


-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: commits-unsubscribe@inlong.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [inlong] Greedyu commented on a diff in pull request #4882: [INLONG-4814][Sort][Manager] Declare the primary key for sink table when the query contains update/delete

Posted by GitBox <gi...@apache.org>.
Greedyu commented on code in PR #4882:
URL: https://github.com/apache/inlong/pull/4882#discussion_r934323340


##########
inlong-sort/sort-connectors/jdbc/src/main/java/org/apache/inlong/sort/jdbc/table/JdbcDialects.java:
##########
@@ -44,6 +45,7 @@ public final class JdbcDialects {
         DIALECTS.add(new TDSQLPostgresDialect());
         DIALECTS.add(new SqlServerDialect());
         DIALECTS.add(new OracleDialect());
+        DIALECTS.add(new ClickHouseDialect());

Review Comment:
   If it is not added to the list, the unit test will be abnormal here
   ![企业微信截图_f539e39a-2dcc-4498-99d7-bb1d593e032f](https://user-images.githubusercontent.com/20356765/182117661-b52daf46-6d1e-4b13-b2fd-f3d07d48f35c.png)
   



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: commits-unsubscribe@inlong.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [inlong] Greedyu commented on a diff in pull request #4882: [INLONG-4814][Sort][Manager] Declare the primary key for sink table when the query contains update/delete

Posted by GitBox <gi...@apache.org>.
Greedyu commented on code in PR #4882:
URL: https://github.com/apache/inlong/pull/4882#discussion_r934396665


##########
inlong-sort/sort-end-to-end-tests/src/test/resources/flinkSql/clickhouse_test.sql:
##########
@@ -0,0 +1,33 @@
+CREATE TABLE test_input1 (
+    `id` INT primary key,
+    name STRING,
+    description STRING
+) WITH (
+    'connector' = 'mysql-cdc-inlong',
+    'hostname' = 'mysql',
+    'port' = '3306',
+    'username' = 'inlong',
+    'password' = 'inlong',
+    'database-name' = 'test',
+    'table-name' = 'test_input1',
+    'scan.incremental.snapshot.chunk.size' = '4',
+    'scan.incremental.snapshot.enabled' = 'false'
+);
+
+CREATE TABLE test_output1 (
+    `id` INT primary key,
+    name STRING,
+    description STRING
+) WITH (
+    'connector' = 'jdbc-inlong',
+    'url' = 'jdbc:clickhouse://clickhouse:8123/default',
+    'table-name' = 'test_output1',
+    'username' = 'default',
+    'password' = ''

Review Comment:
   ok, it's been resolved



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: commits-unsubscribe@inlong.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [inlong] gong commented on a diff in pull request #4882: [INLONG-4814][Sort][Manager] Declare the primary key for sink table when the query contains update/delete

Posted by GitBox <gi...@apache.org>.
gong commented on code in PR #4882:
URL: https://github.com/apache/inlong/pull/4882#discussion_r934339275


##########
inlong-sort/sort-end-to-end-tests/src/test/resources/flinkSql/clickhouse_test.sql:
##########
@@ -0,0 +1,33 @@
+CREATE TABLE test_input1 (
+    `id` INT primary key,
+    name STRING,
+    description STRING
+) WITH (
+    'connector' = 'mysql-cdc-inlong',
+    'hostname' = 'mysql',
+    'port' = '3306',
+    'username' = 'inlong',
+    'password' = 'inlong',
+    'database-name' = 'test',
+    'table-name' = 'test_input1',
+    'scan.incremental.snapshot.chunk.size' = '4',
+    'scan.incremental.snapshot.enabled' = 'false'
+);
+
+CREATE TABLE test_output1 (
+    `id` INT primary key,
+    name STRING,
+    description STRING
+) WITH (
+    'connector' = 'jdbc-inlong',
+    'url' = 'jdbc:clickhouse://clickhouse:8123/default',
+    'table-name' = 'test_output1',
+    'username' = 'default',
+    'password' = ''

Review Comment:
   Add option "dialect-impl"="org.apache.inlong.sort.jdbc.dialect.ClickHouseDialect". It will can find custom dialect



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: commits-unsubscribe@inlong.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [inlong] yunqingmoswu commented on a diff in pull request #4882: [INLONG-4814][Sort][Manager] Declare the primary key for sink table when the query contains update/delete

Posted by GitBox <gi...@apache.org>.
yunqingmoswu commented on code in PR #4882:
URL: https://github.com/apache/inlong/pull/4882#discussion_r934312865


##########
inlong-sort/sort-common/src/main/java/org/apache/inlong/sort/protocol/node/load/ClickHouseLoadNode.java:
##########
@@ -73,12 +85,21 @@ public ClickHouseLoadNode(@JsonProperty("id") String id,
             @Nonnull @JsonProperty("tableName") String tableName,
             @Nonnull @JsonProperty("url") String url,
             @Nonnull @JsonProperty("userName") String userName,
-            @Nonnull @JsonProperty("passWord") String password) {
+            @Nonnull @JsonProperty("passWord") String password,
+            @JsonProperty("engine") String engine,
+            @JsonProperty("partitionBy") String partitionBy,

Review Comment:
   `partitionBy` can be replaced by `partitionFields` such as `HiveLoadNode`.



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: commits-unsubscribe@inlong.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [inlong] thesumery commented on pull request #4882: [INLONG-4814][Sort][Manager] Declare the primary key for sink table when the query contains update/delete

Posted by GitBox <gi...@apache.org>.
thesumery commented on PR #4882:
URL: https://github.com/apache/inlong/pull/4882#issuecomment-1175729688

   If user want to use update scenes, how about this two different solution:
   1. Import `update` ability,but should tell users the limitation(all code and docs should comment it). In additional what you have submitte in this pr, you should also change update and delete statement in `ClickHouseDialect`.
   2. Enable upstream `append-mode`,so upstream produce record is only append stream. Add `time` and `rowkind` field in clickhouse table, those two filed comes from upstream table metadata fileds.Usersdecides how to deduplicate the downstream olap sql.


-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: commits-unsubscribe@inlong.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org