[ 
https://issues.apache.org/jira/browse/KAFKA-4350?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=15621305#comment-15621305
 ] 

Jiangjie Qin commented on KAFKA-4350:
-------------------------------------

[~ecesena] In Kafka the wire protocol compatibility is taken care by the 
brokers. The clients will always send the latest wire protocol they are aware 
of. It is the brokers' responsibility to adapt to the clients protocol version 
and send corresponding response back. For example, if the broker is running at 
0.10 and it receives a request which is sent by a 0.9 client, the broker will 
send the response with the version in 0.9. That means the newer brokers support 
older clients.

However, if a broker is running in 0.9 and receives a request sent by a 0.10 
client. Because the broker does not even know the 0.10 protocol, depending on 
the type of request, different exception might be seen on the client side. In 
your case, it seems the 0.9 broker sent back a metadata response to a 0.10 
client. The 0.10 client is expecting a cluster_id field in the response which 
does not exist in 0.9 metadata response. Therefore a schema exception was 
thrown as you saw. This is the limitation mentioned by [~hachikuji].

So if you want to replicate data between 0.9 and 0.10 brokers, mirror maker in 
0.9 Kafka should be used so that it can talk to both 0.9 and 0.10 brokers.



> Can't mirror from Kafka 0.9 to Kafka 0.10.1
> -------------------------------------------
>
>                 Key: KAFKA-4350
>                 URL: https://issues.apache.org/jira/browse/KAFKA-4350
>             Project: Kafka
>          Issue Type: Bug
>            Reporter: Emanuele Cesena
>
> I'm running 2 clusters: K9 with Kafka 0.9 and K10 with Kafka 0.10.1.
> In K10, I've set up mirror maker to clone a topic from K9 to K10.
> Mirror maker immediately fails while starting, any suggestion? Following 
> error message and configs.
> Error message:
> {code:java} 
> [2016-10-26 23:54:01,663] FATAL [mirrormaker-thread-0] Mirror maker thread 
> failure due to  (kafka.tools.MirrorMaker$MirrorMakerThread)
> org.apache.kafka.common.protocol.types.SchemaException: Error reading field 
> 'cluster_id': Error reading string of length 418, only 43 bytes available
>         at org.apache.kafka.common.protocol.types.Schema.read(Schema.java:73)
>         at 
> org.apache.kafka.clients.NetworkClient.parseResponse(NetworkClient.java:380)
>         at 
> org.apache.kafka.clients.NetworkClient.handleCompletedReceives(NetworkClient.java:449)
>         at org.apache.kafka.clients.NetworkClient.poll(NetworkClient.java:269)
>         at 
> org.apache.kafka.clients.consumer.internals.ConsumerNetworkClient.poll(ConsumerNetworkClient.java:232)
>         at 
> org.apache.kafka.clients.consumer.internals.ConsumerNetworkClient.poll(ConsumerNetworkClient.java:180)
>         at 
> org.apache.kafka.clients.consumer.internals.AbstractCoordinator.ensureCoordinatorReady(AbstractCoordinator.java:193)
>         at 
> org.apache.kafka.clients.consumer.internals.ConsumerCoordinator.poll(ConsumerCoordinator.java:248)
>         at 
> org.apache.kafka.clients.consumer.KafkaConsumer.pollOnce(KafkaConsumer.java:1013)
>         at 
> org.apache.kafka.clients.consumer.KafkaConsumer.poll(KafkaConsumer.java:979)
>         at 
> kafka.tools.MirrorMaker$MirrorMakerNewConsumer.receive(MirrorMaker.scala:582)
>         at 
> kafka.tools.MirrorMaker$MirrorMakerThread.run(MirrorMaker.scala:431)
> [2016-10-26 23:54:01,679] FATAL [mirrormaker-thread-0] Mirror maker thread 
> exited abnormally, stopping the whole mirror maker. 
> (kafka.tools.MirrorMaker$MirrorMakerThread)
> {code} 
> Consumer:
> {code:} 
> group.id=mirrormaker001
> client.id=mirrormaker001
> bootstrap.servers=...K9...
> security.protocol=PLAINTEXT
> auto.offset.reset=earliest
> {code} 
> (note that I first run without client.id, then tried adding a client.id 
> because -- same error in both cases)
> Producer:
> {code:}
> bootstrap.servers=...K10...
> security.protocol=PLAINTEXT
> {code}



--
This message was sent by Atlassian JIRA
(v6.3.4#6332)

Reply via email to