From 173779823dc856de03517d94e04230be5d58c556 Mon Sep 17 00:00:00 2001 From: 2013650523 <1067341434@qq.com> Date: Fri, 22 Jul 2022 16:20:06 +0800 Subject: [PATCH 01/26] 0 --- .../connector-kudu/pom.xml | 30 +++ .../seatunnel/kudu/config/KuduSinkConfig.java | 66 ++++++ .../kudu/config/KuduSourceConfig.java | 34 +++ .../kudu/kuduclient/KuduInputFormat.java | 173 ++++++++++++++++ .../kudu/kuduclient/KuduOutputFormat.java | 165 +++++++++++++++ .../kudu/kuduclient/KuduTypeMapper.java | 104 ++++++++++ .../kudu/sink/KuduAggregatedCommitInfo.java | 31 +++ .../seatunnel/kudu/sink/KuduCommitInfo.java | 32 +++ .../seatunnel/kudu/sink/KuduSink.java | 93 +++++++++ .../sink/KuduSinkAggregatedCommitter.java | 53 +++++ .../seatunnel/kudu/sink/KuduSinkState.java | 29 +++ .../seatunnel/kudu/sink/KuduSinkWriter.java | 82 ++++++++ .../seatunnel/kudu/source/KuduSource.java | 193 ++++++++++++++++++ .../kudu/source/KuduSourceReader.java | 116 +++++++++++ .../kudu/source/KuduSourceSplit.java | 37 ++++ .../source/KuduSourceSplitEnumerator.java | 132 ++++++++++++ .../kudu/source/PartitionParameter.java | 32 +++ .../seatunnel/kudu/state/KuduSinkState.java | 23 +++ .../main/resources/kudu_to_kudu_flink.conf | 60 ++++++ .../main/resources/kudu_to_kudu_spark.conf | 64 ++++++ 20 files changed, 1549 insertions(+) create mode 100644 seatunnel-connectors-v2/connector-kudu/pom.xml create mode 100644 seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/config/KuduSinkConfig.java create mode 100644 seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/config/KuduSourceConfig.java create mode 100644 seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/kuduclient/KuduInputFormat.java create mode 100644 seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/kuduclient/KuduOutputFormat.java create mode 100644 seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/kuduclient/KuduTypeMapper.java create mode 100644 seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/sink/KuduAggregatedCommitInfo.java create mode 100644 seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/sink/KuduCommitInfo.java create mode 100644 seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/sink/KuduSink.java create mode 100644 seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/sink/KuduSinkAggregatedCommitter.java create mode 100644 seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/sink/KuduSinkState.java create mode 100644 seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/sink/KuduSinkWriter.java create mode 100644 seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/source/KuduSource.java create mode 100644 seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/source/KuduSourceReader.java create mode 100644 seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/source/KuduSourceSplit.java create mode 100644 seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/source/KuduSourceSplitEnumerator.java create mode 100644 seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/source/PartitionParameter.java create mode 100644 seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/state/KuduSinkState.java create mode 100644 seatunnel-connectors-v2/connector-kudu/src/main/resources/kudu_to_kudu_flink.conf create mode 100644 seatunnel-connectors-v2/connector-kudu/src/main/resources/kudu_to_kudu_spark.conf diff --git a/seatunnel-connectors-v2/connector-kudu/pom.xml b/seatunnel-connectors-v2/connector-kudu/pom.xml new file mode 100644 index 00000000000..c8d3591abb1 --- /dev/null +++ b/seatunnel-connectors-v2/connector-kudu/pom.xml @@ -0,0 +1,30 @@ + + + + seatunnel-connectors-v2 + org.apache.seatunnel + ${revision} + + 4.0.0 + + connector-kudu + + + + org.apache.seatunnel + seatunnel-api + ${project.version} + + + org.apache.kudu + kudu-client + + + + org.apache.commons + commons-lang3 + + + \ No newline at end of file diff --git a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/config/KuduSinkConfig.java b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/config/KuduSinkConfig.java new file mode 100644 index 00000000000..8f4f3ae44bf --- /dev/null +++ b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/config/KuduSinkConfig.java @@ -0,0 +1,66 @@ +/* + * 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.seatunnel.connectors.seatunnel.kudu.config; + +import lombok.Data; +import lombok.NonNull; + +import org.apache.commons.lang3.StringUtils; +import org.apache.seatunnel.shade.com.typesafe.config.Config; + + + +@Data +public class KuduSinkConfig { + + private static final String KUDU_SAVE_MODE = "save_mode"; + private static final String KUDU_MASTER = "kudu_master"; + private static final String KUDU_TABLE_NAME = "kudu_table"; + + private SaveMode saveMode = SaveMode.APPEND; + + private String kuduMaster; + + /** + * Specifies the name of the table + */ + private String kuduTableName; + + public enum SaveMode { + APPEND(), + OVERWRITE(); + + public static SaveMode fromStr(String str) { + if ("overwrite".equals(str)) { + return OVERWRITE; + } else { + return APPEND; + } + } + } + + public KuduSinkConfig(@NonNull Config pluginConfig) { + + this.saveMode = StringUtils.isBlank(pluginConfig.getString(KUDU_SAVE_MODE)) ? SaveMode.APPEND : SaveMode.fromStr(pluginConfig.getString(KUDU_SAVE_MODE)); + + this.kuduMaster = pluginConfig.getString(KUDU_MASTER); + this.kuduTableName = pluginConfig.getString(KUDU_TABLE_NAME); + + + } +} diff --git a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/config/KuduSourceConfig.java b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/config/KuduSourceConfig.java new file mode 100644 index 00000000000..9b148140c12 --- /dev/null +++ b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/config/KuduSourceConfig.java @@ -0,0 +1,34 @@ +/* + * 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.seatunnel.connectors.seatunnel.kudu.config; + + + +import java.io.Serializable; + +public class KuduSourceConfig implements Serializable { + //kudu master ip + public static final String kuduMaster = "kudu_master"; + + public static final String tableName = "kudu_table"; + + public static final String columnsList = "columnsList"; + + + +} diff --git a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/kuduclient/KuduInputFormat.java b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/kuduclient/KuduInputFormat.java new file mode 100644 index 00000000000..6c9a2e4629c --- /dev/null +++ b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/kuduclient/KuduInputFormat.java @@ -0,0 +1,173 @@ +package org.apache.seatunnel.connectors.seatunnel.kudu.kuduclient; + +import org.apache.kudu.ColumnSchema; +import org.apache.kudu.Schema; +import org.apache.kudu.client.*; +import org.apache.seatunnel.api.common.PrepareFailException; +import org.apache.seatunnel.api.table.type.*; +import org.apache.seatunnel.common.constants.PluginType; +import org.slf4j.Logger; +import org.slf4j.LoggerFactory; + +import java.io.Serializable; +import java.math.BigDecimal; +import java.math.BigInteger; +import java.sql.*; +import java.util.ArrayList; +import java.util.Arrays; +import java.util.List; + +public class KuduInputFormat implements Serializable { + private static final Logger logger = LoggerFactory.getLogger(KuduInputFormat.class); + + public KuduInputFormat(String kuduMaster,String tableName,String columnsList){ + this.kuduMaster=kuduMaster; + this.columnsList=Arrays.asList(columnsList.split(",")); + this.tableName=tableName; + // openInputFormat(); + } + /** + * Declare the global variable KuduClient and use it to manipulate the Kudu table + */ + public KuduClient kuduClient; + + /** + * Specify kuduMaster address + */ + public String kuduMaster; + public List columnsList; + public Schema schema; + public String keyColumn; + + /** + * Specifies the name of the table + */ + public String tableName; + public List getColumnsSchemas(){ + List columns = null; + try { + schema = kuduClient.openTable(tableName).getSchema(); + keyColumn = schema.getPrimaryKeyColumns().get(0).getName(); + columns =schema.getColumns(); + } catch (KuduException e) { + e.printStackTrace(); + } + return columns; + } + + public static SeaTunnelRow getSeaTunnelRowData(RowResult rs, SeaTunnelRowType typeInfo) throws SQLException { + + List fields = new ArrayList<>(); + SeaTunnelDataType[] seaTunnelDataTypes = typeInfo.getFieldTypes(); + + for (int i = 0; i < seaTunnelDataTypes.length; i++) { + Object seatunnelField; + SeaTunnelDataType seaTunnelDataType = seaTunnelDataTypes[i]; + if (null == rs.getObject(i)) { + seatunnelField = null; + } else if (BasicType.BOOLEAN_TYPE.equals(seaTunnelDataType)) { + seatunnelField = rs.getBoolean(i); + } else if (BasicType.BYTE_TYPE.equals(seaTunnelDataType)) { + seatunnelField = rs.getByte(i); + } else if (BasicType.SHORT_TYPE.equals(seaTunnelDataType)) { + seatunnelField = rs.getShort(i); + } else if (BasicType.INT_TYPE.equals(seaTunnelDataType)) { + seatunnelField = rs.getInt(i); + } else if (BasicType.LONG_TYPE.equals(seaTunnelDataType)) { + seatunnelField = rs.getLong(i); + } else if (seaTunnelDataType instanceof DecimalType) { + Object value = rs.getObject(i); + seatunnelField = value instanceof BigInteger ? + new BigDecimal((BigInteger) value, 0) + : value; + } else if (BasicType.FLOAT_TYPE.equals(seaTunnelDataType)) { + seatunnelField = rs.getFloat(i); + } else if (BasicType.DOUBLE_TYPE.equals(seaTunnelDataType)) { + seatunnelField = rs.getDouble(i); + } else if (BasicType.STRING_TYPE.equals(seaTunnelDataType)) { + seatunnelField = rs.getString(i); + } else { + throw new IllegalStateException("Unexpected value: " + seaTunnelDataType); + } + fields.add(seatunnelField); + } + + return new SeaTunnelRow(fields.toArray()); + } + + public SeaTunnelRowType getSeaTunnelRowType(List columnSchemaList) { + + ArrayList> seaTunnelDataTypes = new ArrayList<>(); + ArrayList fieldNames = new ArrayList<>(); + try { + + for (int i = 0; i < columnSchemaList.size(); i++) { + fieldNames.add(columnSchemaList.get(i).getName()); + seaTunnelDataTypes.add(KuduTypeMapper.mapping(columnSchemaList, i)); + } + } catch (Exception e) { + logger .warn("get row type info exception", e); + throw new PrepareFailException("kudu", PluginType.SOURCE, e.toString()); + } + return new SeaTunnelRowType(fieldNames.toArray(new String[fieldNames.size()]), seaTunnelDataTypes.toArray(new SeaTunnelDataType[seaTunnelDataTypes.size()])); + } + + public void openInputFormat() { + + KuduClient.KuduClientBuilder kuduClientBuilder = new + KuduClient.KuduClientBuilder(kuduMaster); + kuduClientBuilder.defaultOperationTimeoutMs(1800000); + + kuduClient = kuduClientBuilder.build(); + + logger.info("The Kudu client is successfully initialized", kuduMaster, kuduClient); + + } + + + /** + * + * @param lowerBound The beginning of each slice + * @param upperBound End of each slice + * @return Get the kuduScanner object for each slice + */ + public KuduScanner getKuduBuildSplit(int lowerBound,int upperBound){ + KuduScanner kuduScanner = null; + try { + KuduScanner.KuduScannerBuilder kuduScannerBuilder = + kuduClient.newScannerBuilder(kuduClient.openTable(tableName)); + + kuduScannerBuilder.setProjectedColumnNames(columnsList); + + KuduPredicate lowerPred = KuduPredicate.newComparisonPredicate( + schema.getColumn(""+keyColumn), + KuduPredicate.ComparisonOp.GREATER_EQUAL, + lowerBound); + + KuduPredicate upperPred = KuduPredicate.newComparisonPredicate( + schema.getColumn(""+keyColumn), + KuduPredicate.ComparisonOp.LESS, + upperBound); + + kuduScanner = kuduScannerBuilder.addPredicate(lowerPred) + .addPredicate(upperPred).build(); + } catch (KuduException e) { + e.printStackTrace(); + logger .warn("get the Kuduscan object for each splice exception", e); + } + return kuduScanner; + } + + public void closeInputFormat() { + if (kuduClient != null) { + try { + kuduClient.close(); + } catch ( KuduException e) { + logger.warn("Kudu Client close failed.", e); + } finally { + kuduClient = null; + } + } + + } +} diff --git a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/kuduclient/KuduOutputFormat.java b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/kuduclient/KuduOutputFormat.java new file mode 100644 index 00000000000..40cd7f1a4b9 --- /dev/null +++ b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/kuduclient/KuduOutputFormat.java @@ -0,0 +1,165 @@ +/* + * 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.seatunnel.connectors.seatunnel.kudu.kuduclient; + + +import org.apache.kudu.ColumnSchema; +import org.apache.kudu.Schema; +import org.apache.kudu.client.*; +import org.apache.seatunnel.api.table.type.SeaTunnelRow; +import org.apache.seatunnel.connectors.seatunnel.kudu.config.KuduSinkConfig; +import org.slf4j.Logger; +import org.slf4j.LoggerFactory; + +import java.io.Serializable; +import java.math.BigDecimal; +import java.nio.ByteBuffer; +import java.sql.Timestamp; + +import static com.google.common.base.Preconditions.checkNotNull; + +/** + * A Kudu outputFormat + */ +public class KuduOutputFormat + implements Serializable { + private static final Logger logger = LoggerFactory.getLogger(KuduOutputFormat.class); + + private String kuduMaster; + private String kuduTableName; + private KuduClient kuduClient; + private KuduSession kuduSession; + private KuduTable kuduTable; + + + public KuduOutputFormat(KuduSinkConfig kuduSinkConfig) { + this.kuduMaster = kuduSinkConfig.getKuduMaster(); + this.kuduTableName = kuduSinkConfig.getKuduTableName(); + init(); + } + + public void write(SeaTunnelRow element) { + + Insert insert = kuduTable.newInsert(); + Schema schema = kuduTable.getSchema(); + + int columnCount = schema.getColumnCount(); + PartialRow row = insert.getRow(); + for (int columnIndex = 0; columnIndex < columnCount; columnIndex++) { + ColumnSchema col = schema.getColumnByIndex(columnIndex); + try { + switch (col.getType()) { + case BOOL: + row.addBoolean(columnIndex, (Boolean) element.getField(columnIndex)); + break; + case INT8: + row.addByte(columnIndex, (Byte) element.getField(columnIndex)); + break; + case INT16: + row.addShort(columnIndex, (Short) element.getField(columnIndex)); + break; + case INT32: + row.addInt(columnIndex, (Integer) element.getField(columnIndex)); + break; + case INT64: + row.addLong(columnIndex, (Long) element.getField(columnIndex)); + break; + case UNIXTIME_MICROS: + if (element.getField(columnIndex) instanceof Timestamp) { + row.addTimestamp(columnIndex, (Timestamp) element.getField(columnIndex)); + } else { + row.addLong(columnIndex, (Long) element.getField(columnIndex)); + } + break; + case FLOAT: + row.addFloat(columnIndex, (Float) element.getField(columnIndex)); + break; + case DOUBLE: + row.addDouble(columnIndex, (Double) element.getField(columnIndex)); + break; + case STRING: + row.addString(columnIndex, element.getField(columnIndex).toString()); + break; + case BINARY: + if (element.getField(columnIndex) instanceof byte[]) { + row.addBinary(columnIndex, (byte[]) element.getField(columnIndex)); + } else { + row.addBinary(columnIndex, (ByteBuffer) element.getField(columnIndex)); + } + break; + case DECIMAL: + row.addDecimal(columnIndex, (BigDecimal) element.getField(columnIndex)); + break; + default: + throw new IllegalArgumentException("Unsupported column type: " + col.getType()); + } + } catch (ClassCastException e) { + e.printStackTrace(); + throw new IllegalArgumentException( + "Value type does not match column type " + col.getType() + + " for column " + col.getName()); + } + + } + + try { + kuduSession.apply(insert); + + } catch (KuduException e) { + e.printStackTrace(); + } + + } + + public void init() { + + + KuduClient.KuduClientBuilder kuduClientBuilder = new + KuduClient.KuduClientBuilder(kuduMaster); + kuduClientBuilder.defaultOperationTimeoutMs(1800000); + + this.kuduClient = kuduClientBuilder.build(); + this.kuduSession = kuduClient.newSession(); + this.kuduSession.setTimeoutMillis(100000); + this.kuduSession.setFlushMode(SessionConfiguration.FlushMode.AUTO_FLUSH_SYNC); + + try { + kuduTable = kuduClient.openTable(kuduTableName); + } catch (KuduException e) { + e.printStackTrace(); + } + + + logger.info("The Kudu client is successfully initialized", kuduMaster, kuduClient); + } + + + public void closeOutputFormat() { + if (kuduClient != null) { + try { + kuduClient.close(); + kuduSession.close(); + } catch ( KuduException e) { + logger.warn("Kudu Client close failed.", e); + } finally { + kuduClient = null; + kuduSession = null; + } + } + } +} diff --git a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/kuduclient/KuduTypeMapper.java b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/kuduclient/KuduTypeMapper.java new file mode 100644 index 00000000000..4f63c61359d --- /dev/null +++ b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/kuduclient/KuduTypeMapper.java @@ -0,0 +1,104 @@ +/* + * 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.seatunnel.connectors.seatunnel.kudu.kuduclient; + +import org.apache.kudu.ColumnSchema; +import org.apache.seatunnel.api.table.type.*; + +import org.slf4j.Logger; +import org.slf4j.LoggerFactory; + +import java.sql.ResultSetMetaData; +import java.sql.SQLException; +import java.util.List; + +public class KuduTypeMapper { + + private static final Logger LOG = LoggerFactory.getLogger(KuduTypeMapper.class); + + // ============================data types===================== + + private static final String KUDU_UNKNOWN = "UNKNOWN"; + private static final String KUDU_BIT = "BOOL"; + + // -------------------------number---------------------------- + private static final String KUDU_TINYINT = "INT8"; + private static final String KUDU_MEDIUMINT = "INT32"; + private static final String KUDU_INT = "INT16"; + private static final String KUDU_BIGINT = "INT64"; + + private static final String KUDU_FLOAT = "FLOAT"; + + private static final String KUDU_DOUBLE = "DOUBLE"; + private static final String KUDU_DECIMAL = "DECIMAL32"; + + + // -------------------------string---------------------------- + + private static final String KUDU_VARCHAR = "STRING"; + + + // ------------------------------time------------------------- + + private static final String KUDU_UNIXTIME_MICROS = "UNIXTIME_MICROS"; + + + // ------------------------------blob------------------------- + + private static final String KUDU_BINARY = "BINARY"; + + + + + public static SeaTunnelDataType mapping(List columnSchemaList, int colIndex) throws SQLException { + String KUDUType = columnSchemaList.get(colIndex).getType().getName().toUpperCase(); + + switch (KUDUType) { + case KUDU_BIT: + return BasicType.BOOLEAN_TYPE; + case KUDU_TINYINT: + case KUDU_MEDIUMINT: + case KUDU_INT: + return BasicType.INT_TYPE; + case KUDU_BIGINT: + return BasicType.LONG_TYPE; + case KUDU_DECIMAL: + return new DecimalType(20, 0); + case KUDU_FLOAT: + return BasicType.FLOAT_TYPE; + case KUDU_DOUBLE: + return BasicType.DOUBLE_TYPE; + + case KUDU_VARCHAR: + return BasicType.STRING_TYPE; + case KUDU_UNIXTIME_MICROS: + return LocalTimeType.LOCAL_DATE_TIME_TYPE; + case KUDU_BINARY: + return PrimitiveByteArrayType.INSTANCE; + + //Doesn't support yet + + case KUDU_UNKNOWN: + default: + throw new UnsupportedOperationException( + String.format( + "Doesn't support KUDU type '%s' .", + KUDUType)); + } + } +} diff --git a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/sink/KuduAggregatedCommitInfo.java b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/sink/KuduAggregatedCommitInfo.java new file mode 100644 index 00000000000..054bdf8f112 --- /dev/null +++ b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/sink/KuduAggregatedCommitInfo.java @@ -0,0 +1,31 @@ +/* + * 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.seatunnel.connectors.seatunnel.kudu.sink; + +import lombok.AllArgsConstructor; +import lombok.Data; + +import java.io.Serializable; +import java.util.Map; + +@Data +@AllArgsConstructor +public class KuduAggregatedCommitInfo implements Serializable { + + +} diff --git a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/sink/KuduCommitInfo.java b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/sink/KuduCommitInfo.java new file mode 100644 index 00000000000..271139fe88e --- /dev/null +++ b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/sink/KuduCommitInfo.java @@ -0,0 +1,32 @@ +/* + * 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.seatunnel.connectors.seatunnel.kudu.sink; + +import lombok.AllArgsConstructor; +import lombok.Data; + +import java.io.Serializable; +import java.util.Map; + +@Data +@AllArgsConstructor +public class KuduCommitInfo implements Serializable { + + + +} diff --git a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/sink/KuduSink.java b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/sink/KuduSink.java new file mode 100644 index 00000000000..6c40252f885 --- /dev/null +++ b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/sink/KuduSink.java @@ -0,0 +1,93 @@ +/* + * 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.seatunnel.connectors.seatunnel.kudu.sink; + +import com.google.auto.service.AutoService; +import org.apache.seatunnel.api.common.PrepareFailException; +import org.apache.seatunnel.api.common.SeaTunnelContext; +import org.apache.seatunnel.api.serialization.DefaultSerializer; +import org.apache.seatunnel.api.serialization.Serializer; +import org.apache.seatunnel.api.sink.SeaTunnelSink; +import org.apache.seatunnel.api.sink.SinkAggregatedCommitter; +import org.apache.seatunnel.api.sink.SinkWriter; +import org.apache.seatunnel.api.table.type.SeaTunnelDataType; +import org.apache.seatunnel.api.table.type.SeaTunnelRow; +import org.apache.seatunnel.api.table.type.SeaTunnelRowType; +import org.apache.seatunnel.shade.com.typesafe.config.Config; + +import java.io.IOException; +import java.util.List; +import java.util.Optional; + +/** + * Kudu Sink implementation by using SeaTunnel sink API. + * This class contains the method to create {@link KuduSinkWriter} and {@link KuduSinkAggregatedCommitter}. + */ +@AutoService(SeaTunnelSink.class) +public class KuduSink implements SeaTunnelSink { + + private Config config; + private SeaTunnelRowType seaTunnelRowType; + + @Override + public String getPluginName() { + return "kuduSink"; + } + + @Override + public void setTypeInfo(SeaTunnelRowType seaTunnelRowType) { + this.seaTunnelRowType = seaTunnelRowType; + } + + @Override + public SeaTunnelDataType getConsumedType() { + return this.seaTunnelRowType; + } + + @Override + public void prepare(Config pluginConfig) throws PrepareFailException { + this.config = pluginConfig; + } + + @Override + public SinkWriter createWriter(SinkWriter.Context context) throws IOException { + return new KuduSinkWriter(seaTunnelRowType, config, context, System.currentTimeMillis()); + } + + @Override + public SinkWriter restoreWriter(SinkWriter.Context context, List states) throws IOException { + return new KuduSinkWriter(seaTunnelRowType, config, context, System.currentTimeMillis()); + } + + @Override + public Optional> getCommitInfoSerializer() { + return Optional.of(new DefaultSerializer<>()); + } + + @Override + public Optional> createAggregatedCommitter() throws IOException { + return Optional.of(new KuduSinkAggregatedCommitter()); + } + + @Override + public Optional> getAggregatedCommitInfoSerializer() { + return Optional.of(new DefaultSerializer<>()); + } + + +} diff --git a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/sink/KuduSinkAggregatedCommitter.java b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/sink/KuduSinkAggregatedCommitter.java new file mode 100644 index 00000000000..f06d5e89919 --- /dev/null +++ b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/sink/KuduSinkAggregatedCommitter.java @@ -0,0 +1,53 @@ +/* + * 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.seatunnel.connectors.seatunnel.kudu.sink; + +import org.apache.seatunnel.api.sink.SinkAggregatedCommitter; + +import org.slf4j.Logger; +import org.slf4j.LoggerFactory; + +import java.io.IOException; +import java.util.ArrayList; +import java.util.HashMap; +import java.util.List; +import java.util.Map; + +public class KuduSinkAggregatedCommitter implements SinkAggregatedCommitter { + private static final Logger LOGGER = LoggerFactory.getLogger(KuduSinkAggregatedCommitter.class); + + + @Override + public List commit(List aggregatedCommitInfo) throws IOException { + return null; + } + + @Override + public KuduAggregatedCommitInfo combine(List commitInfos) { + return null; + } + + @Override + public void abort(List aggregatedCommitInfo) throws Exception { + + } + + @Override + public void close() throws IOException { + } +} diff --git a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/sink/KuduSinkState.java b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/sink/KuduSinkState.java new file mode 100644 index 00000000000..9d32b00354e --- /dev/null +++ b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/sink/KuduSinkState.java @@ -0,0 +1,29 @@ +/* + * 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.seatunnel.connectors.seatunnel.kudu.sink; + +import lombok.AllArgsConstructor; +import lombok.Data; + +import java.io.Serializable; + +@Data +@AllArgsConstructor +public class KuduSinkState implements Serializable { + +} diff --git a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/sink/KuduSinkWriter.java b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/sink/KuduSinkWriter.java new file mode 100644 index 00000000000..1fd9e560aec --- /dev/null +++ b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/sink/KuduSinkWriter.java @@ -0,0 +1,82 @@ +/* + * 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.seatunnel.connectors.seatunnel.kudu.sink; + +import lombok.NonNull; +import org.apache.seatunnel.api.sink.SinkWriter; +import org.apache.seatunnel.api.table.type.SeaTunnelRow; +import org.apache.seatunnel.api.table.type.SeaTunnelRowType; + +import org.apache.seatunnel.connectors.seatunnel.kudu.config.KuduSinkConfig; +import org.apache.seatunnel.connectors.seatunnel.kudu.kuduclient.KuduOutputFormat; +import org.apache.seatunnel.shade.com.typesafe.config.Config; +import org.slf4j.Logger; +import org.slf4j.LoggerFactory; + +import java.io.IOException; +import java.util.Optional; + +public class KuduSinkWriter implements SinkWriter { + private static final Logger LOGGER = LoggerFactory.getLogger(KuduSinkWriter.class); + + private SeaTunnelRowType seaTunnelRowType; + private Config pluginConfig; + private Context context; + private long jobId; + + private KuduOutputFormat fileWriter; + + private KuduSinkConfig kuduSinkConfig; + + public KuduSinkWriter(@NonNull SeaTunnelRowType seaTunnelRowType, + @NonNull Config pluginConfig, + @NonNull Context context, + long jobId) { + this.seaTunnelRowType = seaTunnelRowType; + this.pluginConfig = pluginConfig; + this.context = context; + this.jobId = jobId; + + kuduSinkConfig = new KuduSinkConfig(this.pluginConfig); + fileWriter = new KuduOutputFormat(kuduSinkConfig); + + } + + @Override + public void write(SeaTunnelRow element) throws IOException { + fileWriter.write(element); + } + + @Override + public Optional prepareCommit() throws IOException { + return Optional.empty(); + } + + @Override + public void abortPrepare() { + + } + + + @Override + public void close() throws IOException { + fileWriter.closeOutputFormat(); + } + + +} diff --git a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/source/KuduSource.java b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/source/KuduSource.java new file mode 100644 index 00000000000..0d6f014a444 --- /dev/null +++ b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/source/KuduSource.java @@ -0,0 +1,193 @@ +/* + * 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.seatunnel.connectors.seatunnel.kudu.source; + +import com.google.auto.service.AutoService; +import org.apache.kudu.ColumnSchema; +import org.apache.kudu.client.*; +import org.apache.seatunnel.api.common.PrepareFailException; +import org.apache.seatunnel.api.common.SeaTunnelContext; +import org.apache.seatunnel.api.serialization.DefaultSerializer; +import org.apache.seatunnel.api.serialization.Serializer; +import org.apache.seatunnel.api.source.Boundedness; +import org.apache.seatunnel.api.source.SeaTunnelSource; +import org.apache.seatunnel.api.source.SourceReader; +import org.apache.seatunnel.api.source.SourceSplitEnumerator; +import org.apache.seatunnel.api.table.type.SeaTunnelDataType; +import org.apache.seatunnel.api.table.type.SeaTunnelRow; +import org.apache.seatunnel.api.table.type.SeaTunnelRowType; + +import org.apache.seatunnel.common.constants.PluginType; +import org.apache.seatunnel.connectors.seatunnel.kudu.config.KuduSourceConfig; +import org.apache.seatunnel.connectors.seatunnel.kudu.kuduclient.KuduInputFormat; +import org.apache.seatunnel.connectors.seatunnel.kudu.kuduclient.KuduTypeMapper; +import org.apache.seatunnel.connectors.seatunnel.kudu.state.KuduSinkState; +import org.apache.seatunnel.shade.com.typesafe.config.Config; +import org.slf4j.Logger; +import org.slf4j.LoggerFactory; + +import java.util.ArrayList; +import java.util.Collections; +import java.util.List; + +@AutoService(SeaTunnelSource.class) +public class KuduSource implements SeaTunnelSource { + private static final Logger LOGGER = LoggerFactory.getLogger(KuduSource.class); + + private Config pluginConfig; + private SeaTunnelContext seaTunnelContext; + private SeaTunnelRowType rowTypeInfo; + private KuduInputFormat kuduInputFormat; + private PartitionParameter partitionParameter; + + @Override + public Boundedness getBoundedness() { + return Boundedness.BOUNDED; + } + + @Override + public SeaTunnelRowType getProducedType() { + return this.rowTypeInfo; + } + + @Override + public SourceReader createReader(SourceReader.Context readerContext) { + return new KuduSourceReader(kuduInputFormat,readerContext); + } + + @Override + public Serializer getSplitSerializer() { + return SeaTunnelSource.super.getSplitSerializer(); + } + + @Override + public SourceSplitEnumerator createEnumerator( + SourceSplitEnumerator.Context enumeratorContext) { + return new KuduSourceSplitEnumerator(enumeratorContext,partitionParameter); + } + + @Override + public SourceSplitEnumerator restoreEnumerator( + SourceSplitEnumerator.Context enumeratorContext, KuduSinkState checkpointState) { + // todo: + return new KuduSourceSplitEnumerator(enumeratorContext,partitionParameter); + } + + @Override + public Serializer getEnumeratorStateSerializer() { + return new DefaultSerializer<>(); + } + + @Override + public String getPluginName() { + return "KuduSource"; + } + + @Override + public void prepare(Config config) { + + String kudumaster = config.getString(KuduSourceConfig.kuduMaster); + String tableName = config.getString(KuduSourceConfig.tableName); + String columnslist = config.getString(KuduSourceConfig.columnsList); + kuduInputFormat=new KuduInputFormat(kudumaster,tableName,columnslist); + try { + KuduClient.KuduClientBuilder kuduClientBuilder = new + KuduClient.KuduClientBuilder(kudumaster); + kuduClientBuilder.defaultOperationTimeoutMs(1800000); + + KuduClient kuduClient = kuduClientBuilder.build(); + partitionParameter = initPartitionParameter(kuduClient,tableName); + SeaTunnelRowType seaTunnelRowType =getSeaTunnelRowType(kuduClient.openTable(tableName).getSchema().getColumns()); + rowTypeInfo=seaTunnelRowType; + } catch (KuduException e) { + e.printStackTrace(); + } + } + + private PartitionParameter initPartitionParameter(KuduClient kuduClient,String tableName) { + String keyColumn = null; + int maxKey=0; + int minKey=0; + boolean flag=true; + try { + KuduScanner.KuduScannerBuilder kuduScannerBuilder = + kuduClient.newScannerBuilder(kuduClient.openTable(tableName)); + ArrayList columnsList = new ArrayList(); + keyColumn = kuduClient.openTable(tableName).getSchema().getPrimaryKeyColumns().get(0).getName(); + columnsList.add(""+keyColumn); + kuduScannerBuilder.setProjectedColumnNames(columnsList); + KuduScanner kuduScanner = kuduScannerBuilder.build(); + + + while (kuduScanner.hasMoreRows()) { + RowResultIterator rowResults = kuduScanner.nextRows(); + while (rowResults.hasNext()) { + RowResult row = rowResults.next(); + int id = row.getInt(""+keyColumn); + if (flag){ + maxKey=id; + minKey=id; + flag=false; + }else { + if (id>=maxKey){ + maxKey=id; + } + if (id<=minKey){ + minKey=id; + } + } + } + } + } catch (KuduException e) { + e.printStackTrace(); + } + + + return new PartitionParameter(keyColumn, Long.parseLong(minKey+""), Long.parseLong(maxKey+"")); + } + + + /* @Override + public SeaTunnelContext getSeaTunnelContext() { + return seaTunnelContext; + }*/ + + @Override + public void setSeaTunnelContext(SeaTunnelContext seaTunnelContext) { + this.seaTunnelContext = seaTunnelContext; + } + + + public SeaTunnelRowType getSeaTunnelRowType(List columnSchemaList) { + + ArrayList> seaTunnelDataTypes = new ArrayList<>(); + ArrayList fieldNames = new ArrayList<>(); + try { + + for (int i = 0; i < columnSchemaList.size(); i++) { + fieldNames.add(columnSchemaList.get(i).getName()); + seaTunnelDataTypes.add(KuduTypeMapper.mapping(columnSchemaList, i)); + } + + } catch (Exception e) { + LOGGER.warn("get row type info exception", e); + throw new PrepareFailException("jdbc", PluginType.SOURCE, e.toString()); + } + return new SeaTunnelRowType(fieldNames.toArray(new String[fieldNames.size()]), seaTunnelDataTypes.toArray(new SeaTunnelDataType[seaTunnelDataTypes.size()])); + } +} diff --git a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/source/KuduSourceReader.java b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/source/KuduSourceReader.java new file mode 100644 index 00000000000..b235744f216 --- /dev/null +++ b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/source/KuduSourceReader.java @@ -0,0 +1,116 @@ +/* + * 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.seatunnel.connectors.seatunnel.kudu.source; + +import org.apache.kudu.ColumnSchema; +import org.apache.kudu.client.KuduException; +import org.apache.kudu.client.KuduScanner; +import org.apache.kudu.client.RowResult; +import org.apache.kudu.client.RowResultIterator; +import org.apache.seatunnel.api.source.Boundedness; +import org.apache.seatunnel.api.source.Collector; +import org.apache.seatunnel.api.source.SourceReader; +import org.apache.seatunnel.api.table.type.SeaTunnelRow; +import org.apache.seatunnel.connectors.seatunnel.kudu.kuduclient.KuduInputFormat; +import org.slf4j.Logger; +import org.slf4j.LoggerFactory; + +import java.sql.SQLException; +import java.util.*; +import java.util.concurrent.ThreadLocalRandom; + +public class KuduSourceReader implements SourceReader { + + private static final Logger LOGGER = LoggerFactory.getLogger(KuduSourceReader.class); + + private final Context context; + + private final KuduInputFormat kuduInputFormat; + Deque splits = new LinkedList<>(); + + boolean noMoreSplit; + + public KuduSourceReader(KuduInputFormat kuduInputFormat, Context context) { + this.context = context; + this.kuduInputFormat = kuduInputFormat; + } + + @Override + public void open() { + kuduInputFormat.openInputFormat(); + } + + @Override + public void close() { + kuduInputFormat.closeInputFormat(); + } + + @Override + @SuppressWarnings("magicnumber") + public void pollNext(Collector output) throws InterruptedException, KuduException, SQLException { + KuduSourceSplit split = splits.poll(); + Object[] parameterValues = split.parameterValues; + + int lowerBound=Integer.parseInt(parameterValues[0].toString()); + + int upperBound=Integer.parseInt(parameterValues[1].toString()); + + + List columnSchemaList = kuduInputFormat.getColumnsSchemas(); + KuduScanner kuduScanner = kuduInputFormat.getKuduBuildSplit( lowerBound, upperBound); + // + while (kuduScanner.hasMoreRows()) { + RowResultIterator rowResults = kuduScanner.nextRows(); + while (rowResults.hasNext()) { + RowResult rowResult = rowResults.next(); + SeaTunnelRow seaTunnelRow = KuduInputFormat.getSeaTunnelRowData(rowResult, kuduInputFormat.getSeaTunnelRowType(columnSchemaList)); + output.collect(seaTunnelRow); + } + } + + + // Generate a random number of rows to emit. + + if (Boundedness.BOUNDED.equals(context.getBoundedness())) { + // signal to the source that we have reached the end of the data. + LOGGER.info("Closed the bounded fake source"); + context.signalNoMoreElement(); + } + + } + + @Override + public List snapshotState(long checkpointId) { + return null; + } + + @Override + public void addSplits(List splits) { + this.splits.addAll(splits); + } + + @Override + public void handleNoMoreSplits() { + noMoreSplit = true; + } + + @Override + public void notifyCheckpointComplete(long checkpointId) { + + } +} diff --git a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/source/KuduSourceSplit.java b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/source/KuduSourceSplit.java new file mode 100644 index 00000000000..2c92fb13fbf --- /dev/null +++ b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/source/KuduSourceSplit.java @@ -0,0 +1,37 @@ +/* + * 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.seatunnel.connectors.seatunnel.kudu.source; + +import lombok.AllArgsConstructor; +import lombok.Data; +import org.apache.seatunnel.api.source.SourceSplit; +@Data +@AllArgsConstructor +public class KuduSourceSplit implements SourceSplit { + + private static final long serialVersionUID = -1L; + + Object[] parameterValues; + public final Integer splitId; + + @Override + public String splitId() { + return splitId.toString(); + } + +} diff --git a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/source/KuduSourceSplitEnumerator.java b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/source/KuduSourceSplitEnumerator.java new file mode 100644 index 00000000000..66c8f92dbce --- /dev/null +++ b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/source/KuduSourceSplitEnumerator.java @@ -0,0 +1,132 @@ +/* + * 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.seatunnel.connectors.seatunnel.kudu.source; + +import org.apache.seatunnel.api.source.SourceSplitEnumerator; + +import org.apache.seatunnel.connectors.seatunnel.kudu.state.KuduSinkState; + +import java.io.IOException; +import java.io.Serializable; +import java.util.ArrayList; +import java.util.List; +import java.util.stream.Collectors; + +public class KuduSourceSplitEnumerator implements SourceSplitEnumerator { + + private final Context enumeratorContext; + private PartitionParameter partitionParameter; + List allSplit = new ArrayList<>(); + private Long maxVal; + private Long minVal; + private Long batchSize; + private Integer batchNum; + + public KuduSourceSplitEnumerator(Context enumeratorContext,PartitionParameter partitionParameter) { + this.enumeratorContext = enumeratorContext; + this.partitionParameter=partitionParameter; + } + + @Override + public void open() { + + } + + @Override + public void run() { + + } + + @Override + public void close() throws IOException { + + } + + @Override + public void addSplitsBack(List splits, int subtaskId) { + + } + + @Override + public int currentUnassignedSplitSize() { + return 0; + } + + @Override + public void handleSplitRequest(int subtaskId) { + + } + + @Override + public void registerReader(int subtaskId) { + int parallelism = enumeratorContext.currentParallelism(); + if (allSplit.isEmpty()) { + if (null != partitionParameter) { + Serializable[][] parameterValues = getParameterValues(partitionParameter.minValue, partitionParameter.maxValue,parallelism); + for (int i = 0; i < parameterValues.length; i++) { + allSplit.add(new KuduSourceSplit(parameterValues[i], i)); + } + } else { + allSplit.add(new KuduSourceSplit(null, 0)); + } + } + // Filter the split that the current task needs to run + List splits = allSplit.stream().filter(p -> p.splitId % parallelism == subtaskId).collect(Collectors.toList()); + enumeratorContext.assignSplit(subtaskId, splits); + enumeratorContext.signalNoMoreSplits(subtaskId); + } + + private Serializable[][] getParameterValues(Long minVal, Long maxVal, int parallelism) { + this.maxVal=maxVal; + this.minVal=minVal; + long maxElemCount = (maxVal - minVal) + 1; + batchNum=parallelism; + getBatchSizeAndBatchNum(parallelism); + long bigBatchNum = maxElemCount - (batchSize - 1) * batchNum; + + Serializable[][] parameters = new Serializable[batchNum][2]; + long start = minVal; + for (int i = 0; i < batchNum; i++) { + long end = start + batchSize - 1 - (i >= bigBatchNum ? 1 : 0); + parameters[i] = new Long[] {start, end}; + start = end + 1; + } + return parameters; + + } + + private void getBatchSizeAndBatchNum(int parallelism) { + batchNum=parallelism; + long maxElemCount = (maxVal - minVal) + 1; + if (batchNum > maxElemCount) { + batchNum = (int) maxElemCount; + } + this.batchNum = batchNum; + this.batchSize = new Double(Math.ceil((double) maxElemCount / batchNum)).longValue(); + } + + @Override + public KuduSinkState snapshotState(long checkpointId) throws Exception { + return null; + } + + @Override + public void notifyCheckpointComplete(long checkpointId) throws Exception { + + } +} diff --git a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/source/PartitionParameter.java b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/source/PartitionParameter.java new file mode 100644 index 00000000000..e791164667c --- /dev/null +++ b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/source/PartitionParameter.java @@ -0,0 +1,32 @@ +/* + * 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.seatunnel.connectors.seatunnel.kudu.source; + +import lombok.AllArgsConstructor; +import lombok.Data; + +import java.io.Serializable; + +@Data +@AllArgsConstructor +public class PartitionParameter implements Serializable { + + String partitionColumnName; + Long minValue; + Long maxValue; +} diff --git a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/state/KuduSinkState.java b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/state/KuduSinkState.java new file mode 100644 index 00000000000..f50416ee29a --- /dev/null +++ b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/state/KuduSinkState.java @@ -0,0 +1,23 @@ +/* + * 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.seatunnel.connectors.seatunnel.kudu.state; + +import java.io.Serializable; + +public class KuduSinkState implements Serializable { +} diff --git a/seatunnel-connectors-v2/connector-kudu/src/main/resources/kudu_to_kudu_flink.conf b/seatunnel-connectors-v2/connector-kudu/src/main/resources/kudu_to_kudu_flink.conf new file mode 100644 index 00000000000..b04aeae6898 --- /dev/null +++ b/seatunnel-connectors-v2/connector-kudu/src/main/resources/kudu_to_kudu_flink.conf @@ -0,0 +1,60 @@ +# +# 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. +# +###### +###### This config file is a demonstration of streaming processing in seatunnel config +###### + +env { + # You can set flink configuration here + execution.parallelism = 2 + #job.mode = "BATCH" + #execution.checkpoint.interval = 10000 + #execution.checkpoint.data-uri = "hdfs://localhost:9000/checkpoint" +} + +source { + # This is a example source plugin **only for test and demonstrate the feature source plugin** + KuduSource { + result_table_name = "studentlyh2" + kudu_master = "192.168.88.110:7051" + kudu_table = "studentlyh2" + columnsList = "id,name,age,sex" + } + + # If you would like to get more information about how to configure seatunnel and see full list of source plugins, + # please go to https://seatunnel.apache.org/docs/flink/configuration/source-plugins/Fake +} + +transform { + sql { + sql = "select id,name,age,sex from studentlyh2" + } + + # If you would like to get more information about how to configure seatunnel and see full list of transform plugins, + # please go to https://seatunnel.apache.org/docs/flink/configuration/transform-plugins/Sql +} + +sink { + kuduSink { + kudu_master = "192.168.88.110:7051" + kudu_table = "studentlyhresultflink" + save_mode="append" + } + + # If you would like to get more information about how to configure seatunnel and see full list of sink plugins, + # please go to https://seatunnel.apache.org/docs/flink/configuration/sink-plugins/Console +} \ No newline at end of file diff --git a/seatunnel-connectors-v2/connector-kudu/src/main/resources/kudu_to_kudu_spark.conf b/seatunnel-connectors-v2/connector-kudu/src/main/resources/kudu_to_kudu_spark.conf new file mode 100644 index 00000000000..cb4ecbaa8e5 --- /dev/null +++ b/seatunnel-connectors-v2/connector-kudu/src/main/resources/kudu_to_kudu_spark.conf @@ -0,0 +1,64 @@ +# +# 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. +# +###### +###### This config file is a demonstration of streaming processing in seatunnel config +###### + +env { + # You can set flink configuration here + spark.app.name = "SeaTunnel" + spark.executor.instances = 2 + spark.executor.cores = 2 + spark.executor.memory = "1g" + spark.master = local + #job.mode = "BATCH" + #execution.checkpoint.interval = 10000 + #execution.checkpoint.data-uri = "hdfs://localhost:9000/checkpoint" +} + +source { + # This is a example source plugin **only for test and demonstrate the feature source plugin** + KuduSource { + result_table_name = "studentlyh2" + kudu_master = "192.168.88.110:7051" + kudu_table = "studentlyh2" + columnsList = "id,name,age,sex" + } + + # If you would like to get more information about how to configure seatunnel and see full list of source plugins, + # please go to https://seatunnel.apache.org/docs/flink/configuration/source-plugins/Fake +} + +transform { + sql { + sql = "select id,name,age,sex from studentlyh2" + } + + # If you would like to get more information about how to configure seatunnel and see full list of transform plugins, + # please go to https://seatunnel.apache.org/docs/flink/configuration/transform-plugins/Sql +} + +sink { + kuduSink { + kudu_master = "192.168.88.110:7051" + kudu_table = "studentlyhresult" + save_mode="append" + } + + # If you would like to get more information about how to configure seatunnel and see full list of sink plugins, + # please go to https://seatunnel.apache.org/docs/flink/configuration/sink-plugins/Console +} \ No newline at end of file From eedef2854c265aa87bc00d68e3e04f1c173fc0db Mon Sep 17 00:00:00 2001 From: liuyehan <81779971+2013650523@users.noreply.github.com> Date: Sat, 23 Jul 2022 16:21:28 +0800 Subject: [PATCH 02/26] Update pom.xml add kudu dependency --- pom.xml | 10 ++++++++-- 1 file changed, 8 insertions(+), 2 deletions(-) diff --git a/pom.xml b/pom.xml index 23e15e3c4b3..47296175195 100644 --- a/pom.xml +++ b/pom.xml @@ -142,6 +142,7 @@ 2.2.0 2.6.0 3.4 + 1.11.1 4.4 3.3.0 provided @@ -227,7 +228,12 @@ lz4 1.3.0 - + + + org.apache.kudu + kudu-client + ${kudu.version} + org.apache.flink @@ -1061,4 +1067,4 @@ - \ No newline at end of file + From 146fa249104185ef96ed81c703b3e05e8a00808d Mon Sep 17 00:00:00 2001 From: liuyehan <81779971+2013650523@users.noreply.github.com> Date: Sat, 23 Jul 2022 16:24:43 +0800 Subject: [PATCH 03/26] Update plugin-mapping.properties add kudu config --- plugin-mapping.properties | 2 ++ 1 file changed, 2 insertions(+) diff --git a/plugin-mapping.properties b/plugin-mapping.properties index da3890a70a0..44cce0a7555 100644 --- a/plugin-mapping.properties +++ b/plugin-mapping.properties @@ -101,3 +101,5 @@ seatunnel.sink.Clickhouse = connector-clickhouse seatunnel.sink.ClickhouseFile = connector-clickhouse seatunnel.source.Jdbc = connector-jdbc seatunnel.sink.Jdbc = connector-jdbc +seatunnel.source.Kudu = connector-Kudu +seatunnel.sink.Kudu = connector-Kudu From 9b1f10bdc0e35d53650513f7ccf53435c5354673 Mon Sep 17 00:00:00 2001 From: liuyehan <81779971+2013650523@users.noreply.github.com> Date: Sat, 23 Jul 2022 16:25:44 +0800 Subject: [PATCH 04/26] Update pom.xml add kudu --- seatunnel-connectors-v2/pom.xml | 3 ++- 1 file changed, 2 insertions(+), 1 deletion(-) diff --git a/seatunnel-connectors-v2/pom.xml b/seatunnel-connectors-v2/pom.xml index be62464f81b..75c5f64105b 100644 --- a/seatunnel-connectors-v2/pom.xml +++ b/seatunnel-connectors-v2/pom.xml @@ -43,6 +43,7 @@ connector-pulsar connector-socket connector-assert + connector-kudu @@ -56,4 +57,4 @@ - \ No newline at end of file + From 8c28e806cfadfb6a016a3fff73057eac9327e59e Mon Sep 17 00:00:00 2001 From: 2013650523 <1067341434@qq.com> Date: Sat, 23 Jul 2022 17:44:29 +0800 Subject: [PATCH 05/26] add email sink connector --- .../connector-email/pom.xml | 19 +++++++++++++++++++ 1 file changed, 19 insertions(+) create mode 100644 seatunnel-connectors-v2/connector-email/pom.xml diff --git a/seatunnel-connectors-v2/connector-email/pom.xml b/seatunnel-connectors-v2/connector-email/pom.xml new file mode 100644 index 00000000000..c0c171d5341 --- /dev/null +++ b/seatunnel-connectors-v2/connector-email/pom.xml @@ -0,0 +1,19 @@ + + + + seatunnel + org.apache.seatunnel + 2.1.3-SNAPSHOT + + 4.0.0 + + connector-email + + + 8 + 8 + + + \ No newline at end of file From ed790c81ffdef5f955779a86e11f6d5c245ca60a Mon Sep 17 00:00:00 2001 From: liuyehan <81779971+2013650523@users.noreply.github.com> Date: Sun, 24 Jul 2022 07:10:50 +0800 Subject: [PATCH 06/26] Delete seatunnel-connectors-v2/connector-email directory --- .../connector-email/pom.xml | 19 ------------------- 1 file changed, 19 deletions(-) delete mode 100644 seatunnel-connectors-v2/connector-email/pom.xml diff --git a/seatunnel-connectors-v2/connector-email/pom.xml b/seatunnel-connectors-v2/connector-email/pom.xml deleted file mode 100644 index c0c171d5341..00000000000 --- a/seatunnel-connectors-v2/connector-email/pom.xml +++ /dev/null @@ -1,19 +0,0 @@ - - - - seatunnel - org.apache.seatunnel - 2.1.3-SNAPSHOT - - 4.0.0 - - connector-email - - - 8 - 8 - - - \ No newline at end of file From 5ba8049f20fd72c45d8079b3edd81e3f88b3a742 Mon Sep 17 00:00:00 2001 From: liuyehan <81779971+2013650523@users.noreply.github.com> Date: Sun, 24 Jul 2022 07:19:01 +0800 Subject: [PATCH 07/26] Update plugin-mapping.properties --- plugin-mapping.properties | 4 ++++ 1 file changed, 4 insertions(+) diff --git a/plugin-mapping.properties b/plugin-mapping.properties index 44cce0a7555..7eab74e8225 100644 --- a/plugin-mapping.properties +++ b/plugin-mapping.properties @@ -103,3 +103,7 @@ seatunnel.source.Jdbc = connector-jdbc seatunnel.sink.Jdbc = connector-jdbc seatunnel.source.Kudu = connector-Kudu seatunnel.sink.Kudu = connector-Kudu +seatunnel.sink.HdfsFile = connector-file-hadoop +seatunnel.sink.LocalFile = connector-file-local +seatunnel.source.Pulsar = connector-pulsar +seatunnel.source.Hudi = connector-hudi From 2c1bf2733f71e7aa7f299471009c8137bb2c36c4 Mon Sep 17 00:00:00 2001 From: liuyehan <81779971+2013650523@users.noreply.github.com> Date: Mon, 25 Jul 2022 09:22:24 +0800 Subject: [PATCH 08/26] Update plugin-mapping.properties --- plugin-mapping.properties | 4 ++-- 1 file changed, 2 insertions(+), 2 deletions(-) diff --git a/plugin-mapping.properties b/plugin-mapping.properties index 21a23ae6ca4..bfc4735bbf3 100644 --- a/plugin-mapping.properties +++ b/plugin-mapping.properties @@ -102,8 +102,8 @@ seatunnel.sink.Clickhouse = connector-clickhouse seatunnel.sink.ClickhouseFile = connector-clickhouse seatunnel.source.Jdbc = connector-jdbc seatunnel.sink.Jdbc = connector-jdbc -seatunnel.source.Kudu = connector-Kudu -seatunnel.sink.Kudu = connector-Kudu +seatunnel.source.Kudu = connector-kudu +seatunnel.sink.Kudu = connector-kudu seatunnel.sink.HdfsFile = connector-file-hadoop seatunnel.sink.LocalFile = connector-file-local seatunnel.source.Pulsar = connector-pulsar From 843ba4361bb82f733a7a3b35470e9b22358d19e9 Mon Sep 17 00:00:00 2001 From: liuyehan <81779971+2013650523@users.noreply.github.com> Date: Mon, 25 Jul 2022 09:23:02 +0800 Subject: [PATCH 09/26] Update pom.xml --- .../connector-kudu/pom.xml | 20 ++++++++++++++++++- 1 file changed, 19 insertions(+), 1 deletion(-) diff --git a/seatunnel-connectors-v2/connector-kudu/pom.xml b/seatunnel-connectors-v2/connector-kudu/pom.xml index c8d3591abb1..e66d34f0a3f 100644 --- a/seatunnel-connectors-v2/connector-kudu/pom.xml +++ b/seatunnel-connectors-v2/connector-kudu/pom.xml @@ -1,4 +1,22 @@ + @@ -27,4 +45,4 @@ commons-lang3 - \ No newline at end of file + From 689344799fa00c1381fd398c96062bf64b2d16ca Mon Sep 17 00:00:00 2001 From: liuyehan <81779971+2013650523@users.noreply.github.com> Date: Mon, 25 Jul 2022 09:31:04 +0800 Subject: [PATCH 10/26] Create pom.xml From aa2cf304179b1e27bb88109bfc5574f801915c3e Mon Sep 17 00:00:00 2001 From: 2013650523 <1067341434@qq.com> Date: Fri, 29 Jul 2022 21:37:59 +0800 Subject: [PATCH 11/26] [Connector-V2] Add Kudu source and sink connector --- .../connector-kudu/pom.xml | 5 + .../seatunnel/kudu/config/KuduSinkConfig.java | 9 +- .../kudu/config/KuduSourceConfig.java | 11 +- .../kudu/kuduclient/KuduInputFormat.java | 159 +++++++++++------- .../kudu/kuduclient/KuduOutputFormat.java | 45 +++-- .../kudu/kuduclient/KuduTypeMapper.java | 23 ++- .../kudu/sink/KuduAggregatedCommitInfo.java | 31 ---- .../seatunnel/kudu/sink/KuduCommitInfo.java | 32 ---- .../seatunnel/kudu/sink/KuduSink.java | 42 +---- .../sink/KuduSinkAggregatedCommitter.java | 53 ------ .../seatunnel/kudu/sink/KuduSinkState.java | 29 ---- .../seatunnel/kudu/sink/KuduSinkWriter.java | 32 +--- .../seatunnel/kudu/source/KuduSource.java | 99 +++++------ .../kudu/source/KuduSourceReader.java | 33 ++-- .../kudu/source/KuduSourceSplit.java | 4 +- .../source/KuduSourceSplitEnumerator.java | 27 ++- ...uduSinkState.java => KuduSourceState.java} | 2 +- 17 files changed, 227 insertions(+), 409 deletions(-) delete mode 100644 seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/sink/KuduAggregatedCommitInfo.java delete mode 100644 seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/sink/KuduCommitInfo.java delete mode 100644 seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/sink/KuduSinkAggregatedCommitter.java delete mode 100644 seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/sink/KuduSinkState.java rename seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/state/{KuduSinkState.java => KuduSourceState.java} (94%) diff --git a/seatunnel-connectors-v2/connector-kudu/pom.xml b/seatunnel-connectors-v2/connector-kudu/pom.xml index e66d34f0a3f..febac92c3ac 100644 --- a/seatunnel-connectors-v2/connector-kudu/pom.xml +++ b/seatunnel-connectors-v2/connector-kudu/pom.xml @@ -44,5 +44,10 @@ org.apache.commons commons-lang3 + + org.apache.seatunnel + connector-common + ${project.version} + diff --git a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/config/KuduSinkConfig.java b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/config/KuduSinkConfig.java index 8f4f3ae44bf..017ff5874ac 100644 --- a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/config/KuduSinkConfig.java +++ b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/config/KuduSinkConfig.java @@ -17,13 +17,11 @@ package org.apache.seatunnel.connectors.seatunnel.kudu.config; +import org.apache.seatunnel.shade.com.typesafe.config.Config; + import lombok.Data; import lombok.NonNull; - import org.apache.commons.lang3.StringUtils; -import org.apache.seatunnel.shade.com.typesafe.config.Config; - - @Data public class KuduSinkConfig { @@ -57,10 +55,7 @@ public static SaveMode fromStr(String str) { public KuduSinkConfig(@NonNull Config pluginConfig) { this.saveMode = StringUtils.isBlank(pluginConfig.getString(KUDU_SAVE_MODE)) ? SaveMode.APPEND : SaveMode.fromStr(pluginConfig.getString(KUDU_SAVE_MODE)); - this.kuduMaster = pluginConfig.getString(KUDU_MASTER); this.kuduTableName = pluginConfig.getString(KUDU_TABLE_NAME); - - } } diff --git a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/config/KuduSourceConfig.java b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/config/KuduSourceConfig.java index 9b148140c12..754b53114ae 100644 --- a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/config/KuduSourceConfig.java +++ b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/config/KuduSourceConfig.java @@ -17,18 +17,13 @@ package org.apache.seatunnel.connectors.seatunnel.kudu.config; - - import java.io.Serializable; public class KuduSourceConfig implements Serializable { - //kudu master ip - public static final String kuduMaster = "kudu_master"; - - public static final String tableName = "kudu_table"; - - public static final String columnsList = "columnsList"; + public static final String KUDUMASTER = "kudu_master"; + public static final String TABLENAME = "kudu_table"; + public static final String COLUMNSLIST = "columnsList"; } diff --git a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/kuduclient/KuduInputFormat.java b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/kuduclient/KuduInputFormat.java index 6c9a2e4629c..0f36f7e252d 100644 --- a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/kuduclient/KuduInputFormat.java +++ b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/kuduclient/KuduInputFormat.java @@ -1,31 +1,58 @@ +/* + * 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.seatunnel.connectors.seatunnel.kudu.kuduclient; -import org.apache.kudu.ColumnSchema; -import org.apache.kudu.Schema; -import org.apache.kudu.client.*; import org.apache.seatunnel.api.common.PrepareFailException; -import org.apache.seatunnel.api.table.type.*; +import org.apache.seatunnel.api.table.type.BasicType; +import org.apache.seatunnel.api.table.type.DecimalType; +import org.apache.seatunnel.api.table.type.SeaTunnelDataType; +import org.apache.seatunnel.api.table.type.SeaTunnelRow; +import org.apache.seatunnel.api.table.type.SeaTunnelRowType; import org.apache.seatunnel.common.constants.PluginType; + +import org.apache.kudu.ColumnSchema; +import org.apache.kudu.Schema; +import org.apache.kudu.client.KuduClient; +import org.apache.kudu.client.KuduException; +import org.apache.kudu.client.KuduPredicate; +import org.apache.kudu.client.KuduScanner; +import org.apache.kudu.client.RowResult; import org.slf4j.Logger; import org.slf4j.LoggerFactory; import java.io.Serializable; import java.math.BigDecimal; import java.math.BigInteger; -import java.sql.*; +import java.sql.SQLException; import java.util.ArrayList; import java.util.Arrays; import java.util.List; public class KuduInputFormat implements Serializable { - private static final Logger logger = LoggerFactory.getLogger(KuduInputFormat.class); + private static final Logger LOGGER = LoggerFactory.getLogger(KuduInputFormat.class); + + public KuduInputFormat(String kuduMaster, String tableName, String columnsList) { + this.kuduMaster = kuduMaster; + this.columnsList = Arrays.asList(columnsList.split(",")); + this.tableName = tableName; - public KuduInputFormat(String kuduMaster,String tableName,String columnsList){ - this.kuduMaster=kuduMaster; - this.columnsList=Arrays.asList(columnsList.split(",")); - this.tableName=tableName; - // openInputFormat(); } + /** * Declare the global variable KuduClient and use it to manipulate the Kudu table */ @@ -34,23 +61,26 @@ public KuduInputFormat(String kuduMaster,String tableName,String columnsList){ /** * Specify kuduMaster address */ - public String kuduMaster; - public List columnsList; - public Schema schema; - public String keyColumn; + public String kuduMaster; + public List columnsList; + public Schema schema; + public String keyColumn; + public static final int TIMEOUTMS = 18000; /** * Specifies the name of the table */ - public String tableName; - public List getColumnsSchemas(){ + public String tableName; + + public List getColumnsSchemas() { List columns = null; try { schema = kuduClient.openTable(tableName).getSchema(); keyColumn = schema.getPrimaryKeyColumns().get(0).getName(); - columns =schema.getColumns(); + columns = schema.getColumns(); } catch (KuduException e) { - e.printStackTrace(); + LOGGER.warn("get table Columns Schemas Fail.", e); + throw new RuntimeException("get table Columns Schemas Fail..", e); } return columns; } @@ -63,34 +93,34 @@ public static SeaTunnelRow getSeaTunnelRowData(RowResult rs, SeaTunnelRowType ty for (int i = 0; i < seaTunnelDataTypes.length; i++) { Object seatunnelField; SeaTunnelDataType seaTunnelDataType = seaTunnelDataTypes[i]; - if (null == rs.getObject(i)) { - seatunnelField = null; - } else if (BasicType.BOOLEAN_TYPE.equals(seaTunnelDataType)) { - seatunnelField = rs.getBoolean(i); - } else if (BasicType.BYTE_TYPE.equals(seaTunnelDataType)) { - seatunnelField = rs.getByte(i); - } else if (BasicType.SHORT_TYPE.equals(seaTunnelDataType)) { - seatunnelField = rs.getShort(i); - } else if (BasicType.INT_TYPE.equals(seaTunnelDataType)) { - seatunnelField = rs.getInt(i); - } else if (BasicType.LONG_TYPE.equals(seaTunnelDataType)) { - seatunnelField = rs.getLong(i); - } else if (seaTunnelDataType instanceof DecimalType) { - Object value = rs.getObject(i); - seatunnelField = value instanceof BigInteger ? - new BigDecimal((BigInteger) value, 0) - : value; - } else if (BasicType.FLOAT_TYPE.equals(seaTunnelDataType)) { - seatunnelField = rs.getFloat(i); - } else if (BasicType.DOUBLE_TYPE.equals(seaTunnelDataType)) { - seatunnelField = rs.getDouble(i); - } else if (BasicType.STRING_TYPE.equals(seaTunnelDataType)) { - seatunnelField = rs.getString(i); - } else { - throw new IllegalStateException("Unexpected value: " + seaTunnelDataType); - } - fields.add(seatunnelField); + if (null == rs.getObject(i)) { + seatunnelField = null; + } else if (BasicType.BOOLEAN_TYPE.equals(seaTunnelDataType)) { + seatunnelField = rs.getBoolean(i); + } else if (BasicType.BYTE_TYPE.equals(seaTunnelDataType)) { + seatunnelField = rs.getByte(i); + } else if (BasicType.SHORT_TYPE.equals(seaTunnelDataType)) { + seatunnelField = rs.getShort(i); + } else if (BasicType.INT_TYPE.equals(seaTunnelDataType)) { + seatunnelField = rs.getInt(i); + } else if (BasicType.LONG_TYPE.equals(seaTunnelDataType)) { + seatunnelField = rs.getLong(i); + } else if (seaTunnelDataType instanceof DecimalType) { + Object value = rs.getObject(i); + seatunnelField = value instanceof BigInteger ? + new BigDecimal((BigInteger) value, 0) + : value; + } else if (BasicType.FLOAT_TYPE.equals(seaTunnelDataType)) { + seatunnelField = rs.getFloat(i); + } else if (BasicType.DOUBLE_TYPE.equals(seaTunnelDataType)) { + seatunnelField = rs.getDouble(i); + } else if (BasicType.STRING_TYPE.equals(seaTunnelDataType)) { + seatunnelField = rs.getString(i); + } else { + throw new IllegalStateException("Unexpected value: " + seaTunnelDataType); } + fields.add(seatunnelField); + } return new SeaTunnelRow(fields.toArray()); } @@ -106,32 +136,30 @@ public SeaTunnelRowType getSeaTunnelRowType(List columnSchemaList) seaTunnelDataTypes.add(KuduTypeMapper.mapping(columnSchemaList, i)); } } catch (Exception e) { - logger .warn("get row type info exception", e); + LOGGER.warn("get row type info exception", e); throw new PrepareFailException("kudu", PluginType.SOURCE, e.toString()); } return new SeaTunnelRowType(fieldNames.toArray(new String[fieldNames.size()]), seaTunnelDataTypes.toArray(new SeaTunnelDataType[seaTunnelDataTypes.size()])); } public void openInputFormat() { + KuduClient.KuduClientBuilder kuduClientBuilder = new + KuduClient.KuduClientBuilder(kuduMaster); + kuduClientBuilder.defaultOperationTimeoutMs(TIMEOUTMS); - KuduClient.KuduClientBuilder kuduClientBuilder = new - KuduClient.KuduClientBuilder(kuduMaster); - kuduClientBuilder.defaultOperationTimeoutMs(1800000); + kuduClient = kuduClientBuilder.build(); - kuduClient = kuduClientBuilder.build(); - - logger.info("The Kudu client is successfully initialized", kuduMaster, kuduClient); + LOGGER.info("The Kudu client is successfully initialized", kuduMaster, kuduClient); } - /** - * * @param lowerBound The beginning of each slice - * @param upperBound End of each slice - * @return Get the kuduScanner object for each slice + * @param upperBound End of each slice + * @return Get the kuduScanner object for each slice */ - public KuduScanner getKuduBuildSplit(int lowerBound,int upperBound){ + + public KuduScanner getKuduBuildSplit(int lowerBound, int upperBound) { KuduScanner kuduScanner = null; try { KuduScanner.KuduScannerBuilder kuduScannerBuilder = @@ -140,20 +168,20 @@ public KuduScanner getKuduBuildSplit(int lowerBound,int upperBound){ kuduScannerBuilder.setProjectedColumnNames(columnsList); KuduPredicate lowerPred = KuduPredicate.newComparisonPredicate( - schema.getColumn(""+keyColumn), + schema.getColumn("" + keyColumn), KuduPredicate.ComparisonOp.GREATER_EQUAL, lowerBound); KuduPredicate upperPred = KuduPredicate.newComparisonPredicate( - schema.getColumn(""+keyColumn), + schema.getColumn("" + keyColumn), KuduPredicate.ComparisonOp.LESS, upperBound); - kuduScanner = kuduScannerBuilder.addPredicate(lowerPred) + kuduScanner = kuduScannerBuilder.addPredicate(lowerPred) .addPredicate(upperPred).build(); } catch (KuduException e) { - e.printStackTrace(); - logger .warn("get the Kuduscan object for each splice exception", e); + LOGGER.warn("get the Kuduscan object for each splice exception", e); + throw new RuntimeException("get the Kuduscan object for each splice exception.", e); } return kuduScanner; } @@ -162,8 +190,9 @@ public void closeInputFormat() { if (kuduClient != null) { try { kuduClient.close(); - } catch ( KuduException e) { - logger.warn("Kudu Client close failed.", e); + } catch (KuduException e) { + LOGGER.warn("Kudu Client close failed.", e); + throw new RuntimeException("Kudu Client close failed.", e); } finally { kuduClient = null; } diff --git a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/kuduclient/KuduOutputFormat.java b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/kuduclient/KuduOutputFormat.java index 40cd7f1a4b9..f73b191ca07 100644 --- a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/kuduclient/KuduOutputFormat.java +++ b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/kuduclient/KuduOutputFormat.java @@ -17,12 +17,18 @@ package org.apache.seatunnel.connectors.seatunnel.kudu.kuduclient; +import org.apache.seatunnel.api.table.type.SeaTunnelRow; +import org.apache.seatunnel.connectors.seatunnel.kudu.config.KuduSinkConfig; import org.apache.kudu.ColumnSchema; import org.apache.kudu.Schema; -import org.apache.kudu.client.*; -import org.apache.seatunnel.api.table.type.SeaTunnelRow; -import org.apache.seatunnel.connectors.seatunnel.kudu.config.KuduSinkConfig; +import org.apache.kudu.client.Insert; +import org.apache.kudu.client.KuduClient; +import org.apache.kudu.client.KuduException; +import org.apache.kudu.client.KuduSession; +import org.apache.kudu.client.KuduTable; +import org.apache.kudu.client.PartialRow; +import org.apache.kudu.client.SessionConfiguration; import org.slf4j.Logger; import org.slf4j.LoggerFactory; @@ -31,22 +37,21 @@ import java.nio.ByteBuffer; import java.sql.Timestamp; -import static com.google.common.base.Preconditions.checkNotNull; - /** * A Kudu outputFormat */ public class KuduOutputFormat implements Serializable { - private static final Logger logger = LoggerFactory.getLogger(KuduOutputFormat.class); + + private static final Logger LOGGER = LoggerFactory.getLogger(KuduOutputFormat.class); private String kuduMaster; private String kuduTableName; private KuduClient kuduClient; private KuduSession kuduSession; private KuduTable kuduTable; - - + public static final long TIMEOUTMS = 18000; + public static final long SESSIONTIMEOUTMS = 100000; public KuduOutputFormat(KuduSinkConfig kuduSinkConfig) { this.kuduMaster = kuduSinkConfig.getKuduMaster(); this.kuduTableName = kuduSinkConfig.getKuduTableName(); @@ -119,43 +124,37 @@ public void write(SeaTunnelRow element) { try { kuduSession.apply(insert); - } catch (KuduException e) { - e.printStackTrace(); + LOGGER.warn("kudu session insert data fail.", e); + throw new RuntimeException("kudu session insert data fail.", e); } } public void init() { - - KuduClient.KuduClientBuilder kuduClientBuilder = new KuduClient.KuduClientBuilder(kuduMaster); - kuduClientBuilder.defaultOperationTimeoutMs(1800000); - + kuduClientBuilder.defaultOperationTimeoutMs(TIMEOUTMS); this.kuduClient = kuduClientBuilder.build(); this.kuduSession = kuduClient.newSession(); - this.kuduSession.setTimeoutMillis(100000); + this.kuduSession.setTimeoutMillis(SESSIONTIMEOUTMS); this.kuduSession.setFlushMode(SessionConfiguration.FlushMode.AUTO_FLUSH_SYNC); - try { kuduTable = kuduClient.openTable(kuduTableName); } catch (KuduException e) { - e.printStackTrace(); + LOGGER.warn("Failed to initialize the Kudu client.", e); + throw new RuntimeException("Failed to initialize the Kudu client.", e); } - - - logger.info("The Kudu client is successfully initialized", kuduMaster, kuduClient); + LOGGER.info("The Kudu client is successfully initialized", kuduMaster, kuduClient); } - public void closeOutputFormat() { if (kuduClient != null) { try { kuduClient.close(); kuduSession.close(); - } catch ( KuduException e) { - logger.warn("Kudu Client close failed.", e); + } catch (KuduException e) { + LOGGER.warn("Kudu Client close failed.", e); } finally { kuduClient = null; kuduSession = null; diff --git a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/kuduclient/KuduTypeMapper.java b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/kuduclient/KuduTypeMapper.java index 4f63c61359d..cea22dfb2f1 100644 --- a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/kuduclient/KuduTypeMapper.java +++ b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/kuduclient/KuduTypeMapper.java @@ -17,13 +17,16 @@ package org.apache.seatunnel.connectors.seatunnel.kudu.kuduclient; -import org.apache.kudu.ColumnSchema; -import org.apache.seatunnel.api.table.type.*; +import org.apache.seatunnel.api.table.type.BasicType; +import org.apache.seatunnel.api.table.type.DecimalType; +import org.apache.seatunnel.api.table.type.LocalTimeType; +import org.apache.seatunnel.api.table.type.PrimitiveByteArrayType; +import org.apache.seatunnel.api.table.type.SeaTunnelDataType; +import org.apache.kudu.ColumnSchema; import org.slf4j.Logger; import org.slf4j.LoggerFactory; -import java.sql.ResultSetMetaData; import java.sql.SQLException; import java.util.List; @@ -61,14 +64,10 @@ public class KuduTypeMapper { // ------------------------------blob------------------------- private static final String KUDU_BINARY = "BINARY"; - - - - + private static final int PRECISION = 20; public static SeaTunnelDataType mapping(List columnSchemaList, int colIndex) throws SQLException { - String KUDUType = columnSchemaList.get(colIndex).getType().getName().toUpperCase(); - - switch (KUDUType) { + String kuduType = columnSchemaList.get(colIndex).getType().getName().toUpperCase(); + switch (kuduType) { case KUDU_BIT: return BasicType.BOOLEAN_TYPE; case KUDU_TINYINT: @@ -78,7 +77,7 @@ public static SeaTunnelDataType mapping(List columnSchemaList, case KUDU_BIGINT: return BasicType.LONG_TYPE; case KUDU_DECIMAL: - return new DecimalType(20, 0); + return new DecimalType(PRECISION, 0); case KUDU_FLOAT: return BasicType.FLOAT_TYPE; case KUDU_DOUBLE: @@ -98,7 +97,7 @@ public static SeaTunnelDataType mapping(List columnSchemaList, throw new UnsupportedOperationException( String.format( "Doesn't support KUDU type '%s' .", - KUDUType)); + kuduType)); } } } diff --git a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/sink/KuduAggregatedCommitInfo.java b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/sink/KuduAggregatedCommitInfo.java deleted file mode 100644 index 054bdf8f112..00000000000 --- a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/sink/KuduAggregatedCommitInfo.java +++ /dev/null @@ -1,31 +0,0 @@ -/* - * 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.seatunnel.connectors.seatunnel.kudu.sink; - -import lombok.AllArgsConstructor; -import lombok.Data; - -import java.io.Serializable; -import java.util.Map; - -@Data -@AllArgsConstructor -public class KuduAggregatedCommitInfo implements Serializable { - - -} diff --git a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/sink/KuduCommitInfo.java b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/sink/KuduCommitInfo.java deleted file mode 100644 index 271139fe88e..00000000000 --- a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/sink/KuduCommitInfo.java +++ /dev/null @@ -1,32 +0,0 @@ -/* - * 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.seatunnel.connectors.seatunnel.kudu.sink; - -import lombok.AllArgsConstructor; -import lombok.Data; - -import java.io.Serializable; -import java.util.Map; - -@Data -@AllArgsConstructor -public class KuduCommitInfo implements Serializable { - - - -} diff --git a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/sink/KuduSink.java b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/sink/KuduSink.java index 6c40252f885..2f907ae49e2 100644 --- a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/sink/KuduSink.java +++ b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/sink/KuduSink.java @@ -17,29 +17,27 @@ package org.apache.seatunnel.connectors.seatunnel.kudu.sink; -import com.google.auto.service.AutoService; import org.apache.seatunnel.api.common.PrepareFailException; -import org.apache.seatunnel.api.common.SeaTunnelContext; -import org.apache.seatunnel.api.serialization.DefaultSerializer; -import org.apache.seatunnel.api.serialization.Serializer; import org.apache.seatunnel.api.sink.SeaTunnelSink; -import org.apache.seatunnel.api.sink.SinkAggregatedCommitter; import org.apache.seatunnel.api.sink.SinkWriter; import org.apache.seatunnel.api.table.type.SeaTunnelDataType; import org.apache.seatunnel.api.table.type.SeaTunnelRow; import org.apache.seatunnel.api.table.type.SeaTunnelRowType; +import org.apache.seatunnel.connectors.seatunnel.common.sink.AbstractSimpleSink; +import org.apache.seatunnel.connectors.seatunnel.common.sink.AbstractSinkWriter; + import org.apache.seatunnel.shade.com.typesafe.config.Config; +import com.google.auto.service.AutoService; + import java.io.IOException; -import java.util.List; -import java.util.Optional; /** * Kudu Sink implementation by using SeaTunnel sink API. - * This class contains the method to create {@link KuduSinkWriter} and {@link KuduSinkAggregatedCommitter}. + * This class contains the method to create {@link AbstractSimpleSink}. */ @AutoService(SeaTunnelSink.class) -public class KuduSink implements SeaTunnelSink { +public class KuduSink extends AbstractSimpleSink { private Config config; private SeaTunnelRowType seaTunnelRowType; @@ -65,29 +63,7 @@ public void prepare(Config pluginConfig) throws PrepareFailException { } @Override - public SinkWriter createWriter(SinkWriter.Context context) throws IOException { - return new KuduSinkWriter(seaTunnelRowType, config, context, System.currentTimeMillis()); - } - - @Override - public SinkWriter restoreWriter(SinkWriter.Context context, List states) throws IOException { - return new KuduSinkWriter(seaTunnelRowType, config, context, System.currentTimeMillis()); - } - - @Override - public Optional> getCommitInfoSerializer() { - return Optional.of(new DefaultSerializer<>()); + public AbstractSinkWriter createWriter(SinkWriter.Context context) throws IOException { + return new KuduSinkWriter(seaTunnelRowType, config); } - - @Override - public Optional> createAggregatedCommitter() throws IOException { - return Optional.of(new KuduSinkAggregatedCommitter()); - } - - @Override - public Optional> getAggregatedCommitInfoSerializer() { - return Optional.of(new DefaultSerializer<>()); - } - - } diff --git a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/sink/KuduSinkAggregatedCommitter.java b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/sink/KuduSinkAggregatedCommitter.java deleted file mode 100644 index f06d5e89919..00000000000 --- a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/sink/KuduSinkAggregatedCommitter.java +++ /dev/null @@ -1,53 +0,0 @@ -/* - * 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.seatunnel.connectors.seatunnel.kudu.sink; - -import org.apache.seatunnel.api.sink.SinkAggregatedCommitter; - -import org.slf4j.Logger; -import org.slf4j.LoggerFactory; - -import java.io.IOException; -import java.util.ArrayList; -import java.util.HashMap; -import java.util.List; -import java.util.Map; - -public class KuduSinkAggregatedCommitter implements SinkAggregatedCommitter { - private static final Logger LOGGER = LoggerFactory.getLogger(KuduSinkAggregatedCommitter.class); - - - @Override - public List commit(List aggregatedCommitInfo) throws IOException { - return null; - } - - @Override - public KuduAggregatedCommitInfo combine(List commitInfos) { - return null; - } - - @Override - public void abort(List aggregatedCommitInfo) throws Exception { - - } - - @Override - public void close() throws IOException { - } -} diff --git a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/sink/KuduSinkState.java b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/sink/KuduSinkState.java deleted file mode 100644 index 9d32b00354e..00000000000 --- a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/sink/KuduSinkState.java +++ /dev/null @@ -1,29 +0,0 @@ -/* - * 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.seatunnel.connectors.seatunnel.kudu.sink; - -import lombok.AllArgsConstructor; -import lombok.Data; - -import java.io.Serializable; - -@Data -@AllArgsConstructor -public class KuduSinkState implements Serializable { - -} diff --git a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/sink/KuduSinkWriter.java b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/sink/KuduSinkWriter.java index 1fd9e560aec..d78f486187e 100644 --- a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/sink/KuduSinkWriter.java +++ b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/sink/KuduSinkWriter.java @@ -17,40 +17,35 @@ package org.apache.seatunnel.connectors.seatunnel.kudu.sink; -import lombok.NonNull; -import org.apache.seatunnel.api.sink.SinkWriter; import org.apache.seatunnel.api.table.type.SeaTunnelRow; import org.apache.seatunnel.api.table.type.SeaTunnelRowType; - +import org.apache.seatunnel.connectors.seatunnel.common.sink.AbstractSinkWriter; import org.apache.seatunnel.connectors.seatunnel.kudu.config.KuduSinkConfig; import org.apache.seatunnel.connectors.seatunnel.kudu.kuduclient.KuduOutputFormat; + import org.apache.seatunnel.shade.com.typesafe.config.Config; + +import lombok.NonNull; import org.slf4j.Logger; import org.slf4j.LoggerFactory; import java.io.IOException; -import java.util.Optional; -public class KuduSinkWriter implements SinkWriter { +public class KuduSinkWriter extends AbstractSinkWriter { private static final Logger LOGGER = LoggerFactory.getLogger(KuduSinkWriter.class); private SeaTunnelRowType seaTunnelRowType; private Config pluginConfig; - private Context context; - private long jobId; + private KuduOutputFormat fileWriter; private KuduSinkConfig kuduSinkConfig; public KuduSinkWriter(@NonNull SeaTunnelRowType seaTunnelRowType, - @NonNull Config pluginConfig, - @NonNull Context context, - long jobId) { + @NonNull Config pluginConfig) { this.seaTunnelRowType = seaTunnelRowType; this.pluginConfig = pluginConfig; - this.context = context; - this.jobId = jobId; kuduSinkConfig = new KuduSinkConfig(this.pluginConfig); fileWriter = new KuduOutputFormat(kuduSinkConfig); @@ -62,21 +57,8 @@ public void write(SeaTunnelRow element) throws IOException { fileWriter.write(element); } - @Override - public Optional prepareCommit() throws IOException { - return Optional.empty(); - } - - @Override - public void abortPrepare() { - - } - - @Override public void close() throws IOException { fileWriter.closeOutputFormat(); } - - } diff --git a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/source/KuduSource.java b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/source/KuduSource.java index 0d6f014a444..ddb144b1c61 100644 --- a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/source/KuduSource.java +++ b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/source/KuduSource.java @@ -17,9 +17,6 @@ package org.apache.seatunnel.connectors.seatunnel.kudu.source; -import com.google.auto.service.AutoService; -import org.apache.kudu.ColumnSchema; -import org.apache.kudu.client.*; import org.apache.seatunnel.api.common.PrepareFailException; import org.apache.seatunnel.api.common.SeaTunnelContext; import org.apache.seatunnel.api.serialization.DefaultSerializer; @@ -31,29 +28,36 @@ import org.apache.seatunnel.api.table.type.SeaTunnelDataType; import org.apache.seatunnel.api.table.type.SeaTunnelRow; import org.apache.seatunnel.api.table.type.SeaTunnelRowType; - import org.apache.seatunnel.common.constants.PluginType; import org.apache.seatunnel.connectors.seatunnel.kudu.config.KuduSourceConfig; import org.apache.seatunnel.connectors.seatunnel.kudu.kuduclient.KuduInputFormat; import org.apache.seatunnel.connectors.seatunnel.kudu.kuduclient.KuduTypeMapper; -import org.apache.seatunnel.connectors.seatunnel.kudu.state.KuduSinkState; +import org.apache.seatunnel.connectors.seatunnel.kudu.state.KuduSourceState; + import org.apache.seatunnel.shade.com.typesafe.config.Config; + +import com.google.auto.service.AutoService; +import org.apache.kudu.ColumnSchema; +import org.apache.kudu.client.KuduClient; +import org.apache.kudu.client.KuduException; +import org.apache.kudu.client.KuduScanner; +import org.apache.kudu.client.RowResult; +import org.apache.kudu.client.RowResultIterator; import org.slf4j.Logger; import org.slf4j.LoggerFactory; import java.util.ArrayList; -import java.util.Collections; import java.util.List; @AutoService(SeaTunnelSource.class) -public class KuduSource implements SeaTunnelSource { +public class KuduSource implements SeaTunnelSource { private static final Logger LOGGER = LoggerFactory.getLogger(KuduSource.class); - private Config pluginConfig; private SeaTunnelContext seaTunnelContext; private SeaTunnelRowType rowTypeInfo; private KuduInputFormat kuduInputFormat; private PartitionParameter partitionParameter; + public static final int TIMEOUTMS = 18000; @Override public Boundedness getBoundedness() { @@ -67,7 +71,7 @@ public SeaTunnelRowType getProducedType() { @Override public SourceReader createReader(SourceReader.Context readerContext) { - return new KuduSourceReader(kuduInputFormat,readerContext); + return new KuduSourceReader(kuduInputFormat, readerContext); } @Override @@ -76,20 +80,20 @@ public Serializer getSplitSerializer() { } @Override - public SourceSplitEnumerator createEnumerator( + public SourceSplitEnumerator createEnumerator( SourceSplitEnumerator.Context enumeratorContext) { - return new KuduSourceSplitEnumerator(enumeratorContext,partitionParameter); + return new KuduSourceSplitEnumerator(enumeratorContext, partitionParameter); } @Override - public SourceSplitEnumerator restoreEnumerator( - SourceSplitEnumerator.Context enumeratorContext, KuduSinkState checkpointState) { + public SourceSplitEnumerator restoreEnumerator( + SourceSplitEnumerator.Context enumeratorContext, KuduSourceState checkpointState) { // todo: - return new KuduSourceSplitEnumerator(enumeratorContext,partitionParameter); + return new KuduSourceSplitEnumerator(enumeratorContext, partitionParameter); } @Override - public Serializer getEnumeratorStateSerializer() { + public Serializer getEnumeratorStateSerializer() { return new DefaultSerializer<>(); } @@ -100,81 +104,68 @@ public String getPluginName() { @Override public void prepare(Config config) { - - String kudumaster = config.getString(KuduSourceConfig.kuduMaster); - String tableName = config.getString(KuduSourceConfig.tableName); - String columnslist = config.getString(KuduSourceConfig.columnsList); - kuduInputFormat=new KuduInputFormat(kudumaster,tableName,columnslist); + String kudumaster = config.getString(KuduSourceConfig.KUDUMASTER); + String tableName = config.getString(KuduSourceConfig.TABLENAME); + String columnslist = config.getString(KuduSourceConfig.COLUMNSLIST); + kuduInputFormat = new KuduInputFormat(kudumaster, tableName, columnslist); try { KuduClient.KuduClientBuilder kuduClientBuilder = new KuduClient.KuduClientBuilder(kudumaster); - kuduClientBuilder.defaultOperationTimeoutMs(1800000); + kuduClientBuilder.defaultOperationTimeoutMs(TIMEOUTMS); KuduClient kuduClient = kuduClientBuilder.build(); - partitionParameter = initPartitionParameter(kuduClient,tableName); - SeaTunnelRowType seaTunnelRowType =getSeaTunnelRowType(kuduClient.openTable(tableName).getSchema().getColumns()); - rowTypeInfo=seaTunnelRowType; + partitionParameter = initPartitionParameter(kuduClient, tableName); + SeaTunnelRowType seaTunnelRowType = getSeaTunnelRowType(kuduClient.openTable(tableName).getSchema().getColumns()); + rowTypeInfo = seaTunnelRowType; } catch (KuduException e) { - e.printStackTrace(); + throw new RuntimeException("Parameters in the preparation phase fail to be generated", e); } } - private PartitionParameter initPartitionParameter(KuduClient kuduClient,String tableName) { + private PartitionParameter initPartitionParameter(KuduClient kuduClient, String tableName) { String keyColumn = null; - int maxKey=0; - int minKey=0; - boolean flag=true; + int maxKey = 0; + int minKey = 0; + boolean flag = true; try { KuduScanner.KuduScannerBuilder kuduScannerBuilder = kuduClient.newScannerBuilder(kuduClient.openTable(tableName)); ArrayList columnsList = new ArrayList(); keyColumn = kuduClient.openTable(tableName).getSchema().getPrimaryKeyColumns().get(0).getName(); - columnsList.add(""+keyColumn); + columnsList.add("" + keyColumn); kuduScannerBuilder.setProjectedColumnNames(columnsList); KuduScanner kuduScanner = kuduScannerBuilder.build(); - - while (kuduScanner.hasMoreRows()) { RowResultIterator rowResults = kuduScanner.nextRows(); while (rowResults.hasNext()) { RowResult row = rowResults.next(); - int id = row.getInt(""+keyColumn); + int id = row.getInt("" + keyColumn); if (flag){ - maxKey=id; - minKey=id; - flag=false; - }else { - if (id>=maxKey){ - maxKey=id; + maxKey = id; + minKey = id; + flag = false; + } else { + if (id >= maxKey){ + maxKey = id; } - if (id<=minKey){ - minKey=id; + if (id <= minKey){ + minKey = id; } } } } } catch (KuduException e) { - e.printStackTrace(); + throw new RuntimeException("Failed to generate upper and lower limits for each partition", e); } - - - return new PartitionParameter(keyColumn, Long.parseLong(minKey+""), Long.parseLong(maxKey+"")); + return new PartitionParameter(keyColumn, Long.parseLong(minKey + ""), Long.parseLong(maxKey + "")); } - - /* @Override - public SeaTunnelContext getSeaTunnelContext() { - return seaTunnelContext; - }*/ - @Override public void setSeaTunnelContext(SeaTunnelContext seaTunnelContext) { this.seaTunnelContext = seaTunnelContext; } - public SeaTunnelRowType getSeaTunnelRowType(List columnSchemaList) { - ArrayList> seaTunnelDataTypes = new ArrayList<>(); ArrayList fieldNames = new ArrayList<>(); try { @@ -186,7 +177,7 @@ public SeaTunnelRowType getSeaTunnelRowType(List columnSchemaList) } catch (Exception e) { LOGGER.warn("get row type info exception", e); - throw new PrepareFailException("jdbc", PluginType.SOURCE, e.toString()); + throw new PrepareFailException("kudu", PluginType.SOURCE, e.toString()); } return new SeaTunnelRowType(fieldNames.toArray(new String[fieldNames.size()]), seaTunnelDataTypes.toArray(new SeaTunnelDataType[seaTunnelDataTypes.size()])); } diff --git a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/source/KuduSourceReader.java b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/source/KuduSourceReader.java index b235744f216..ad23a6e3909 100644 --- a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/source/KuduSourceReader.java +++ b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/source/KuduSourceReader.java @@ -17,22 +17,22 @@ package org.apache.seatunnel.connectors.seatunnel.kudu.source; -import org.apache.kudu.ColumnSchema; -import org.apache.kudu.client.KuduException; -import org.apache.kudu.client.KuduScanner; -import org.apache.kudu.client.RowResult; -import org.apache.kudu.client.RowResultIterator; import org.apache.seatunnel.api.source.Boundedness; import org.apache.seatunnel.api.source.Collector; import org.apache.seatunnel.api.source.SourceReader; import org.apache.seatunnel.api.table.type.SeaTunnelRow; import org.apache.seatunnel.connectors.seatunnel.kudu.kuduclient.KuduInputFormat; + +import org.apache.kudu.ColumnSchema; +import org.apache.kudu.client.KuduScanner; +import org.apache.kudu.client.RowResult; +import org.apache.kudu.client.RowResultIterator; import org.slf4j.Logger; import org.slf4j.LoggerFactory; -import java.sql.SQLException; -import java.util.*; -import java.util.concurrent.ThreadLocalRandom; +import java.util.Deque; +import java.util.LinkedList; +import java.util.List; public class KuduSourceReader implements SourceReader { @@ -61,18 +61,13 @@ public void close() { } @Override - @SuppressWarnings("magicnumber") - public void pollNext(Collector output) throws InterruptedException, KuduException, SQLException { + public void pollNext(Collector output) throws Exception { KuduSourceSplit split = splits.poll(); Object[] parameterValues = split.parameterValues; - - int lowerBound=Integer.parseInt(parameterValues[0].toString()); - - int upperBound=Integer.parseInt(parameterValues[1].toString()); - - + int lowerBound = Integer.parseInt(parameterValues[0].toString()); + int upperBound = Integer.parseInt(parameterValues[1].toString()); List columnSchemaList = kuduInputFormat.getColumnsSchemas(); - KuduScanner kuduScanner = kuduInputFormat.getKuduBuildSplit( lowerBound, upperBound); + KuduScanner kuduScanner = kuduInputFormat.getKuduBuildSplit(lowerBound, upperBound); // while (kuduScanner.hasMoreRows()) { RowResultIterator rowResults = kuduScanner.nextRows(); @@ -82,10 +77,6 @@ public void pollNext(Collector output) throws InterruptedException output.collect(seaTunnelRow); } } - - - // Generate a random number of rows to emit. - if (Boundedness.BOUNDED.equals(context.getBoundedness())) { // signal to the source that we have reached the end of the data. LOGGER.info("Closed the bounded fake source"); diff --git a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/source/KuduSourceSplit.java b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/source/KuduSourceSplit.java index 2c92fb13fbf..4f1320f7312 100644 --- a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/source/KuduSourceSplit.java +++ b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/source/KuduSourceSplit.java @@ -17,9 +17,11 @@ package org.apache.seatunnel.connectors.seatunnel.kudu.source; +import org.apache.seatunnel.api.source.SourceSplit; + import lombok.AllArgsConstructor; import lombok.Data; -import org.apache.seatunnel.api.source.SourceSplit; + @Data @AllArgsConstructor public class KuduSourceSplit implements SourceSplit { diff --git a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/source/KuduSourceSplitEnumerator.java b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/source/KuduSourceSplitEnumerator.java index 66c8f92dbce..aceb7b2e645 100644 --- a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/source/KuduSourceSplitEnumerator.java +++ b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/source/KuduSourceSplitEnumerator.java @@ -18,8 +18,7 @@ package org.apache.seatunnel.connectors.seatunnel.kudu.source; import org.apache.seatunnel.api.source.SourceSplitEnumerator; - -import org.apache.seatunnel.connectors.seatunnel.kudu.state.KuduSinkState; +import org.apache.seatunnel.connectors.seatunnel.kudu.state.KuduSourceState; import java.io.IOException; import java.io.Serializable; @@ -27,19 +26,19 @@ import java.util.List; import java.util.stream.Collectors; -public class KuduSourceSplitEnumerator implements SourceSplitEnumerator { +public class KuduSourceSplitEnumerator implements SourceSplitEnumerator { private final Context enumeratorContext; - private PartitionParameter partitionParameter; + private PartitionParameter partitionParameter; List allSplit = new ArrayList<>(); private Long maxVal; private Long minVal; private Long batchSize; private Integer batchNum; - public KuduSourceSplitEnumerator(Context enumeratorContext,PartitionParameter partitionParameter) { + public KuduSourceSplitEnumerator(Context enumeratorContext, PartitionParameter partitionParameter) { this.enumeratorContext = enumeratorContext; - this.partitionParameter=partitionParameter; + this.partitionParameter = partitionParameter; } @Override @@ -77,7 +76,7 @@ public void registerReader(int subtaskId) { int parallelism = enumeratorContext.currentParallelism(); if (allSplit.isEmpty()) { if (null != partitionParameter) { - Serializable[][] parameterValues = getParameterValues(partitionParameter.minValue, partitionParameter.maxValue,parallelism); + Serializable[][] parameterValues = getParameterValues(partitionParameter.minValue, partitionParameter.maxValue, parallelism); for (int i = 0; i < parameterValues.length; i++) { allSplit.add(new KuduSourceSplit(parameterValues[i], i)); } @@ -92,18 +91,18 @@ public void registerReader(int subtaskId) { } private Serializable[][] getParameterValues(Long minVal, Long maxVal, int parallelism) { - this.maxVal=maxVal; - this.minVal=minVal; + this.maxVal = maxVal; + this.minVal = minVal; long maxElemCount = (maxVal - minVal) + 1; - batchNum=parallelism; - getBatchSizeAndBatchNum(parallelism); + batchNum = parallelism; + getBatchSizeAndBatchNum(parallelism); long bigBatchNum = maxElemCount - (batchSize - 1) * batchNum; Serializable[][] parameters = new Serializable[batchNum][2]; long start = minVal; for (int i = 0; i < batchNum; i++) { long end = start + batchSize - 1 - (i >= bigBatchNum ? 1 : 0); - parameters[i] = new Long[] {start, end}; + parameters[i] = new Long[]{start, end}; start = end + 1; } return parameters; @@ -111,7 +110,7 @@ private Serializable[][] getParameterValues(Long minVal, Long maxVal, int parall } private void getBatchSizeAndBatchNum(int parallelism) { - batchNum=parallelism; + batchNum = parallelism; long maxElemCount = (maxVal - minVal) + 1; if (batchNum > maxElemCount) { batchNum = (int) maxElemCount; @@ -121,7 +120,7 @@ private void getBatchSizeAndBatchNum(int parallelism) { } @Override - public KuduSinkState snapshotState(long checkpointId) throws Exception { + public KuduSourceState snapshotState(long checkpointId) throws Exception { return null; } diff --git a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/state/KuduSinkState.java b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/state/KuduSourceState.java similarity index 94% rename from seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/state/KuduSinkState.java rename to seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/state/KuduSourceState.java index f50416ee29a..317bf375fc1 100644 --- a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/state/KuduSinkState.java +++ b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/state/KuduSourceState.java @@ -19,5 +19,5 @@ import java.io.Serializable; -public class KuduSinkState implements Serializable { +public class KuduSourceState implements Serializable { } From 42ea0f0d39cbf148c6480b88b9f8e6b4d26158d6 Mon Sep 17 00:00:00 2001 From: 2013650523 <1067341434@qq.com> Date: Fri, 29 Jul 2022 22:05:38 +0800 Subject: [PATCH 12/26] [Connector-V2] Add Kudu source and sink connector --- .../seatunnel/kudu/config/KuduSinkConfig.java | 11 +++++---- .../seatunnel/kudu/source/KuduSource.java | 23 ++++++++++++------- 2 files changed, 22 insertions(+), 12 deletions(-) diff --git a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/config/KuduSinkConfig.java b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/config/KuduSinkConfig.java index 017ff5874ac..b44a3e730e6 100644 --- a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/config/KuduSinkConfig.java +++ b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/config/KuduSinkConfig.java @@ -53,9 +53,12 @@ public static SaveMode fromStr(String str) { } public KuduSinkConfig(@NonNull Config pluginConfig) { - - this.saveMode = StringUtils.isBlank(pluginConfig.getString(KUDU_SAVE_MODE)) ? SaveMode.APPEND : SaveMode.fromStr(pluginConfig.getString(KUDU_SAVE_MODE)); - this.kuduMaster = pluginConfig.getString(KUDU_MASTER); - this.kuduTableName = pluginConfig.getString(KUDU_TABLE_NAME); + if (pluginConfig.hasPath(KUDU_SAVE_MODE) && pluginConfig.hasPath(KUDU_MASTER) && pluginConfig.hasPath(KUDU_TABLE_NAME)) { + this.saveMode = StringUtils.isBlank(pluginConfig.getString(KUDU_SAVE_MODE)) ? SaveMode.APPEND : SaveMode.fromStr(pluginConfig.getString(KUDU_SAVE_MODE)); + this.kuduMaster = pluginConfig.getString(KUDU_MASTER); + this.kuduTableName = pluginConfig.getString(KUDU_TABLE_NAME); + } else { + throw new RuntimeException("Missing Sink configuration parameters"); + } } } diff --git a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/source/KuduSource.java b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/source/KuduSource.java index ddb144b1c61..65b004a6a36 100644 --- a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/source/KuduSource.java +++ b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/source/KuduSource.java @@ -66,7 +66,7 @@ public Boundedness getBoundedness() { @Override public SeaTunnelRowType getProducedType() { - return this.rowTypeInfo; + return this.rowTypeInfo; } @Override @@ -104,10 +104,17 @@ public String getPluginName() { @Override public void prepare(Config config) { - String kudumaster = config.getString(KuduSourceConfig.KUDUMASTER); - String tableName = config.getString(KuduSourceConfig.TABLENAME); - String columnslist = config.getString(KuduSourceConfig.COLUMNSLIST); - kuduInputFormat = new KuduInputFormat(kudumaster, tableName, columnslist); + String kudumaster = ""; + String tableName = ""; + String columnslist = ""; + if (config.hasPath(KuduSourceConfig.KUDUMASTER) && config.hasPath(KuduSourceConfig.KUDUMASTER) && config.hasPath(KuduSourceConfig.KUDUMASTER)) { + kudumaster = config.getString(KuduSourceConfig.KUDUMASTER); + tableName = config.getString(KuduSourceConfig.TABLENAME); + columnslist = config.getString(KuduSourceConfig.COLUMNSLIST); + kuduInputFormat = new KuduInputFormat(kudumaster, tableName, columnslist); + } else { + throw new RuntimeException("Missing Source configuration parameters"); + } try { KuduClient.KuduClientBuilder kuduClientBuilder = new KuduClient.KuduClientBuilder(kudumaster); @@ -140,15 +147,15 @@ private PartitionParameter initPartitionParameter(KuduClient kuduClient, String while (rowResults.hasNext()) { RowResult row = rowResults.next(); int id = row.getInt("" + keyColumn); - if (flag){ + if (flag) { maxKey = id; minKey = id; flag = false; } else { - if (id >= maxKey){ + if (id >= maxKey) { maxKey = id; } - if (id <= minKey){ + if (id <= minKey) { minKey = id; } } From 097492cdd03529b993066b33c14433b6a313a43c Mon Sep 17 00:00:00 2001 From: 2013650523 <1067341434@qq.com> Date: Sat, 30 Jul 2022 16:20:55 +0800 Subject: [PATCH 13/26] [Connector-V2] Add Email sink connector --- pom.xml | 7 + .../connector-email/pom.xml | 31 ++++ .../email/config/EmailSinkConfig.java | 56 +++++++ .../seatunnel/email/sink/EmailSink.java | 63 +++++++ .../seatunnel/email/sink/EmailSinkWriter.java | 158 ++++++++++++++++++ seatunnel-connectors-v2/pom.xml | 1 + 6 files changed, 316 insertions(+) create mode 100644 seatunnel-connectors-v2/connector-email/pom.xml create mode 100644 seatunnel-connectors-v2/connector-email/src/main/java/org/apache/seatunnel/connectors/seatunnel/email/config/EmailSinkConfig.java create mode 100644 seatunnel-connectors-v2/connector-email/src/main/java/org/apache/seatunnel/connectors/seatunnel/email/sink/EmailSink.java create mode 100644 seatunnel-connectors-v2/connector-email/src/main/java/org/apache/seatunnel/connectors/seatunnel/email/sink/EmailSinkWriter.java diff --git a/pom.xml b/pom.xml index 9578970ef13..99d2f912c0b 100644 --- a/pom.xml +++ b/pom.xml @@ -170,6 +170,7 @@ 2.6.0 3.4 1.11.1 + 1.4.7 4.4 3.3.0 provided @@ -268,6 +269,12 @@ kudu-client ${kudu.version} + + + com.sun.mail + javax.mail + ${email.version} + org.apache.flink diff --git a/seatunnel-connectors-v2/connector-email/pom.xml b/seatunnel-connectors-v2/connector-email/pom.xml new file mode 100644 index 00000000000..f30c49f76ae --- /dev/null +++ b/seatunnel-connectors-v2/connector-email/pom.xml @@ -0,0 +1,31 @@ + + + + seatunnel-connectors-v2 + org.apache.seatunnel + ${revision} + + 4.0.0 + + connector-email + + + + org.apache.seatunnel + connector-common + ${project.version} + + + org.apache.seatunnel + seatunnel-api + ${project.version} + + + com.sun.mail + javax.mail + + + + \ No newline at end of file diff --git a/seatunnel-connectors-v2/connector-email/src/main/java/org/apache/seatunnel/connectors/seatunnel/email/config/EmailSinkConfig.java b/seatunnel-connectors-v2/connector-email/src/main/java/org/apache/seatunnel/connectors/seatunnel/email/config/EmailSinkConfig.java new file mode 100644 index 00000000000..d09dcca3607 --- /dev/null +++ b/seatunnel-connectors-v2/connector-email/src/main/java/org/apache/seatunnel/connectors/seatunnel/email/config/EmailSinkConfig.java @@ -0,0 +1,56 @@ +/* + * 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.seatunnel.connectors.seatunnel.email.config; + +import org.apache.seatunnel.shade.com.typesafe.config.Config; + +import lombok.Data; +import lombok.NonNull; + +@Data +public class EmailSinkConfig { + + private static final String EMAIL_FROM_ADDRESS = "email_from_address"; + private static final String EMAIL_TO_ADDRESS = "email_to_address"; + private static final String EMAIL_AUTHORIZATION_CODE = "email_authorization_code"; + private static final String EMAIL_MESSAGE_HEADLINE = "email_message_headline"; + private static final String EMAIL_MESSAGE_CONTENT = "email_message_content"; + private String emailFromAddress; + private String emailToAddress; + private String emailAuthorizationCode; + private String emailMessageHeadline; + private String emailMessageContent; + + public EmailSinkConfig(@NonNull Config pluginConfig) { + if (pluginConfig.hasPath(EMAIL_FROM_ADDRESS)) { + this.emailFromAddress = pluginConfig.getString(EMAIL_FROM_ADDRESS); + } + if (pluginConfig.hasPath(EMAIL_TO_ADDRESS)) { + this.emailToAddress = pluginConfig.getString(EMAIL_TO_ADDRESS); + } + if (pluginConfig.hasPath(EMAIL_AUTHORIZATION_CODE)) { + this.emailAuthorizationCode = pluginConfig.getString(EMAIL_AUTHORIZATION_CODE); + } + if (pluginConfig.hasPath(EMAIL_MESSAGE_HEADLINE)) { + this.emailMessageHeadline = pluginConfig.getString(EMAIL_MESSAGE_HEADLINE); + } + if (pluginConfig.hasPath(EMAIL_MESSAGE_CONTENT)) { + this.emailMessageContent = pluginConfig.getString(EMAIL_MESSAGE_CONTENT); + } + } +} diff --git a/seatunnel-connectors-v2/connector-email/src/main/java/org/apache/seatunnel/connectors/seatunnel/email/sink/EmailSink.java b/seatunnel-connectors-v2/connector-email/src/main/java/org/apache/seatunnel/connectors/seatunnel/email/sink/EmailSink.java new file mode 100644 index 00000000000..bd486054267 --- /dev/null +++ b/seatunnel-connectors-v2/connector-email/src/main/java/org/apache/seatunnel/connectors/seatunnel/email/sink/EmailSink.java @@ -0,0 +1,63 @@ +/* + * 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.seatunnel.connectors.seatunnel.email.sink; + +import org.apache.seatunnel.api.sink.SeaTunnelSink; +import org.apache.seatunnel.api.sink.SinkWriter; +import org.apache.seatunnel.api.table.type.SeaTunnelDataType; +import org.apache.seatunnel.api.table.type.SeaTunnelRow; +import org.apache.seatunnel.api.table.type.SeaTunnelRowType; +import org.apache.seatunnel.connectors.seatunnel.common.sink.AbstractSimpleSink; +import org.apache.seatunnel.connectors.seatunnel.common.sink.AbstractSinkWriter; + +import org.apache.seatunnel.shade.com.typesafe.config.Config; + +import com.google.auto.service.AutoService; + +@AutoService(SeaTunnelSink.class) +public class EmailSink extends AbstractSimpleSink { + + private Config pluginConfig; + private SeaTunnelRowType seaTunnelRowType; + + @Override + public void setTypeInfo(SeaTunnelRowType seaTunnelRowType) { + this.seaTunnelRowType = seaTunnelRowType; + } + + @Override + public SeaTunnelDataType getConsumedType() { + return this.seaTunnelRowType; + } + + @Override + public AbstractSinkWriter createWriter(SinkWriter.Context context) { + return new EmailSinkWriter(seaTunnelRowType, pluginConfig); + } + + @Override + public String getPluginName() { + return "EmailSink"; + } + + @Override + public void prepare(Config pluginConfig) { + this.pluginConfig = pluginConfig; + } + +} diff --git a/seatunnel-connectors-v2/connector-email/src/main/java/org/apache/seatunnel/connectors/seatunnel/email/sink/EmailSinkWriter.java b/seatunnel-connectors-v2/connector-email/src/main/java/org/apache/seatunnel/connectors/seatunnel/email/sink/EmailSinkWriter.java new file mode 100644 index 00000000000..731b92c39c7 --- /dev/null +++ b/seatunnel-connectors-v2/connector-email/src/main/java/org/apache/seatunnel/connectors/seatunnel/email/sink/EmailSinkWriter.java @@ -0,0 +1,158 @@ +/* + * 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.seatunnel.connectors.seatunnel.email.sink; + +import org.apache.seatunnel.api.table.type.SeaTunnelRow; +import org.apache.seatunnel.api.table.type.SeaTunnelRowType; +import org.apache.seatunnel.connectors.seatunnel.common.sink.AbstractSinkWriter; +import org.apache.seatunnel.connectors.seatunnel.email.config.EmailSinkConfig; + +import org.apache.seatunnel.shade.com.typesafe.config.Config; + +import com.sun.mail.util.MailSSLSocketFactory; +import org.slf4j.Logger; +import org.slf4j.LoggerFactory; + +import javax.activation.DataHandler; +import javax.activation.DataSource; +import javax.activation.FileDataSource; +import javax.mail.Authenticator; +import javax.mail.BodyPart; +import javax.mail.Message; +import javax.mail.Multipart; +import javax.mail.PasswordAuthentication; +import javax.mail.Session; +import javax.mail.Transport; +import javax.mail.internet.InternetAddress; +import javax.mail.internet.MimeBodyPart; +import javax.mail.internet.MimeMessage; +import javax.mail.internet.MimeMultipart; + +import java.io.File; +import java.io.FileWriter; +import java.io.IOException; +import java.util.Properties; + +public class EmailSinkWriter extends AbstractSinkWriter { + + private static final Logger LOGGER = LoggerFactory.getLogger(EmailSinkWriter.class); + + private final SeaTunnelRowType seaTunnelRowType; + private EmailSinkConfig config; + private StringBuffer stringBuffer; + + public EmailSinkWriter(SeaTunnelRowType seaTunnelRowType, Config pluginConfig) { + this.seaTunnelRowType = seaTunnelRowType; + this.config = new EmailSinkConfig(pluginConfig); + this.stringBuffer = new StringBuffer(); + } + + @Override + public void write(SeaTunnelRow element) { + Object[] fields = element.getFields(); + + for (Object field : fields) { + stringBuffer.append(field.toString() + ","); + } + stringBuffer.deleteCharAt(fields.length - 1); + stringBuffer.append("\n"); + + } + + @Override + public void close() { + createFile(); + Properties properties = new Properties(); + + properties.setProperty("mail.host", "smtp.qq.com"); + + properties.setProperty("mail.transport.protocol", "smtp"); + + properties.setProperty("mail.smtp.auth", "true"); + + try { + MailSSLSocketFactory sf = new MailSSLSocketFactory(); + sf.setTrustAllHosts(true); + properties.put("mail.smtp.ssl.enable", "true"); + properties.put("mail.smtp.ssl.socketFactory", sf); + Session session = Session.getDefaultInstance(properties, new Authenticator() { + @Override + protected PasswordAuthentication getPasswordAuthentication() { + return new PasswordAuthentication(config.getEmailFromAddress(), config.getEmailAuthorizationCode()); + } + }); + //Create the default MimeMessage object + MimeMessage message = new MimeMessage(session); + + // Set the email address + message.setFrom(new InternetAddress(config.getEmailFromAddress())); + + // Set the recipient email address + message.addRecipient(Message.RecipientType.TO, + new InternetAddress(config.getEmailToAddress())); + + // Setting the Email subject + message.setSubject(config.getEmailMessageHeadline()); + + // Create Message + BodyPart messageBodyPart = new MimeBodyPart(); + + // Set Message content + messageBodyPart.setText(config.getEmailMessageContent()); + + // Create multiple messages + Multipart multipart = new MimeMultipart(); + // Set up the text message section + multipart.addBodyPart(messageBodyPart); + // accessory + messageBodyPart = new MimeBodyPart(); + String filename = "emailsink.csv"; + DataSource source = new FileDataSource(filename); + messageBodyPart.setDataHandler(new DataHandler(source)); + messageBodyPart.setFileName(filename); + multipart.addBodyPart(messageBodyPart); + message.setContent(multipart); + + // send a message + Transport.send(message); + LOGGER.info("Sent message successfully...."); + } catch (Exception e) { + LOGGER.warn("send email Fail.", e); + throw new RuntimeException("send email Fail.", e); + } + } + + public void createFile() { + try { + String data = stringBuffer.toString(); + File file = new File("emailsink.csv"); + //if file doesnt exists, then create it + if (!file.exists()) { + file.createNewFile(); + } + FileWriter fileWritter = new FileWriter(file.getName()); + fileWritter.write(data); + fileWritter.close(); + LOGGER.info("Create File successfully...."); + } catch (IOException e) { + LOGGER.warn("Create File Fail.", e); + throw new RuntimeException("Create File Fail.", e); + } + + } +} diff --git a/seatunnel-connectors-v2/pom.xml b/seatunnel-connectors-v2/pom.xml index 56483f8acc3..75903d3789e 100644 --- a/seatunnel-connectors-v2/pom.xml +++ b/seatunnel-connectors-v2/pom.xml @@ -45,6 +45,7 @@ connector-hudi connector-assert connector-kudu + connector-email From a0107832d7ceff6208dff8072a45a0eb59c9b02e Mon Sep 17 00:00:00 2001 From: liuyehan <81779971+2013650523@users.noreply.github.com> Date: Sat, 30 Jul 2022 16:30:30 +0800 Subject: [PATCH 14/26] Update pom.xml --- seatunnel-connectors-v2/pom.xml | 1 - 1 file changed, 1 deletion(-) diff --git a/seatunnel-connectors-v2/pom.xml b/seatunnel-connectors-v2/pom.xml index 75903d3789e..a242ac7ff44 100644 --- a/seatunnel-connectors-v2/pom.xml +++ b/seatunnel-connectors-v2/pom.xml @@ -44,7 +44,6 @@ connector-file connector-hudi connector-assert - connector-kudu connector-email From 25847b4c3f89d8e2d7678305d6b2f47ee0662075 Mon Sep 17 00:00:00 2001 From: liuyehan <81779971+2013650523@users.noreply.github.com> Date: Sat, 30 Jul 2022 16:31:40 +0800 Subject: [PATCH 15/26] Update pom.xml --- pom.xml | 7 ------- 1 file changed, 7 deletions(-) diff --git a/pom.xml b/pom.xml index 99d2f912c0b..86fee22459d 100644 --- a/pom.xml +++ b/pom.xml @@ -169,7 +169,6 @@ 2.2.0 2.6.0 3.4 - 1.11.1 1.4.7 4.4 3.3.0 @@ -262,12 +261,6 @@ net.jpountz.lz4 lz4 1.3.0 - - - - org.apache.kudu - kudu-client - ${kudu.version} From 3cf9ef5083b2979b17df5c80d67ebdfb47a47981 Mon Sep 17 00:00:00 2001 From: liuyehan <81779971+2013650523@users.noreply.github.com> Date: Sat, 30 Jul 2022 16:35:13 +0800 Subject: [PATCH 16/26] Delete seatunnel-connectors-v2/connector-kudu directory --- .../connector-kudu/pom.xml | 53 ----- .../seatunnel/kudu/config/KuduSinkConfig.java | 64 ------ .../kudu/config/KuduSourceConfig.java | 29 --- .../kudu/kuduclient/KuduInputFormat.java | 202 ------------------ .../kudu/kuduclient/KuduOutputFormat.java | 164 -------------- .../kudu/kuduclient/KuduTypeMapper.java | 103 --------- .../seatunnel/kudu/sink/KuduSink.java | 69 ------ .../seatunnel/kudu/sink/KuduSinkWriter.java | 64 ------ .../seatunnel/kudu/source/KuduSource.java | 191 ----------------- .../kudu/source/KuduSourceReader.java | 107 ---------- .../kudu/source/KuduSourceSplit.java | 39 ---- .../source/KuduSourceSplitEnumerator.java | 131 ------------ .../kudu/source/PartitionParameter.java | 32 --- .../seatunnel/kudu/state/KuduSourceState.java | 23 -- .../main/resources/kudu_to_kudu_flink.conf | 60 ------ .../main/resources/kudu_to_kudu_spark.conf | 64 ------ 16 files changed, 1395 deletions(-) delete mode 100644 seatunnel-connectors-v2/connector-kudu/pom.xml delete mode 100644 seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/config/KuduSinkConfig.java delete mode 100644 seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/config/KuduSourceConfig.java delete mode 100644 seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/kuduclient/KuduInputFormat.java delete mode 100644 seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/kuduclient/KuduOutputFormat.java delete mode 100644 seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/kuduclient/KuduTypeMapper.java delete mode 100644 seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/sink/KuduSink.java delete mode 100644 seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/sink/KuduSinkWriter.java delete mode 100644 seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/source/KuduSource.java delete mode 100644 seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/source/KuduSourceReader.java delete mode 100644 seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/source/KuduSourceSplit.java delete mode 100644 seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/source/KuduSourceSplitEnumerator.java delete mode 100644 seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/source/PartitionParameter.java delete mode 100644 seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/state/KuduSourceState.java delete mode 100644 seatunnel-connectors-v2/connector-kudu/src/main/resources/kudu_to_kudu_flink.conf delete mode 100644 seatunnel-connectors-v2/connector-kudu/src/main/resources/kudu_to_kudu_spark.conf diff --git a/seatunnel-connectors-v2/connector-kudu/pom.xml b/seatunnel-connectors-v2/connector-kudu/pom.xml deleted file mode 100644 index febac92c3ac..00000000000 --- a/seatunnel-connectors-v2/connector-kudu/pom.xml +++ /dev/null @@ -1,53 +0,0 @@ - - - - - seatunnel-connectors-v2 - org.apache.seatunnel - ${revision} - - 4.0.0 - - connector-kudu - - - - org.apache.seatunnel - seatunnel-api - ${project.version} - - - org.apache.kudu - kudu-client - - - - org.apache.commons - commons-lang3 - - - org.apache.seatunnel - connector-common - ${project.version} - - - diff --git a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/config/KuduSinkConfig.java b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/config/KuduSinkConfig.java deleted file mode 100644 index b44a3e730e6..00000000000 --- a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/config/KuduSinkConfig.java +++ /dev/null @@ -1,64 +0,0 @@ -/* - * 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.seatunnel.connectors.seatunnel.kudu.config; - -import org.apache.seatunnel.shade.com.typesafe.config.Config; - -import lombok.Data; -import lombok.NonNull; -import org.apache.commons.lang3.StringUtils; - -@Data -public class KuduSinkConfig { - - private static final String KUDU_SAVE_MODE = "save_mode"; - private static final String KUDU_MASTER = "kudu_master"; - private static final String KUDU_TABLE_NAME = "kudu_table"; - - private SaveMode saveMode = SaveMode.APPEND; - - private String kuduMaster; - - /** - * Specifies the name of the table - */ - private String kuduTableName; - - public enum SaveMode { - APPEND(), - OVERWRITE(); - - public static SaveMode fromStr(String str) { - if ("overwrite".equals(str)) { - return OVERWRITE; - } else { - return APPEND; - } - } - } - - public KuduSinkConfig(@NonNull Config pluginConfig) { - if (pluginConfig.hasPath(KUDU_SAVE_MODE) && pluginConfig.hasPath(KUDU_MASTER) && pluginConfig.hasPath(KUDU_TABLE_NAME)) { - this.saveMode = StringUtils.isBlank(pluginConfig.getString(KUDU_SAVE_MODE)) ? SaveMode.APPEND : SaveMode.fromStr(pluginConfig.getString(KUDU_SAVE_MODE)); - this.kuduMaster = pluginConfig.getString(KUDU_MASTER); - this.kuduTableName = pluginConfig.getString(KUDU_TABLE_NAME); - } else { - throw new RuntimeException("Missing Sink configuration parameters"); - } - } -} diff --git a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/config/KuduSourceConfig.java b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/config/KuduSourceConfig.java deleted file mode 100644 index 754b53114ae..00000000000 --- a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/config/KuduSourceConfig.java +++ /dev/null @@ -1,29 +0,0 @@ -/* - * 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.seatunnel.connectors.seatunnel.kudu.config; - -import java.io.Serializable; - -public class KuduSourceConfig implements Serializable { - - public static final String KUDUMASTER = "kudu_master"; - public static final String TABLENAME = "kudu_table"; - public static final String COLUMNSLIST = "columnsList"; - - -} diff --git a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/kuduclient/KuduInputFormat.java b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/kuduclient/KuduInputFormat.java deleted file mode 100644 index 0f36f7e252d..00000000000 --- a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/kuduclient/KuduInputFormat.java +++ /dev/null @@ -1,202 +0,0 @@ -/* - * 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.seatunnel.connectors.seatunnel.kudu.kuduclient; - -import org.apache.seatunnel.api.common.PrepareFailException; -import org.apache.seatunnel.api.table.type.BasicType; -import org.apache.seatunnel.api.table.type.DecimalType; -import org.apache.seatunnel.api.table.type.SeaTunnelDataType; -import org.apache.seatunnel.api.table.type.SeaTunnelRow; -import org.apache.seatunnel.api.table.type.SeaTunnelRowType; -import org.apache.seatunnel.common.constants.PluginType; - -import org.apache.kudu.ColumnSchema; -import org.apache.kudu.Schema; -import org.apache.kudu.client.KuduClient; -import org.apache.kudu.client.KuduException; -import org.apache.kudu.client.KuduPredicate; -import org.apache.kudu.client.KuduScanner; -import org.apache.kudu.client.RowResult; -import org.slf4j.Logger; -import org.slf4j.LoggerFactory; - -import java.io.Serializable; -import java.math.BigDecimal; -import java.math.BigInteger; -import java.sql.SQLException; -import java.util.ArrayList; -import java.util.Arrays; -import java.util.List; - -public class KuduInputFormat implements Serializable { - private static final Logger LOGGER = LoggerFactory.getLogger(KuduInputFormat.class); - - public KuduInputFormat(String kuduMaster, String tableName, String columnsList) { - this.kuduMaster = kuduMaster; - this.columnsList = Arrays.asList(columnsList.split(",")); - this.tableName = tableName; - - } - - /** - * Declare the global variable KuduClient and use it to manipulate the Kudu table - */ - public KuduClient kuduClient; - - /** - * Specify kuduMaster address - */ - public String kuduMaster; - public List columnsList; - public Schema schema; - public String keyColumn; - public static final int TIMEOUTMS = 18000; - - /** - * Specifies the name of the table - */ - public String tableName; - - public List getColumnsSchemas() { - List columns = null; - try { - schema = kuduClient.openTable(tableName).getSchema(); - keyColumn = schema.getPrimaryKeyColumns().get(0).getName(); - columns = schema.getColumns(); - } catch (KuduException e) { - LOGGER.warn("get table Columns Schemas Fail.", e); - throw new RuntimeException("get table Columns Schemas Fail..", e); - } - return columns; - } - - public static SeaTunnelRow getSeaTunnelRowData(RowResult rs, SeaTunnelRowType typeInfo) throws SQLException { - - List fields = new ArrayList<>(); - SeaTunnelDataType[] seaTunnelDataTypes = typeInfo.getFieldTypes(); - - for (int i = 0; i < seaTunnelDataTypes.length; i++) { - Object seatunnelField; - SeaTunnelDataType seaTunnelDataType = seaTunnelDataTypes[i]; - if (null == rs.getObject(i)) { - seatunnelField = null; - } else if (BasicType.BOOLEAN_TYPE.equals(seaTunnelDataType)) { - seatunnelField = rs.getBoolean(i); - } else if (BasicType.BYTE_TYPE.equals(seaTunnelDataType)) { - seatunnelField = rs.getByte(i); - } else if (BasicType.SHORT_TYPE.equals(seaTunnelDataType)) { - seatunnelField = rs.getShort(i); - } else if (BasicType.INT_TYPE.equals(seaTunnelDataType)) { - seatunnelField = rs.getInt(i); - } else if (BasicType.LONG_TYPE.equals(seaTunnelDataType)) { - seatunnelField = rs.getLong(i); - } else if (seaTunnelDataType instanceof DecimalType) { - Object value = rs.getObject(i); - seatunnelField = value instanceof BigInteger ? - new BigDecimal((BigInteger) value, 0) - : value; - } else if (BasicType.FLOAT_TYPE.equals(seaTunnelDataType)) { - seatunnelField = rs.getFloat(i); - } else if (BasicType.DOUBLE_TYPE.equals(seaTunnelDataType)) { - seatunnelField = rs.getDouble(i); - } else if (BasicType.STRING_TYPE.equals(seaTunnelDataType)) { - seatunnelField = rs.getString(i); - } else { - throw new IllegalStateException("Unexpected value: " + seaTunnelDataType); - } - fields.add(seatunnelField); - } - - return new SeaTunnelRow(fields.toArray()); - } - - public SeaTunnelRowType getSeaTunnelRowType(List columnSchemaList) { - - ArrayList> seaTunnelDataTypes = new ArrayList<>(); - ArrayList fieldNames = new ArrayList<>(); - try { - - for (int i = 0; i < columnSchemaList.size(); i++) { - fieldNames.add(columnSchemaList.get(i).getName()); - seaTunnelDataTypes.add(KuduTypeMapper.mapping(columnSchemaList, i)); - } - } catch (Exception e) { - LOGGER.warn("get row type info exception", e); - throw new PrepareFailException("kudu", PluginType.SOURCE, e.toString()); - } - return new SeaTunnelRowType(fieldNames.toArray(new String[fieldNames.size()]), seaTunnelDataTypes.toArray(new SeaTunnelDataType[seaTunnelDataTypes.size()])); - } - - public void openInputFormat() { - KuduClient.KuduClientBuilder kuduClientBuilder = new - KuduClient.KuduClientBuilder(kuduMaster); - kuduClientBuilder.defaultOperationTimeoutMs(TIMEOUTMS); - - kuduClient = kuduClientBuilder.build(); - - LOGGER.info("The Kudu client is successfully initialized", kuduMaster, kuduClient); - - } - - /** - * @param lowerBound The beginning of each slice - * @param upperBound End of each slice - * @return Get the kuduScanner object for each slice - */ - - public KuduScanner getKuduBuildSplit(int lowerBound, int upperBound) { - KuduScanner kuduScanner = null; - try { - KuduScanner.KuduScannerBuilder kuduScannerBuilder = - kuduClient.newScannerBuilder(kuduClient.openTable(tableName)); - - kuduScannerBuilder.setProjectedColumnNames(columnsList); - - KuduPredicate lowerPred = KuduPredicate.newComparisonPredicate( - schema.getColumn("" + keyColumn), - KuduPredicate.ComparisonOp.GREATER_EQUAL, - lowerBound); - - KuduPredicate upperPred = KuduPredicate.newComparisonPredicate( - schema.getColumn("" + keyColumn), - KuduPredicate.ComparisonOp.LESS, - upperBound); - - kuduScanner = kuduScannerBuilder.addPredicate(lowerPred) - .addPredicate(upperPred).build(); - } catch (KuduException e) { - LOGGER.warn("get the Kuduscan object for each splice exception", e); - throw new RuntimeException("get the Kuduscan object for each splice exception.", e); - } - return kuduScanner; - } - - public void closeInputFormat() { - if (kuduClient != null) { - try { - kuduClient.close(); - } catch (KuduException e) { - LOGGER.warn("Kudu Client close failed.", e); - throw new RuntimeException("Kudu Client close failed.", e); - } finally { - kuduClient = null; - } - } - - } -} diff --git a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/kuduclient/KuduOutputFormat.java b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/kuduclient/KuduOutputFormat.java deleted file mode 100644 index f73b191ca07..00000000000 --- a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/kuduclient/KuduOutputFormat.java +++ /dev/null @@ -1,164 +0,0 @@ -/* - * 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.seatunnel.connectors.seatunnel.kudu.kuduclient; - -import org.apache.seatunnel.api.table.type.SeaTunnelRow; -import org.apache.seatunnel.connectors.seatunnel.kudu.config.KuduSinkConfig; - -import org.apache.kudu.ColumnSchema; -import org.apache.kudu.Schema; -import org.apache.kudu.client.Insert; -import org.apache.kudu.client.KuduClient; -import org.apache.kudu.client.KuduException; -import org.apache.kudu.client.KuduSession; -import org.apache.kudu.client.KuduTable; -import org.apache.kudu.client.PartialRow; -import org.apache.kudu.client.SessionConfiguration; -import org.slf4j.Logger; -import org.slf4j.LoggerFactory; - -import java.io.Serializable; -import java.math.BigDecimal; -import java.nio.ByteBuffer; -import java.sql.Timestamp; - -/** - * A Kudu outputFormat - */ -public class KuduOutputFormat - implements Serializable { - - private static final Logger LOGGER = LoggerFactory.getLogger(KuduOutputFormat.class); - - private String kuduMaster; - private String kuduTableName; - private KuduClient kuduClient; - private KuduSession kuduSession; - private KuduTable kuduTable; - public static final long TIMEOUTMS = 18000; - public static final long SESSIONTIMEOUTMS = 100000; - public KuduOutputFormat(KuduSinkConfig kuduSinkConfig) { - this.kuduMaster = kuduSinkConfig.getKuduMaster(); - this.kuduTableName = kuduSinkConfig.getKuduTableName(); - init(); - } - - public void write(SeaTunnelRow element) { - - Insert insert = kuduTable.newInsert(); - Schema schema = kuduTable.getSchema(); - - int columnCount = schema.getColumnCount(); - PartialRow row = insert.getRow(); - for (int columnIndex = 0; columnIndex < columnCount; columnIndex++) { - ColumnSchema col = schema.getColumnByIndex(columnIndex); - try { - switch (col.getType()) { - case BOOL: - row.addBoolean(columnIndex, (Boolean) element.getField(columnIndex)); - break; - case INT8: - row.addByte(columnIndex, (Byte) element.getField(columnIndex)); - break; - case INT16: - row.addShort(columnIndex, (Short) element.getField(columnIndex)); - break; - case INT32: - row.addInt(columnIndex, (Integer) element.getField(columnIndex)); - break; - case INT64: - row.addLong(columnIndex, (Long) element.getField(columnIndex)); - break; - case UNIXTIME_MICROS: - if (element.getField(columnIndex) instanceof Timestamp) { - row.addTimestamp(columnIndex, (Timestamp) element.getField(columnIndex)); - } else { - row.addLong(columnIndex, (Long) element.getField(columnIndex)); - } - break; - case FLOAT: - row.addFloat(columnIndex, (Float) element.getField(columnIndex)); - break; - case DOUBLE: - row.addDouble(columnIndex, (Double) element.getField(columnIndex)); - break; - case STRING: - row.addString(columnIndex, element.getField(columnIndex).toString()); - break; - case BINARY: - if (element.getField(columnIndex) instanceof byte[]) { - row.addBinary(columnIndex, (byte[]) element.getField(columnIndex)); - } else { - row.addBinary(columnIndex, (ByteBuffer) element.getField(columnIndex)); - } - break; - case DECIMAL: - row.addDecimal(columnIndex, (BigDecimal) element.getField(columnIndex)); - break; - default: - throw new IllegalArgumentException("Unsupported column type: " + col.getType()); - } - } catch (ClassCastException e) { - e.printStackTrace(); - throw new IllegalArgumentException( - "Value type does not match column type " + col.getType() + - " for column " + col.getName()); - } - - } - - try { - kuduSession.apply(insert); - } catch (KuduException e) { - LOGGER.warn("kudu session insert data fail.", e); - throw new RuntimeException("kudu session insert data fail.", e); - } - - } - - public void init() { - KuduClient.KuduClientBuilder kuduClientBuilder = new - KuduClient.KuduClientBuilder(kuduMaster); - kuduClientBuilder.defaultOperationTimeoutMs(TIMEOUTMS); - this.kuduClient = kuduClientBuilder.build(); - this.kuduSession = kuduClient.newSession(); - this.kuduSession.setTimeoutMillis(SESSIONTIMEOUTMS); - this.kuduSession.setFlushMode(SessionConfiguration.FlushMode.AUTO_FLUSH_SYNC); - try { - kuduTable = kuduClient.openTable(kuduTableName); - } catch (KuduException e) { - LOGGER.warn("Failed to initialize the Kudu client.", e); - throw new RuntimeException("Failed to initialize the Kudu client.", e); - } - LOGGER.info("The Kudu client is successfully initialized", kuduMaster, kuduClient); - } - - public void closeOutputFormat() { - if (kuduClient != null) { - try { - kuduClient.close(); - kuduSession.close(); - } catch (KuduException e) { - LOGGER.warn("Kudu Client close failed.", e); - } finally { - kuduClient = null; - kuduSession = null; - } - } - } -} diff --git a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/kuduclient/KuduTypeMapper.java b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/kuduclient/KuduTypeMapper.java deleted file mode 100644 index cea22dfb2f1..00000000000 --- a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/kuduclient/KuduTypeMapper.java +++ /dev/null @@ -1,103 +0,0 @@ -/* - * 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.seatunnel.connectors.seatunnel.kudu.kuduclient; - -import org.apache.seatunnel.api.table.type.BasicType; -import org.apache.seatunnel.api.table.type.DecimalType; -import org.apache.seatunnel.api.table.type.LocalTimeType; -import org.apache.seatunnel.api.table.type.PrimitiveByteArrayType; -import org.apache.seatunnel.api.table.type.SeaTunnelDataType; - -import org.apache.kudu.ColumnSchema; -import org.slf4j.Logger; -import org.slf4j.LoggerFactory; - -import java.sql.SQLException; -import java.util.List; - -public class KuduTypeMapper { - - private static final Logger LOG = LoggerFactory.getLogger(KuduTypeMapper.class); - - // ============================data types===================== - - private static final String KUDU_UNKNOWN = "UNKNOWN"; - private static final String KUDU_BIT = "BOOL"; - - // -------------------------number---------------------------- - private static final String KUDU_TINYINT = "INT8"; - private static final String KUDU_MEDIUMINT = "INT32"; - private static final String KUDU_INT = "INT16"; - private static final String KUDU_BIGINT = "INT64"; - - private static final String KUDU_FLOAT = "FLOAT"; - - private static final String KUDU_DOUBLE = "DOUBLE"; - private static final String KUDU_DECIMAL = "DECIMAL32"; - - - // -------------------------string---------------------------- - - private static final String KUDU_VARCHAR = "STRING"; - - - // ------------------------------time------------------------- - - private static final String KUDU_UNIXTIME_MICROS = "UNIXTIME_MICROS"; - - - // ------------------------------blob------------------------- - - private static final String KUDU_BINARY = "BINARY"; - private static final int PRECISION = 20; - public static SeaTunnelDataType mapping(List columnSchemaList, int colIndex) throws SQLException { - String kuduType = columnSchemaList.get(colIndex).getType().getName().toUpperCase(); - switch (kuduType) { - case KUDU_BIT: - return BasicType.BOOLEAN_TYPE; - case KUDU_TINYINT: - case KUDU_MEDIUMINT: - case KUDU_INT: - return BasicType.INT_TYPE; - case KUDU_BIGINT: - return BasicType.LONG_TYPE; - case KUDU_DECIMAL: - return new DecimalType(PRECISION, 0); - case KUDU_FLOAT: - return BasicType.FLOAT_TYPE; - case KUDU_DOUBLE: - return BasicType.DOUBLE_TYPE; - - case KUDU_VARCHAR: - return BasicType.STRING_TYPE; - case KUDU_UNIXTIME_MICROS: - return LocalTimeType.LOCAL_DATE_TIME_TYPE; - case KUDU_BINARY: - return PrimitiveByteArrayType.INSTANCE; - - //Doesn't support yet - - case KUDU_UNKNOWN: - default: - throw new UnsupportedOperationException( - String.format( - "Doesn't support KUDU type '%s' .", - kuduType)); - } - } -} diff --git a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/sink/KuduSink.java b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/sink/KuduSink.java deleted file mode 100644 index 2f907ae49e2..00000000000 --- a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/sink/KuduSink.java +++ /dev/null @@ -1,69 +0,0 @@ -/* - * 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.seatunnel.connectors.seatunnel.kudu.sink; - -import org.apache.seatunnel.api.common.PrepareFailException; -import org.apache.seatunnel.api.sink.SeaTunnelSink; -import org.apache.seatunnel.api.sink.SinkWriter; -import org.apache.seatunnel.api.table.type.SeaTunnelDataType; -import org.apache.seatunnel.api.table.type.SeaTunnelRow; -import org.apache.seatunnel.api.table.type.SeaTunnelRowType; -import org.apache.seatunnel.connectors.seatunnel.common.sink.AbstractSimpleSink; -import org.apache.seatunnel.connectors.seatunnel.common.sink.AbstractSinkWriter; - -import org.apache.seatunnel.shade.com.typesafe.config.Config; - -import com.google.auto.service.AutoService; - -import java.io.IOException; - -/** - * Kudu Sink implementation by using SeaTunnel sink API. - * This class contains the method to create {@link AbstractSimpleSink}. - */ -@AutoService(SeaTunnelSink.class) -public class KuduSink extends AbstractSimpleSink { - - private Config config; - private SeaTunnelRowType seaTunnelRowType; - - @Override - public String getPluginName() { - return "kuduSink"; - } - - @Override - public void setTypeInfo(SeaTunnelRowType seaTunnelRowType) { - this.seaTunnelRowType = seaTunnelRowType; - } - - @Override - public SeaTunnelDataType getConsumedType() { - return this.seaTunnelRowType; - } - - @Override - public void prepare(Config pluginConfig) throws PrepareFailException { - this.config = pluginConfig; - } - - @Override - public AbstractSinkWriter createWriter(SinkWriter.Context context) throws IOException { - return new KuduSinkWriter(seaTunnelRowType, config); - } -} diff --git a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/sink/KuduSinkWriter.java b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/sink/KuduSinkWriter.java deleted file mode 100644 index d78f486187e..00000000000 --- a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/sink/KuduSinkWriter.java +++ /dev/null @@ -1,64 +0,0 @@ -/* - * 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.seatunnel.connectors.seatunnel.kudu.sink; - -import org.apache.seatunnel.api.table.type.SeaTunnelRow; -import org.apache.seatunnel.api.table.type.SeaTunnelRowType; -import org.apache.seatunnel.connectors.seatunnel.common.sink.AbstractSinkWriter; -import org.apache.seatunnel.connectors.seatunnel.kudu.config.KuduSinkConfig; -import org.apache.seatunnel.connectors.seatunnel.kudu.kuduclient.KuduOutputFormat; - -import org.apache.seatunnel.shade.com.typesafe.config.Config; - -import lombok.NonNull; -import org.slf4j.Logger; -import org.slf4j.LoggerFactory; - -import java.io.IOException; - -public class KuduSinkWriter extends AbstractSinkWriter { - private static final Logger LOGGER = LoggerFactory.getLogger(KuduSinkWriter.class); - - private SeaTunnelRowType seaTunnelRowType; - private Config pluginConfig; - - - private KuduOutputFormat fileWriter; - - private KuduSinkConfig kuduSinkConfig; - - public KuduSinkWriter(@NonNull SeaTunnelRowType seaTunnelRowType, - @NonNull Config pluginConfig) { - this.seaTunnelRowType = seaTunnelRowType; - this.pluginConfig = pluginConfig; - - kuduSinkConfig = new KuduSinkConfig(this.pluginConfig); - fileWriter = new KuduOutputFormat(kuduSinkConfig); - - } - - @Override - public void write(SeaTunnelRow element) throws IOException { - fileWriter.write(element); - } - - @Override - public void close() throws IOException { - fileWriter.closeOutputFormat(); - } -} diff --git a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/source/KuduSource.java b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/source/KuduSource.java deleted file mode 100644 index 65b004a6a36..00000000000 --- a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/source/KuduSource.java +++ /dev/null @@ -1,191 +0,0 @@ -/* - * 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.seatunnel.connectors.seatunnel.kudu.source; - -import org.apache.seatunnel.api.common.PrepareFailException; -import org.apache.seatunnel.api.common.SeaTunnelContext; -import org.apache.seatunnel.api.serialization.DefaultSerializer; -import org.apache.seatunnel.api.serialization.Serializer; -import org.apache.seatunnel.api.source.Boundedness; -import org.apache.seatunnel.api.source.SeaTunnelSource; -import org.apache.seatunnel.api.source.SourceReader; -import org.apache.seatunnel.api.source.SourceSplitEnumerator; -import org.apache.seatunnel.api.table.type.SeaTunnelDataType; -import org.apache.seatunnel.api.table.type.SeaTunnelRow; -import org.apache.seatunnel.api.table.type.SeaTunnelRowType; -import org.apache.seatunnel.common.constants.PluginType; -import org.apache.seatunnel.connectors.seatunnel.kudu.config.KuduSourceConfig; -import org.apache.seatunnel.connectors.seatunnel.kudu.kuduclient.KuduInputFormat; -import org.apache.seatunnel.connectors.seatunnel.kudu.kuduclient.KuduTypeMapper; -import org.apache.seatunnel.connectors.seatunnel.kudu.state.KuduSourceState; - -import org.apache.seatunnel.shade.com.typesafe.config.Config; - -import com.google.auto.service.AutoService; -import org.apache.kudu.ColumnSchema; -import org.apache.kudu.client.KuduClient; -import org.apache.kudu.client.KuduException; -import org.apache.kudu.client.KuduScanner; -import org.apache.kudu.client.RowResult; -import org.apache.kudu.client.RowResultIterator; -import org.slf4j.Logger; -import org.slf4j.LoggerFactory; - -import java.util.ArrayList; -import java.util.List; - -@AutoService(SeaTunnelSource.class) -public class KuduSource implements SeaTunnelSource { - private static final Logger LOGGER = LoggerFactory.getLogger(KuduSource.class); - - private SeaTunnelContext seaTunnelContext; - private SeaTunnelRowType rowTypeInfo; - private KuduInputFormat kuduInputFormat; - private PartitionParameter partitionParameter; - public static final int TIMEOUTMS = 18000; - - @Override - public Boundedness getBoundedness() { - return Boundedness.BOUNDED; - } - - @Override - public SeaTunnelRowType getProducedType() { - return this.rowTypeInfo; - } - - @Override - public SourceReader createReader(SourceReader.Context readerContext) { - return new KuduSourceReader(kuduInputFormat, readerContext); - } - - @Override - public Serializer getSplitSerializer() { - return SeaTunnelSource.super.getSplitSerializer(); - } - - @Override - public SourceSplitEnumerator createEnumerator( - SourceSplitEnumerator.Context enumeratorContext) { - return new KuduSourceSplitEnumerator(enumeratorContext, partitionParameter); - } - - @Override - public SourceSplitEnumerator restoreEnumerator( - SourceSplitEnumerator.Context enumeratorContext, KuduSourceState checkpointState) { - // todo: - return new KuduSourceSplitEnumerator(enumeratorContext, partitionParameter); - } - - @Override - public Serializer getEnumeratorStateSerializer() { - return new DefaultSerializer<>(); - } - - @Override - public String getPluginName() { - return "KuduSource"; - } - - @Override - public void prepare(Config config) { - String kudumaster = ""; - String tableName = ""; - String columnslist = ""; - if (config.hasPath(KuduSourceConfig.KUDUMASTER) && config.hasPath(KuduSourceConfig.KUDUMASTER) && config.hasPath(KuduSourceConfig.KUDUMASTER)) { - kudumaster = config.getString(KuduSourceConfig.KUDUMASTER); - tableName = config.getString(KuduSourceConfig.TABLENAME); - columnslist = config.getString(KuduSourceConfig.COLUMNSLIST); - kuduInputFormat = new KuduInputFormat(kudumaster, tableName, columnslist); - } else { - throw new RuntimeException("Missing Source configuration parameters"); - } - try { - KuduClient.KuduClientBuilder kuduClientBuilder = new - KuduClient.KuduClientBuilder(kudumaster); - kuduClientBuilder.defaultOperationTimeoutMs(TIMEOUTMS); - - KuduClient kuduClient = kuduClientBuilder.build(); - partitionParameter = initPartitionParameter(kuduClient, tableName); - SeaTunnelRowType seaTunnelRowType = getSeaTunnelRowType(kuduClient.openTable(tableName).getSchema().getColumns()); - rowTypeInfo = seaTunnelRowType; - } catch (KuduException e) { - throw new RuntimeException("Parameters in the preparation phase fail to be generated", e); - } - } - - private PartitionParameter initPartitionParameter(KuduClient kuduClient, String tableName) { - String keyColumn = null; - int maxKey = 0; - int minKey = 0; - boolean flag = true; - try { - KuduScanner.KuduScannerBuilder kuduScannerBuilder = - kuduClient.newScannerBuilder(kuduClient.openTable(tableName)); - ArrayList columnsList = new ArrayList(); - keyColumn = kuduClient.openTable(tableName).getSchema().getPrimaryKeyColumns().get(0).getName(); - columnsList.add("" + keyColumn); - kuduScannerBuilder.setProjectedColumnNames(columnsList); - KuduScanner kuduScanner = kuduScannerBuilder.build(); - while (kuduScanner.hasMoreRows()) { - RowResultIterator rowResults = kuduScanner.nextRows(); - while (rowResults.hasNext()) { - RowResult row = rowResults.next(); - int id = row.getInt("" + keyColumn); - if (flag) { - maxKey = id; - minKey = id; - flag = false; - } else { - if (id >= maxKey) { - maxKey = id; - } - if (id <= minKey) { - minKey = id; - } - } - } - } - } catch (KuduException e) { - throw new RuntimeException("Failed to generate upper and lower limits for each partition", e); - } - return new PartitionParameter(keyColumn, Long.parseLong(minKey + ""), Long.parseLong(maxKey + "")); - } - - @Override - public void setSeaTunnelContext(SeaTunnelContext seaTunnelContext) { - this.seaTunnelContext = seaTunnelContext; - } - - public SeaTunnelRowType getSeaTunnelRowType(List columnSchemaList) { - ArrayList> seaTunnelDataTypes = new ArrayList<>(); - ArrayList fieldNames = new ArrayList<>(); - try { - - for (int i = 0; i < columnSchemaList.size(); i++) { - fieldNames.add(columnSchemaList.get(i).getName()); - seaTunnelDataTypes.add(KuduTypeMapper.mapping(columnSchemaList, i)); - } - - } catch (Exception e) { - LOGGER.warn("get row type info exception", e); - throw new PrepareFailException("kudu", PluginType.SOURCE, e.toString()); - } - return new SeaTunnelRowType(fieldNames.toArray(new String[fieldNames.size()]), seaTunnelDataTypes.toArray(new SeaTunnelDataType[seaTunnelDataTypes.size()])); - } -} diff --git a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/source/KuduSourceReader.java b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/source/KuduSourceReader.java deleted file mode 100644 index ad23a6e3909..00000000000 --- a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/source/KuduSourceReader.java +++ /dev/null @@ -1,107 +0,0 @@ -/* - * 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.seatunnel.connectors.seatunnel.kudu.source; - -import org.apache.seatunnel.api.source.Boundedness; -import org.apache.seatunnel.api.source.Collector; -import org.apache.seatunnel.api.source.SourceReader; -import org.apache.seatunnel.api.table.type.SeaTunnelRow; -import org.apache.seatunnel.connectors.seatunnel.kudu.kuduclient.KuduInputFormat; - -import org.apache.kudu.ColumnSchema; -import org.apache.kudu.client.KuduScanner; -import org.apache.kudu.client.RowResult; -import org.apache.kudu.client.RowResultIterator; -import org.slf4j.Logger; -import org.slf4j.LoggerFactory; - -import java.util.Deque; -import java.util.LinkedList; -import java.util.List; - -public class KuduSourceReader implements SourceReader { - - private static final Logger LOGGER = LoggerFactory.getLogger(KuduSourceReader.class); - - private final Context context; - - private final KuduInputFormat kuduInputFormat; - Deque splits = new LinkedList<>(); - - boolean noMoreSplit; - - public KuduSourceReader(KuduInputFormat kuduInputFormat, Context context) { - this.context = context; - this.kuduInputFormat = kuduInputFormat; - } - - @Override - public void open() { - kuduInputFormat.openInputFormat(); - } - - @Override - public void close() { - kuduInputFormat.closeInputFormat(); - } - - @Override - public void pollNext(Collector output) throws Exception { - KuduSourceSplit split = splits.poll(); - Object[] parameterValues = split.parameterValues; - int lowerBound = Integer.parseInt(parameterValues[0].toString()); - int upperBound = Integer.parseInt(parameterValues[1].toString()); - List columnSchemaList = kuduInputFormat.getColumnsSchemas(); - KuduScanner kuduScanner = kuduInputFormat.getKuduBuildSplit(lowerBound, upperBound); - // - while (kuduScanner.hasMoreRows()) { - RowResultIterator rowResults = kuduScanner.nextRows(); - while (rowResults.hasNext()) { - RowResult rowResult = rowResults.next(); - SeaTunnelRow seaTunnelRow = KuduInputFormat.getSeaTunnelRowData(rowResult, kuduInputFormat.getSeaTunnelRowType(columnSchemaList)); - output.collect(seaTunnelRow); - } - } - if (Boundedness.BOUNDED.equals(context.getBoundedness())) { - // signal to the source that we have reached the end of the data. - LOGGER.info("Closed the bounded fake source"); - context.signalNoMoreElement(); - } - - } - - @Override - public List snapshotState(long checkpointId) { - return null; - } - - @Override - public void addSplits(List splits) { - this.splits.addAll(splits); - } - - @Override - public void handleNoMoreSplits() { - noMoreSplit = true; - } - - @Override - public void notifyCheckpointComplete(long checkpointId) { - - } -} diff --git a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/source/KuduSourceSplit.java b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/source/KuduSourceSplit.java deleted file mode 100644 index 4f1320f7312..00000000000 --- a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/source/KuduSourceSplit.java +++ /dev/null @@ -1,39 +0,0 @@ -/* - * 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.seatunnel.connectors.seatunnel.kudu.source; - -import org.apache.seatunnel.api.source.SourceSplit; - -import lombok.AllArgsConstructor; -import lombok.Data; - -@Data -@AllArgsConstructor -public class KuduSourceSplit implements SourceSplit { - - private static final long serialVersionUID = -1L; - - Object[] parameterValues; - public final Integer splitId; - - @Override - public String splitId() { - return splitId.toString(); - } - -} diff --git a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/source/KuduSourceSplitEnumerator.java b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/source/KuduSourceSplitEnumerator.java deleted file mode 100644 index aceb7b2e645..00000000000 --- a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/source/KuduSourceSplitEnumerator.java +++ /dev/null @@ -1,131 +0,0 @@ -/* - * 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.seatunnel.connectors.seatunnel.kudu.source; - -import org.apache.seatunnel.api.source.SourceSplitEnumerator; -import org.apache.seatunnel.connectors.seatunnel.kudu.state.KuduSourceState; - -import java.io.IOException; -import java.io.Serializable; -import java.util.ArrayList; -import java.util.List; -import java.util.stream.Collectors; - -public class KuduSourceSplitEnumerator implements SourceSplitEnumerator { - - private final Context enumeratorContext; - private PartitionParameter partitionParameter; - List allSplit = new ArrayList<>(); - private Long maxVal; - private Long minVal; - private Long batchSize; - private Integer batchNum; - - public KuduSourceSplitEnumerator(Context enumeratorContext, PartitionParameter partitionParameter) { - this.enumeratorContext = enumeratorContext; - this.partitionParameter = partitionParameter; - } - - @Override - public void open() { - - } - - @Override - public void run() { - - } - - @Override - public void close() throws IOException { - - } - - @Override - public void addSplitsBack(List splits, int subtaskId) { - - } - - @Override - public int currentUnassignedSplitSize() { - return 0; - } - - @Override - public void handleSplitRequest(int subtaskId) { - - } - - @Override - public void registerReader(int subtaskId) { - int parallelism = enumeratorContext.currentParallelism(); - if (allSplit.isEmpty()) { - if (null != partitionParameter) { - Serializable[][] parameterValues = getParameterValues(partitionParameter.minValue, partitionParameter.maxValue, parallelism); - for (int i = 0; i < parameterValues.length; i++) { - allSplit.add(new KuduSourceSplit(parameterValues[i], i)); - } - } else { - allSplit.add(new KuduSourceSplit(null, 0)); - } - } - // Filter the split that the current task needs to run - List splits = allSplit.stream().filter(p -> p.splitId % parallelism == subtaskId).collect(Collectors.toList()); - enumeratorContext.assignSplit(subtaskId, splits); - enumeratorContext.signalNoMoreSplits(subtaskId); - } - - private Serializable[][] getParameterValues(Long minVal, Long maxVal, int parallelism) { - this.maxVal = maxVal; - this.minVal = minVal; - long maxElemCount = (maxVal - minVal) + 1; - batchNum = parallelism; - getBatchSizeAndBatchNum(parallelism); - long bigBatchNum = maxElemCount - (batchSize - 1) * batchNum; - - Serializable[][] parameters = new Serializable[batchNum][2]; - long start = minVal; - for (int i = 0; i < batchNum; i++) { - long end = start + batchSize - 1 - (i >= bigBatchNum ? 1 : 0); - parameters[i] = new Long[]{start, end}; - start = end + 1; - } - return parameters; - - } - - private void getBatchSizeAndBatchNum(int parallelism) { - batchNum = parallelism; - long maxElemCount = (maxVal - minVal) + 1; - if (batchNum > maxElemCount) { - batchNum = (int) maxElemCount; - } - this.batchNum = batchNum; - this.batchSize = new Double(Math.ceil((double) maxElemCount / batchNum)).longValue(); - } - - @Override - public KuduSourceState snapshotState(long checkpointId) throws Exception { - return null; - } - - @Override - public void notifyCheckpointComplete(long checkpointId) throws Exception { - - } -} diff --git a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/source/PartitionParameter.java b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/source/PartitionParameter.java deleted file mode 100644 index e791164667c..00000000000 --- a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/source/PartitionParameter.java +++ /dev/null @@ -1,32 +0,0 @@ -/* - * 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.seatunnel.connectors.seatunnel.kudu.source; - -import lombok.AllArgsConstructor; -import lombok.Data; - -import java.io.Serializable; - -@Data -@AllArgsConstructor -public class PartitionParameter implements Serializable { - - String partitionColumnName; - Long minValue; - Long maxValue; -} diff --git a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/state/KuduSourceState.java b/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/state/KuduSourceState.java deleted file mode 100644 index 317bf375fc1..00000000000 --- a/seatunnel-connectors-v2/connector-kudu/src/main/java/org/apache/seatunnel/connectors/seatunnel/kudu/state/KuduSourceState.java +++ /dev/null @@ -1,23 +0,0 @@ -/* - * 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.seatunnel.connectors.seatunnel.kudu.state; - -import java.io.Serializable; - -public class KuduSourceState implements Serializable { -} diff --git a/seatunnel-connectors-v2/connector-kudu/src/main/resources/kudu_to_kudu_flink.conf b/seatunnel-connectors-v2/connector-kudu/src/main/resources/kudu_to_kudu_flink.conf deleted file mode 100644 index b04aeae6898..00000000000 --- a/seatunnel-connectors-v2/connector-kudu/src/main/resources/kudu_to_kudu_flink.conf +++ /dev/null @@ -1,60 +0,0 @@ -# -# 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. -# -###### -###### This config file is a demonstration of streaming processing in seatunnel config -###### - -env { - # You can set flink configuration here - execution.parallelism = 2 - #job.mode = "BATCH" - #execution.checkpoint.interval = 10000 - #execution.checkpoint.data-uri = "hdfs://localhost:9000/checkpoint" -} - -source { - # This is a example source plugin **only for test and demonstrate the feature source plugin** - KuduSource { - result_table_name = "studentlyh2" - kudu_master = "192.168.88.110:7051" - kudu_table = "studentlyh2" - columnsList = "id,name,age,sex" - } - - # If you would like to get more information about how to configure seatunnel and see full list of source plugins, - # please go to https://seatunnel.apache.org/docs/flink/configuration/source-plugins/Fake -} - -transform { - sql { - sql = "select id,name,age,sex from studentlyh2" - } - - # If you would like to get more information about how to configure seatunnel and see full list of transform plugins, - # please go to https://seatunnel.apache.org/docs/flink/configuration/transform-plugins/Sql -} - -sink { - kuduSink { - kudu_master = "192.168.88.110:7051" - kudu_table = "studentlyhresultflink" - save_mode="append" - } - - # If you would like to get more information about how to configure seatunnel and see full list of sink plugins, - # please go to https://seatunnel.apache.org/docs/flink/configuration/sink-plugins/Console -} \ No newline at end of file diff --git a/seatunnel-connectors-v2/connector-kudu/src/main/resources/kudu_to_kudu_spark.conf b/seatunnel-connectors-v2/connector-kudu/src/main/resources/kudu_to_kudu_spark.conf deleted file mode 100644 index cb4ecbaa8e5..00000000000 --- a/seatunnel-connectors-v2/connector-kudu/src/main/resources/kudu_to_kudu_spark.conf +++ /dev/null @@ -1,64 +0,0 @@ -# -# 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. -# -###### -###### This config file is a demonstration of streaming processing in seatunnel config -###### - -env { - # You can set flink configuration here - spark.app.name = "SeaTunnel" - spark.executor.instances = 2 - spark.executor.cores = 2 - spark.executor.memory = "1g" - spark.master = local - #job.mode = "BATCH" - #execution.checkpoint.interval = 10000 - #execution.checkpoint.data-uri = "hdfs://localhost:9000/checkpoint" -} - -source { - # This is a example source plugin **only for test and demonstrate the feature source plugin** - KuduSource { - result_table_name = "studentlyh2" - kudu_master = "192.168.88.110:7051" - kudu_table = "studentlyh2" - columnsList = "id,name,age,sex" - } - - # If you would like to get more information about how to configure seatunnel and see full list of source plugins, - # please go to https://seatunnel.apache.org/docs/flink/configuration/source-plugins/Fake -} - -transform { - sql { - sql = "select id,name,age,sex from studentlyh2" - } - - # If you would like to get more information about how to configure seatunnel and see full list of transform plugins, - # please go to https://seatunnel.apache.org/docs/flink/configuration/transform-plugins/Sql -} - -sink { - kuduSink { - kudu_master = "192.168.88.110:7051" - kudu_table = "studentlyhresult" - save_mode="append" - } - - # If you would like to get more information about how to configure seatunnel and see full list of sink plugins, - # please go to https://seatunnel.apache.org/docs/flink/configuration/sink-plugins/Console -} \ No newline at end of file From a936e3c2c3eaba41e43cb634b624a10993f901dc Mon Sep 17 00:00:00 2001 From: liuyehan <81779971+2013650523@users.noreply.github.com> Date: Sat, 30 Jul 2022 16:37:24 +0800 Subject: [PATCH 17/26] Update pom.xml --- seatunnel-connectors-v2/connector-email/pom.xml | 7 +------ 1 file changed, 1 insertion(+), 6 deletions(-) diff --git a/seatunnel-connectors-v2/connector-email/pom.xml b/seatunnel-connectors-v2/connector-email/pom.xml index f30c49f76ae..8c716947695 100644 --- a/seatunnel-connectors-v2/connector-email/pom.xml +++ b/seatunnel-connectors-v2/connector-email/pom.xml @@ -17,15 +17,10 @@ connector-common ${project.version} - - org.apache.seatunnel - seatunnel-api - ${project.version} - com.sun.mail javax.mail - \ No newline at end of file + From f938db43413a51d1fd69d9584c1837befaac6461 Mon Sep 17 00:00:00 2001 From: liuyehan <81779971+2013650523@users.noreply.github.com> Date: Sat, 30 Jul 2022 16:38:23 +0800 Subject: [PATCH 18/26] Update plugin-mapping.properties --- plugin-mapping.properties | 3 +-- 1 file changed, 1 insertion(+), 2 deletions(-) diff --git a/plugin-mapping.properties b/plugin-mapping.properties index bfc4735bbf3..6f2eb5df2ee 100644 --- a/plugin-mapping.properties +++ b/plugin-mapping.properties @@ -102,8 +102,7 @@ seatunnel.sink.Clickhouse = connector-clickhouse seatunnel.sink.ClickhouseFile = connector-clickhouse seatunnel.source.Jdbc = connector-jdbc seatunnel.sink.Jdbc = connector-jdbc -seatunnel.source.Kudu = connector-kudu -seatunnel.sink.Kudu = connector-kudu +seatunnel.sink.Email = connector-email seatunnel.sink.HdfsFile = connector-file-hadoop seatunnel.sink.LocalFile = connector-file-local seatunnel.source.Pulsar = connector-pulsar From 83d532c1b907b6bf532b40c47b5f8efb2c1f7c09 Mon Sep 17 00:00:00 2001 From: liuyehan <81779971+2013650523@users.noreply.github.com> Date: Sat, 30 Jul 2022 17:29:27 +0800 Subject: [PATCH 19/26] Update pom.xml --- seatunnel-connectors-v2/connector-email/pom.xml | 17 +++++++++++++++++ 1 file changed, 17 insertions(+) diff --git a/seatunnel-connectors-v2/connector-email/pom.xml b/seatunnel-connectors-v2/connector-email/pom.xml index 8c716947695..e54ac23dd01 100644 --- a/seatunnel-connectors-v2/connector-email/pom.xml +++ b/seatunnel-connectors-v2/connector-email/pom.xml @@ -1,3 +1,20 @@ +/* + * 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. + */ + Date: Mon, 1 Aug 2022 10:15:54 +0800 Subject: [PATCH 20/26] [Connector-V2] update codestyle pom.xml --- .../connector-email/pom.xml | 40 +++++++++++-------- 1 file changed, 23 insertions(+), 17 deletions(-) diff --git a/seatunnel-connectors-v2/connector-email/pom.xml b/seatunnel-connectors-v2/connector-email/pom.xml index e54ac23dd01..80414be3edb 100644 --- a/seatunnel-connectors-v2/connector-email/pom.xml +++ b/seatunnel-connectors-v2/connector-email/pom.xml @@ -1,21 +1,22 @@ -/* - * 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. - */ - + @@ -38,6 +39,11 @@ com.sun.mail javax.mail + + org.apache.seatunnel + seatunnel-api + ${project.version} + From e7c67cc8c07c6fbffd7fe17ceef5a97c31adef0e Mon Sep 17 00:00:00 2001 From: 2013650523 <1067341434@qq.com> Date: Tue, 2 Aug 2022 10:41:05 +0800 Subject: [PATCH 21/26] [Connector-V2] update config --- .../seatunnel/email/config/EmailSinkConfig.java | 15 +++++++++++++++ .../seatunnel/email/sink/EmailSinkWriter.java | 6 +++--- 2 files changed, 18 insertions(+), 3 deletions(-) diff --git a/seatunnel-connectors-v2/connector-email/src/main/java/org/apache/seatunnel/connectors/seatunnel/email/config/EmailSinkConfig.java b/seatunnel-connectors-v2/connector-email/src/main/java/org/apache/seatunnel/connectors/seatunnel/email/config/EmailSinkConfig.java index d09dcca3607..f7b69c1ff2e 100644 --- a/seatunnel-connectors-v2/connector-email/src/main/java/org/apache/seatunnel/connectors/seatunnel/email/config/EmailSinkConfig.java +++ b/seatunnel-connectors-v2/connector-email/src/main/java/org/apache/seatunnel/connectors/seatunnel/email/config/EmailSinkConfig.java @@ -30,11 +30,17 @@ public class EmailSinkConfig { private static final String EMAIL_AUTHORIZATION_CODE = "email_authorization_code"; private static final String EMAIL_MESSAGE_HEADLINE = "email_message_headline"; private static final String EMAIL_MESSAGE_CONTENT = "email_message_content"; + private static final String EMAIL_HOST = "email_host"; + private static final String EMAIL_TRANSPORT_PROTOCOL = "email_transport_protocol"; + private static final String EMAIL_SMTP_AUTH = "email_smtp_auth"; private String emailFromAddress; private String emailToAddress; private String emailAuthorizationCode; private String emailMessageHeadline; private String emailMessageContent; + private String emailHost; + private String emailTransportProtocol; + private String emailSmtpAuth; public EmailSinkConfig(@NonNull Config pluginConfig) { if (pluginConfig.hasPath(EMAIL_FROM_ADDRESS)) { @@ -52,5 +58,14 @@ public EmailSinkConfig(@NonNull Config pluginConfig) { if (pluginConfig.hasPath(EMAIL_MESSAGE_CONTENT)) { this.emailMessageContent = pluginConfig.getString(EMAIL_MESSAGE_CONTENT); } + if (pluginConfig.hasPath(EMAIL_HOST)) { + this.emailHost = pluginConfig.getString(EMAIL_HOST); + } + if (pluginConfig.hasPath(EMAIL_TRANSPORT_PROTOCOL)) { + this.emailTransportProtocol = pluginConfig.getString(EMAIL_TRANSPORT_PROTOCOL); + } + if (pluginConfig.hasPath(EMAIL_SMTP_AUTH)) { + this.emailSmtpAuth = pluginConfig.getString(EMAIL_SMTP_AUTH); + } } } diff --git a/seatunnel-connectors-v2/connector-email/src/main/java/org/apache/seatunnel/connectors/seatunnel/email/sink/EmailSinkWriter.java b/seatunnel-connectors-v2/connector-email/src/main/java/org/apache/seatunnel/connectors/seatunnel/email/sink/EmailSinkWriter.java index 731b92c39c7..1a86dce2256 100644 --- a/seatunnel-connectors-v2/connector-email/src/main/java/org/apache/seatunnel/connectors/seatunnel/email/sink/EmailSinkWriter.java +++ b/seatunnel-connectors-v2/connector-email/src/main/java/org/apache/seatunnel/connectors/seatunnel/email/sink/EmailSinkWriter.java @@ -79,11 +79,11 @@ public void close() { createFile(); Properties properties = new Properties(); - properties.setProperty("mail.host", "smtp.qq.com"); + properties.setProperty("mail.host", config.getEmailHost()); - properties.setProperty("mail.transport.protocol", "smtp"); + properties.setProperty("mail.transport.protocol", config.getEmailTransportProtocol()); - properties.setProperty("mail.smtp.auth", "true"); + properties.setProperty("mail.smtp.auth", config.getEmailSmtpAuth()); try { MailSSLSocketFactory sf = new MailSSLSocketFactory(); From db3fdbfcef39be1a240d0be3cdafef43be1bd37e Mon Sep 17 00:00:00 2001 From: 2013650523 <1067341434@qq.com> Date: Tue, 2 Aug 2022 14:00:54 +0800 Subject: [PATCH 22/26] [Connector-V2] update license --- seatunnel-dist/release-docs/LICENSE | 1 + seatunnel-dist/release-docs/NOTICE | 12 ++ .../release-docs/licenses/LICENSE-email.txt | 191 ++++++++++++++++++ 3 files changed, 204 insertions(+) create mode 100644 seatunnel-dist/release-docs/licenses/LICENSE-email.txt diff --git a/seatunnel-dist/release-docs/LICENSE b/seatunnel-dist/release-docs/LICENSE index 8191c55a083..74d398d87fe 100644 --- a/seatunnel-dist/release-docs/LICENSE +++ b/seatunnel-dist/release-docs/LICENSE @@ -395,6 +395,7 @@ The text of each license is the standard Apache 2.0 license. (Apache License, Version 2.0) Apache Hadoop Auth (org.apache.hadoop:hadoop-auth:2.7.4 - no url defined) (Apache License, Version 2.0) Apache Hadoop Auth (org.apache.hadoop:hadoop-auth:3.0.0 - no url defined) (Apache License, Version 2.0) Apache Hadoop Client (org.apache.hadoop:hadoop-client:2.6.5 - no url defined) + (Apache License, Version 2.0) Email (com.sun.mail:javax.mail:7.4.1 - no url defined) (Apache License, Version 2.0) Apache Hadoop Client (org.apache.hadoop:hadoop-client:3.0.0 - no url defined) (Apache License, Version 2.0) Apache Hadoop Common (org.apache.hadoop:hadoop-common:2.6.5 - no url defined) (Apache License, Version 2.0) Apache Hadoop Common (org.apache.hadoop:hadoop-common:2.7.7 - no url defined) diff --git a/seatunnel-dist/release-docs/NOTICE b/seatunnel-dist/release-docs/NOTICE index 2489bae1edb..eb5285dcc38 100644 --- a/seatunnel-dist/release-docs/NOTICE +++ b/seatunnel-dist/release-docs/NOTICE @@ -4445,4 +4445,16 @@ Copyright 2000-2019 The Apache Software Foundation This product includes software developed at The Apache Software Foundation (http://www.apache.org/). +========================================================================= + +Email NOTICE + +========================================================================= + +Email +Copyright 2000-2019 The Apache Software Foundation + +This product includes software developed at +The Apache Software Foundation (http://www.apache.org/). + ========================================================================= \ No newline at end of file diff --git a/seatunnel-dist/release-docs/licenses/LICENSE-email.txt b/seatunnel-dist/release-docs/licenses/LICENSE-email.txt new file mode 100644 index 00000000000..321704f42a1 --- /dev/null +++ b/seatunnel-dist/release-docs/licenses/LICENSE-email.txt @@ -0,0 +1,191 @@ + + + + Apache License + Version 2.0, January 2004 + http://www.apache.org/licenses/ + + TERMS AND CONDITIONS FOR USE, REPRODUCTION, AND DISTRIBUTION + + 1. Definitions. + + "License" shall mean the terms and conditions for use, reproduction, + and distribution as defined by Sections 1 through 9 of this document. + + "Licensor" shall mean the copyright owner or entity authorized by + the copyright owner that is granting the License. + + "Legal Entity" shall mean the union of the acting entity and all + other entities that control, are controlled by, or are under common + control with that entity. For the purposes of this definition, + "control" means (i) the power, direct or indirect, to cause the + direction or management of such entity, whether by contract or + otherwise, or (ii) ownership of fifty percent (50%) or more of the + outstanding shares, or (iii) beneficial ownership of such entity. + + "You" (or "Your") shall mean an individual or Legal Entity + exercising permissions granted by this License. + + "Source" form shall mean the preferred form for making modifications, + including but not limited to software source code, documentation + source, and configuration files. + + "Object" form shall mean any form resulting from mechanical + transformation or translation of a Source form, including but + not limited to compiled object code, generated documentation, + and conversions to other media types. + + "Work" shall mean the work of authorship, whether in Source or + Object form, made available under the License, as indicated by a + copyright notice that is included in or attached to the work + (an example is provided in the Appendix below). + + "Derivative Works" shall mean any work, whether in Source or Object + form, that is based on (or derived from) the Work and for which the + editorial revisions, annotations, elaborations, or other modifications + represent, as a whole, an original work of authorship. For the purposes + of this License, Derivative Works shall not include works that remain + separable from, or merely link (or bind by name) to the interfaces of, + the Work and Derivative Works thereof. + + "Contribution" shall mean any work of authorship, including + the original version of the Work and any modifications or additions + to that Work or Derivative Works thereof, that is intentionally + submitted to Licensor for inclusion in the Work by the copyright owner + or by an individual or Legal Entity authorized to submit on behalf of + the copyright owner. For the purposes of this definition, "submitted" + means any form of electronic, verbal, or written communication sent + to the Licensor or its representatives, including but not limited to + communication on electronic mailing lists, source code control systems, + and issue tracking systems that are managed by, or on behalf of, the + Licensor for the purpose of discussing and improving the Work, but + excluding communication that is conspicuously marked or otherwise + designated in writing by the copyright owner as "Not a Contribution." + + "Contributor" shall mean Licensor and any individual or Legal Entity + on behalf of whom a Contribution has been received by Licensor and + subsequently incorporated within the Work. + + 2. Grant of Copyright License. Subject to the terms and conditions of + this License, each Contributor hereby grants to You a perpetual, + worldwide, non-exclusive, no-charge, royalty-free, irrevocable + copyright license to reproduce, prepare Derivative Works of, + publicly display, publicly perform, sublicense, and distribute the + Work and such Derivative Works in Source or Object form. + + 3. Grant of Patent License. Subject to the terms and conditions of + this License, each Contributor hereby grants to You a perpetual, + worldwide, non-exclusive, no-charge, royalty-free, irrevocable + (except as stated in this section) patent license to make, have made, + use, offer to sell, sell, import, and otherwise transfer the Work, + where such license applies only to those patent claims licensable + by such Contributor that are necessarily infringed by their + Contribution(s) alone or by combination of their Contribution(s) + with the Work to which such Contribution(s) was submitted. If You + institute patent litigation against any entity (including a + cross-claim or counterclaim in a lawsuit) alleging that the Work + or a Contribution incorporated within the Work constitutes direct + or contributory patent infringement, then any patent licenses + granted to You under this License for that Work shall terminate + as of the date such litigation is filed. + + 4. Redistribution. You may reproduce and distribute copies of the + Work or Derivative Works thereof in any medium, with or without + modifications, and in Source or Object form, provided that You + meet the following conditions: + + (a) You must give any other recipients of the Work or + Derivative Works a copy of this License; and + + (b) You must cause any modified files to carry prominent notices + stating that You changed the files; and + + (c) You must retain, in the Source form of any Derivative Works + that You distribute, all copyright, patent, trademark, and + attribution notices from the Source form of the Work, + excluding those notices that do not pertain to any part of + the Derivative Works; and + + (d) If the Work includes a "NOTICE" text file as part of its + distribution, then any Derivative Works that You distribute must + include a readable copy of the attribution notices contained + within such NOTICE file, excluding those notices that do not + pertain to any part of the Derivative Works, in at least one + of the following places: within a NOTICE text file distributed + as part of the Derivative Works; within the Source form or + documentation, if provided along with the Derivative Works; or, + within a display generated by the Derivative Works, if and + wherever such third-party notices normally appear. The contents + of the NOTICE file are for informational purposes only and + do not modify the License. You may add Your own attribution + notices within Derivative Works that You distribute, alongside + or as an addendum to the NOTICE text from the Work, provided + that such additional attribution notices cannot be construed + as modifying the License. + + You may add Your own copyright statement to Your modifications and + may provide additional or different license terms and conditions + for use, reproduction, or distribution of Your modifications, or + for any such Derivative Works as a whole, provided Your use, + reproduction, and distribution of the Work otherwise complies with + the conditions stated in this License. + + 5. Submission of Contributions. Unless You explicitly state otherwise, + any Contribution intentionally submitted for inclusion in the Work + by You to the Licensor shall be under the terms and conditions of + this License, without any additional terms or conditions. + Notwithstanding the above, nothing herein shall supersede or modify + the terms of any separate license agreement you may have executed + with Licensor regarding such Contributions. + + 6. Trademarks. This License does not grant permission to use the trade + names, trademarks, service marks, or product names of the Licensor, + except as required for reasonable and customary use in describing the + origin of the Work and reproducing the content of the NOTICE file. + + 7. Disclaimer of Warranty. Unless required by applicable law or + agreed to in writing, Licensor provides the Work (and each + Contributor provides its Contributions) on an "AS IS" BASIS, + WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or + implied, including, without limitation, any warranties or conditions + of TITLE, NON-INFRINGEMENT, MERCHANTABILITY, or FITNESS FOR A + PARTICULAR PURPOSE. You are solely responsible for determining the + appropriateness of using or redistributing the Work and assume any + risks associated with Your exercise of permissions under this License. + + 8. Limitation of Liability. In no event and under no legal theory, + whether in tort (including negligence), contract, or otherwise, + unless required by applicable law (such as deliberate and grossly + negligent acts) or agreed to in writing, shall any Contributor be + liable to You for damages, including any direct, indirect, special, + incidental, or consequential damages of any character arising as a + result of this License or out of the use or inability to use the + Work (including but not limited to damages for loss of goodwill, + work stoppage, computer failure or malfunction, or any and all + other commercial damages or losses), even if such Contributor + has been advised of the possibility of such damages. + + 9. Accepting Warranty or Additional Liability. While redistributing + the Work or Derivative Works thereof, You may choose to offer, + and charge a fee for, acceptance of support, warranty, indemnity, + or other liability obligations and/or rights consistent with this + License. However, in accepting such obligations, You may act only + on Your own behalf and on Your sole responsibility, not on behalf + of any other Contributor, and only if You agree to indemnify, + defend, and hold each Contributor harmless for any liability + incurred by, or claims asserted against, such Contributor by reason + of your accepting any such warranty or additional liability. + + END OF TERMS AND CONDITIONS + +Additional License files can be found in the 'licenses' folder located in the same directory as the LICENSE file (i.e. this file) + +- Software produced outside the ASF which is available under other licenses (not Apache-2.0) + +MIT +* cobyism:html5shiv:3.7.2 +* font-awesome:font-awesome-code:4.2.0 +* gridsim:gridsim: +* jekyll:jekyll: +* normalize:normalize:3.0.2 +* respond:respond:1.4.2 \ No newline at end of file From 01c71ec81ecc17a176f33e907358f0ff1010459a Mon Sep 17 00:00:00 2001 From: 2013650523 <1067341434@qq.com> Date: Tue, 2 Aug 2022 15:01:33 +0800 Subject: [PATCH 23/26] [Connector-V2] update email package version 1.5.6 --- pom.xml | 2 +- seatunnel-dist/release-docs/LICENSE | 1 - seatunnel-dist/release-docs/NOTICE | 12 -- .../release-docs/licenses/LICENSE-email.txt | 191 ------------------ 4 files changed, 1 insertion(+), 205 deletions(-) delete mode 100644 seatunnel-dist/release-docs/licenses/LICENSE-email.txt diff --git a/pom.xml b/pom.xml index 86fee22459d..42b57aa4fc4 100644 --- a/pom.xml +++ b/pom.xml @@ -169,7 +169,7 @@ 2.2.0 2.6.0 3.4 - 1.4.7 + 1.5.6 4.4 3.3.0 provided diff --git a/seatunnel-dist/release-docs/LICENSE b/seatunnel-dist/release-docs/LICENSE index 74d398d87fe..8191c55a083 100644 --- a/seatunnel-dist/release-docs/LICENSE +++ b/seatunnel-dist/release-docs/LICENSE @@ -395,7 +395,6 @@ The text of each license is the standard Apache 2.0 license. (Apache License, Version 2.0) Apache Hadoop Auth (org.apache.hadoop:hadoop-auth:2.7.4 - no url defined) (Apache License, Version 2.0) Apache Hadoop Auth (org.apache.hadoop:hadoop-auth:3.0.0 - no url defined) (Apache License, Version 2.0) Apache Hadoop Client (org.apache.hadoop:hadoop-client:2.6.5 - no url defined) - (Apache License, Version 2.0) Email (com.sun.mail:javax.mail:7.4.1 - no url defined) (Apache License, Version 2.0) Apache Hadoop Client (org.apache.hadoop:hadoop-client:3.0.0 - no url defined) (Apache License, Version 2.0) Apache Hadoop Common (org.apache.hadoop:hadoop-common:2.6.5 - no url defined) (Apache License, Version 2.0) Apache Hadoop Common (org.apache.hadoop:hadoop-common:2.7.7 - no url defined) diff --git a/seatunnel-dist/release-docs/NOTICE b/seatunnel-dist/release-docs/NOTICE index eb5285dcc38..2489bae1edb 100644 --- a/seatunnel-dist/release-docs/NOTICE +++ b/seatunnel-dist/release-docs/NOTICE @@ -4445,16 +4445,4 @@ Copyright 2000-2019 The Apache Software Foundation This product includes software developed at The Apache Software Foundation (http://www.apache.org/). -========================================================================= - -Email NOTICE - -========================================================================= - -Email -Copyright 2000-2019 The Apache Software Foundation - -This product includes software developed at -The Apache Software Foundation (http://www.apache.org/). - ========================================================================= \ No newline at end of file diff --git a/seatunnel-dist/release-docs/licenses/LICENSE-email.txt b/seatunnel-dist/release-docs/licenses/LICENSE-email.txt deleted file mode 100644 index 321704f42a1..00000000000 --- a/seatunnel-dist/release-docs/licenses/LICENSE-email.txt +++ /dev/null @@ -1,191 +0,0 @@ - - - - Apache License - Version 2.0, January 2004 - http://www.apache.org/licenses/ - - TERMS AND CONDITIONS FOR USE, REPRODUCTION, AND DISTRIBUTION - - 1. Definitions. - - "License" shall mean the terms and conditions for use, reproduction, - and distribution as defined by Sections 1 through 9 of this document. - - "Licensor" shall mean the copyright owner or entity authorized by - the copyright owner that is granting the License. - - "Legal Entity" shall mean the union of the acting entity and all - other entities that control, are controlled by, or are under common - control with that entity. For the purposes of this definition, - "control" means (i) the power, direct or indirect, to cause the - direction or management of such entity, whether by contract or - otherwise, or (ii) ownership of fifty percent (50%) or more of the - outstanding shares, or (iii) beneficial ownership of such entity. - - "You" (or "Your") shall mean an individual or Legal Entity - exercising permissions granted by this License. - - "Source" form shall mean the preferred form for making modifications, - including but not limited to software source code, documentation - source, and configuration files. - - "Object" form shall mean any form resulting from mechanical - transformation or translation of a Source form, including but - not limited to compiled object code, generated documentation, - and conversions to other media types. - - "Work" shall mean the work of authorship, whether in Source or - Object form, made available under the License, as indicated by a - copyright notice that is included in or attached to the work - (an example is provided in the Appendix below). - - "Derivative Works" shall mean any work, whether in Source or Object - form, that is based on (or derived from) the Work and for which the - editorial revisions, annotations, elaborations, or other modifications - represent, as a whole, an original work of authorship. For the purposes - of this License, Derivative Works shall not include works that remain - separable from, or merely link (or bind by name) to the interfaces of, - the Work and Derivative Works thereof. - - "Contribution" shall mean any work of authorship, including - the original version of the Work and any modifications or additions - to that Work or Derivative Works thereof, that is intentionally - submitted to Licensor for inclusion in the Work by the copyright owner - or by an individual or Legal Entity authorized to submit on behalf of - the copyright owner. For the purposes of this definition, "submitted" - means any form of electronic, verbal, or written communication sent - to the Licensor or its representatives, including but not limited to - communication on electronic mailing lists, source code control systems, - and issue tracking systems that are managed by, or on behalf of, the - Licensor for the purpose of discussing and improving the Work, but - excluding communication that is conspicuously marked or otherwise - designated in writing by the copyright owner as "Not a Contribution." - - "Contributor" shall mean Licensor and any individual or Legal Entity - on behalf of whom a Contribution has been received by Licensor and - subsequently incorporated within the Work. - - 2. Grant of Copyright License. Subject to the terms and conditions of - this License, each Contributor hereby grants to You a perpetual, - worldwide, non-exclusive, no-charge, royalty-free, irrevocable - copyright license to reproduce, prepare Derivative Works of, - publicly display, publicly perform, sublicense, and distribute the - Work and such Derivative Works in Source or Object form. - - 3. Grant of Patent License. Subject to the terms and conditions of - this License, each Contributor hereby grants to You a perpetual, - worldwide, non-exclusive, no-charge, royalty-free, irrevocable - (except as stated in this section) patent license to make, have made, - use, offer to sell, sell, import, and otherwise transfer the Work, - where such license applies only to those patent claims licensable - by such Contributor that are necessarily infringed by their - Contribution(s) alone or by combination of their Contribution(s) - with the Work to which such Contribution(s) was submitted. If You - institute patent litigation against any entity (including a - cross-claim or counterclaim in a lawsuit) alleging that the Work - or a Contribution incorporated within the Work constitutes direct - or contributory patent infringement, then any patent licenses - granted to You under this License for that Work shall terminate - as of the date such litigation is filed. - - 4. Redistribution. You may reproduce and distribute copies of the - Work or Derivative Works thereof in any medium, with or without - modifications, and in Source or Object form, provided that You - meet the following conditions: - - (a) You must give any other recipients of the Work or - Derivative Works a copy of this License; and - - (b) You must cause any modified files to carry prominent notices - stating that You changed the files; and - - (c) You must retain, in the Source form of any Derivative Works - that You distribute, all copyright, patent, trademark, and - attribution notices from the Source form of the Work, - excluding those notices that do not pertain to any part of - the Derivative Works; and - - (d) If the Work includes a "NOTICE" text file as part of its - distribution, then any Derivative Works that You distribute must - include a readable copy of the attribution notices contained - within such NOTICE file, excluding those notices that do not - pertain to any part of the Derivative Works, in at least one - of the following places: within a NOTICE text file distributed - as part of the Derivative Works; within the Source form or - documentation, if provided along with the Derivative Works; or, - within a display generated by the Derivative Works, if and - wherever such third-party notices normally appear. The contents - of the NOTICE file are for informational purposes only and - do not modify the License. You may add Your own attribution - notices within Derivative Works that You distribute, alongside - or as an addendum to the NOTICE text from the Work, provided - that such additional attribution notices cannot be construed - as modifying the License. - - You may add Your own copyright statement to Your modifications and - may provide additional or different license terms and conditions - for use, reproduction, or distribution of Your modifications, or - for any such Derivative Works as a whole, provided Your use, - reproduction, and distribution of the Work otherwise complies with - the conditions stated in this License. - - 5. Submission of Contributions. Unless You explicitly state otherwise, - any Contribution intentionally submitted for inclusion in the Work - by You to the Licensor shall be under the terms and conditions of - this License, without any additional terms or conditions. - Notwithstanding the above, nothing herein shall supersede or modify - the terms of any separate license agreement you may have executed - with Licensor regarding such Contributions. - - 6. Trademarks. This License does not grant permission to use the trade - names, trademarks, service marks, or product names of the Licensor, - except as required for reasonable and customary use in describing the - origin of the Work and reproducing the content of the NOTICE file. - - 7. Disclaimer of Warranty. Unless required by applicable law or - agreed to in writing, Licensor provides the Work (and each - Contributor provides its Contributions) on an "AS IS" BASIS, - WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or - implied, including, without limitation, any warranties or conditions - of TITLE, NON-INFRINGEMENT, MERCHANTABILITY, or FITNESS FOR A - PARTICULAR PURPOSE. You are solely responsible for determining the - appropriateness of using or redistributing the Work and assume any - risks associated with Your exercise of permissions under this License. - - 8. Limitation of Liability. In no event and under no legal theory, - whether in tort (including negligence), contract, or otherwise, - unless required by applicable law (such as deliberate and grossly - negligent acts) or agreed to in writing, shall any Contributor be - liable to You for damages, including any direct, indirect, special, - incidental, or consequential damages of any character arising as a - result of this License or out of the use or inability to use the - Work (including but not limited to damages for loss of goodwill, - work stoppage, computer failure or malfunction, or any and all - other commercial damages or losses), even if such Contributor - has been advised of the possibility of such damages. - - 9. Accepting Warranty or Additional Liability. While redistributing - the Work or Derivative Works thereof, You may choose to offer, - and charge a fee for, acceptance of support, warranty, indemnity, - or other liability obligations and/or rights consistent with this - License. However, in accepting such obligations, You may act only - on Your own behalf and on Your sole responsibility, not on behalf - of any other Contributor, and only if You agree to indemnify, - defend, and hold each Contributor harmless for any liability - incurred by, or claims asserted against, such Contributor by reason - of your accepting any such warranty or additional liability. - - END OF TERMS AND CONDITIONS - -Additional License files can be found in the 'licenses' folder located in the same directory as the LICENSE file (i.e. this file) - -- Software produced outside the ASF which is available under other licenses (not Apache-2.0) - -MIT -* cobyism:html5shiv:3.7.2 -* font-awesome:font-awesome-code:4.2.0 -* gridsim:gridsim: -* jekyll:jekyll: -* normalize:normalize:3.0.2 -* respond:respond:1.4.2 \ No newline at end of file From ba84ee167e88f257f5248881537191877a45f00d Mon Sep 17 00:00:00 2001 From: 2013650523 <1067341434@qq.com> Date: Tue, 2 Aug 2022 16:13:11 +0800 Subject: [PATCH 24/26] [Connector-V2] fix problem on code review --- seatunnel-connectors-v2-dist/pom.xml | 5 +++++ 1 file changed, 5 insertions(+) diff --git a/seatunnel-connectors-v2-dist/pom.xml b/seatunnel-connectors-v2-dist/pom.xml index fe37965a1b8..31283c554be 100644 --- a/seatunnel-connectors-v2-dist/pom.xml +++ b/seatunnel-connectors-v2-dist/pom.xml @@ -96,6 +96,11 @@ connector-hudi ${project.version} + + org.apache.seatunnel + connector-email + ${project.version} + From b7181b637a784cd6a09c25b1b97a64f5183bd073 Mon Sep 17 00:00:00 2001 From: 2013650523 <1067341434@qq.com> Date: Wed, 3 Aug 2022 10:06:21 +0800 Subject: [PATCH 25/26] [Connector-V2] add Email usage document --- docs/en/connector-v2/source/Email.md | 71 +++++++++++++++++++ .../resources/fake_to_emailsink_flink.conf | 63 ++++++++++++++++ 2 files changed, 134 insertions(+) create mode 100644 docs/en/connector-v2/source/Email.md create mode 100644 seatunnel-connectors-v2/connector-email/src/main/resources/fake_to_emailsink_flink.conf diff --git a/docs/en/connector-v2/source/Email.md b/docs/en/connector-v2/source/Email.md new file mode 100644 index 00000000000..24593992074 --- /dev/null +++ b/docs/en/connector-v2/source/Email.md @@ -0,0 +1,71 @@ +# Email + +## Description + +Send the data as a file to email. + + The tested kudu version is 1.5.6. + +## Options + +| name | type | required | default value | +|--------------------------|---------|----------|---------------| +| email_from_address | string | yes | - | +| email_to_address | string | yes | - | +| email_host | string | yes | - | +| email_transport_protocol | string | yes | - | +| email_smtp_auth | string | yes | - | +| email_authorization_code | string | yes | - | +| email_message_headline | string | yes | - | +| email_message_content | string | yes | - | + + +### email_from_address [string] + +Sender Email Address . + +### email_to_address [string] + +Address to receive mail. + +### email_host [string] + +SMTP server to connect to. + +### email_transport_protocol [string] + +The protocol to load the session . + +### email_smtp_auth [string] + +Whether to authenticate the customer. + +### email_authorization_code [string] + +authorization code,You can obtain the authorization code from the mailbox Settings. + +### email_message_headline [string] + +The subject line of the entire message. + +### email_message_content [string] + +The body of the entire message. + + +## Example + +```bash + + EmailSink { + email_from_address = "xxxxxx@qq.com" + email_to_address = "xxxxxx@163.com" + email_host="smtp.qq.com" + email_transport_protocol="smtp" + email_smtp_auth="true" + email_authorization_code="" + email_message_headline="这个是标题" + email_message_content="这个是内容" + } + +``` diff --git a/seatunnel-connectors-v2/connector-email/src/main/resources/fake_to_emailsink_flink.conf b/seatunnel-connectors-v2/connector-email/src/main/resources/fake_to_emailsink_flink.conf new file mode 100644 index 00000000000..8655a11d328 --- /dev/null +++ b/seatunnel-connectors-v2/connector-email/src/main/resources/fake_to_emailsink_flink.conf @@ -0,0 +1,63 @@ +# +# 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. +# +###### +###### This config file is a demonstration of streaming processing in seatunnel config +###### + +env { + # You can set flink configuration here + execution.parallelism = 1 + #job.mode = "BATCH" + #execution.checkpoint.interval = 10000 + #execution.checkpoint.data-uri = "hdfs://localhost:9000/checkpoint" +} + +source { + # This is a example source plugin **only for test and demonstrate the feature source plugin** + FakeSource { + result_table_name = "fake" + field_name = "name,age" + } + + # If you would like to get more information about how to configure seatunnel and see full list of source plugins, + # please go to https://seatunnel.apache.org/docs/flink/configuration/source-plugins/Fake +} + +transform { + sql { + sql = "select name,age from fake" + } + + # If you would like to get more information about how to configure seatunnel and see full list of transform plugins, + # please go to https://seatunnel.apache.org/docs/flink/configuration/transform-plugins/Sql +} + +sink { + EmailSink { + email_from_address = "xxxxxx@qq.com" + email_to_address = "xxxxxx@163.com" + email_host="smtp.qq.com" + email_transport_protocol="smtp" + email_smtp_auth="true" + email_authorization_code="" + email_message_headline="这个是标题" + email_message_content="这个是内容" + } + + # If you would like to get more information about how to configure seatunnel and see full list of sink plugins, + # please go to https://seatunnel.apache.org/docs/flink/configuration/sink-plugins/Console +} \ No newline at end of file From a21ac08ae0a5548827c7ce43f2cca7f5d54bb6b8 Mon Sep 17 00:00:00 2001 From: liuyehan <81779971+2013650523@users.noreply.github.com> Date: Wed, 3 Aug 2022 12:13:42 +0800 Subject: [PATCH 26/26] Update Email.md --- docs/en/connector-v2/source/Email.md | 6 +++--- 1 file changed, 3 insertions(+), 3 deletions(-) diff --git a/docs/en/connector-v2/source/Email.md b/docs/en/connector-v2/source/Email.md index 24593992074..0ae3706c5df 100644 --- a/docs/en/connector-v2/source/Email.md +++ b/docs/en/connector-v2/source/Email.md @@ -4,7 +4,7 @@ Send the data as a file to email. - The tested kudu version is 1.5.6. + The tested email version is 1.5.6. ## Options @@ -64,8 +64,8 @@ The body of the entire message. email_transport_protocol="smtp" email_smtp_auth="true" email_authorization_code="" - email_message_headline="这个是标题" - email_message_content="这个是内容" + email_message_headline="" + email_message_content="" } ```