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

Debezium Cassandra 4 Connector not working with 1.9.4 release BUT works with 1.9.2 release

    XMLWordPrintable

Details

    • False
    • None
    • False

    Description

      Bug report

      CDC records are not generated on 1.9.4 version of Debezium Cassandra 4 release when run on a clustered setup(3 Cassandra nodes).

      But 1.9.2 version of  Debezium Cassandra 4 release works perfectly on the same Cassandra setup.

      What Debezium connector do you use and what version?

      Non-Working Version: Debezium Cassandra 1.9.4 connector: https://repo1.maven.org/maven2/io/debezium/debezium-connector-cassandra-4/1.9.4.Final/debezium-connector-cassandra-4-1.9.4.Final-jar-with-dependencies.jar

      Working Version: Debezium Cassandra4 1.9.2 connector: https://repo1.maven.org/maven2/io/debezium/debezium-connector-cassandra-4/1.9.2.Final/debezium-connector-cassandra-4-1.9.2.Final-jar-with-dependencies.jar

      What is the connector configuration?

      connector.name=test_connector

      commit.log.relocation.dir=/tmp/debezium-connector-cassandra/test_dir/relocation/

      http.port=8000

      cassandra.config=/app/debezium/cassandra.yaml

      cassandra.hosts=127.0.0.1

      cassandra.port=9042

      cassandra.driver.config.file=/app/debezium/application.conf

      kafka.producer.bootstrap.servers=<Kafka Broker Addresses>

      kafka.topic.prefix=test

      kafka.producer.security.protocol=SSL

      kafka.producer.ssl.keystore.location=<Accesible JKS Location>

      kafka.producer.ssl.keystore.password=<passwords>

      kafka.producer.ssl.truststore.location=<Accesible JKS Location>

      kafka.producer.ssl.truststore.password=<passwords>

      kafka.producer.ssl.key.password=<passwords>

      key.converter=org.apache.kafka.connect.json.JsonConverter

      value.converter=org.apache.kafka.connect.json.JsonConverter

      offset.backing.store.dir=/tmp/debezium-connector-cassandra/test_dir/

      offset.flush.interval.ms=60000

      max.offset.flush.size=5000

      cdc.dir.poll.interval.ms=2000

      snapshot.consistency=ONE

      snapshot.mode=NEVER

      latest.commit.log.only=false

      What is the captured database version and mode of deployment?

      Cassandra 4.0.3 running on premises 

      No of nodes: 3

      Replication factor: 3

      Java version installed on cassandra node(s)

      openjdk version "11.0.12" 2021-07-20 LTS

      OpenJDK Runtime Environment Zulu11.50+20-SA (build 11.0.12+7-LTS)

      OpenJDK 64-Bit Server VM Zulu11.50+20-SA (build 11.0.12+7-LTS, mixed mode)

      What behaviour do you expect?

      Expectation is to see CDC messages sent to Kafka, but when when we upgrade to 1.9.4 messages are not sent to kafka.

      What behaviour do you see?

      CDC messages are not sent to Kafka from any node with Debezium Cassandra4 1.9.4 release. 

      The Cassandra4 connector identifies Log file which is marked "COMPLETED" and sends this file to CommitLogReader But the handleMutation never gets called, the CommitLogReader returns and file is moved to archive/deleted. This causes no messages to be sent to Kafka. 

      Relevant portions of logs with 1.9.4 version are below:

      18:26:53.461 [pool-2-thread-1] INFO io.debezium.connector.cassandra.Cassandra4CommitLogProcessor$CommitLogProcessingCallable - Processing commit log /mnt/resource/cassandra/commitlog/cdc_raw/CommitLog-7-1652287118530.log

      18:26:53.462 [pool-2-thread-1] INFO io.debezium.connector.cassandra.Cassandra4CommitLogProcessor$CommitLogProcessingCallable - LogicalCommitLog

      {commitLogPosition=CommitLogPosition(segmentId=1652287118530, position=0), synced=33552621, completed=true, log=/mnt/resource/cassandra/commitlog/cdc_raw/*CommitLog-7-1652287118530*.log, index=/mnt/resource/cassandra/commitlog/cdc_raw/*CommitLog-7-1652287118530*_cdc.idx, commitLogSegmentId=1652287118530}

      18:26:53.468 [pool-2-thread-1] DEBUG org.apache.cassandra.db.commitlog.CommitLogReader - Reading /mnt/resource/cassandra/commitlog/cdc_raw/CommitLog-7-1652287118530.log (CL version 7, messaging version 12, compression null)

      18:26:54.539 [pool-2-thread-1] INFO org.apache.cassandra.db.commitlog.CommitLogReader - Finished reading /mnt/resource/cassandra/commitlog/cdc_raw/CommitLog-7-1652287118530.log

      18:26:54.540 [pool-2-thread-1] INFO io.debezium.connector.cassandra.Cassandra4CommitLogProcessor$CommitLogProcessingCallable - ProcessingResult{commitLog=LogicalCommitLog{commitLogPosition=CommitLogPosition(segmentId=1652287118530, position=0), synced=33552621, completed=true, log=/mnt/resource/cassandra/commitlog/cdc_raw/*CommitLog-7-1652287118530*.log, index=/mnt/resource/cassandra/commitlog/cdc_raw/*CommitLog-7-1652287118530*_cdc.idx, commitLogSegmentId=1652287118530}

      , result=OK, ex=none}

      18:26:55.451 [pool-4-thread-3] INFO io.debezium.connector.cassandra.QueueProcessor - Encountered EOF event for CommitLog-7-1652287118530.log ...

      18:26:55.494 [pool-4-thread-3] INFO io.debezium.connector.cassandra.CommitLogUtil - Moved CommitLog file /mnt/resource/cassandra/commitlog/cdc_raw/CommitLog-7-1652287118530.log to /tmp/debezium-connector-cassandra/test_dir/relocation/archive.

       

      In 1.9.2 release, we see the logs below which correctly parses the Changed Data:

      18:21:31.615 [pool-2-thread-1] INFO io.debezium.connector.cassandra.Cassandra4CommitLogProcessor$CommitLogProcessingCallable - Processing commit log /mnt/resource/cassandra/commitlog/cdc_raw/CommitLog-7-1652287118530.log

      18:21:31.615 [pool-2-thread-1] INFO io.debezium.connector.cassandra.Cassandra4CommitLogProcessor$CommitLogProcessingCallable - LogicalCommitLog

      {commitLogPosition=CommitLogPosition(segmentId=1652287118530, position=0), synced=33552621, completed=true, log=/mnt/resource/cassandra/commitlog/cdc_raw/*CommitLog-7-1652287118530*.log, index=/mnt/resource/cassandra/commitlog/cdc_raw/*CommitLog-7-1652287118530*_cdc.idx, commitLogSegmentId=1652287118530}

      18:21:31.621 [pool-2-thread-1] DEBUG org.apache.cassandra.db.commitlog.CommitLogReader - Reading /mnt/resource/cassandra/commitlog/cdc_raw/CommitLog-7-1652287118530.log (CL version 7, messaging version 12, compression null)

      ...

      18:21:32.137 [pool-2-thread-1] DEBUG io.debezium.connector.base.ChangeEventQueue - Enqueuing source record 'Record{source={cluster=<Record>

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

      Tested with 2.0.0.Alpha2 and it has same issue. 

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

      Yes attaching DEBUG logs for non working version.

      194NonWorkingVersion.log

      Please grep for CommitLog-7-1652287118530 in the log file. 

      How to reproduce the issue using our tutorial deployment?

      How to run debezium:

      java -Dcom.sun.management.jmxremote -Dcom.sun.management.jmxremote.authenticate=false -Dcom.sun.management.jmxremote.ssl=false -Dcom.sun.management.jmxremote.authenticate=false -Dcom.sun.management.jmxremote.port=7198 -Dcassandra.storagedir=/app/cassandra/data add-exports java.base/jdk.internal.misc=ALL-UNNAMED add-exports java.base/jdk.internal.ref=ALL-UNNAMED add-exports java.base/sun.nio.ch=ALL-UNNAMED add-exports java.management.rmi/com.sun.jmx.remote.internal.rmi=ALL-UNNAMED add-exports java.rmi/sun.rmi.registry=ALL-UNNAMED add-exports java.rmi/sun.rmi.server=ALL-UNNAMED add-exports java.sql/java.sql=ALL-UNNAMED add-opens java.base/java.lang.module=ALL-UNNAMED add-opens java.base/jdk.internal.loader=ALL-UNNAMED add-opens java.base/jdk.internal.ref=ALL-UNNAMED add-opens java.base/jdk.internal.reflect=ALL-UNNAMED add-opens java.base/jdk.internal.math=ALL-UNNAMED add-opens java.base/jdk.internal.module=ALL-UNNAMED add-opens java.base/jdk.internal.util.jar=ALL-UNNAMED -add-opens jdk.management/com.sun.management.internal=ALL-UNNAMED -jar debezium-connector-cassandra-4-1.9.4.Final-jar-with-dependencies.jar debezium.properties

      Attachments

        Activity

          People

            Unassigned Unassigned
            nitin.nitt Nitin Chhabra (Inactive)
            Votes:
            0 Vote for this issue
            Watchers:
            8 Start watching this issue

            Dates

              Created:
              Updated:
              Resolved: