mirror of https://github.com/apache/kafka.git
KAFKA-3597; Query ConsoleConsumer and VerifiableProducer if they shutdown cleanly
Even if a test calls stop() on console_consumer or verifiable_producer, it is still possible that producer/consumer will not shutdown cleanly, and will be killed forcefully after a timeout. It will be useful for some tests to know whether a clean shutdown happened or not. This PR adds methods to console_consumer and verifiable_producer to query whether clean shutdown happened or not. hachikuji and/or granders Please review. Author: Anna Povzner <anna@confluent.io> Reviewers: Jason Gustafson, Geoff Anderson, Gwen Shapira Closes #1278 from apovzner/kafka-3597
This commit is contained in:
parent
eb50d2f6ca
commit
e29eac4bbb
|
@ -103,6 +103,10 @@ object ConsoleConsumer extends Logging {
|
||||||
consumer.stop()
|
consumer.stop()
|
||||||
|
|
||||||
shutdownLatch.await()
|
shutdownLatch.await()
|
||||||
|
|
||||||
|
if (conf.enableSystestEventsLogging) {
|
||||||
|
System.out.println("shutdown_complete")
|
||||||
|
}
|
||||||
}
|
}
|
||||||
})
|
})
|
||||||
}
|
}
|
||||||
|
@ -253,6 +257,9 @@ object ConsoleConsumer extends Logging {
|
||||||
.withRequiredArg
|
.withRequiredArg
|
||||||
.describedAs("deserializer for values")
|
.describedAs("deserializer for values")
|
||||||
.ofType(classOf[String])
|
.ofType(classOf[String])
|
||||||
|
val enableSystestEventsLoggingOpt = parser.accepts("enable-systest-events",
|
||||||
|
"Log lifecycle events of the consumer in addition to logging consumed " +
|
||||||
|
"messages. (This is specific for system tests.)")
|
||||||
|
|
||||||
if (args.length == 0)
|
if (args.length == 0)
|
||||||
CommandLineUtils.printUsageAndDie(parser, "The console consumer is a tool that reads data from Kafka and outputs it to standard output.")
|
CommandLineUtils.printUsageAndDie(parser, "The console consumer is a tool that reads data from Kafka and outputs it to standard output.")
|
||||||
|
@ -260,6 +267,7 @@ object ConsoleConsumer extends Logging {
|
||||||
var groupIdPassed = true
|
var groupIdPassed = true
|
||||||
val options: OptionSet = tryParse(parser, args)
|
val options: OptionSet = tryParse(parser, args)
|
||||||
val useNewConsumer = options.has(useNewConsumerOpt)
|
val useNewConsumer = options.has(useNewConsumerOpt)
|
||||||
|
val enableSystestEventsLogging = options.has(enableSystestEventsLoggingOpt)
|
||||||
|
|
||||||
// If using old consumer, exactly one of whitelist/blacklist/topic is required.
|
// If using old consumer, exactly one of whitelist/blacklist/topic is required.
|
||||||
// If using new consumer, topic must be specified.
|
// If using new consumer, topic must be specified.
|
||||||
|
|
|
@ -123,6 +123,7 @@ class ConsoleConsumer(JmxMixin, BackgroundThreadService):
|
||||||
self.from_beginning = from_beginning
|
self.from_beginning = from_beginning
|
||||||
self.message_validator = message_validator
|
self.message_validator = message_validator
|
||||||
self.messages_consumed = {idx: [] for idx in range(1, num_nodes + 1)}
|
self.messages_consumed = {idx: [] for idx in range(1, num_nodes + 1)}
|
||||||
|
self.clean_shutdown_nodes = set()
|
||||||
self.client_id = client_id
|
self.client_id = client_id
|
||||||
self.print_key = print_key
|
self.print_key = print_key
|
||||||
self.log_level = "TRACE"
|
self.log_level = "TRACE"
|
||||||
|
@ -185,6 +186,7 @@ class ConsoleConsumer(JmxMixin, BackgroundThreadService):
|
||||||
if node.version > LATEST_0_9:
|
if node.version > LATEST_0_9:
|
||||||
cmd+=" --formatter kafka.tools.LoggingMessageFormatter"
|
cmd+=" --formatter kafka.tools.LoggingMessageFormatter"
|
||||||
|
|
||||||
|
cmd += " --enable-systest-events"
|
||||||
cmd += " 2>> %(stderr)s | tee -a %(stdout)s &" % args
|
cmd += " 2>> %(stderr)s | tee -a %(stdout)s &" % args
|
||||||
return cmd
|
return cmd
|
||||||
|
|
||||||
|
@ -226,10 +228,15 @@ class ConsoleConsumer(JmxMixin, BackgroundThreadService):
|
||||||
|
|
||||||
for line in itertools.chain([first_line], consumer_output):
|
for line in itertools.chain([first_line], consumer_output):
|
||||||
msg = line.strip()
|
msg = line.strip()
|
||||||
if self.message_validator is not None:
|
if msg == "shutdown_complete":
|
||||||
msg = self.message_validator(msg)
|
if node in self.clean_shutdown_nodes:
|
||||||
if msg is not None:
|
raise Exception("Unexpected shutdown event from consumer, already shutdown. Consumer index: %d" % idx)
|
||||||
self.messages_consumed[idx].append(msg)
|
self.clean_shutdown_nodes.add(node)
|
||||||
|
else:
|
||||||
|
if self.message_validator is not None:
|
||||||
|
msg = self.message_validator(msg)
|
||||||
|
if msg is not None:
|
||||||
|
self.messages_consumed[idx].append(msg)
|
||||||
|
|
||||||
self.read_jmx_output(idx, node)
|
self.read_jmx_output(idx, node)
|
||||||
|
|
||||||
|
|
|
@ -71,6 +71,7 @@ class VerifiableProducer(BackgroundThreadService):
|
||||||
self.acked_values = []
|
self.acked_values = []
|
||||||
self.not_acked_values = []
|
self.not_acked_values = []
|
||||||
self.produced_count = {}
|
self.produced_count = {}
|
||||||
|
self.clean_shutdown_nodes = set()
|
||||||
self.acks = acks
|
self.acks = acks
|
||||||
|
|
||||||
|
|
||||||
|
@ -139,6 +140,11 @@ class VerifiableProducer(BackgroundThreadService):
|
||||||
last_produced_time = t
|
last_produced_time = t
|
||||||
prev_msg = data
|
prev_msg = data
|
||||||
|
|
||||||
|
elif data["name"] == "shutdown_complete":
|
||||||
|
if node in self.clean_shutdown_nodes:
|
||||||
|
raise Exception("Unexpected shutdown event from producer, already shutdown. Producer index: %d" % idx)
|
||||||
|
self.clean_shutdown_nodes.add(node)
|
||||||
|
|
||||||
def start_cmd(self, node, idx):
|
def start_cmd(self, node, idx):
|
||||||
|
|
||||||
cmd = ""
|
cmd = ""
|
||||||
|
|
|
@ -247,6 +247,14 @@ public class VerifiableProducer {
|
||||||
/** Close the producer to flush any remaining messages. */
|
/** Close the producer to flush any remaining messages. */
|
||||||
public void close() {
|
public void close() {
|
||||||
producer.close();
|
producer.close();
|
||||||
|
System.out.println(shutdownString());
|
||||||
|
}
|
||||||
|
|
||||||
|
String shutdownString() {
|
||||||
|
Map<String, Object> data = new HashMap<>();
|
||||||
|
data.put("class", this.getClass().toString());
|
||||||
|
data.put("name", "shutdown_complete");
|
||||||
|
return toJsonString(data);
|
||||||
}
|
}
|
||||||
|
|
||||||
/**
|
/**
|
||||||
|
|
Loading…
Reference in New Issue