forked from apache/kafka
-
Notifications
You must be signed in to change notification settings - Fork 0
/
verifiable_consumer.py
485 lines (405 loc) · 21.1 KB
/
verifiable_consumer.py
1
2
3
4
5
6
7
8
9
10
11
12
13
14
15
16
17
18
19
20
21
22
23
24
25
26
27
28
29
30
31
32
33
34
35
36
37
38
39
40
41
42
43
44
45
46
47
48
49
50
51
52
53
54
55
56
57
58
59
60
61
62
63
64
65
66
67
68
69
70
71
72
73
74
75
76
77
78
79
80
81
82
83
84
85
86
87
88
89
90
91
92
93
94
95
96
97
98
99
100
101
102
103
104
105
106
107
108
109
110
111
112
113
114
115
116
117
118
119
120
121
122
123
124
125
126
127
128
129
130
131
132
133
134
135
136
137
138
139
140
141
142
143
144
145
146
147
148
149
150
151
152
153
154
155
156
157
158
159
160
161
162
163
164
165
166
167
168
169
170
171
172
173
174
175
176
177
178
179
180
181
182
183
184
185
186
187
188
189
190
191
192
193
194
195
196
197
198
199
200
201
202
203
204
205
206
207
208
209
210
211
212
213
214
215
216
217
218
219
220
221
222
223
224
225
226
227
228
229
230
231
232
233
234
235
236
237
238
239
240
241
242
243
244
245
246
247
248
249
250
251
252
253
254
255
256
257
258
259
260
261
262
263
264
265
266
267
268
269
270
271
272
273
274
275
276
277
278
279
280
281
282
283
284
285
286
287
288
289
290
291
292
293
294
295
296
297
298
299
300
301
302
303
304
305
306
307
308
309
310
311
312
313
314
315
316
317
318
319
320
321
322
323
324
325
326
327
328
329
330
331
332
333
334
335
336
337
338
339
340
341
342
343
344
345
346
347
348
349
350
351
352
353
354
355
356
357
358
359
360
361
362
363
364
365
366
367
368
369
370
371
372
373
374
375
376
377
378
379
380
381
382
383
384
385
386
387
388
389
390
391
392
393
394
395
396
397
398
399
400
401
402
403
404
405
406
407
408
409
410
411
412
413
414
415
416
417
418
419
420
421
422
423
424
425
426
427
428
429
430
431
432
433
434
435
436
437
438
439
440
441
442
443
444
445
446
447
448
449
450
451
452
453
454
455
456
457
458
459
460
461
462
463
464
465
466
467
468
469
470
471
472
473
474
475
476
477
478
479
480
481
482
483
484
485
# Licensed to the Apache Software Foundation (ASF) under one or more
# contributor license agreements. See the NOTICE file distributed with
# this work for additional information regarding copyright ownership.
# The ASF licenses this file to You under the Apache License, Version 2.0
# (the "License"); you may not use this file except in compliance with
# the License. You may obtain a copy of the License at
#
# http://www.apache.org/licenses/LICENSE-2.0
#
# Unless required by applicable law or agreed to in writing, software
# distributed under the License is distributed on an "AS IS" BASIS,
# WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
# See the License for the specific language governing permissions and
# limitations under the License.
import json
import os
from ducktape.services.background_thread import BackgroundThreadService
from kafkatest.directory_layout.kafka_path import KafkaPathResolverMixin
from kafkatest.services.kafka import TopicPartition, consumer_group
from kafkatest.services.verifiable_client import VerifiableClientMixin
from kafkatest.version import DEV_BRANCH, V_2_3_0, V_2_3_1, V_3_7_0, V_0_10_0_0
class ConsumerState:
Started = 1
Dead = 2
Rebalancing = 3
Joined = 4
def _create_partition_from_dict(d):
topic = d["topic"]
partition = d["partition"]
return TopicPartition(topic, partition)
class ConsumerEventHandler(object):
def __init__(self, node, verify_offsets, idx):
self.node = node
self.idx = idx
self.state = ConsumerState.Dead
self.revoked_count = 0
self.assigned_count = 0
self.assignment = []
self.position = {}
self.committed = {}
self.total_consumed = 0
self.verify_offsets = verify_offsets
def handle_shutdown_complete(self, node=None, logger=None):
self.state = ConsumerState.Dead
self.assignment = []
self.position = {}
if node is not None and logger is not None:
logger.debug("Shut down %s" % node.account.hostname)
def handle_startup_complete(self, node, logger):
self.state = ConsumerState.Started
logger.debug("Started %s" % node.account.hostname)
def handle_offsets_committed(self, event, node, logger):
if event["success"]:
for offset_commit in event["offsets"]:
if offset_commit.get("error", "") != "":
logger.debug("%s: Offset commit failed for: %s" % (str(node.account), offset_commit))
continue
tp = _create_partition_from_dict(offset_commit)
offset = offset_commit["offset"]
assert tp in self.assignment, \
"Committed offsets for partition %s not assigned (current assignment: %s)" % \
(str(tp), str(self.assignment))
assert tp in self.position, "No previous position for %s: %s" % (str(tp), event)
assert self.position[tp] >= offset, \
"The committed offset %d was greater than the current position %d for partition %s" % \
(offset, self.position[tp], str(tp))
self.committed[tp] = offset
logger.debug("Offsets committed for %s" % node.account.hostname)
def handle_records_consumed(self, event, logger):
assert self.state == ConsumerState.Joined, \
"Consumed records should only be received when joined (current state: %s)" % str(self.state)
for record_batch in event["partitions"]:
tp = _create_partition_from_dict(record_batch)
min_offset = record_batch["minOffset"]
max_offset = record_batch["maxOffset"]
assert tp in self.assignment, \
"Consumed records for partition %s which is not assigned (current assignment: %s)" % \
(str(tp), str(self.assignment))
if tp not in self.position or self.position[tp] == min_offset:
self.position[tp] = max_offset + 1
else:
msg = "Consumed from an unexpected offset (%d, %d) for partition %s" % \
(self.position.get(tp), min_offset, str(tp))
if self.verify_offsets:
raise AssertionError(msg)
else:
if tp in self.position:
self.position[tp] = max_offset + 1
logger.warn(msg)
self.total_consumed += event["count"]
def handle_partitions_revoked(self, event, node, logger):
self.revoked_count += 1
self.state = ConsumerState.Rebalancing
self.position = {}
logger.debug("All partitions revoked from %s" % node.account.hostname)
def handle_partitions_assigned(self, event, node, logger):
self.assigned_count += 1
self.state = ConsumerState.Joined
assignment = []
for topic_partition in event["partitions"]:
tp = _create_partition_from_dict(topic_partition)
assignment.append(tp)
logger.debug("Partitions %s assigned to %s" % (assignment, node.account.hostname))
self.assignment = assignment
def handle_kill_process(self, clean_shutdown):
# if the shutdown was clean, then we expect the explicit
# shutdown event from the consumer
if not clean_shutdown:
self.handle_shutdown_complete()
def current_assignment(self):
return list(self.assignment)
def current_position(self, tp):
if tp in self.position:
return self.position[tp]
else:
return None
def last_commit(self, tp):
if tp in self.committed:
return self.committed[tp]
else:
return None
# This needs to be used for cooperative and consumer protocol
class IncrementalAssignmentConsumerEventHandler(ConsumerEventHandler):
def __init__(self, node, verify_offsets, idx):
super().__init__(node, verify_offsets, idx)
def handle_partitions_revoked(self, event, node, logger):
self.revoked_count += 1
self.state = ConsumerState.Rebalancing
self.position = {}
revoked = []
for topic_partition in event["partitions"]:
tp = _create_partition_from_dict(topic_partition)
if tp in self.assignment:
self.assignment.remove(tp)
revoked.append(tp)
else:
logger.warn("Could not remove topic partition %s from assignment as it was not previously assigned to %s" % (tp, node.account.hostname))
logger.debug("Partitions %s revoked from %s" % (revoked, node.account.hostname))
def handle_partitions_assigned(self, event, node, logger):
self.assigned_count += 1
self.state = ConsumerState.Joined
assignment = []
for topic_partition in event["partitions"]:
tp = _create_partition_from_dict(topic_partition)
assignment.append(tp)
logger.debug("Partitions %s assigned to %s" % (assignment, node.account.hostname))
self.assignment.extend(assignment)
class VerifiableConsumer(KafkaPathResolverMixin, VerifiableClientMixin, BackgroundThreadService):
"""This service wraps org.apache.kafka.tools.VerifiableConsumer for use in
system testing.
NOTE: this class should be treated as a PUBLIC API. Downstream users use
this service both directly and through class extension, so care must be
taken to ensure compatibility.
"""
PERSISTENT_ROOT = "/mnt/verifiable_consumer"
STDOUT_CAPTURE = os.path.join(PERSISTENT_ROOT, "verifiable_consumer.stdout")
STDERR_CAPTURE = os.path.join(PERSISTENT_ROOT, "verifiable_consumer.stderr")
LOG_DIR = os.path.join(PERSISTENT_ROOT, "logs")
LOG_FILE = os.path.join(LOG_DIR, "verifiable_consumer.log")
LOG4J_CONFIG = os.path.join(PERSISTENT_ROOT, "tools-log4j.properties")
CONFIG_FILE = os.path.join(PERSISTENT_ROOT, "verifiable_consumer.properties")
logs = {
"verifiable_consumer_stdout": {
"path": STDOUT_CAPTURE,
"collect_default": False},
"verifiable_consumer_stderr": {
"path": STDERR_CAPTURE,
"collect_default": False},
"verifiable_consumer_log": {
"path": LOG_FILE,
"collect_default": True}
}
def __init__(self, context, num_nodes, kafka, topic, group_id,
static_membership=False, max_messages=-1, session_timeout_sec=30, enable_autocommit=False,
assignment_strategy=None, group_protocol=None, group_remote_assignor=None,
version=DEV_BRANCH, stop_timeout_sec=30, log_level="INFO", jaas_override_variables=None,
on_record_consumed=None, reset_policy="earliest", verify_offsets=True):
"""
:param jaas_override_variables: A dict of variables to be used in the jaas.conf template file
"""
super(VerifiableConsumer, self).__init__(context, num_nodes)
self.log_level = log_level
self.kafka = kafka
self.topic = topic
self.group_protocol = group_protocol
self.group_remote_assignor = group_remote_assignor
self.group_id = group_id
self.reset_policy = reset_policy
self.static_membership = static_membership
self.max_messages = max_messages
self.session_timeout_sec = session_timeout_sec
self.enable_autocommit = enable_autocommit
self.assignment_strategy = assignment_strategy
self.group_protocol = group_protocol
self.group_remote_assignor = group_remote_assignor
self.prop_file = ""
self.stop_timeout_sec = stop_timeout_sec
self.on_record_consumed = on_record_consumed
self.verify_offsets = verify_offsets
self.event_handlers = {}
self.global_position = {}
self.global_committed = {}
self.jaas_override_variables = jaas_override_variables or {}
for node in self.nodes:
node.version = version
def java_class_name(self):
return "VerifiableConsumer"
def _worker(self, idx, node):
with self.lock:
if node not in self.event_handlers:
if self._isEager():
self.event_handlers[node] = ConsumerEventHandler(node, self.verify_offsets, idx)
else:
self.event_handlers[node] = IncrementalAssignmentConsumerEventHandler(node, self.verify_offsets, idx)
handler = self.event_handlers[node]
node.account.ssh("mkdir -p %s" % VerifiableConsumer.PERSISTENT_ROOT, allow_fail=False)
# Create and upload log properties
log_config = self.render('tools_log4j.properties', log_file=VerifiableConsumer.LOG_FILE)
node.account.create_file(VerifiableConsumer.LOG4J_CONFIG, log_config)
# Create and upload config file
self.security_config = self.kafka.security_config.client_config(self.prop_file, node,
self.jaas_override_variables)
self.security_config.setup_node(node)
self.prop_file += str(self.security_config)
self.logger.info("verifiable_consumer.properties:")
self.logger.info(self.prop_file)
node.account.create_file(VerifiableConsumer.CONFIG_FILE, self.prop_file)
self.security_config.setup_node(node)
# apply group.instance.id to the node for static membership validation
node.group_instance_id = None
if self.static_membership:
assert node.version >= V_2_3_0, \
"Version %s does not support static membership (must be 2.3 or higher)" % str(node.version)
node.group_instance_id = self.group_id + "-instance-" + str(idx)
if self.assignment_strategy:
assert node.version >= V_0_10_0_0, \
"Version %s does not setting an assignment strategy (must be 0.10.0 or higher)" % str(node.version)
cmd = self.start_cmd(node)
self.logger.debug("VerifiableConsumer %d command: %s" % (idx, cmd))
for line in node.account.ssh_capture(cmd):
event = self.try_parse_json(node, line.strip())
if event is not None:
with self.lock:
name = event["name"]
if name == "shutdown_complete":
handler.handle_shutdown_complete(node, self.logger)
elif name == "startup_complete":
handler.handle_startup_complete(node, self.logger)
elif name == "offsets_committed":
handler.handle_offsets_committed(event, node, self.logger)
self._update_global_committed(event)
elif name == "records_consumed":
handler.handle_records_consumed(event, self.logger)
self._update_global_position(event, node)
elif name == "record_data" and self.on_record_consumed:
self.on_record_consumed(event, node)
elif name == "partitions_revoked":
handler.handle_partitions_revoked(event, node, self.logger)
elif name == "partitions_assigned":
handler.handle_partitions_assigned(event, node, self.logger)
else:
self.logger.debug("%s: ignoring unknown event: %s" % (str(node.account), event))
def _isEager(self):
return self.group_protocol == consumer_group.classic_group_protocol and self.assignment_strategy != "org.apache.kafka.clients.consumer.CooperativeStickyAssignor"
def _update_global_position(self, consumed_event, node):
for consumed_partition in consumed_event["partitions"]:
tp = _create_partition_from_dict(consumed_partition)
if tp in self.global_committed:
# verify that the position never gets behind the current commit.
if self.global_committed[tp] > consumed_partition["minOffset"]:
msg = "Consumed position %d is behind the current committed offset %d for partition %s" % \
(consumed_partition["minOffset"], self.global_committed[tp], str(tp))
if self.verify_offsets:
raise AssertionError(msg)
else:
self.logger.warn(msg)
# the consumer cannot generally guarantee that the position increases monotonically
# without gaps in the face of hard failures, so we only log a warning when this happens
if tp in self.global_position and self.global_position[tp] != consumed_partition["minOffset"]:
self.logger.warn("%s: Expected next consumed offset of %d for partition %s, but instead saw %d" %
(str(node.account), self.global_position[tp], str(tp), consumed_partition["minOffset"]))
self.global_position[tp] = consumed_partition["maxOffset"] + 1
def _update_global_committed(self, commit_event):
if commit_event["success"]:
for offset_commit in commit_event["offsets"]:
tp = _create_partition_from_dict(offset_commit)
offset = offset_commit["offset"]
assert self.global_position[tp] >= offset, \
"Committed offset %d for partition %s is ahead of the current position %d" % \
(offset, str(tp), self.global_position[tp])
self.global_committed[tp] = offset
def start_cmd(self, node):
cmd = ""
cmd += "export LOG_DIR=%s;" % VerifiableConsumer.LOG_DIR
cmd += " export KAFKA_OPTS=%s;" % self.security_config.kafka_opts
cmd += " export KAFKA_LOG4J_OPTS=\"-Dlog4j.configuration=file:%s\"; " % VerifiableConsumer.LOG4J_CONFIG
cmd += self.impl.exec_cmd(node)
if self.on_record_consumed:
cmd += " --verbose"
if node.group_instance_id:
cmd += " --group-instance-id %s" % node.group_instance_id
elif node.version == V_2_3_0 or node.version == V_2_3_1:
# In 2.3, --group-instance-id was required, but would be left empty
# if `None` is passed as the argument value
cmd += " --group-instance-id None"
# 3.7.0 includes support for KIP-848 which introduced a new implementation of the consumer group protocol.
# The two implementations use slightly different configuration, hence these arguments are conditional.
#
# See the Java class/method VerifiableConsumer.createFromArgs() for how the command line arguments are
# parsed and used as configuration in the runner.
if node.version >= V_3_7_0 and self.is_consumer_group_protocol_enabled():
cmd += " --group-protocol %s" % self.group_protocol
if self.group_remote_assignor:
cmd += " --group-remote-assignor %s" % self.group_remote_assignor
else:
# Either we're an older consumer version or we're using the old consumer group protocol.
if self.assignment_strategy:
cmd += " --assignment-strategy %s" % self.assignment_strategy
if self.enable_autocommit:
cmd += " --enable-autocommit "
cmd += " --reset-policy %s --group-id %s --topic %s --broker-list %s --session-timeout %s" % \
(self.reset_policy, self.group_id, self.topic,
self.kafka.bootstrap_servers(self.security_config.security_protocol),
self.session_timeout_sec*1000)
if self.max_messages > 0:
cmd += " --max-messages %s" % str(self.max_messages)
cmd += " --consumer.config %s" % VerifiableConsumer.CONFIG_FILE
cmd += " 2>> %s | tee -a %s &" % (VerifiableConsumer.STDOUT_CAPTURE, VerifiableConsumer.STDOUT_CAPTURE)
return cmd
def pids(self, node):
return self.impl.pids(node)
def try_parse_json(self, node, string):
"""Try to parse a string as json. Return None if not parseable."""
try:
return json.loads(string)
except ValueError:
self.logger.debug("%s: Could not parse as json: %s" % (str(node.account), str(string)))
return None
def stop_all(self):
for node in self.nodes:
self.stop_node(node)
def kill_node(self, node, clean_shutdown=True, allow_fail=False):
sig = self.impl.kill_signal(clean_shutdown)
for pid in self.pids(node):
node.account.signal(pid, sig, allow_fail)
with self.lock:
self.event_handlers[node].handle_kill_process(clean_shutdown)
def stop_node(self, node, clean_shutdown=True):
self.kill_node(node, clean_shutdown=clean_shutdown)
stopped = self.wait_node(node, timeout_sec=self.stop_timeout_sec)
assert stopped, "Node %s: did not stop within the specified timeout of %s seconds" % \
(str(node.account), str(self.stop_timeout_sec))
def clean_node(self, node):
self.kill_node(node, clean_shutdown=False)
node.account.ssh("rm -rf " + self.PERSISTENT_ROOT, allow_fail=False)
self.security_config.clean_node(node)
def current_assignment(self):
with self.lock:
return { handler.node: handler.current_assignment() for handler in self.event_handlers.values() }
def current_position(self, tp):
with self.lock:
if tp in self.global_position:
return self.global_position[tp]
else:
return None
def owner(self, tp):
with self.lock:
for handler in self.event_handlers.values():
if tp in handler.current_assignment():
return handler.node
return None
def last_commit(self, tp):
with self.lock:
if tp in self.global_committed:
return self.global_committed[tp]
else:
return None
def total_consumed(self):
with self.lock:
return sum(handler.total_consumed for handler in self.event_handlers.values())
def num_rebalances(self):
with self.lock:
return max(handler.assigned_count for handler in self.event_handlers.values())
def num_revokes_for_alive(self, keep_alive=1):
with self.lock:
return max(handler.revoked_count for handler in self.event_handlers.values()
if handler.idx <= keep_alive)
def joined_nodes(self):
with self.lock:
return [handler.node for handler in self.event_handlers.values()
if handler.state == ConsumerState.Joined]
def rebalancing_nodes(self):
with self.lock:
return [handler.node for handler in self.event_handlers.values()
if handler.state == ConsumerState.Rebalancing]
def dead_nodes(self):
with self.lock:
return [handler.node for handler in self.event_handlers.values()
if handler.state == ConsumerState.Dead]
def alive_nodes(self):
with self.lock:
return [handler.node for handler in self.event_handlers.values()
if handler.state != ConsumerState.Dead]
def is_consumer_group_protocol_enabled(self):
return self.group_protocol and self.group_protocol.upper() == "CONSUMER"