Maxim Ivanov created BEAM-11851:
-----------------------------------

             Summary: ConfluentSchemaRegistryProvider fails when authentication 
is required
                 Key: BEAM-11851
                 URL: https://issues.apache.org/jira/browse/BEAM-11851
             Project: Beam
          Issue Type: Bug
          Components: io-java-kafka
    Affects Versions: 2.27.0
            Reporter: Maxim Ivanov


When configuring KafkaIO.reader with ConfluentSchemaRegistryProvider as value 
deserializer with `basic.auth.credentials.source=AUTH_INO` and 
`basic.auth.user.info=user:password` properties, it fails to start the pipepline


{code:java}
[error] java.lang.RuntimeException: Unable to get latest schema metadata for 
subject: identity_users_v2-value
[error]   at 
org.apache.beam.sdk.io.kafka.ConfluentSchemaRegistryDeserializerProvider.getSchemaMetadata(ConfluentSchemaRegistryDeserializerProvider.java:119)
[error]   at 
org.apache.beam.sdk.io.kafka.ConfluentSchemaRegistryDeserializerProvider.getAvroSchema(ConfluentSchemaRegistryDeserializerProvider.java:110)
[error]   at 
org.apache.beam.sdk.io.kafka.ConfluentSchemaRegistryDeserializerProvider.getCoder(ConfluentSchemaRegistryDeserializerProvider.java:106)
[error]   at 
org.apache.beam.sdk.io.kafka.KafkaIO$Read.getValueCoder(KafkaIO.java:1147)
[error]   at org.apache.beam.sdk.io.kafka.KafkaIO$Read.expand(KafkaIO.java:1052)
[error]   at org.apache.beam.sdk.io.kafka.KafkaIO$Read.expand(KafkaIO.java:481)
[error]   at org.apache.beam.sdk.Pipeline.applyInternal(Pipeline.java:547)
[error]   at org.apache.beam.sdk.Pipeline.applyTransform(Pipeline.java:498)
[error]   at org.apache.beam.sdk.values.PBegin.apply(PBegin.java:56)
[error]   at org.apache.beam.sdk.Pipeline.apply(Pipeline.java:191) {code}
 

I suspect that it because it reconfigured KafkaAvroDeserializer only in 
getDeserializer, but doesn't do so in getCoder.



--
This message was sent by Atlassian Jira
(v8.3.4#803005)

Reply via email to