Skip to content

Commit 2001668

Browse files
committed
[FLINK-32522][connectors/kafka] Kafka connector should depend on commons-collections instead of inheriting from flink
1 parent b0f15f2 commit 2001668

File tree

5 files changed

+19
-3
lines changed

5 files changed

+19
-3
lines changed

flink-connector-kafka/pom.xml

Lines changed: 8 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -72,6 +72,14 @@ under the License.
7272
<version>${kafka.version}</version>
7373
</dependency>
7474

75+
<!-- Others -->
76+
77+
<dependency>
78+
<groupId>org.apache.commons</groupId>
79+
<artifactId>commons-collections4</artifactId>
80+
<version>4.4</version>
81+
</dependency>
82+
7583
<!-- Tests -->
7684

7785
<dependency>

flink-connector-kafka/src/main/java/org/apache/flink/streaming/connectors/kafka/FlinkKafkaConsumerBase.java

Lines changed: 1 addition & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -59,7 +59,7 @@
5959
import org.apache.flink.util.ExceptionUtils;
6060
import org.apache.flink.util.SerializedValue;
6161

62-
import org.apache.commons.collections.map.LinkedMap;
62+
import org.apache.commons.collections4.map.LinkedMap;
6363
import org.apache.kafka.clients.consumer.ConsumerConfig;
6464
import org.slf4j.Logger;
6565
import org.slf4j.LoggerFactory;

flink-connector-kafka/src/test/java/org/apache/flink/streaming/connectors/kafka/KafkaTestEnvironmentImpl.java

Lines changed: 4 additions & 2 deletions
Original file line numberDiff line numberDiff line change
@@ -30,7 +30,7 @@
3030
import org.apache.flink.streaming.util.serialization.KeyedSerializationSchema;
3131
import org.apache.flink.util.DockerImageVersions;
3232

33-
import org.apache.commons.collections.list.UnmodifiableList;
33+
import org.apache.commons.collections4.list.UnmodifiableList;
3434
import org.apache.kafka.clients.admin.AdminClient;
3535
import org.apache.kafka.clients.admin.NewTopic;
3636
import org.apache.kafka.clients.admin.TopicDescription;
@@ -250,7 +250,9 @@ public <T> KafkaSourceBuilder<T> getSourceBuilder(
250250
@SuppressWarnings("unchecked")
251251
public <K, V> Collection<ConsumerRecord<K, V>> getAllRecordsFromTopic(
252252
Properties properties, String topic) {
253-
return UnmodifiableList.decorate(KafkaUtil.drainAllRecordsFromTopic(topic, properties));
253+
List<ConsumerRecord<byte[], byte[]>> records =
254+
KafkaUtil.drainAllRecordsFromTopic(topic, properties);
255+
return UnmodifiableList.unmodifiableList((List) records);
254256
}
255257

256258
@Override

flink-sql-connector-kafka/pom.xml

Lines changed: 5 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -70,6 +70,7 @@ under the License.
7070
<include>org.apache.flink:flink-connector-base</include>
7171
<include>org.apache.flink:flink-connector-kafka</include>
7272
<include>org.apache.kafka:*</include>
73+
<include>org.apache.commons:commons-collections4</include>
7374
</includes>
7475
</artifactSet>
7576
<filters>
@@ -91,6 +92,10 @@ under the License.
9192
<pattern>org.apache.kafka</pattern>
9293
<shadedPattern>org.apache.flink.kafka.shaded.org.apache.kafka</shadedPattern>
9394
</relocation>
95+
<relocation>
96+
<pattern>org.apache.commons</pattern>
97+
<shadedPattern>org.apache.flink.kafka.shaded.org.apache.commons</shadedPattern>
98+
</relocation>
9499
</relocations>
95100
</configuration>
96101
</execution>

flink-sql-connector-kafka/src/main/resources/META-INF/NOTICE

Lines changed: 1 addition & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -7,3 +7,4 @@ The Apache Software Foundation (http://www.apache.org/).
77
This project bundles the following dependencies under the Apache Software License 2.0. (http://www.apache.org/licenses/LICENSE-2.0.txt)
88

99
- org.apache.kafka:kafka-clients:3.4.0
10+
- commons-collections:commons-collections:3.2.2

0 commit comments

Comments
 (0)