From bd260f14dd280f464e5dcea76b3735782312b4ae Mon Sep 17 00:00:00 2001 From: Martijn Visser Date: Tue, 10 Oct 2023 13:11:38 +0200 Subject: [PATCH] [FLINK-33191][Connector/Kafka] Make the `testKafkaValueDeserializationSchemaWrapper` test use `flink-shaded-jackson` since it tests `flink-shaded-jackson` ObjectNodes. Co-authored-by: zentol This closes #57. --- .../KafkaRecordDeserializationSchemaTest.java | 20 +++++++++++++------ 1 file changed, 14 insertions(+), 6 deletions(-) diff --git a/flink-connector-kafka/src/test/java/org/apache/flink/connector/kafka/source/reader/deserializer/KafkaRecordDeserializationSchemaTest.java b/flink-connector-kafka/src/test/java/org/apache/flink/connector/kafka/source/reader/deserializer/KafkaRecordDeserializationSchemaTest.java index d61b7f834..b0ca63161 100644 --- a/flink-connector-kafka/src/test/java/org/apache/flink/connector/kafka/source/reader/deserializer/KafkaRecordDeserializationSchemaTest.java +++ b/flink-connector-kafka/src/test/java/org/apache/flink/connector/kafka/source/reader/deserializer/KafkaRecordDeserializationSchemaTest.java @@ -81,16 +81,24 @@ public void testKafkaDeserializationSchemaWrapper() throws Exception { @Test public void testKafkaValueDeserializationSchemaWrapper() throws Exception { final ConsumerRecord consumerRecord = getConsumerRecord(); - KafkaRecordDeserializationSchema schema = - KafkaRecordDeserializationSchema.valueOnly( - new JsonDeserializationSchema<>(ObjectNode.class)); + KafkaRecordDeserializationSchema< + org.apache.flink.shaded.jackson2.com.fasterxml.jackson.databind.node + .ObjectNode> + schema = + KafkaRecordDeserializationSchema.valueOnly( + new JsonDeserializationSchema<>( + org.apache.flink.shaded.jackson2.com.fasterxml.jackson + .databind.node.ObjectNode.class)); schema.open(new DummyInitializationContext()); - SimpleCollector collector = new SimpleCollector<>(); + SimpleCollector< + org.apache.flink.shaded.jackson2.com.fasterxml.jackson.databind.node + .ObjectNode> + collector = new SimpleCollector<>(); schema.deserialize(consumerRecord, collector); assertThat(collector.list).hasSize(1); - ObjectNode deserializedValue = collector.list.get(0); - + org.apache.flink.shaded.jackson2.com.fasterxml.jackson.databind.node.ObjectNode + deserializedValue = collector.list.get(0); assertThat(deserializedValue.get("word").asText()).isEqualTo("world"); assertThat(deserializedValue.get("key")).isNull(); assertThat(deserializedValue.get("metadata")).isNull();