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/paimon.git
The following commit(s) were added to refs/heads/master by this push: new 8eaa706e94 [flink] Bug fix for non-JSON deserialization in Flink CDC (#6116) 8eaa706e94 is described below commit 8eaa706e942d120eadad67cd242dee73c97f6922 Author: Arnav Balyan <60175178+arnavbal...@users.noreply.github.com> AuthorDate: Wed Aug 27 07:39:22 2025 +0530 [flink] Bug fix for non-JSON deserialization in Flink CDC (#6116) --- .../KafkaDebeziumJsonDeserializationSchema.java | 7 ++- ...KafkaDebeziumJsonDeserializationSchemaTest.java | 58 ++++++++++++++++++++++ 2 files changed, 64 insertions(+), 1 deletion(-) diff --git a/paimon-flink/paimon-flink-cdc/src/main/java/org/apache/paimon/flink/action/cdc/kafka/KafkaDebeziumJsonDeserializationSchema.java b/paimon-flink/paimon-flink-cdc/src/main/java/org/apache/paimon/flink/action/cdc/kafka/KafkaDebeziumJsonDeserializationSchema.java index a56608e3f7..887af5f606 100644 --- a/paimon-flink/paimon-flink-cdc/src/main/java/org/apache/paimon/flink/action/cdc/kafka/KafkaDebeziumJsonDeserializationSchema.java +++ b/paimon-flink/paimon-flink-cdc/src/main/java/org/apache/paimon/flink/action/cdc/kafka/KafkaDebeziumJsonDeserializationSchema.java @@ -67,7 +67,12 @@ public class KafkaDebeziumJsonDeserializationSchema byte[] key = message.key(); JsonNode keyNode = null; if (key != null && key.length > 0) { - keyNode = objectMapper.readValue(key, JsonNode.class); + try { + keyNode = objectMapper.readValue(key, JsonNode.class); + } catch (Exception ignore) { + // If the key is not valid JSON, ignore it to avoid failing deserialization. + // The CDC pipeline only relies on the JSON value payload. + } } JsonNode valueNode = objectMapper.readValue(message.value(), JsonNode.class); diff --git a/paimon-flink/paimon-flink-cdc/src/test/java/org/apache/paimon/flink/action/cdc/kafka/KafkaDebeziumJsonDeserializationSchemaTest.java b/paimon-flink/paimon-flink-cdc/src/test/java/org/apache/paimon/flink/action/cdc/kafka/KafkaDebeziumJsonDeserializationSchemaTest.java new file mode 100644 index 0000000000..be29047ab3 --- /dev/null +++ b/paimon-flink/paimon-flink-cdc/src/test/java/org/apache/paimon/flink/action/cdc/kafka/KafkaDebeziumJsonDeserializationSchemaTest.java @@ -0,0 +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.paimon.flink.action.cdc.kafka; + +import org.apache.paimon.flink.action.cdc.CdcSourceRecord; + +import org.apache.paimon.shade.jackson2.com.fasterxml.jackson.databind.JsonNode; + +import org.apache.kafka.clients.consumer.ConsumerRecord; +import org.junit.jupiter.api.Assertions; +import org.junit.jupiter.api.Test; + +import java.nio.charset.StandardCharsets; + +/** + * Unit tests for {@link KafkaDebeziumJsonDeserializationSchema}. Ensures that deserialization + * succeeds when Kafka keys are not valid JSON and verifies that the value payload is still parsed + * correctly. + */ +public class KafkaDebeziumJsonDeserializationSchemaTest { + + @Test + public void testDeserializeWithNonJsonKey() throws Exception { + KafkaDebeziumJsonDeserializationSchema schema = + new KafkaDebeziumJsonDeserializationSchema(); + + byte[] rawKey = "non-json-key".getBytes(StandardCharsets.UTF_8); + byte[] jsonValue = "{\"after\":{\"id\":1},\"op\":\"c\"}".getBytes(StandardCharsets.UTF_8); + + CdcSourceRecord record = + schema.deserialize(new ConsumerRecord<>("topic", 0, 0L, rawKey, jsonValue)); + + Assertions.assertNotNull(record, "Deserialization should succeed and return a record"); + Assertions.assertNull(record.getKey(), "Key should be null when the Kafka key is not JSON"); + + JsonNode valueNode = (JsonNode) record.getValue(); + Assertions.assertEquals( + 1, + valueNode.get("after").get("id").asInt(), + "Value JSON should be parsed correctly"); + } +}