This is an automated email from the ASF dual-hosted git repository.
lzljs3620320 pushed a commit to branch master
in repository https://gitbox.apache.org/repos/asf/flink-table-store.git
The following commit(s) were added to refs/heads/master by this push:
new f65157d [hotfix] Revert "[FLINK-26677]": shade flink-connector-base
instead of using connectors
f65157d is described below
commit f65157d9131603d551f5a94cc7d44b054c5e126d
Author: JingsongLi <[email protected]>
AuthorDate: Thu Mar 24 12:09:17 2022 +0800
[hotfix] Revert "[FLINK-26677]": shade flink-connector-base instead of
using connectors
---
.../java/org/apache/flink/table/store/connector/TableStore.java | 2 +-
.../flink/table/store/connector/source/FileStoreSourceReader.java | 2 +-
.../table/store/connector/source/FileStoreSourceSplitReader.java | 8 ++++----
.../table/store/connector/source/LogHybridSourceFactory.java | 8 ++++----
.../store/connector/source/FileStoreSourceSplitReaderTest.java | 6 +++---
.../java/org/apache/flink/table/store/file/utils/FileUtils.java | 2 +-
.../java/org/apache/flink/table/store/log/LogSourceProvider.java | 2 +-
flink-table-store-dist/pom.xml | 1 +
.../org/apache/flink/table/store/kafka/KafkaLogSinkProvider.java | 2 +-
9 files changed, 17 insertions(+), 16 deletions(-)
diff --git
a/flink-table-store-connector/src/main/java/org/apache/flink/table/store/connector/TableStore.java
b/flink-table-store-connector/src/main/java/org/apache/flink/table/store/connector/TableStore.java
index 051fd0e..a33c06f 100644
---
a/flink-table-store-connector/src/main/java/org/apache/flink/table/store/connector/TableStore.java
+++
b/flink-table-store-connector/src/main/java/org/apache/flink/table/store/connector/TableStore.java
@@ -25,7 +25,7 @@ import org.apache.flink.api.connector.source.Boundedness;
import org.apache.flink.api.connector.source.Source;
import org.apache.flink.configuration.Configuration;
import org.apache.flink.configuration.DelegatingConfiguration;
-import
org.apache.flink.connector.files.shaded.org.apache.flink.connector.base.source.hybrid.HybridSource;
+import org.apache.flink.connector.base.source.hybrid.HybridSource;
import org.apache.flink.streaming.api.datastream.DataStream;
import org.apache.flink.streaming.api.datastream.DataStreamSink;
import org.apache.flink.streaming.api.datastream.DataStreamSource;
diff --git
a/flink-table-store-connector/src/main/java/org/apache/flink/table/store/connector/source/FileStoreSourceReader.java
b/flink-table-store-connector/src/main/java/org/apache/flink/table/store/connector/source/FileStoreSourceReader.java
index 6a5a263..4d2e1a4 100644
---
a/flink-table-store-connector/src/main/java/org/apache/flink/table/store/connector/source/FileStoreSourceReader.java
+++
b/flink-table-store-connector/src/main/java/org/apache/flink/table/store/connector/source/FileStoreSourceReader.java
@@ -20,8 +20,8 @@ package org.apache.flink.table.store.connector.source;
import org.apache.flink.api.connector.source.SourceReader;
import org.apache.flink.api.connector.source.SourceReaderContext;
+import
org.apache.flink.connector.base.source.reader.SingleThreadMultiplexSourceReaderBase;
import org.apache.flink.connector.file.src.util.RecordAndPosition;
-import
org.apache.flink.connector.files.shaded.org.apache.flink.connector.base.source.reader.SingleThreadMultiplexSourceReaderBase;
import org.apache.flink.table.data.RowData;
import org.apache.flink.table.store.file.operation.FileStoreRead;
diff --git
a/flink-table-store-connector/src/main/java/org/apache/flink/table/store/connector/source/FileStoreSourceSplitReader.java
b/flink-table-store-connector/src/main/java/org/apache/flink/table/store/connector/source/FileStoreSourceSplitReader.java
index 82931d3..9361615 100644
---
a/flink-table-store-connector/src/main/java/org/apache/flink/table/store/connector/source/FileStoreSourceSplitReader.java
+++
b/flink-table-store-connector/src/main/java/org/apache/flink/table/store/connector/source/FileStoreSourceSplitReader.java
@@ -19,15 +19,15 @@
package org.apache.flink.table.store.connector.source;
import org.apache.flink.annotation.VisibleForTesting;
+import org.apache.flink.connector.base.source.reader.RecordsWithSplitIds;
+import org.apache.flink.connector.base.source.reader.splitreader.SplitReader;
+import
org.apache.flink.connector.base.source.reader.splitreader.SplitsAddition;
+import org.apache.flink.connector.base.source.reader.splitreader.SplitsChange;
import org.apache.flink.connector.file.src.impl.FileRecords;
import org.apache.flink.connector.file.src.reader.BulkFormat;
import org.apache.flink.connector.file.src.util.MutableRecordAndPosition;
import org.apache.flink.connector.file.src.util.Pool;
import org.apache.flink.connector.file.src.util.RecordAndPosition;
-import
org.apache.flink.connector.files.shaded.org.apache.flink.connector.base.source.reader.RecordsWithSplitIds;
-import
org.apache.flink.connector.files.shaded.org.apache.flink.connector.base.source.reader.splitreader.SplitReader;
-import
org.apache.flink.connector.files.shaded.org.apache.flink.connector.base.source.reader.splitreader.SplitsAddition;
-import
org.apache.flink.connector.files.shaded.org.apache.flink.connector.base.source.reader.splitreader.SplitsChange;
import org.apache.flink.table.data.RowData;
import org.apache.flink.table.data.utils.ProjectedRowData;
import org.apache.flink.table.store.file.KeyValue;
diff --git
a/flink-table-store-connector/src/main/java/org/apache/flink/table/store/connector/source/LogHybridSourceFactory.java
b/flink-table-store-connector/src/main/java/org/apache/flink/table/store/connector/source/LogHybridSourceFactory.java
index 59c5008..0f892fc 100644
---
a/flink-table-store-connector/src/main/java/org/apache/flink/table/store/connector/source/LogHybridSourceFactory.java
+++
b/flink-table-store-connector/src/main/java/org/apache/flink/table/store/connector/source/LogHybridSourceFactory.java
@@ -19,17 +19,17 @@
package org.apache.flink.table.store.connector.source;
import org.apache.flink.api.connector.source.Source;
-import
org.apache.flink.connector.files.shaded.org.apache.flink.connector.base.source.hybrid.HybridSource;
+import org.apache.flink.connector.base.source.hybrid.HybridSource;
+import
org.apache.flink.connector.base.source.hybrid.HybridSource.SourceFactory;
import org.apache.flink.table.data.RowData;
import org.apache.flink.table.store.file.Snapshot;
import org.apache.flink.table.store.log.LogSourceProvider;
import java.util.Map;
-/** Log {@link HybridSource.SourceFactory} from {@link
StaticFileStoreSplitEnumerator}. */
+/** Log {@link SourceFactory} from {@link StaticFileStoreSplitEnumerator}. */
public class LogHybridSourceFactory
- implements HybridSource.SourceFactory<
- RowData, Source<RowData, ?, ?>,
StaticFileStoreSplitEnumerator> {
+ implements SourceFactory<RowData, Source<RowData, ?, ?>,
StaticFileStoreSplitEnumerator> {
private final LogSourceProvider provider;
diff --git
a/flink-table-store-connector/src/test/java/org/apache/flink/table/store/connector/source/FileStoreSourceSplitReaderTest.java
b/flink-table-store-connector/src/test/java/org/apache/flink/table/store/connector/source/FileStoreSourceSplitReaderTest.java
index e8d3921..f5dc625 100644
---
a/flink-table-store-connector/src/test/java/org/apache/flink/table/store/connector/source/FileStoreSourceSplitReaderTest.java
+++
b/flink-table-store-connector/src/test/java/org/apache/flink/table/store/connector/source/FileStoreSourceSplitReaderTest.java
@@ -19,10 +19,10 @@
package org.apache.flink.table.store.connector.source;
import org.apache.flink.api.java.tuple.Tuple2;
+import org.apache.flink.connector.base.source.reader.RecordsWithSplitIds;
+import
org.apache.flink.connector.base.source.reader.splitreader.SplitsAddition;
+import org.apache.flink.connector.base.source.reader.splitreader.SplitsChange;
import org.apache.flink.connector.file.src.util.RecordAndPosition;
-import
org.apache.flink.connector.files.shaded.org.apache.flink.connector.base.source.reader.RecordsWithSplitIds;
-import
org.apache.flink.connector.files.shaded.org.apache.flink.connector.base.source.reader.splitreader.SplitsAddition;
-import
org.apache.flink.connector.files.shaded.org.apache.flink.connector.base.source.reader.splitreader.SplitsChange;
import org.apache.flink.table.data.GenericRowData;
import org.apache.flink.table.data.RowData;
import org.apache.flink.table.store.file.ValueKind;
diff --git
a/flink-table-store-core/src/main/java/org/apache/flink/table/store/file/utils/FileUtils.java
b/flink-table-store-core/src/main/java/org/apache/flink/table/store/file/utils/FileUtils.java
index d4e0c44..e797a28 100644
---
a/flink-table-store-core/src/main/java/org/apache/flink/table/store/file/utils/FileUtils.java
+++
b/flink-table-store-core/src/main/java/org/apache/flink/table/store/file/utils/FileUtils.java
@@ -19,10 +19,10 @@
package org.apache.flink.table.store.file.utils;
import org.apache.flink.configuration.Configuration;
+import org.apache.flink.connector.base.source.reader.SourceReaderOptions;
import org.apache.flink.connector.file.src.FileSourceSplit;
import org.apache.flink.connector.file.src.reader.BulkFormat;
import org.apache.flink.connector.file.src.util.Utils;
-import
org.apache.flink.connector.files.shaded.org.apache.flink.connector.base.source.reader.SourceReaderOptions;
import org.apache.flink.core.fs.FSDataInputStream;
import org.apache.flink.core.fs.FSDataOutputStream;
import org.apache.flink.core.fs.FileSystem;
diff --git
a/flink-table-store-core/src/main/java/org/apache/flink/table/store/log/LogSourceProvider.java
b/flink-table-store-core/src/main/java/org/apache/flink/table/store/log/LogSourceProvider.java
index 12ece82..94dd604 100644
---
a/flink-table-store-core/src/main/java/org/apache/flink/table/store/log/LogSourceProvider.java
+++
b/flink-table-store-core/src/main/java/org/apache/flink/table/store/log/LogSourceProvider.java
@@ -19,7 +19,7 @@
package org.apache.flink.table.store.log;
import org.apache.flink.api.connector.source.Source;
-import
org.apache.flink.connector.files.shaded.org.apache.flink.connector.base.source.hybrid.HybridSource;
+import org.apache.flink.connector.base.source.hybrid.HybridSource;
import org.apache.flink.table.data.RowData;
import javax.annotation.Nullable;
diff --git a/flink-table-store-dist/pom.xml b/flink-table-store-dist/pom.xml
index c8837ca..f745d85 100644
--- a/flink-table-store-dist/pom.xml
+++ b/flink-table-store-dist/pom.xml
@@ -118,6 +118,7 @@ under the License.
<include>org.apache.flink:flink-table-store-core</include>
<include>org.apache.flink:flink-table-store-format</include>
<include>org.apache.flink:flink-table-store-kafka</include>
+
<include>org.apache.flink:flink-connector-base</include>
<include>org.apache.flink:flink-connector-files</include>
<include>org.apache.flink:flink-sql-connector-kafka</include>
<include>org.apache.flink:flink-sql-avro</include>
diff --git
a/flink-table-store-kafka/src/main/java/org/apache/flink/table/store/kafka/KafkaLogSinkProvider.java
b/flink-table-store-kafka/src/main/java/org/apache/flink/table/store/kafka/KafkaLogSinkProvider.java
index 273de54..85b9cc0 100644
---
a/flink-table-store-kafka/src/main/java/org/apache/flink/table/store/kafka/KafkaLogSinkProvider.java
+++
b/flink-table-store-kafka/src/main/java/org/apache/flink/table/store/kafka/KafkaLogSinkProvider.java
@@ -20,7 +20,7 @@ package org.apache.flink.table.store.kafka;
import org.apache.flink.annotation.VisibleForTesting;
import org.apache.flink.api.common.serialization.SerializationSchema;
-import
org.apache.flink.connector.kafka.shaded.org.apache.flink.connector.base.DeliveryGuarantee;
+import org.apache.flink.connector.base.DeliveryGuarantee;
import org.apache.flink.connector.kafka.sink.KafkaSink;
import org.apache.flink.connector.kafka.sink.KafkaSinkBuilder;
import org.apache.flink.table.data.RowData;