flink-issues mailing list archives

Site index · List index
Message view « Date » · « Thread »
Top « Date » · « Thread »
From "ASF GitHub Bot (JIRA)" <j...@apache.org>
Subject [jira] [Commented] (FLINK-5368) Let Kafka consumer show something when it fails to read one topic out of topic list
Date Thu, 05 Jan 2017 10:52:58 GMT

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

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

Github user tzulitai commented on a diff in the pull request:

    https://github.com/apache/flink/pull/3036#discussion_r94748197
  
    --- Diff: flink-connectors/flink-connector-kafka-0.9/src/main/java/org/apache/flink/streaming/connectors/kafka/FlinkKafkaConsumer09.java
---
    @@ -208,13 +208,12 @@ public FlinkKafkaConsumer09(List<String> topics, KeyedDeserializationSchema<T>
d
     				if (partitionsForTopic != null) {
     					partitions.addAll(convertToFlinkKafkaTopicPartition(partitionsForTopic));
     				}
    +				else{
    +					LOG.info("Unable to retrieve any partitions for the requested topic: {}", topic);
    +				}
     			}
     		}
     
    -		if (partitions.isEmpty()) {
    -			throw new RuntimeException("Unable to retrieve any partitions for the requested topics
" + topics);
    -		}
    --- End diff --
    
    I think we still want to fail if there are no partitions at all for ALL topics, correct?
    This change here will remove that behaviour.


> Let Kafka consumer show something when it fails to read one topic out of topic list
> -----------------------------------------------------------------------------------
>
>                 Key: FLINK-5368
>                 URL: https://issues.apache.org/jira/browse/FLINK-5368
>             Project: Flink
>          Issue Type: Improvement
>          Components: Kafka Connector
>            Reporter: Sendoh
>            Assignee: Sendoh
>            Priority: Critical
>
> As a developer when reading data from many topics, I want Kafka consumer to show something
if any topic is not available. The motivation is we read many topics as list at one time,
and sometimes we fail to recognize that one or two topics' names have been changed or deprecated,
and Flink Kafka connector doesn't show the error.
> My proposed change would be either to throw RuntimeException or to use LOG.error(topic
+ "doesn't have any partition") if partitionsForTopic is null at this function. 
> https://github.com/apache/flink/blob/master/flink-connectors/flink-connector-kafka-0.9/src/main/java/org/apache/flink/streaming/connectors/kafka/FlinkKafkaConsumer09.java#L208
> Any suggestion is welcome.



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

Mime
View raw message