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

ASF GitHub Bot commented on FLINK-10843:
----------------------------------------

pnowojski commented on a change in pull request #7087: [FLINK-10843] 
[connectors] Make Kafka table factory versioning more flexible
URL: https://github.com/apache/flink/pull/7087#discussion_r233097356
 
 

 ##########
 File path: 
flink-connectors/flink-connector-kafka-base/src/main/java/org/apache/flink/streaming/connectors/kafka/KafkaTableSourceSinkFactoryBase.java
 ##########
 @@ -249,7 +264,7 @@ private DescriptorProperties 
getValidatedProperties(Map<String, String> properti
 
                // allow Kafka timestamps to be used, watermarks can not be 
received from source
                new SchemaValidator(true, supportsKafkaTimestamps(), 
false).validate(descriptorProperties);
-               new KafkaValidator().validate(descriptorProperties);
+               new 
KafkaValidator(kafkaVersionPattern()).validate(descriptorProperties);
 
 Review comment:
   It's kind of strange that we validate the kafka version after the fact that 
we have already selected some kafka version (after all `this` instance has 
already a known Kafka version). This seems like either a duplicated logic or a 
kind of unit test that is perform during the runtime. Maybe it could be 
avoided/removed?

----------------------------------------------------------------
This is an automated message from the Apache Git Service.
To respond to the message, please log on GitHub and use the
URL above to go to the specific comment.
 
For queries about this service, please contact Infrastructure at:
[email protected]


> Make Kafka version definition more flexible for new Kafka table factory
> -----------------------------------------------------------------------
>
>                 Key: FLINK-10843
>                 URL: https://issues.apache.org/jira/browse/FLINK-10843
>             Project: Flink
>          Issue Type: Bug
>          Components: Kafka Connector, Table API &amp; SQL
>    Affects Versions: 1.7.0
>            Reporter: Timo Walther
>            Assignee: Timo Walther
>            Priority: Major
>              Labels: pull-request-available
>
> Currently, a user has to specify a specific version for a Kafka connector 
> like:
> {code}
> connector:
>   type: kafka
>   version: "0.11"     # required: valid connector versions are "0.8", "0.9", 
> "0.10", and "0.11"
>   topic: ...          # required: topic name from which the table is read
> {code}
> However, the new Kafka connector aims to be universal, thus, at least for 1.x 
> and 2.x versions which we should support those as parameters as well. 
> Currently, {{2.0}} is the only accepted string for the factory.



--
This message was sent by Atlassian JIRA
(v7.6.3#76005)

Reply via email to