mirror of https://github.com/apache/kafka.git
KAFKA-14753: Improve kafka producer example (#13354)
Reviewers: Guozhang Wang <wangguoz@gmail.com>
This commit is contained in:
parent
b19ae7857b
commit
4527e54647
|
@ -27,7 +27,13 @@ import org.apache.kafka.common.serialization.StringSerializer;
|
||||||
import java.util.Properties;
|
import java.util.Properties;
|
||||||
import java.util.concurrent.CountDownLatch;
|
import java.util.concurrent.CountDownLatch;
|
||||||
import java.util.concurrent.ExecutionException;
|
import java.util.concurrent.ExecutionException;
|
||||||
|
import java.util.concurrent.Future;
|
||||||
|
|
||||||
|
/**
|
||||||
|
* Demo producer that demonstrate two modes of KafkaProducer.
|
||||||
|
* If the user uses the Async mode: The messages will be printed to stdout upon successful completion
|
||||||
|
* If the user uses the sync mode (isAsync = false): Each send loop will block until completion.
|
||||||
|
*/
|
||||||
public class Producer extends Thread {
|
public class Producer extends Thread {
|
||||||
private final KafkaProducer<Integer, String> producer;
|
private final KafkaProducer<Integer, String> producer;
|
||||||
private final String topic;
|
private final String topic;
|
||||||
|
@ -54,8 +60,8 @@ public class Producer extends Thread {
|
||||||
props.put(ProducerConfig.TRANSACTIONAL_ID_CONFIG, transactionalId);
|
props.put(ProducerConfig.TRANSACTIONAL_ID_CONFIG, transactionalId);
|
||||||
}
|
}
|
||||||
props.put(ProducerConfig.ENABLE_IDEMPOTENCE_CONFIG, enableIdempotency);
|
props.put(ProducerConfig.ENABLE_IDEMPOTENCE_CONFIG, enableIdempotency);
|
||||||
|
|
||||||
producer = new KafkaProducer<>(props);
|
producer = new KafkaProducer<>(props);
|
||||||
|
|
||||||
this.topic = topic;
|
this.topic = topic;
|
||||||
this.isAsync = isAsync;
|
this.isAsync = isAsync;
|
||||||
this.numRecords = numRecords;
|
this.numRecords = numRecords;
|
||||||
|
@ -70,28 +76,45 @@ public class Producer extends Thread {
|
||||||
public void run() {
|
public void run() {
|
||||||
int messageKey = 0;
|
int messageKey = 0;
|
||||||
int recordsSent = 0;
|
int recordsSent = 0;
|
||||||
while (recordsSent < numRecords) {
|
try {
|
||||||
String messageStr = "Message_" + messageKey;
|
while (recordsSent < numRecords) {
|
||||||
long startTime = System.currentTimeMillis();
|
final long currentTimeMs = System.currentTimeMillis();
|
||||||
if (isAsync) { // Send asynchronously
|
produceOnce(messageKey, recordsSent, currentTimeMs);
|
||||||
producer.send(new ProducerRecord<>(topic,
|
messageKey += 2;
|
||||||
messageKey,
|
recordsSent += 1;
|
||||||
messageStr), new DemoCallBack(startTime, messageKey, messageStr));
|
|
||||||
} else { // Send synchronously
|
|
||||||
try {
|
|
||||||
producer.send(new ProducerRecord<>(topic,
|
|
||||||
messageKey,
|
|
||||||
messageStr)).get();
|
|
||||||
System.out.println("Sent message: (" + messageKey + ", " + messageStr + ")");
|
|
||||||
} catch (InterruptedException | ExecutionException e) {
|
|
||||||
e.printStackTrace();
|
|
||||||
}
|
|
||||||
}
|
}
|
||||||
messageKey += 2;
|
} catch (Exception e) {
|
||||||
recordsSent += 1;
|
System.out.println("Producer encountered exception:" + e);
|
||||||
|
} finally {
|
||||||
|
System.out.println("Producer sent " + numRecords + " records successfully");
|
||||||
|
this.producer.close();
|
||||||
|
latch.countDown();
|
||||||
}
|
}
|
||||||
System.out.println("Producer sent " + numRecords + " records successfully");
|
}
|
||||||
latch.countDown();
|
|
||||||
|
private void produceOnce(final int messageKey, final int recordsSent, final long currentTimeMs) throws ExecutionException, InterruptedException {
|
||||||
|
String messageStr = "Message_" + messageKey;
|
||||||
|
|
||||||
|
if (isAsync) { // Send asynchronously
|
||||||
|
sendAsync(messageKey, messageStr, currentTimeMs);
|
||||||
|
return;
|
||||||
|
}
|
||||||
|
Future<RecordMetadata> future = send(messageKey, messageStr);
|
||||||
|
future.get();
|
||||||
|
System.out.println("Sent message: (" + messageKey + ", " + messageStr + ")");
|
||||||
|
}
|
||||||
|
|
||||||
|
private void sendAsync(final int messageKey, final String messageStr, final long currentTimeMs) {
|
||||||
|
this.producer.send(new ProducerRecord<>(topic,
|
||||||
|
messageKey,
|
||||||
|
messageStr),
|
||||||
|
new DemoCallBack(currentTimeMs, messageKey, messageStr));
|
||||||
|
}
|
||||||
|
|
||||||
|
private Future<RecordMetadata> send(final int messageKey, final String messageStr) {
|
||||||
|
return producer.send(new ProducerRecord<>(topic,
|
||||||
|
messageKey,
|
||||||
|
messageStr));
|
||||||
}
|
}
|
||||||
}
|
}
|
||||||
|
|
||||||
|
|
Loading…
Reference in New Issue