Uploaded image for project: 'Debezium'
  1. Debezium
  2. DBZ-9378

Proper Kafka producer exception not logged due to record.key serialisation error

XMLWordPrintable

      In order to make your issue reports as actionable as possible, please provide the following information, depending on the issue type.

      Bug report

      For bug reports, provide this information, please:

      What Debezium connector do you use and what version?

      Debezium Server 3.2.1.Final (standalone)

      What is the connector configuration?

       

      debezium.format.header.schemas.enable=false
      debezium.format.key.schemas.enable=false
      debezium.format.value.schemas.enable=false
      debezium.sink.kafka.producer.bootstrap.servers=*******
      debezium.sink.kafka.producer.compression.type=zstd
      debezium.sink.kafka.producer.key.serializer=org.apache.kafka.common.serialization.StringSerializer
      debezium.sink.kafka.producer.sasl.jaas.config=org.apache.kafka.common.security.scram.ScramLoginModule required username="${DEBEZIUM_SINK_KAFKA_PRODUCER_SASL_USERNAME}" password="${DEBEZIUM_SINK_KAFKA_PRODUCER_SASL_PASSWORD}";
      debezium.sink.kafka.producer.sasl.mechanism=SCRAM-SHA-512
      debezium.sink.kafka.producer.security.protocol=SASL_SSL
      debezium.sink.kafka.producer.value.serializer=org.apache.kafka.common.serialization.StringSerializer
      debezium.sink.type=kafka
      debezium.source.capture.mode=change_streams_update_full_with_pre_image
      debezium.source.collection.include.list=.*\.documents
      debezium.source.connector.class=io.debezium.connector.mongodb.MongoDbConnector
      debezium.source.mongodb.connection.string=mongodb+srv://*******/?retryWrites=true&w=majority
      debezium.source.offset.flush.interval.ms=0
      debezium.source.offset.storage.file.filename=/debezium/data/offsets.dat
      debezium.source.snapshot.mode=no_data
      debezium.source.tombstones.on.delete=true
      debezium.source.topic.prefix=cdc
      debezium.transforms.router.regex=(.*)\.documents
      debezium.transforms.router.replacement=cdc.documents
      debezium.transforms.router.type=org.apache.kafka.connect.transforms.RegexRouter
      debezium.transforms=router
      quarkus.http.port=8080
      quarkus.log.console.json=true 

       

      What is the captured database version and mode of deployment?

      (E.g. on-premises, with a specific cloud provider, etc.)

      MongoDB Atlas 8.0.12
      Kafka 3.8.x AWS MSK 

      What behavior do you expect?

      An actual error message is logged.

      What behavior do you see?

      ```
      class java.lang.String cannot be cast to class [B (java.lang.String and [B are in module java.base of loader 'bootstrap')
      ```

      This is happening while we're trying to log the actual error message:

      https://github.com/debezium/debezium-server/blob/v3.2.1.Final/debezium-server-kafka/src/main/java/io/debezium/server/kafka/KafkaChangeConsumer.java#L107

      Do you see the same behaviour using the latest released Debezium version?

      (Ideally, also verify with latest Alpha/Beta/CR version)

      Same code can be seen on `main`.

      Do you have the connector logs, ideally from start till finish?

      (You might be asked later to provide DEBUG/TRACE level log)

      Only partial:

      {
        "timestamp": "2025-08-22T11:39:33.83375018Z",
        "sequence": 1028,
        "loggerClassName": "org.slf4j.impl.Slf4jLogger",
        "loggerName": "io.debezium.embedded.async.AsyncEmbeddedEngine",
        "level": "ERROR",
        "message": "Engine has failed with ",
        "threadName": "pool-7-thread-1",
        "threadId": 34,
        "mdc": {},
        "ndc": "",
        "hostName": "studio-cdc-79f5b9548d-cd7cc",
        "processName": "/usr/lib/jvm/java-21-openjdk-21.0.8.0.9-1.el8.x86_64/bin/java",
        "processId": 1,
        "exception": {
          "refId": 1,
          "exceptionType": "java.util.concurrent.ExecutionException",
          "message": "io.debezium.DebeziumException: java.lang.ClassCastException: class java.lang.String cannot be cast to class [B (java.lang.String and [B are in module java.base of loader 'bootstrap')",
          "frames": [
           
      {         "class": "java.util.concurrent.FutureTask",         "method": "report",         "line": 122       }
      ,
           
      {         "class": "java.util.concurrent.FutureTask",         "method": "get",         "line": 191       }
      ,
           
      {         "class": "io.debezium.embedded.async.AsyncEmbeddedEngine",         "method": "runTasksPolling",         "line": 511       }
      ,
           
      {         "class": "io.debezium.embedded.async.AsyncEmbeddedEngine",         "method": "run",         "line": 221       }
      ,
           
      {         "class": "io.debezium.server.DebeziumServer",         "method": "lambda$start$1",         "line": 182       }
      ,
           
      {         "class": "java.util.concurrent.ThreadPoolExecutor",         "method": "runWorker",         "line": 1144       }
      ,
           
      {         "class": "java.util.concurrent.ThreadPoolExecutor$Worker",         "method": "run",         "line": 642       }
      ,
           
      {         "class": "java.lang.Thread",         "method": "run",         "line": 1583       }
          ],
          "causedBy": {
            "exception": {
              "refId": 2,
              "exceptionType": "io.debezium.DebeziumException",
              "message": "java.lang.ClassCastException: class java.lang.String cannot be cast to class [B (java.lang.String and [B are in module java.base of loader 'bootstrap')",
              "frames": [
               
      {             "class": "io.debezium.server.kafka.KafkaChangeConsumer",             "method": "handleBatch",             "line": 117           }
      ,
               
      {             "class": "io.debezium.embedded.async.ParallelSmtAndConvertBatchProcessor",             "method": "processRecords",             "line": 56           }
      ,
               
      {             "class": "io.debezium.embedded.async.AsyncEmbeddedEngine$PollRecords",             "method": "doCall",             "line": 1222           }
      ,
               
      {             "class": "io.debezium.embedded.async.AsyncEmbeddedEngine$PollRecords",             "method": "doCall",             "line": 1202           }
      ,
               
      {             "class": "io.debezium.embedded.async.RetryingCallable",             "method": "call",             "line": 47           }
      ,
               
      {             "class": "java.util.concurrent.FutureTask",             "method": "run",             "line": 317           }
      ,
               
      {             "class": "java.util.concurrent.Executors$RunnableAdapter",             "method": "call",             "line": 572           }
      ,
               
      {             "class": "java.util.concurrent.FutureTask",             "method": "run",             "line": 317           }
      ,
               
      {             "class": "java.util.concurrent.ThreadPoolExecutor",             "method": "runWorker",             "line": 1144           }
      ,
               
      {             "class": "java.util.concurrent.ThreadPoolExecutor$Worker",             "method": "run",             "line": 642           }
      ,
               
      {             "class": "java.lang.Thread",             "method": "run",             "line": 1583           }
              ],
              "causedBy": {
                "exception": {
                  "refId": 3,
                  "exceptionType": "java.lang.ClassCastException",
                  "message": "class java.lang.String cannot be cast to class [B (java.lang.String and [B are in module java.base of loader 'bootstrap')",
                  "frames": [
                   
      {                 "class": "io.debezium.server.kafka.KafkaChangeConsumer",                 "method": "lambda$handleBatch$0",                 "line": 107               }
      ,
                   
      {                 "class": "org.apache.kafka.clients.producer.KafkaProducer",                 "method": "doSend",                 "line": 1108               }
      ,
                   
      {                 "class": "org.apache.kafka.clients.producer.KafkaProducer",                 "method": "send",                 "line": 993               }
      ,
                   
      {                 "class": "io.debezium.server.kafka.KafkaChangeConsumer",                 "method": "handleBatch",                 "line": 104               }
      ,
                   
      {                 "class": "io.debezium.embedded.async.ParallelSmtAndConvertBatchProcessor",                 "method": "processRecords",                 "line": 56               }
      ,
                   
      {                 "class": "io.debezium.embedded.async.AsyncEmbeddedEngine$PollRecords",                 "method": "doCall",                 "line": 1222               }
      ,
                   
      {                 "class": "io.debezium.embedded.async.AsyncEmbeddedEngine$PollRecords",                 "method": "doCall",                 "line": 1202               }
      ,
                   
      {                 "class": "io.debezium.embedded.async.RetryingCallable",                 "method": "call",                 "line": 47               }
      ,
                   
      {                 "class": "java.util.concurrent.FutureTask",                 "method": "run",                 "line": 317               }
      ,
                   
      {                 "class": "java.util.concurrent.Executors$RunnableAdapter",                 "method": "call",                 "line": 572               }
      ,
                   
      {                 "class": "java.util.concurrent.FutureTask",                 "method": "run",                 "line": 317               }
      ,
                   
      {                 "class": "java.util.concurrent.ThreadPoolExecutor",                 "method": "runWorker",                 "line": 1144               }
      ,
                   
      {                 "class": "java.util.concurrent.ThreadPoolExecutor$Worker",                 "method": "run",                 "line": 642               }
      ,
                   
      {                 "class": "java.lang.Thread",                 "method": "run",                 "line": 1583               }
                  ]
                }
              }
            }
          }
        }
      } 

      How to reproduce the issue using our tutorial deployment?

      <Your answer>

              Unassigned Unassigned
              sylwester.kardziejonek Sylwester Kardziejonek (Inactive)
              Votes:
              0 Vote for this issue
              Watchers:
              3 Start watching this issue

                Created:
                Updated:
                Resolved: