-
Notifications
You must be signed in to change notification settings - Fork 29k
[SPARK-16776][STREAMING] Replace deprecated API in KafkaTestUtils for 0.10.0. #14416
New issue
Have a question about this project? Sign up for a free GitHub account to open an issue and contact its maintainers and the community.
By clicking “Sign up for GitHub”, you agree to our terms of service and privacy statement. We’ll occasionally send you account related emails.
Already on GitHub? Sign in to your account
Conversation
|
Could you please take a look, @koeninger and @holdenk ? |
|
Please let me cc @srowen as well because it is anyway about building. |
| producer.send(messages.map { new KeyedMessage[String, String](topic, _ ) }: _*) | ||
| producer = new KafkaProducer[String, String](producerConfiguration) | ||
| val records = messages.map { new ProducerRecord[String, String](topic, _) } | ||
| records.map(producer.send) |
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
super minor: could use foreach instead of map since we don't care about the return value.
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
Oh, thank you.
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
This type of thing is even essential for correctness in some cases, like https://issues.apache.org/jira/browse/SPARK-16664
|
This looks reasonable pending tests and verifying no more kafka deprecation warnings in the build logs for this file :) |
|
Test build #63033 has finished for PR 14416 at commit
|
|
Test build #63038 has finished for PR 14416 at commit
|
| producer.send(messages.map { new KeyedMessage[String, String](topic, _ ) }: _*) | ||
| producer = new KafkaProducer[String, String](producerConfiguration) | ||
| val records = messages.map { new ProducerRecord[String, String](topic, _) } | ||
| records.foreach(producer.send) |
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
I think the point here was that you don't need to map at all; messages.foreach should suffice.
|
LGTM, minor comments about map notwithstanding. |
|
Test build #63044 has finished for PR 14416 at commit
|
|
Merged to master/2.0 |
… 0.10.0. ## What changes were proposed in this pull request? This PR replaces the old Kafka API to 0.10.0 ones in `KafkaTestUtils`. The change include: - `Producer` to `KafkaProducer` - Change configurations to equalvant ones. (I referred [here](http://kafka.apache.org/documentation.html#producerconfigs) for 0.10.0 and [here](http://kafka.apache.org/082/documentation.html#producerconfigs ) for old, 0.8.2). This PR will remove the build warning as below: ```scala [WARNING] .../spark/external/kafka-0-10/src/main/scala/org/apache/spark/streaming/kafka010/KafkaTestUtils.scala:71: class Producer in package producer is deprecated: This class has been deprecated and will be removed in a future release. Please use org.apache.kafka.clients.producer.KafkaProducer instead. [WARNING] private var producer: Producer[String, String] = _ [WARNING] ^ [WARNING] .../spark/external/kafka-0-10/src/main/scala/org/apache/spark/streaming/kafka010/KafkaTestUtils.scala:181: class Producer in package producer is deprecated: This class has been deprecated and will be removed in a future release. Please use org.apache.kafka.clients.producer.KafkaProducer instead. [WARNING] producer = new Producer[String, String](new ProducerConfig(producerConfiguration)) [WARNING] ^ [WARNING] .../spark/streaming/kafka010/KafkaTestUtils.scala:181: class ProducerConfig in package producer is deprecated: This class has been deprecated and will be removed in a future release. Please use org.apache.kafka.clients.producer.ProducerConfig instead. [WARNING] producer = new Producer[String, String](new ProducerConfig(producerConfiguration)) [WARNING] ^ [WARNING] .../spark/external/kafka-0-10/src/main/scala/org/apache/spark/streaming/kafka010/KafkaTestUtils.scala:182: class KeyedMessage in package producer is deprecated: This class has been deprecated and will be removed in a future release. Please use org.apache.kafka.clients.producer.ProducerRecord instead. [WARNING] producer.send(messages.map { new KeyedMessage[String, String](topic, _ ) }: _*) [WARNING] ^ [WARNING] four warnings found [WARNING] warning: [options] bootstrap class path not set in conjunction with -source 1.7 [WARNING] 1 warning ``` ## How was this patch tested? Existing tests that use `KafkaTestUtils` should cover this. Author: hyukjinkwon <[email protected]> Closes #14416 from HyukjinKwon/SPARK-16776. (cherry picked from commit f93ad4f) Signed-off-by: Sean Owen <[email protected]>
What changes were proposed in this pull request?
This PR replaces the old Kafka API to 0.10.0 ones in
KafkaTestUtils.The change include:
ProducertoKafkaProducerThis PR will remove the build warning as below:
How was this patch tested?
Existing tests that use
KafkaTestUtilsshould cover this.