From 2c6d7db00dbab4842735d42f83a80087f95d4c54 Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Mon, 11 Jan 2016 16:16:55 -0800 Subject: [PATCH 01/70] basic implementation of joingrouprequest and tests --- pykafka/protocol.py | 117 +++++++++++++++++++++++++++++++++ tests/pykafka/test_protocol.py | 20 ++++++ 2 files changed, 137 insertions(+) diff --git a/pykafka/protocol.py b/pykafka/protocol.py index ad46035dd..bec45d2f4 100644 --- a/pykafka/protocol.py +++ b/pykafka/protocol.py @@ -1188,3 +1188,120 @@ def __init__(self, buff): partition[2], partition[3]) self.topics[topic_name][partition[0]] = pres + + +### +# Group Membership API +### +class ConsumerGroupProtocolMetadata(object): + """ + Protocol specification:: + + ProtocolMetadata => Version Subscription UserData + Version => int16 + Subscription => [Topic] + Topic => string + UserData => bytes + """ + def __init__(self): + self.version = 1 + self.topic_names = [b"dummytopic"] + self.user_data = b"testuserdata" + + def __len__(self): + # version + len(topic names) + size = 2 + 4 + for topic_name in self.topic_names: + # len(topic_name) + topic_name + size += 2 + len(topic_name) + # len(user data) + user data + size += 4 + len(self.user_data) + return size + + def get_bytes(self): + output = bytearray(len(self)) + offset = 0 + fmt = '!hi' + struct.pack_into(fmt, output, offset, self.version, len(self.topic_names)) + offset += struct.calcsize(fmt) + for topic_name in self.topic_names: + fmt = '!h%ds' % len(topic_name) + struct.pack_into(fmt, output, offset, len(topic_name), topic_name) + offset += struct.calcsize(fmt) + fmt = '!i%ds' % len(self.user_data) + struct.pack_into(fmt, output, offset, len(self.user_data), self.user_data) + offset += struct.calcsize(fmt) + return output + + +GroupMembershipProtocol = namedtuple( + 'GroupMembershipProtocol', ['protocol_type', 'protocol_name', 'metadata'] +) + + +ConsumerGroupProtocol = GroupMembershipProtocol("consumer", "dummyassignmentstrategy", + ConsumerGroupProtocolMetadata()) + + +class JoinGroupRequest(Request): + """A group join request + + Specification:: + + JoinGroupRequest => GroupId SessionTimeout MemberId ProtocolType GroupProtocols + GroupId => string + SessionTimeout => int32 + MemberId => string + ProtocolType => string + GroupProtocols => [ProtocolName ProtocolMetadata] + ProtocolName => string + ProtocolMetadata => bytes + """ + def __init__(self, member_id, session_timeout=30000): + """Create a new group join request""" + self.protocol = ConsumerGroupProtocol + self.group_id = "dummygroup" + self.session_timeout = session_timeout + self.member_id = member_id + self.protocol_type = self.protocol.protocol_type + self.group_protocols = [(self.protocol.protocol_name, + self.protocol.metadata.get_bytes())] + + def __len__(self): + """Length of the serialized message, in bytes""" + # Header + group id + session timeout + size = self.HEADER_LEN + 2 + len(self.group_id) + 4 + # + member id + protocol type + len(group protocols) + size += 2 + len(self.member_id) + 2 + len(self.protocol_type) + 4 + # metadata tuples + for name, metadata in self.group_protocols: + size += 2 + len(name) + 4 + len(metadata) + return size + + @property + def API_KEY(self): + """API_KEY for this request, from the Kafka docs""" + return 11 + + def get_bytes(self): + """Serialize the message + + :returns: Serialized message + :rtype: :class:`bytearray` + """ + output = bytearray(len(self)) + self._write_header(output, api_version=1) + offset = self.HEADER_LEN + fmt = '!h%dsih%dsh%dsi' % (len(self.group_id), len(self.member_id), + len(self.protocol_type)) + struct.pack_into(fmt, output, offset, len(self.group_id), self.group_id, + self.session_timeout, len(self.member_id), self.member_id, + len(self.protocol_type), self.protocol_type, + len(self.group_protocols)) + offset += struct.calcsize(fmt) + for protocol_name, protocol_metadata in self.group_protocols: + fmt = '!h%dsi%ds' % (len(protocol_name), len(protocol_metadata)) + struct.pack_into(fmt, output, offset, len(protocol_name), protocol_name, + len(protocol_metadata), bytes(protocol_metadata)) + offset += struct.calcsize(fmt) + return output diff --git a/tests/pykafka/test_protocol.py b/tests/pykafka/test_protocol.py index 7fc120761..af78bc311 100644 --- a/tests/pykafka/test_protocol.py +++ b/tests/pykafka/test_protocol.py @@ -256,5 +256,25 @@ def test_offset_fetch_response(self): self.assertEqual(response.topics[b'emmett.dummy'][0].offset, 1) +class TestGroupMembershipAPI(unittest2.TestCase): + maxDiff = None + + def test_consumer_group_protocol_metadata(self): + meta = protocol.ConsumerGroupProtocolMetadata() + msg = meta.get_bytes() + self.assertEqual( + msg, + bytearray(b'\x00\x01\x00\x00\x00\x01\x00\ndummytopic\x00\x00\x00\x0ctestuserdata') + ) + + def test_join_group_request(self): + req = protocol.JoinGroupRequest(b'testmember') + msg = req.get_bytes() + self.assertEqual( + msg, + bytearray(b'\x00\x00\x00z\x00\x0b\x00\x01\x00\x00\x00\x00\x00\x07pykafka\x00\ndummygroup\x00\x00u0\x00\ntestmember\x00\x08consumer\x00\x00\x00\x01\x00\x17dummyassignmentstrategy\x00\x00\x00"\x00\x01\x00\x00\x00\x01\x00\ndummytopic\x00\x00\x00\x0ctestuserdata') + ) + + if __name__ == '__main__': unittest2.main() From fc1ce362184d6c526fbc6808c7fe11074d934050 Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Wed, 13 Jan 2016 16:26:36 -0800 Subject: [PATCH 02/70] implement syncgrouprequest and joingroupresponse --- pykafka/protocol.py | 148 +++++++++++++++++++++++++++++++++++++++++++- 1 file changed, 146 insertions(+), 2 deletions(-) diff --git a/pykafka/protocol.py b/pykafka/protocol.py index bec45d2f4..5ff6f20d5 100644 --- a/pykafka/protocol.py +++ b/pykafka/protocol.py @@ -1265,7 +1265,7 @@ def __init__(self, member_id, session_timeout=30000): self.member_id = member_id self.protocol_type = self.protocol.protocol_type self.group_protocols = [(self.protocol.protocol_name, - self.protocol.metadata.get_bytes())] + bytes(self.protocol.metadata.get_bytes()))] def __len__(self): """Length of the serialized message, in bytes""" @@ -1302,6 +1302,150 @@ def get_bytes(self): for protocol_name, protocol_metadata in self.group_protocols: fmt = '!h%dsi%ds' % (len(protocol_name), len(protocol_metadata)) struct.pack_into(fmt, output, offset, len(protocol_name), protocol_name, - len(protocol_metadata), bytes(protocol_metadata)) + len(protocol_metadata), protocol_metadata) + offset += struct.calcsize(fmt) + return output + + +class JoinGroupResponse(Response): + """A group join response + + Specification:: + + JoinGroupResponse => ErrorCode GenerationId GroupProtocol LeaderId MemberId Members + ErrorCode => int16 + GenerationId => int32 + GroupProtocol => string + LeaderId => string + MemberId => string + Members => [MemberId MemberMetadata] + MemberId => string + MemberMetadata => bytes + """ + def __init__(self, buff): + """Deserialize into a new Response + + :param buff: Serialized message + :type buff: :class:`bytearray` + """ + fmt = 'hiSSS[SS ]' + response = struct_helpers.unpack_from(fmt, buff, 0) + + error_code = response[0] + if error_code != 0: + self.raise_error(error_code, response) + self.generation_id = response[1] + self.group_protocol = response[2] + self.leader_id = response[3] + self.member_id = response[4] + # TODO - parse metadata bytestring into ConsumerGroupProtocolMetadata? + self.members = {_id: meta for _id, meta in iteritems(response[5])} + + +class MemberAssignment(object): + """ + Protocol specification:: + + MemberAssignment => Version PartitionAssignment + Version => int16 + PartitionAssignment => [Topic [Partition]] + Topic => string + Partition => int32 + UserData => bytes + """ + def __init__(self, partition_assignment): + self.version = 1 + self.partition_assignment = { + topic.name: [partition.id for partition in partition_assignment[topic]] + for topic in partition_assignment} + self.user_data = b"testuserdata" + + def __len__(self): + # version + len(partition assignment) + size = 2 + 4 + for topic_name, partitions in self.partition_assignment: + # len(topic_name) + topic_name + size += 2 + len(topic_name) + size += 4 * len(partitions) + # len(user data) + user data + size += 4 + len(self.user_data) + return size + + def get_bytes(self): + output = bytearray(len(self)) + offset = 0 + fmt = '!hi' + struct.pack_into(fmt, output, offset, self.version, len(self.partition_assignment)) + offset += struct.calcsize(fmt) + for topic_name, partitions in iteritems(self.partition_assignment): + fmt = '!h%ds' % len(topic_name) + struct.pack_into(fmt, output, offset, len(topic_name), topic_name) + offset += struct.calcsize(fmt) + for partition_id in partitions: + fmt = '!i' + struct.pack_into(fmt, output, offset, partition_id) + offset += struct.calcsize(fmt) + fmt = '!i%ds' % len(self.user_data) + struct.pack_into(fmt, output, offset, len(self.user_data), self.user_data) + offset += struct.calcsize(fmt) + return output + + +class SyncGroupRequest(Request): + """A group sync request + + Specification:: + + SyncGroupRequest => GroupId GenerationId MemberId GroupAssignment + GroupId => string + GenerationId => int32 + MemberId => string + GroupAssignment => [MemberId MemberAssignment] + MemberId => string + MemberAssignment => bytes + """ + def __init__(self, generation_id, member_id, group_assignment): + """Create a new group join request""" + self.group_id = "dummygroup" + self.generation_id = generation_id + self.member_id = member_id + self.group_assignment = group_assignment + + def __len__(self): + """Length of the serialized message, in bytes""" + # Header + len(group id) + group id + generation id + size = self.HEADER_LEN + 2 + len(self.group_id) + 4 + # + len(member id) + member id + len(group assignment) + size += 2 + len(self.member_id) + 4 + # group assignment tuples + for member_id, member_assignment in self.group_assignment: + # + len(member id) + member id + len(member assignment) + member assignment + size += 2 + len(member_id) + 4 + len(member_assignment) + return size + + @property + def API_KEY(self): + """API_KEY for this request, from the Kafka docs""" + return 14 + + def get_bytes(self): + """Serialize the message + + :returns: Serialized message + :rtype: :class:`bytearray` + """ + output = bytearray(len(self)) + self._write_header(output, api_version=1) + offset = self.HEADER_LEN + fmt = '!h%dsih%dsi' % (len(self.group_id), len(self.member_id)) + struct.pack_into(fmt, output, offset, len(self.group_id), self.group_id, + self.generation_id, len(self.member_id), self.member_id, + len(self.group_assignment)) + offset += struct.calcsize(fmt) + for member_id, member_assignment in self.group_assignment: + assignment_bytes = bytes(member_assignment.get_bytes()) + fmt = '!h%dsi%ds' % (len(member_id), len(assignment_bytes)) + struct.pack_into(fmt, output, offset, len(member_id), member_id, + len(assignment_bytes), assignment_bytes) offset += struct.calcsize(fmt) return output From 433ce9a62b5559266317c49140031e175671ba31 Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Fri, 15 Jan 2016 10:08:34 -0800 Subject: [PATCH 03/70] implement SyncGroupResponse by allowing MemberAssignment construction from a bytestring --- pykafka/protocol.py | 43 +++++++++++++++++++++++++++++++++++++++---- 1 file changed, 39 insertions(+), 4 deletions(-) diff --git a/pykafka/protocol.py b/pykafka/protocol.py index 5ff6f20d5..3874a9ee7 100644 --- a/pykafka/protocol.py +++ b/pykafka/protocol.py @@ -1353,12 +1353,23 @@ class MemberAssignment(object): Partition => int32 UserData => bytes """ - def __init__(self, partition_assignment): - self.version = 1 + def __init__(self, partition_assignment, version=1, user_data=b""): + self.version = version self.partition_assignment = { - topic.name: [partition.id for partition in partition_assignment[topic]] + topic: [partition for partition in partition_assignment[topic]] for topic in partition_assignment} - self.user_data = b"testuserdata" + self.user_data = user_data + + @classmethod + def from_bytestring(cls, buff): + fmt = 'h [S [i ] ] S' + response = struct_helpers.unpack_from(fmt, buff, 0) + + version = response[0] + partition_assignment = {topic: partitions + for topic, partitions in iteritems(response[1])} + user_data = response[2] + return cls(partition_assignment, version=version, user_data=user_data) def __len__(self): # version + len(partition assignment) @@ -1449,3 +1460,27 @@ def get_bytes(self): len(assignment_bytes), assignment_bytes) offset += struct.calcsize(fmt) return output + + +class SyncGroupResponse(Response): + """A group sync response + + Specification:: + + SyncGroupResponse => ErrorCode MemberAssignment + ErrorCode => int16 + MemberAssignment => bytes + """ + def __init__(self, buff): + """Deserialize into a new Response + + :param buff: Serialized message + :type buff: :class:`bytearray` + """ + fmt = 'hS' + response = struct_helpers.unpack_from(fmt, buff, 0) + + error_code = response[0] + if error_code != 0: + self.raise_error(error_code, response) + self.member_assignment = MemberAssignment.from_bytestring(response[1]) From a177badd33d1a19ba732805066519d468f59d2e1 Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Fri, 15 Jan 2016 10:25:00 -0800 Subject: [PATCH 04/70] add LeaveGroup and Heartbeat request/response implementations --- pykafka/protocol.py | 134 +++++++++++++++++++++++++++++++++++++++++++- 1 file changed, 133 insertions(+), 1 deletion(-) diff --git a/pykafka/protocol.py b/pykafka/protocol.py index 3874a9ee7..0a3ffad2a 100644 --- a/pykafka/protocol.py +++ b/pykafka/protocol.py @@ -1417,7 +1417,7 @@ class SyncGroupRequest(Request): """ def __init__(self, generation_id, member_id, group_assignment): """Create a new group join request""" - self.group_id = "dummygroup" + self.group_id = b"dummygroup" self.generation_id = generation_id self.member_id = member_id self.group_assignment = group_assignment @@ -1484,3 +1484,135 @@ def __init__(self, buff): if error_code != 0: self.raise_error(error_code, response) self.member_assignment = MemberAssignment.from_bytestring(response[1]) + + +class HeartbeatRequest(Request): + """A group heartbeat request + + Specification:: + + HeartbeatRequest => GroupId GenerationId MemberId + GroupId => string + GenerationId => int32 + MemberId => string + """ + def __init__(self, group_id, generation_id, member_id): + """Create a new group join request""" + self.group_id = group_id + self.generation_id = generation_id + self.member_id = member_id + + def __len__(self): + """Length of the serialized message, in bytes""" + # Header + len(group id) + group id + generation id + size = self.HEADER_LEN + 2 + len(self.group_id) + 4 + # + len(member id) + member id + size += 2 + len(self.member_id) + return size + + @property + def API_KEY(self): + """API_KEY for this request, from the Kafka docs""" + return 12 + + def get_bytes(self): + """Serialize the message + + :returns: Serialized message + :rtype: :class:`bytearray` + """ + output = bytearray(len(self)) + self._write_header(output, api_version=1) + offset = self.HEADER_LEN + fmt = '!h%dsih%ds' % (len(self.group_id), len(self.member_id)) + struct.pack_into(fmt, output, offset, len(self.group_id), self.group_id, + self.generation_id, len(self.member_id), self.member_id) + offset += struct.calcsize(fmt) + return output + + +class HeartbeatResponse(Response): + """A group heartbeat response + + Specification:: + + HeartbeatResponse => ErrorCode + ErrorCode => int16 + """ + def __init__(self, buff): + """Deserialize into a new Response + + :param buff: Serialized message + :type buff: :class:`bytearray` + """ + fmt = 'h' + response = struct_helpers.unpack_from(fmt, buff, 0) + + error_code = response[0] + if error_code != 0: + self.raise_error(error_code, response) + + +class LeaveGroupRequest(Request): + """A group exit request + + Specification:: + + LeaveGroupRequest => GroupId MemberId + GroupId => string + MemberId => string + """ + def __init__(self, group_id, member_id): + """Create a new group join request""" + self.group_id = group_id + self.member_id = member_id + + def __len__(self): + """Length of the serialized message, in bytes""" + # Header + len(group id) + group id + size = self.HEADER_LEN + 2 + len(self.group_id) + # + len(member id) + member id + size += 2 + len(self.member_id) + return size + + @property + def API_KEY(self): + """API_KEY for this request, from the Kafka docs""" + return 13 + + def get_bytes(self): + """Serialize the message + + :returns: Serialized message + :rtype: :class:`bytearray` + """ + output = bytearray(len(self)) + self._write_header(output, api_version=1) + offset = self.HEADER_LEN + fmt = '!h%dsh%ds' % (len(self.group_id), len(self.member_id)) + struct.pack_into(fmt, output, offset, len(self.group_id), self.group_id, + len(self.member_id), self.member_id) + offset += struct.calcsize(fmt) + return output + + +class LeaveGroupResponse(Response): + """A group exit response + + Specification:: + + LeaveGroupResponse => ErrorCode + ErrorCode => int16 + """ + def __init__(self, buff): + """Deserialize into a new Response + + :param buff: Serialized message + :type buff: :class:`bytearray` + """ + fmt = 'h' + response = struct_helpers.unpack_from(fmt, buff, 0) + + error_code = response[0] + if error_code != 0: + self.raise_error(error_code, response) From a01d883eeb5a63db689363391f982b4b9ea96f5f Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Fri, 15 Jan 2016 10:54:43 -0800 Subject: [PATCH 05/70] first-run tests for a few request types --- pykafka/protocol.py | 13 +++++-------- tests/pykafka/test_protocol.py | 24 ++++++++++++++++++++++++ 2 files changed, 29 insertions(+), 8 deletions(-) diff --git a/pykafka/protocol.py b/pykafka/protocol.py index 0a3ffad2a..38d53060e 100644 --- a/pykafka/protocol.py +++ b/pykafka/protocol.py @@ -1355,9 +1355,7 @@ class MemberAssignment(object): """ def __init__(self, partition_assignment, version=1, user_data=b""): self.version = version - self.partition_assignment = { - topic: [partition for partition in partition_assignment[topic]] - for topic in partition_assignment} + self.partition_assignment = partition_assignment self.user_data = user_data @classmethod @@ -1366,8 +1364,7 @@ def from_bytestring(cls, buff): response = struct_helpers.unpack_from(fmt, buff, 0) version = response[0] - partition_assignment = {topic: partitions - for topic, partitions in iteritems(response[1])} + partition_assignment = response[1] user_data = response[2] return cls(partition_assignment, version=version, user_data=user_data) @@ -1388,7 +1385,7 @@ def get_bytes(self): fmt = '!hi' struct.pack_into(fmt, output, offset, self.version, len(self.partition_assignment)) offset += struct.calcsize(fmt) - for topic_name, partitions in iteritems(self.partition_assignment): + for topic_name, partitions in self.partition_assignment: fmt = '!h%ds' % len(topic_name) struct.pack_into(fmt, output, offset, len(topic_name), topic_name) offset += struct.calcsize(fmt) @@ -1415,9 +1412,9 @@ class SyncGroupRequest(Request): MemberId => string MemberAssignment => bytes """ - def __init__(self, generation_id, member_id, group_assignment): + def __init__(self, group_id, generation_id, member_id, group_assignment): """Create a new group join request""" - self.group_id = b"dummygroup" + self.group_id = group_id self.generation_id = generation_id self.member_id = member_id self.group_assignment = group_assignment diff --git a/tests/pykafka/test_protocol.py b/tests/pykafka/test_protocol.py index af78bc311..a4445cd49 100644 --- a/tests/pykafka/test_protocol.py +++ b/tests/pykafka/test_protocol.py @@ -275,6 +275,30 @@ def test_join_group_request(self): bytearray(b'\x00\x00\x00z\x00\x0b\x00\x01\x00\x00\x00\x00\x00\x07pykafka\x00\ndummygroup\x00\x00u0\x00\ntestmember\x00\x08consumer\x00\x00\x00\x01\x00\x17dummyassignmentstrategy\x00\x00\x00"\x00\x01\x00\x00\x00\x01\x00\ndummytopic\x00\x00\x00\x0ctestuserdata') ) + def test_member_assignment_construction(self): + assignment = protocol.MemberAssignment([(b"mytopic1", [3, 5, 7, 9]), + (b"mytopic2", [2, 4, 6, 8])]) + msg = assignment.get_bytes() + self.assertEqual( + msg, + bytearray(b'\x00\x01\x00\x00\x00\x02\x00\x08mytopic1\x00\x00\x00\x03\x00\x00\x00\x05\x00\x00\x00\x07\x00\x00\x00\t\x00\x08mytopic2\x00\x00\x00\x02\x00\x00\x00\x04\x00\x00\x00\x06\x00\x00\x00\x08\x00\x00\x00\x00') + ) + + def test_sync_group_request(self): + req = protocol.SyncGroupRequest( + b'dummygroup', 1, b'testmember1', + [ + ('testmember1', protocol.MemberAssignment([(b"mytopic1", [3, 5, 7, 9]), + (b"mytopic2", [3, 5, 7, 9])])), + ('testmember2', protocol.MemberAssignment([(b"mytopic1", [2, 4, 6, 8]), + (b"mytopic2", [2, 4, 6, 8])])) + ]) + msg = req.get_bytes() + self.assertEqual( + msg, + bytearray(b'\x00\x00\x00\xd0\x00\x0e\x00\x01\x00\x00\x00\x00\x00\x07pykafka\x00\ndummygroup\x00\x00\x00\x01\x00\x0btestmember1\x00\x00\x00\x02\x00\x0btestmember1\x00\x00\x00>\x00\x01\x00\x00\x00\x02\x00\x08mytopic1\x00\x00\x00\x03\x00\x00\x00\x05\x00\x00\x00\x07\x00\x00\x00\t\x00\x08mytopic2\x00\x00\x00\x03\x00\x00\x00\x05\x00\x00\x00\x07\x00\x00\x00\t\x00\x00\x00\x00\x00\x0btestmember2\x00\x00\x00>\x00\x01\x00\x00\x00\x02\x00\x08mytopic1\x00\x00\x00\x02\x00\x00\x00\x04\x00\x00\x00\x06\x00\x00\x00\x08\x00\x08mytopic2\x00\x00\x00\x02\x00\x00\x00\x04\x00\x00\x00\x06\x00\x00\x00\x08\x00\x00\x00\x00') + ) + if __name__ == '__main__': unittest2.main() From f6d184bb17b013218e3d5401b628b52b0a6c1850 Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Fri, 15 Jan 2016 10:57:56 -0800 Subject: [PATCH 06/70] test remaining request types --- tests/pykafka/test_protocol.py | 16 ++++++++++++++++ 1 file changed, 16 insertions(+) diff --git a/tests/pykafka/test_protocol.py b/tests/pykafka/test_protocol.py index a4445cd49..1bf99c983 100644 --- a/tests/pykafka/test_protocol.py +++ b/tests/pykafka/test_protocol.py @@ -299,6 +299,22 @@ def test_sync_group_request(self): bytearray(b'\x00\x00\x00\xd0\x00\x0e\x00\x01\x00\x00\x00\x00\x00\x07pykafka\x00\ndummygroup\x00\x00\x00\x01\x00\x0btestmember1\x00\x00\x00\x02\x00\x0btestmember1\x00\x00\x00>\x00\x01\x00\x00\x00\x02\x00\x08mytopic1\x00\x00\x00\x03\x00\x00\x00\x05\x00\x00\x00\x07\x00\x00\x00\t\x00\x08mytopic2\x00\x00\x00\x03\x00\x00\x00\x05\x00\x00\x00\x07\x00\x00\x00\t\x00\x00\x00\x00\x00\x0btestmember2\x00\x00\x00>\x00\x01\x00\x00\x00\x02\x00\x08mytopic1\x00\x00\x00\x02\x00\x00\x00\x04\x00\x00\x00\x06\x00\x00\x00\x08\x00\x08mytopic2\x00\x00\x00\x02\x00\x00\x00\x04\x00\x00\x00\x06\x00\x00\x00\x08\x00\x00\x00\x00') ) + def test_heartbeat_request(self): + req = protocol.HeartbeatRequest(b'dummygroup', 1, b'testmember') + msg = req.get_bytes() + self.assertEqual( + msg, + bytearray(b'\x00\x00\x00-\x00\x0c\x00\x01\x00\x00\x00\x00\x00\x07pykafka\x00\ndummygroup\x00\x00\x00\x01\x00\ntestmember') + ) + + def test_leave_group_request(self): + req = protocol.LeaveGroupRequest(b'dummygroup', b'testmember') + msg = req.get_bytes() + self.assertEqual( + msg, + bytearray(b'\x00\x00\x00)\x00\r\x00\x01\x00\x00\x00\x00\x00\x07pykafka\x00\ndummygroup\x00\ntestmember') + ) + if __name__ == '__main__': unittest2.main() From 02235816b9f4a43a9d2ee656b6ae6ce2fb3981db Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Fri, 15 Jan 2016 12:41:03 -0800 Subject: [PATCH 07/70] update naming of API key 10 for 0.9 --- pykafka/cli/kafka_tools.py | 2 +- pykafka/cluster.py | 8 ++++---- pykafka/protocol.py | 14 +++++++------- pykafka/simpleconsumer.py | 2 +- tests/pykafka/test_protocol.py | 2 +- 5 files changed, 14 insertions(+), 14 deletions(-) diff --git a/pykafka/cli/kafka_tools.py b/pykafka/cli/kafka_tools.py index dafd4d2c8..37a28ccc7 100644 --- a/pykafka/cli/kafka_tools.py +++ b/pykafka/cli/kafka_tools.py @@ -227,7 +227,7 @@ def reset_offsets(client, args): for partition_id, res in offsets.iteritems()] # Send them to the appropriate broker. - broker = client.cluster.get_offset_manager(args.consumer_group) + broker = client.cluster.get_group_coordinator(args.consumer_group) broker.commit_consumer_group_offsets( args.consumer_group, 1, 'kafka-tools', reqs ) diff --git a/pykafka/cluster.py b/pykafka/cluster.py index d73ead3e8..c60355a01 100644 --- a/pykafka/cluster.py +++ b/pykafka/cluster.py @@ -32,7 +32,7 @@ NoBrokersAvailableError, SocketDisconnectedError, LeaderNotAvailable) -from .protocol import ConsumerMetadataRequest, ConsumerMetadataResponse +from .protocol import GroupCoordinatorRequest, ConsumerMetadataResponse from .topic import Topic from .utils.compat import iteritems, range @@ -366,8 +366,8 @@ def _update_brokers(self, broker_metadata): # needed. raise Exception('Broker host/port change detected! %s', broker) - def get_offset_manager(self, consumer_group): - """Get the broker designated as the offset manager for this consumer group. + def get_group_coordinator(self, consumer_group): + """Get the broker designated as the group coordinator for this consumer group. Based on Step 1 at https://cwiki.apache.org/confluence/display/KAFKA/Committing+and+fetching+consumer+offsets+in+Kafka @@ -385,7 +385,7 @@ def get_offset_manager(self, consumer_group): log.debug("Retrying offset manager discovery") time.sleep(i * 2) - req = ConsumerMetadataRequest(consumer_group) + req = GroupCoordinatorRequest(consumer_group) future = broker.handler.request(req) try: res = future.get(ConsumerMetadataResponse) diff --git a/pykafka/protocol.py b/pykafka/protocol.py index 38d53060e..3b1688b78 100644 --- a/pykafka/protocol.py +++ b/pykafka/protocol.py @@ -49,7 +49,7 @@ "OffsetRequest", "OffsetResponse", "OffsetCommitRequest", "FetchRequest", "FetchResponse", "PartitionFetchRequest", "OffsetCommitResponse", "OffsetFetchRequest", "OffsetFetchResponse", - "PartitionOffsetRequest", "ConsumerMetadataRequest", + "PartitionOffsetRequest", "GroupCoordinatorRequest", "ConsumerMetadataResponse", "PartitionOffsetCommitRequest", "PartitionOffsetFetchRequest", "Request", "Response", "Message", "MessageSet" @@ -866,16 +866,16 @@ def __init__(self, buff): partition[2], partition[1]) -class ConsumerMetadataRequest(Request): +class GroupCoordinatorRequest(Request): """A consumer metadata request Specification:: - ConsumerMetadataRequest => ConsumerGroup + GroupCoordinatorRequest => ConsumerGroup ConsumerGroup => string """ def __init__(self, consumer_group): - """Create a new consumer metadata request""" + """Create a new group coordinator request""" self.consumer_group = consumer_group def __len__(self): @@ -902,12 +902,12 @@ def get_bytes(self): return output -class ConsumerMetadataResponse(Response): - """A consumer metadata response +class GroupCoordinatorResponse(Response): + """A group coordinator response Specification:: - ConsumerMetadataResponse => ErrorCode CoordinatorId CoordinatorHost CoordinatorPort + GroupCoordinatorResponse => ErrorCode CoordinatorId CoordinatorHost CoordinatorPort ErrorCode => int16 CoordinatorId => int32 CoordinatorHost => string diff --git a/pykafka/simpleconsumer.py b/pykafka/simpleconsumer.py index a56d4e684..470e18a88 100644 --- a/pykafka/simpleconsumer.py +++ b/pykafka/simpleconsumer.py @@ -287,7 +287,7 @@ def _discover_offset_manager(self): If a consumer group is not supplied to __init__, this method does nothing """ if self._consumer_group is not None: - self._offset_manager = self._cluster.get_offset_manager(self._consumer_group) + self._offset_manager = self._cluster.get_group_coordinator(self._consumer_group) @property def topic(self): diff --git a/tests/pykafka/test_protocol.py b/tests/pykafka/test_protocol.py index 1bf99c983..3a0b2359a 100644 --- a/tests/pykafka/test_protocol.py +++ b/tests/pykafka/test_protocol.py @@ -207,7 +207,7 @@ class TestOffsetCommitFetchAPI(unittest2.TestCase): maxDiff = None def test_consumer_metadata_request(self): - req = protocol.ConsumerMetadataRequest(b'test') + req = protocol.GroupCoordinatorRequest(b'test') msg = req.get_bytes() self.assertEqual( msg, From 7b7baf6ef8e2f08695a53c67dc0246e5ff487358 Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Fri, 15 Jan 2016 13:45:39 -0800 Subject: [PATCH 08/70] add joingrouprequest stub to broker --- pykafka/broker.py | 16 +++++++++++----- pykafka/protocol.py | 4 ++-- tests/pykafka/test_protocol.py | 2 +- 3 files changed, 14 insertions(+), 8 deletions(-) diff --git a/pykafka/broker.py b/pykafka/broker.py index 821fd4156..83aa74bdf 100644 --- a/pykafka/broker.py +++ b/pykafka/broker.py @@ -24,11 +24,9 @@ from .exceptions import LeaderNotAvailable, SocketDisconnectedError from .handlers import RequestHandler from .protocol import ( - FetchRequest, FetchResponse, OffsetRequest, - OffsetResponse, MetadataRequest, MetadataResponse, - OffsetCommitRequest, OffsetCommitResponse, - OffsetFetchRequest, OffsetFetchResponse, - ProduceResponse) + FetchRequest, FetchResponse, OffsetRequest, OffsetResponse, MetadataRequest, + MetadataResponse, OffsetCommitRequest, OffsetCommitResponse, OffsetFetchRequest, + OffsetFetchResponse, ProduceResponse, JoinGroupRequest, JoinGroupResponse) from .utils.compat import range, iteritems log = logging.getLogger(__name__) @@ -348,3 +346,11 @@ def fetch_consumer_group_offsets(self, consumer_group, preqs): self.connect_offsets_channel() req = OffsetFetchRequest(consumer_group, partition_requests=preqs) return self._offsets_channel_req_handler.request(req).get(OffsetFetchResponse) + + ########################## + # Group Membership API # + ########################## + + def join_managed_consumer_group(self, consumer_group): + future = self._req_handler.request(JoinGroupRequest(self._consumer_group)) + return future.get(JoinGroupResponse) diff --git a/pykafka/protocol.py b/pykafka/protocol.py index 3b1688b78..857a3f6ef 100644 --- a/pykafka/protocol.py +++ b/pykafka/protocol.py @@ -1257,10 +1257,10 @@ class JoinGroupRequest(Request): ProtocolName => string ProtocolMetadata => bytes """ - def __init__(self, member_id, session_timeout=30000): + def __init__(self, group_id, member_id=b'', session_timeout=30000): """Create a new group join request""" self.protocol = ConsumerGroupProtocol - self.group_id = "dummygroup" + self.group_id = group_id self.session_timeout = session_timeout self.member_id = member_id self.protocol_type = self.protocol.protocol_type diff --git a/tests/pykafka/test_protocol.py b/tests/pykafka/test_protocol.py index 3a0b2359a..33fe14f9a 100644 --- a/tests/pykafka/test_protocol.py +++ b/tests/pykafka/test_protocol.py @@ -268,7 +268,7 @@ def test_consumer_group_protocol_metadata(self): ) def test_join_group_request(self): - req = protocol.JoinGroupRequest(b'testmember') + req = protocol.JoinGroupRequest(b'dummygroup', member_id=b'testmember') msg = req.get_bytes() self.assertEqual( msg, From a68222b288418c130ad098be5c500869649589fc Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Fri, 15 Jan 2016 13:46:05 -0800 Subject: [PATCH 09/70] update wording in simpleconsumer --- pykafka/simpleconsumer.py | 18 +++++++++--------- 1 file changed, 9 insertions(+), 9 deletions(-) diff --git a/pykafka/simpleconsumer.py b/pykafka/simpleconsumer.py index 470e18a88..fd14353b1 100644 --- a/pykafka/simpleconsumer.py +++ b/pykafka/simpleconsumer.py @@ -169,7 +169,7 @@ def __init__(self, self._worker_trace_logged = False self._update_lock = self._cluster.handler.Lock() - self._discover_offset_manager() + self._discover_group_coordinator() if partitions is not None: self._partitions = {p: OwnedPartition(p, self._cluster.handler, @@ -219,7 +219,7 @@ def _update(self): with self._update_lock: self._cluster.update() self._setup_partitions_by_leader() - self._discover_offset_manager() + self._discover_group_coordinator() def start(self): """Begin communicating with Kafka, including setting up worker threads @@ -281,13 +281,13 @@ def _handle_GroupLoadInProgress(parts): GroupLoadInProgress.ERROR_CODE: _handle_GroupLoadInProgress } - def _discover_offset_manager(self): - """Set the offset manager for this consumer. + def _discover_group_coordinator(self): + """Set the group coordinator for this consumer. If a consumer group is not supplied to __init__, this method does nothing """ if self._consumer_group is not None: - self._offset_manager = self._cluster.get_group_coordinator(self._consumer_group) + self._group_coordinator = self._cluster.get_group_coordinator(self._consumer_group) @property def topic(self): @@ -426,14 +426,14 @@ def commit_offsets(self): reqs = [p.build_offset_commit_request() for p in self._partitions.values()] log.debug("Committing offsets for %d partitions to broker id %s", len(reqs), - self._offset_manager.id) + self._group_coordinator.id) for i in range(self._offsets_commit_max_retries): if i > 0: log.debug("Retrying") self._cluster.handler.sleep(i * (self._offsets_channel_backoff_ms / 1000)) try: - response = self._offset_manager.commit_consumer_group_offsets( + response = self._group_coordinator.commit_consumer_group_offsets( self._consumer_group, -1, b'pykafka', reqs) except (SocketDisconnectedError, IOError): log.error("Error committing offsets for topic '%s' " @@ -509,13 +509,13 @@ def _handle_success(parts): success_responses = [] log.debug("Fetching offsets for %d partitions from broker id %s", len(reqs), - self._offset_manager.id) + self._group_coordinator.id) for i in range(self._offsets_fetch_max_retries): if i > 0: log.debug("Retrying offset fetch") - res = self._offset_manager.fetch_consumer_group_offsets(self._consumer_group, reqs) + res = self._group_coordinator.fetch_consumer_group_offsets(self._consumer_group, reqs) parts_by_error = handle_partition_responses( self._default_error_handlers, response=res, From 56fb0f1d090f95b91202e04971ba787a2d885e2a Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Fri, 15 Jan 2016 13:46:46 -0800 Subject: [PATCH 10/70] start basic implementation of managed balanced consumer not yet certain if this will end up as its own class, but using it as a workspace to figure out the right abstraction --- pykafka/managedbalancedconsumer.py | 79 ++++++++++++++++++++++++++++++ 1 file changed, 79 insertions(+) create mode 100644 pykafka/managedbalancedconsumer.py diff --git a/pykafka/managedbalancedconsumer.py b/pykafka/managedbalancedconsumer.py new file mode 100644 index 000000000..e4e753bdf --- /dev/null +++ b/pykafka/managedbalancedconsumer.py @@ -0,0 +1,79 @@ +""" +Author: Emmett Butler +""" +__license__ = """ +Copyright 2016 Parse.ly, Inc. + +Licensed 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. +""" +__all__ = ["ManagedBalancedConsumer"] +import logging + +from .common import OffsetType + +log = logging.getLogger(__name__) + + +class ManagedBalancedConsumer(object): + def __init__(self, + topic, + cluster, + consumer_group, + fetch_message_max_bytes=1024 * 1024, + num_consumer_fetchers=1, + auto_commit_enable=False, + auto_commit_interval_ms=60 * 1000, + queued_max_messages=2000, + fetch_min_bytes=1, + fetch_wait_max_ms=100, + offsets_channel_backoff_ms=1000, + offsets_commit_max_retries=5, + auto_offset_reset=OffsetType.EARLIEST, + consumer_timeout_ms=-1, + auto_start=True, + reset_offset_on_start=False): + self._cluster = cluster + if not isinstance(consumer_group, bytes): + raise TypeError("consumer_group must be a bytes object") + self._consumer_group = consumer_group + self._topic = topic + + self._auto_commit_enable = auto_commit_enable + self._auto_commit_interval_ms = auto_commit_interval_ms + self._fetch_message_max_bytes = fetch_message_max_bytes + self._fetch_min_bytes = fetch_min_bytes + self._num_consumer_fetchers = num_consumer_fetchers + self._queued_max_messages = queued_max_messages + self._fetch_wait_max_ms = fetch_wait_max_ms + self._consumer_timeout_ms = consumer_timeout_ms + self._offsets_channel_backoff_ms = offsets_channel_backoff_ms + self._offsets_commit_max_retries = offsets_commit_max_retries + self._auto_offset_reset = auto_offset_reset + self._reset_offset_on_start = reset_offset_on_start + self._running = False + + self._consumer = None + self._consumer_id = None + + self._discover_group_coordinator() + self._join_group() + + if auto_start is True: + self.start() + + def _discover_group_coordinator(self): + """Set the group coordinator for this consumer.""" + self._group_coordinator = self._cluster.get_group_coordinator(self._consumer_group) + + def _join_group(self): + res = self._group_coordinator.join_managed_consumer_group(self._consumer_group) From 7b8283d1a1740f7adb12d330ff245c13751010a6 Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Fri, 15 Jan 2016 14:17:20 -0800 Subject: [PATCH 11/70] basic managed group join flow --- pykafka/broker.py | 10 ++++++++-- pykafka/managedbalancedconsumer.py | 17 +++++++++++++++++ 2 files changed, 25 insertions(+), 2 deletions(-) diff --git a/pykafka/broker.py b/pykafka/broker.py index 83aa74bdf..b56085e7d 100644 --- a/pykafka/broker.py +++ b/pykafka/broker.py @@ -26,7 +26,8 @@ from .protocol import ( FetchRequest, FetchResponse, OffsetRequest, OffsetResponse, MetadataRequest, MetadataResponse, OffsetCommitRequest, OffsetCommitResponse, OffsetFetchRequest, - OffsetFetchResponse, ProduceResponse, JoinGroupRequest, JoinGroupResponse) + OffsetFetchResponse, ProduceResponse, JoinGroupRequest, JoinGroupResponse, + SyncGroupRequest, SyncGroupResponse) from .utils.compat import range, iteritems log = logging.getLogger(__name__) @@ -352,5 +353,10 @@ def fetch_consumer_group_offsets(self, consumer_group, preqs): ########################## def join_managed_consumer_group(self, consumer_group): - future = self._req_handler.request(JoinGroupRequest(self._consumer_group)) + future = self._req_handler.request(JoinGroupRequest(consumer_group)) return future.get(JoinGroupResponse) + + def sync_group(self, consumer_group, generation_id, member_id, group_assignment): + future = self._req_handler.request( + SyncGroupRequest(consumer_group, generation_id, member_id, group_assignment)) + return future.get(SyncGroupResponse) diff --git a/pykafka/managedbalancedconsumer.py b/pykafka/managedbalancedconsumer.py index e4e753bdf..10eaf0f5e 100644 --- a/pykafka/managedbalancedconsumer.py +++ b/pykafka/managedbalancedconsumer.py @@ -20,6 +20,8 @@ import logging from .common import OffsetType +from .protocol import MemberAssignment +from .utils.compat import iteritems log = logging.getLogger(__name__) @@ -64,6 +66,7 @@ def __init__(self, self._consumer = None self._consumer_id = None + self._is_group_leader = False self._discover_group_coordinator() self._join_group() @@ -77,3 +80,17 @@ def _discover_group_coordinator(self): def _join_group(self): res = self._group_coordinator.join_managed_consumer_group(self._consumer_group) + self._generation_id = res.generation_id + self._consumer_id = res.member_id + if len(res.members) > 0: + self._is_group_leader = True + group_assignment = [] + for member_id, metadata in iteritems(res.members): + partitions = self._decide_partitions(member_id, metadata) + group_assignment.append( + (member_id, MemberAssignment([(self._topic.name, [partitions])]))) + res = self._group_coordinator.sync_group(self._consumer_group, + self._generation_id, + self._consumer_id, + group_assignment) + self._setup_internal_consumer(res.member_assignment.partition_assignment) From 78d4449745fc62ed3e26100de37bf3290de3c22c Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Fri, 15 Jan 2016 14:27:46 -0800 Subject: [PATCH 12/70] fix import error --- pykafka/cluster.py | 4 ++-- 1 file changed, 2 insertions(+), 2 deletions(-) diff --git a/pykafka/cluster.py b/pykafka/cluster.py index c60355a01..7ded01993 100644 --- a/pykafka/cluster.py +++ b/pykafka/cluster.py @@ -32,7 +32,7 @@ NoBrokersAvailableError, SocketDisconnectedError, LeaderNotAvailable) -from .protocol import GroupCoordinatorRequest, ConsumerMetadataResponse +from .protocol import GroupCoordinatorRequest, GroupCoordinatorResponse from .topic import Topic from .utils.compat import iteritems, range @@ -388,7 +388,7 @@ def get_group_coordinator(self, consumer_group): req = GroupCoordinatorRequest(consumer_group) future = broker.handler.request(req) try: - res = future.get(ConsumerMetadataResponse) + res = future.get(GroupCoordinatorResponse) except ConsumerCoordinatorNotAvailable: log.error('Error discovering offset manager.') if i == self._max_connection_retries - 1: From 4bd69f0b32d8ea8264945dd34698e207014c4d71 Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Fri, 15 Jan 2016 21:07:25 -0800 Subject: [PATCH 13/70] fix __all__ error --- pykafka/protocol.py | 2 +- 1 file changed, 1 insertion(+), 1 deletion(-) diff --git a/pykafka/protocol.py b/pykafka/protocol.py index 857a3f6ef..5fb4eb85d 100644 --- a/pykafka/protocol.py +++ b/pykafka/protocol.py @@ -50,7 +50,7 @@ "FetchRequest", "FetchResponse", "PartitionFetchRequest", "OffsetCommitResponse", "OffsetFetchRequest", "OffsetFetchResponse", "PartitionOffsetRequest", "GroupCoordinatorRequest", - "ConsumerMetadataResponse", "PartitionOffsetCommitRequest", + "GroupCoordinatorResponse", "PartitionOffsetCommitRequest", "PartitionOffsetFetchRequest", "Request", "Response", "Message", "MessageSet" ] From 979d3f047fc6370b5e2fe99e4ee4512602f31fa7 Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Fri, 15 Jan 2016 21:07:32 -0800 Subject: [PATCH 14/70] fix version trace --- pykafka/protocol.py | 2 +- 1 file changed, 1 insertion(+), 1 deletion(-) diff --git a/pykafka/protocol.py b/pykafka/protocol.py index 5fb4eb85d..eb82778e0 100644 --- a/pykafka/protocol.py +++ b/pykafka/protocol.py @@ -1204,7 +1204,7 @@ class ConsumerGroupProtocolMetadata(object): UserData => bytes """ def __init__(self): - self.version = 1 + self.version = 0 self.topic_names = [b"dummytopic"] self.user_data = b"testuserdata" From 6855d9c66391beb9ca4470a6bfbe37bb3d6ed310 Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Fri, 15 Jan 2016 21:15:48 -0800 Subject: [PATCH 15/70] fix incorrect api version --- pykafka/protocol.py | 8 ++++---- 1 file changed, 4 insertions(+), 4 deletions(-) diff --git a/pykafka/protocol.py b/pykafka/protocol.py index eb82778e0..a3acb41bd 100644 --- a/pykafka/protocol.py +++ b/pykafka/protocol.py @@ -1290,7 +1290,7 @@ def get_bytes(self): :rtype: :class:`bytearray` """ output = bytearray(len(self)) - self._write_header(output, api_version=1) + self._write_header(output) offset = self.HEADER_LEN fmt = '!h%dsih%dsh%dsi' % (len(self.group_id), len(self.member_id), len(self.protocol_type)) @@ -1443,7 +1443,7 @@ def get_bytes(self): :rtype: :class:`bytearray` """ output = bytearray(len(self)) - self._write_header(output, api_version=1) + self._write_header(output) offset = self.HEADER_LEN fmt = '!h%dsih%dsi' % (len(self.group_id), len(self.member_id)) struct.pack_into(fmt, output, offset, len(self.group_id), self.group_id, @@ -1519,7 +1519,7 @@ def get_bytes(self): :rtype: :class:`bytearray` """ output = bytearray(len(self)) - self._write_header(output, api_version=1) + self._write_header(output) offset = self.HEADER_LEN fmt = '!h%dsih%ds' % (len(self.group_id), len(self.member_id)) struct.pack_into(fmt, output, offset, len(self.group_id), self.group_id, @@ -1584,7 +1584,7 @@ def get_bytes(self): :rtype: :class:`bytearray` """ output = bytearray(len(self)) - self._write_header(output, api_version=1) + self._write_header(output) offset = self.HEADER_LEN fmt = '!h%dsh%ds' % (len(self.group_id), len(self.member_id)) struct.pack_into(fmt, output, offset, len(self.group_id), self.group_id, From 93b6a9b3227bd5ac82e9f16b3767ef267d4bef46 Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Fri, 15 Jan 2016 21:15:59 -0800 Subject: [PATCH 16/70] protocol fixes --- pykafka/protocol.py | 4 ++-- 1 file changed, 2 insertions(+), 2 deletions(-) diff --git a/pykafka/protocol.py b/pykafka/protocol.py index a3acb41bd..1d4420bb5 100644 --- a/pykafka/protocol.py +++ b/pykafka/protocol.py @@ -1257,7 +1257,7 @@ class JoinGroupRequest(Request): ProtocolName => string ProtocolMetadata => bytes """ - def __init__(self, group_id, member_id=b'', session_timeout=30000): + def __init__(self, group_id, member_id=b'', session_timeout=10000): """Create a new group join request""" self.protocol = ConsumerGroupProtocol self.group_id = group_id @@ -1339,7 +1339,7 @@ def __init__(self, buff): self.leader_id = response[3] self.member_id = response[4] # TODO - parse metadata bytestring into ConsumerGroupProtocolMetadata? - self.members = {_id: meta for _id, meta in iteritems(response[5])} + self.members = {_id: meta for _id, meta in response[5]} class MemberAssignment(object): From 4593f780f98f93a1e4abdf5f9cac107d264ee4c4 Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Fri, 15 Jan 2016 21:26:48 -0800 Subject: [PATCH 17/70] fix syncgroup request errors --- pykafka/managedbalancedconsumer.py | 5 ++++- pykafka/protocol.py | 2 ++ 2 files changed, 6 insertions(+), 1 deletion(-) diff --git a/pykafka/managedbalancedconsumer.py b/pykafka/managedbalancedconsumer.py index 10eaf0f5e..a54bb779b 100644 --- a/pykafka/managedbalancedconsumer.py +++ b/pykafka/managedbalancedconsumer.py @@ -88,9 +88,12 @@ def _join_group(self): for member_id, metadata in iteritems(res.members): partitions = self._decide_partitions(member_id, metadata) group_assignment.append( - (member_id, MemberAssignment([(self._topic.name, [partitions])]))) + (member_id, MemberAssignment([(self._topic.name, partitions)]))) res = self._group_coordinator.sync_group(self._consumer_group, self._generation_id, self._consumer_id, group_assignment) self._setup_internal_consumer(res.member_assignment.partition_assignment) + + def _decide_partitions(self, member_id, metadata): + return [1, 2, 3, 4, 5] diff --git a/pykafka/protocol.py b/pykafka/protocol.py index 1d4420bb5..1e92163d7 100644 --- a/pykafka/protocol.py +++ b/pykafka/protocol.py @@ -1360,6 +1360,8 @@ def __init__(self, partition_assignment, version=1, user_data=b""): @classmethod def from_bytestring(cls, buff): + if len(buff) == 0: + return cls([]) fmt = 'h [S [i ] ] S' response = struct_helpers.unpack_from(fmt, buff, 0) From 07cc0c385e78e711caf7b641271656200c8e8114 Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Fri, 15 Jan 2016 21:34:39 -0800 Subject: [PATCH 18/70] joingroupresponse test --- tests/pykafka/test_protocol.py | 13 +++++++++++++ 1 file changed, 13 insertions(+) diff --git a/tests/pykafka/test_protocol.py b/tests/pykafka/test_protocol.py index 33fe14f9a..567e8f787 100644 --- a/tests/pykafka/test_protocol.py +++ b/tests/pykafka/test_protocol.py @@ -275,6 +275,19 @@ def test_join_group_request(self): bytearray(b'\x00\x00\x00z\x00\x0b\x00\x01\x00\x00\x00\x00\x00\x07pykafka\x00\ndummygroup\x00\x00u0\x00\ntestmember\x00\x08consumer\x00\x00\x00\x01\x00\x17dummyassignmentstrategy\x00\x00\x00"\x00\x01\x00\x00\x00\x01\x00\ndummytopic\x00\x00\x00\x0ctestuserdata') ) + def test_join_group_response(self): + response = protocol.JoinGroupResponse( + bytearray('\x00\x00\x00\x00\x00\x01\x00\x17dummyassignmentstrategy\x00,pykafka-b2361322-674c-4e26-9194-305962636e57\x00,pykafka-b2361322-674c-4e26-9194-305962636e57\x00\x00\x00\x01\x00,pykafka-b2361322-674c-4e26-9194-305962636e57\x00\x00\x00"\x00\x00\x00\x00\x00\x01\x00\ndummytopic\x00\x00\x00\x0ctestuserdata\x00\x00\x00\x00') + ) + self.assertEqual(response.generation_id, 0) + self.assertEqual(response.group_protocol, 'dummyassignmentstrategy') + self.assertEqual(response.leader_id, + 'pykafka-b2361322-674c-4e26-9194-305962636e57') + self.assertEqual(response.member_id, + 'pykafka-b2361322-674c-4e26-9194-305962636e57') + self.assertEqual(response.members, + {'pykafka-b2361322-674c-4e26-9194-305962636e57': ''}) + def test_member_assignment_construction(self): assignment = protocol.MemberAssignment([(b"mytopic1", [3, 5, 7, 9]), (b"mytopic2", [2, 4, 6, 8])]) From cfa9e316a03cc1eb44fe6c92df3f0b5a5fdd9f2d Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Fri, 22 Jan 2016 15:43:32 -0800 Subject: [PATCH 19/70] use existing decide_partitions in new group implementation --- pykafka/managedbalancedconsumer.py | 58 +++++++++++++++++++++++------- 1 file changed, 45 insertions(+), 13 deletions(-) diff --git a/pykafka/managedbalancedconsumer.py b/pykafka/managedbalancedconsumer.py index a54bb779b..8f30eb71f 100644 --- a/pykafka/managedbalancedconsumer.py +++ b/pykafka/managedbalancedconsumer.py @@ -17,16 +17,18 @@ limitations under the License. """ __all__ = ["ManagedBalancedConsumer"] +import itertools import logging +from .balancedconsumer import BalancedConsumer from .common import OffsetType from .protocol import MemberAssignment -from .utils.compat import iteritems +from .utils.compat import iteritems, itervalues log = logging.getLogger(__name__) -class ManagedBalancedConsumer(object): +class ManagedBalancedConsumer(BalancedConsumer): def __init__(self, topic, cluster, @@ -62,6 +64,8 @@ def __init__(self, self._offsets_commit_max_retries = offsets_commit_max_retries self._auto_offset_reset = auto_offset_reset self._reset_offset_on_start = reset_offset_on_start + self._use_rdkafka = False + self._worker_exception = None self._running = False self._consumer = None @@ -71,9 +75,6 @@ def __init__(self, self._discover_group_coordinator() self._join_group() - if auto_start is True: - self.start() - def _discover_group_coordinator(self): """Set the group coordinator for this consumer.""" self._group_coordinator = self._cluster.get_group_coordinator(self._consumer_group) @@ -82,18 +83,49 @@ def _join_group(self): res = self._group_coordinator.join_managed_consumer_group(self._consumer_group) self._generation_id = res.generation_id self._consumer_id = res.member_id + all_members = [] if len(res.members) > 0: self._is_group_leader = True - group_assignment = [] + all_members = [member_id for member_id, _ in iteritems(res.members)] + group_assignments = [] + leader_assignment = None for member_id, metadata in iteritems(res.members): - partitions = self._decide_partitions(member_id, metadata) - group_assignment.append( - (member_id, MemberAssignment([(self._topic.name, partitions)]))) + partitions = self._decide_partitions(all_members, member_id) + assignment = (self._topic.name, [p.id for p in partitions]) + if member_id == self._consumer_id and self._is_group_leader: + leader_assignment = assignment + group_assignments.append((member_id, MemberAssignment([assignment]))) res = self._group_coordinator.sync_group(self._consumer_group, self._generation_id, self._consumer_id, - group_assignment) - self._setup_internal_consumer(res.member_assignment.partition_assignment) + group_assignments) + if self._is_group_leader: + assignment = leader_assignment[1] + else: + assignment = res.member_assignment.partition_assignment[1] + self._setup_internal_consumer( + partitions=[p for p in itervalues(self._topic.partitions) + if p.id in assignment]) + + def _decide_partitions(self, participants, member_id): + # Freeze and sort partitions so we always have the same results + p_to_str = lambda p: '-'.join([str(p.topic.name), str(p.leader.id), str(p.id)]) + all_parts = self._topic.partitions.values() + all_parts = sorted(all_parts, key=p_to_str) + + # get start point, # of partitions, and remainder + participants = sorted(participants) # just make sure it's sorted. + idx = participants.index(member_id) + parts_per_consumer = len(all_parts) // len(participants) + remainder_ppc = len(all_parts) % len(participants) + + start = parts_per_consumer * idx + min(idx, remainder_ppc) + num_parts = parts_per_consumer + (0 if (idx + 1 > remainder_ppc) else 1) - def _decide_partitions(self, member_id, metadata): - return [1, 2, 3, 4, 5] + # assign partitions from i*N to (i+1)*N - 1 to consumer Ci + new_partitions = itertools.islice(all_parts, start, start + num_parts) + new_partitions = set(new_partitions) + log.info('Balancing %i participants for %i partitions.\nOwning %i partitions.', + len(participants), len(all_parts), len(new_partitions)) + log.debug('My partitions: %s', [p_to_str(p) for p in new_partitions]) + return new_partitions From 6e1971fea14aabaf4ce4b81eaf6c9df174968935 Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Wed, 24 Feb 2016 11:40:36 -0800 Subject: [PATCH 20/70] pass necessary parameters to commit_offsets --- pykafka/balancedconsumer.py | 5 ++++- pykafka/managedbalancedconsumer.py | 4 +++- pykafka/simpleconsumer.py | 13 +++++++++++-- 3 files changed, 18 insertions(+), 4 deletions(-) diff --git a/pykafka/balancedconsumer.py b/pykafka/balancedconsumer.py index 39a2ac57b..31212cc97 100644 --- a/pykafka/balancedconsumer.py +++ b/pykafka/balancedconsumer.py @@ -210,6 +210,7 @@ def __init__(self, self._zookeeper_connection_timeout_ms = zookeeper_connection_timeout_ms self._reset_offset_on_start = reset_offset_on_start self._post_rebalance_callback = post_rebalance_callback + self._generation_id = -1 self._running = False self._worker_exception = None self._worker_trace_logged = False @@ -378,7 +379,9 @@ def _get_internal_consumer(self, partitions=None, start=True): auto_offset_reset=self._auto_offset_reset, reset_offset_on_start=reset_offset_on_start, auto_start=start, - compacted_topic=self._is_compacted_topic + compacted_topic=self._is_compacted_topic, + generation_id=self._generation_id, + consumer_id=self._consumer_id ) def _decide_partitions(self, participants): diff --git a/pykafka/managedbalancedconsumer.py b/pykafka/managedbalancedconsumer.py index 8f30eb71f..fcde415c4 100644 --- a/pykafka/managedbalancedconsumer.py +++ b/pykafka/managedbalancedconsumer.py @@ -45,7 +45,8 @@ def __init__(self, auto_offset_reset=OffsetType.EARLIEST, consumer_timeout_ms=-1, auto_start=True, - reset_offset_on_start=False): + reset_offset_on_start=False, + compacted_topic=True): self._cluster = cluster if not isinstance(consumer_group, bytes): raise TypeError("consumer_group must be a bytes object") @@ -64,6 +65,7 @@ def __init__(self, self._offsets_commit_max_retries = offsets_commit_max_retries self._auto_offset_reset = auto_offset_reset self._reset_offset_on_start = reset_offset_on_start + self._is_compacted_topic = compacted_topic self._use_rdkafka = False self._worker_exception = None self._running = False diff --git a/pykafka/simpleconsumer.py b/pykafka/simpleconsumer.py index fd14353b1..00bf993ae 100644 --- a/pykafka/simpleconsumer.py +++ b/pykafka/simpleconsumer.py @@ -67,7 +67,9 @@ def __init__(self, consumer_timeout_ms=-1, auto_start=True, reset_offset_on_start=False, - compacted_topic=False): + compacted_topic=False, + generation_id=-1, + consumer_id=None): """Create a SimpleConsumer. Settings and default values are taken from the Scala @@ -136,6 +138,11 @@ def __init__(self, consumer to use less stringent message ordering logic because compacted topics do not provide offsets in stict incrementing order. :type compacted_topic: bool + :param generation_id: The generation id with which to make group requests + :type generation_id: int + :param consumer_id: The identifying string to use for this consumer on group + requests + :type consumer_id: str """ self._cluster = cluster if not (isinstance(consumer_group, bytes) or consumer_group is None): @@ -157,6 +164,8 @@ def __init__(self, self._auto_start = auto_start self._reset_offset_on_start = reset_offset_on_start self._is_compacted_topic = compacted_topic + self._generation_id = generation_id + self._consumer_id = consumer_id # incremented for any message arrival from any partition # the initial value is 0 (no messages waiting) @@ -434,7 +443,7 @@ def commit_offsets(self): try: response = self._group_coordinator.commit_consumer_group_offsets( - self._consumer_group, -1, b'pykafka', reqs) + self._consumer_group, self._generation_id, self._consumer_id, reqs) except (SocketDisconnectedError, IOError): log.error("Error committing offsets for topic '%s' " "(SocketDisconnectedError)", From 7f24303c2f3af7fab9ac510ec434f0fb987c2204 Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Wed, 24 Feb 2016 15:10:48 -0800 Subject: [PATCH 21/70] add basic heartbeats --- pykafka/broker.py | 11 ++++-- pykafka/exceptions.py | 26 ++++++++++++- pykafka/managedbalancedconsumer.py | 59 ++++++++++++++++++++++++++---- pykafka/protocol.py | 7 +--- 4 files changed, 87 insertions(+), 16 deletions(-) diff --git a/pykafka/broker.py b/pykafka/broker.py index b56085e7d..667dffd03 100644 --- a/pykafka/broker.py +++ b/pykafka/broker.py @@ -27,7 +27,7 @@ FetchRequest, FetchResponse, OffsetRequest, OffsetResponse, MetadataRequest, MetadataResponse, OffsetCommitRequest, OffsetCommitResponse, OffsetFetchRequest, OffsetFetchResponse, ProduceResponse, JoinGroupRequest, JoinGroupResponse, - SyncGroupRequest, SyncGroupResponse) + SyncGroupRequest, SyncGroupResponse, HeartbeatRequest, HeartbeatResponse) from .utils.compat import range, iteritems log = logging.getLogger(__name__) @@ -352,11 +352,16 @@ def fetch_consumer_group_offsets(self, consumer_group, preqs): # Group Membership API # ########################## - def join_managed_consumer_group(self, consumer_group): - future = self._req_handler.request(JoinGroupRequest(consumer_group)) + def join_managed_consumer_group(self, consumer_group, member_id): + future = self._req_handler.request(JoinGroupRequest(consumer_group, member_id)) return future.get(JoinGroupResponse) def sync_group(self, consumer_group, generation_id, member_id, group_assignment): future = self._req_handler.request( SyncGroupRequest(consumer_group, generation_id, member_id, group_assignment)) return future.get(SyncGroupResponse) + + def group_heartbeat(self, consumer_group, generation_id, member_id): + future = self._req_handler.request( + HeartbeatRequest(consumer_group, generation_id, member_id)) + return future.get(HeartbeatResponse) diff --git a/pykafka/exceptions.py b/pykafka/exceptions.py index e39c655be..67c1e7f14 100644 --- a/pykafka/exceptions.py +++ b/pykafka/exceptions.py @@ -173,6 +173,27 @@ class NotCoordinatorForConsumer(ProtocolClientError): ERROR_CODE = 16 +class IllegalGeneration(ProtocolClientError): + """Returned from group membership requests (such as heartbeats) when the generation + id provided in the request is not the current generation + """ + ERROR_CODE = 22 + + +class UnknownMemberId(ProtocolClientError): + """Returned from group requests (offset commits/fetches, heartbeats, etc) when the + memberId is not in the current generation. + """ + ERROR_CODE = 25 + + +class RebalanceInProgress(ProtocolClientError): + """Returned in heartbeat requests when the coordinator has begun rebalancing the + group. This indicates to the client that it should rejoin the group. + """ + ERROR_CODE = 27 + + ERROR_CODES = dict( (exc.ERROR_CODE, exc) for exc in (UnknownError, @@ -187,7 +208,10 @@ class NotCoordinatorForConsumer(ProtocolClientError): OffsetMetadataTooLarge, GroupLoadInProgress, ConsumerCoordinatorNotAvailable, - NotCoordinatorForConsumer) + NotCoordinatorForConsumer, + IllegalGeneration, + UnknownMemberId, + RebalanceInProgress) ) diff --git a/pykafka/managedbalancedconsumer.py b/pykafka/managedbalancedconsumer.py index fcde415c4..6a0f80576 100644 --- a/pykafka/managedbalancedconsumer.py +++ b/pykafka/managedbalancedconsumer.py @@ -19,9 +19,12 @@ __all__ = ["ManagedBalancedConsumer"] import itertools import logging +import sys +import weakref from .balancedconsumer import BalancedConsumer from .common import OffsetType +from .exceptions import IllegalGeneration, RebalanceInProgress, UnknownMemberId from .protocol import MemberAssignment from .utils.compat import iteritems, itervalues @@ -46,7 +49,8 @@ def __init__(self, consumer_timeout_ms=-1, auto_start=True, reset_offset_on_start=False, - compacted_topic=True): + compacted_topic=True, + heartbeat_interval_ms=3000): self._cluster = cluster if not isinstance(consumer_group, bytes): raise TypeError("consumer_group must be a bytes object") @@ -66,27 +70,68 @@ def __init__(self, self._auto_offset_reset = auto_offset_reset self._reset_offset_on_start = reset_offset_on_start self._is_compacted_topic = compacted_topic + self._heartbeat_interval_ms = heartbeat_interval_ms self._use_rdkafka = False - self._worker_exception = None - self._running = False + self._running = True self._consumer = None - self._consumer_id = None + self._consumer_id = b'' self._is_group_leader = False + self._worker_trace_logged = False + self._worker_exception = None self._discover_group_coordinator() self._join_group() + self._setup_heartbeat_worker() def _discover_group_coordinator(self): """Set the group coordinator for this consumer.""" - self._group_coordinator = self._cluster.get_group_coordinator(self._consumer_group) + self._group_coordinator = self._cluster.get_group_coordinator( + self._consumer_group) + + def _setup_heartbeat_worker(self): + """Start the heartbeat worker""" + self = weakref.proxy(self) + + def fetcher(): + while True: + try: + if not self._running: + break + self._send_heartbeat() + self._cluster.handler.sleep(self._heartbeat_interval_ms / 1000) + except ReferenceError: + break + except Exception: + # surface all exceptions to the main thread + self._worker_exception = sys.exc_info() + break + log.debug("Heartbeat worker exiting") + log.info("Starting heartbeat worker") + return self._cluster.handler.spawn( + fetcher, name="pykafka.ManagedBalancedConsumer.heartbeats") + + def _send_heartbeat(self): + res = self._group_coordinator.group_heartbeat( + self._consumer_group, + self._generation_id, + self._consumer_id + ) + log.debug(res.error_code) + if res.error_code == IllegalGeneration.ERROR_CODE: + self._join_group() + elif res.error_code == RebalanceInProgress.ERROR_CODE: + self._join_group() + elif res.error_code == UnknownMemberId.ERROR_CODE: + self._join_group() def _join_group(self): - res = self._group_coordinator.join_managed_consumer_group(self._consumer_group) + res = self._group_coordinator.join_managed_consumer_group( + self._consumer_group, self._consumer_id) self._generation_id = res.generation_id self._consumer_id = res.member_id all_members = [] - if len(res.members) > 0: + if len(res.members) > 1: self._is_group_leader = True all_members = [member_id for member_id, _ in iteritems(res.members)] group_assignments = [] diff --git a/pykafka/protocol.py b/pykafka/protocol.py index 1e92163d7..0a04795d2 100644 --- a/pykafka/protocol.py +++ b/pykafka/protocol.py @@ -1257,7 +1257,7 @@ class JoinGroupRequest(Request): ProtocolName => string ProtocolMetadata => bytes """ - def __init__(self, group_id, member_id=b'', session_timeout=10000): + def __init__(self, group_id, member_id, session_timeout=30000): """Create a new group join request""" self.protocol = ConsumerGroupProtocol self.group_id = group_id @@ -1546,10 +1546,7 @@ def __init__(self, buff): """ fmt = 'h' response = struct_helpers.unpack_from(fmt, buff, 0) - - error_code = response[0] - if error_code != 0: - self.raise_error(error_code, response) + self.error_code = response[0] class LeaveGroupRequest(Request): From 333d027a93f133ae9986ccd919e65c3f6c0a72f6 Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Thu, 25 Feb 2016 11:22:00 -0800 Subject: [PATCH 22/70] fix protocol errors --- pykafka/managedbalancedconsumer.py | 2 +- pykafka/protocol.py | 36 ++++++++++++------------------ 2 files changed, 15 insertions(+), 23 deletions(-) diff --git a/pykafka/managedbalancedconsumer.py b/pykafka/managedbalancedconsumer.py index 6a0f80576..5f5656981 100644 --- a/pykafka/managedbalancedconsumer.py +++ b/pykafka/managedbalancedconsumer.py @@ -131,7 +131,7 @@ def _join_group(self): self._generation_id = res.generation_id self._consumer_id = res.member_id all_members = [] - if len(res.members) > 1: + if res.leader_id == self._consumer_id: self._is_group_leader = True all_members = [member_id for member_id, _ in iteritems(res.members)] group_assignments = [] diff --git a/pykafka/protocol.py b/pykafka/protocol.py index 0a04795d2..442624696 100644 --- a/pykafka/protocol.py +++ b/pykafka/protocol.py @@ -1328,7 +1328,7 @@ def __init__(self, buff): :param buff: Serialized message :type buff: :class:`bytearray` """ - fmt = 'hiSSS[SS ]' + fmt = 'hiSSS[SY ]' response = struct_helpers.unpack_from(fmt, buff, 0) error_code = response[0] @@ -1353,51 +1353,46 @@ class MemberAssignment(object): Partition => int32 UserData => bytes """ - def __init__(self, partition_assignment, version=1, user_data=b""): + def __init__(self, partition_assignment, version=1): self.version = version self.partition_assignment = partition_assignment - self.user_data = user_data @classmethod def from_bytestring(cls, buff): if len(buff) == 0: - return cls([]) - fmt = 'h [S [i ] ] S' + return cls(tuple()) + fmt = 'h [S [i ] ]' response = struct_helpers.unpack_from(fmt, buff, 0) version = response[0] partition_assignment = response[1] - user_data = response[2] - return cls(partition_assignment, version=version, user_data=user_data) + return cls(partition_assignment, version=version) def __len__(self): # version + len(partition assignment) size = 2 + 4 for topic_name, partitions in self.partition_assignment: - # len(topic_name) + topic_name - size += 2 + len(topic_name) + # len(topic_name) + topic_name + len(partitions) + size += 2 + len(topic_name) + 4 size += 4 * len(partitions) - # len(user data) + user data - size += 4 + len(self.user_data) return size def get_bytes(self): output = bytearray(len(self)) offset = 0 fmt = '!hi' - struct.pack_into(fmt, output, offset, self.version, len(self.partition_assignment)) + struct.pack_into(fmt, output, offset, self.version, + len(self.partition_assignment)) offset += struct.calcsize(fmt) for topic_name, partitions in self.partition_assignment: - fmt = '!h%ds' % len(topic_name) - struct.pack_into(fmt, output, offset, len(topic_name), topic_name) + fmt = '!h%dsi' % len(topic_name) + struct.pack_into(fmt, output, offset, len(topic_name), topic_name, + len(partitions)) offset += struct.calcsize(fmt) for partition_id in partitions: fmt = '!i' struct.pack_into(fmt, output, offset, partition_id) offset += struct.calcsize(fmt) - fmt = '!i%ds' % len(self.user_data) - struct.pack_into(fmt, output, offset, len(self.user_data), self.user_data) - offset += struct.calcsize(fmt) return output @@ -1476,12 +1471,9 @@ def __init__(self, buff): :param buff: Serialized message :type buff: :class:`bytearray` """ - fmt = 'hS' + fmt = 'hY' response = struct_helpers.unpack_from(fmt, buff, 0) - - error_code = response[0] - if error_code != 0: - self.raise_error(error_code, response) + self.error_code = response[0] self.member_assignment = MemberAssignment.from_bytestring(response[1]) From 384004a910fc24ef1097a094c8c5432f0e241af8 Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Thu, 25 Feb 2016 11:22:38 -0800 Subject: [PATCH 23/70] properly extract partition assignment --- pykafka/managedbalancedconsumer.py | 3 ++- 1 file changed, 2 insertions(+), 1 deletion(-) diff --git a/pykafka/managedbalancedconsumer.py b/pykafka/managedbalancedconsumer.py index 5f5656981..17fab90d4 100644 --- a/pykafka/managedbalancedconsumer.py +++ b/pykafka/managedbalancedconsumer.py @@ -149,10 +149,11 @@ def _join_group(self): if self._is_group_leader: assignment = leader_assignment[1] else: - assignment = res.member_assignment.partition_assignment[1] + assignment = res.member_assignment.partition_assignment[0][1] self._setup_internal_consumer( partitions=[p for p in itervalues(self._topic.partitions) if p.id in assignment]) + self._raise_worker_exceptions() def _decide_partitions(self, participants, member_id): # Freeze and sort partitions so we always have the same results From bf0c52917258b2a13690f59e6959f18b932d7767 Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Thu, 25 Feb 2016 11:35:31 -0800 Subject: [PATCH 24/70] explicitly leave group on stop() --- pykafka/broker.py | 7 ++++++- pykafka/managedbalancedconsumer.py | 7 +++++++ 2 files changed, 13 insertions(+), 1 deletion(-) diff --git a/pykafka/broker.py b/pykafka/broker.py index 667dffd03..5cfa28fd5 100644 --- a/pykafka/broker.py +++ b/pykafka/broker.py @@ -27,7 +27,8 @@ FetchRequest, FetchResponse, OffsetRequest, OffsetResponse, MetadataRequest, MetadataResponse, OffsetCommitRequest, OffsetCommitResponse, OffsetFetchRequest, OffsetFetchResponse, ProduceResponse, JoinGroupRequest, JoinGroupResponse, - SyncGroupRequest, SyncGroupResponse, HeartbeatRequest, HeartbeatResponse) + SyncGroupRequest, SyncGroupResponse, HeartbeatRequest, HeartbeatResponse, + LeaveGroupRequest, LeaveGroupResponse) from .utils.compat import range, iteritems log = logging.getLogger(__name__) @@ -356,6 +357,10 @@ def join_managed_consumer_group(self, consumer_group, member_id): future = self._req_handler.request(JoinGroupRequest(consumer_group, member_id)) return future.get(JoinGroupResponse) + def leave_managed_consumer_group(self, consumer_group, member_id): + future = self._req_handler.request(LeaveGroupRequest(consumer_group, member_id)) + return future.get(LeaveGroupResponse) + def sync_group(self, consumer_group, generation_id, member_id, group_assignment): future = self._req_handler.request( SyncGroupRequest(consumer_group, generation_id, member_id, group_assignment)) diff --git a/pykafka/managedbalancedconsumer.py b/pykafka/managedbalancedconsumer.py index 17fab90d4..0ccf3e18e 100644 --- a/pykafka/managedbalancedconsumer.py +++ b/pykafka/managedbalancedconsumer.py @@ -177,3 +177,10 @@ def _decide_partitions(self, participants, member_id): len(participants), len(all_parts), len(new_partitions)) log.debug('My partitions: %s', [p_to_str(p) for p in new_partitions]) return new_partitions + + def stop(self): + self._running = False + if self._consumer is not None: + self._consumer.stop() + self._group_coordinator.leave_managed_consumer_group(self._consumer_group, + self._consumer_id) From d90a943dbce7f6ecfa4b4232a93448ff05f20b2f Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Thu, 25 Feb 2016 11:42:51 -0800 Subject: [PATCH 25/70] allow managed consumers to be created via Topic' --- pykafka/topic.py | 18 +++++++++++++----- 1 file changed, 13 insertions(+), 5 deletions(-) diff --git a/pykafka/topic.py b/pykafka/topic.py index 526e177db..9625ac434 100644 --- a/pykafka/topic.py +++ b/pykafka/topic.py @@ -24,6 +24,7 @@ from .common import OffsetType from .exceptions import LeaderNotAvailable from .handlers import GEventHandler +from .managedbalancedconsumer import ManagedBalancedConsumer from .partition import Partition from .producer import Producer from .protocol import PartitionOffsetRequest @@ -186,13 +187,20 @@ def get_simple_consumer(self, consumer_group=consumer_group, **kwargs) - def get_balanced_consumer(self, consumer_group, **kwargs): + def get_balanced_consumer(self, consumer_group, managed=False, **kwargs): """Return a BalancedConsumer of this topic :param consumer_group: The name of the consumer group to join :type consumer_group: bytes + :param managed: If True, manage the consumer group with Kafka using the 0.9 + group management api (requires Kafka >=0.9)) + :type managed: bool """ - if "zookeeper_connect" not in kwargs and \ - self._cluster._zookeeper_connect is not None: - kwargs['zookeeper_connect'] = self._cluster._zookeeper_connect - return BalancedConsumer(self, self._cluster, consumer_group, **kwargs) + if not managed: + if "zookeeper_connect" not in kwargs and \ + self._cluster._zookeeper_connect is not None: + kwargs['zookeeper_connect'] = self._cluster._zookeeper_connect + cls = BalancedConsumer + else: + cls = ManagedBalancedConsumer + return cls(self, self._cluster, consumer_group, **kwargs) From b6d277072825713cebbaddd4f905a3e7ed9ded11 Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Thu, 25 Feb 2016 11:56:22 -0800 Subject: [PATCH 26/70] fix existing protocol tests --- pykafka/protocol.py | 2 +- tests/pykafka/test_protocol.py | 18 +++++++++--------- 2 files changed, 10 insertions(+), 10 deletions(-) diff --git a/pykafka/protocol.py b/pykafka/protocol.py index 442624696..a24d2ac42 100644 --- a/pykafka/protocol.py +++ b/pykafka/protocol.py @@ -1488,7 +1488,7 @@ class HeartbeatRequest(Request): MemberId => string """ def __init__(self, group_id, generation_id, member_id): - """Create a new group join request""" + """Create a new heartbeat request""" self.group_id = group_id self.generation_id = generation_id self.member_id = member_id diff --git a/tests/pykafka/test_protocol.py b/tests/pykafka/test_protocol.py index 567e8f787..2b529ccad 100644 --- a/tests/pykafka/test_protocol.py +++ b/tests/pykafka/test_protocol.py @@ -79,7 +79,7 @@ def test_snappy_compression(self): def test_partition_error(self): # Response has a UnknownTopicOrPartition error for test/0 response = protocol.ProduceResponse( - buffer(b'\x00\x00\x00\x01\x00\x04test\x00\x00\x00\x01\x00\x00\x00\x00\x00\x03\x00\x00\x00\x00\x00\x00\x00\x02') + buffer(b'\x00\x00\x00\x01\x00\x04test\x00\x00\x00\x01\x00\x00\x00\x00\x00\x03\x00\x00\x00\x00\x00\x00\x00\x02') ) self.assertEqual(response.topics[b'test'][0].err, 3) @@ -264,7 +264,7 @@ def test_consumer_group_protocol_metadata(self): msg = meta.get_bytes() self.assertEqual( msg, - bytearray(b'\x00\x01\x00\x00\x00\x01\x00\ndummytopic\x00\x00\x00\x0ctestuserdata') + bytearray(b'\x00\x00\x00\x00\x00\x01\x00\ndummytopic\x00\x00\x00\x0ctestuserdata') ) def test_join_group_request(self): @@ -272,21 +272,21 @@ def test_join_group_request(self): msg = req.get_bytes() self.assertEqual( msg, - bytearray(b'\x00\x00\x00z\x00\x0b\x00\x01\x00\x00\x00\x00\x00\x07pykafka\x00\ndummygroup\x00\x00u0\x00\ntestmember\x00\x08consumer\x00\x00\x00\x01\x00\x17dummyassignmentstrategy\x00\x00\x00"\x00\x01\x00\x00\x00\x01\x00\ndummytopic\x00\x00\x00\x0ctestuserdata') + bytearray(b'\x00\x00\x00z\x00\x0b\x00\x00\x00\x00\x00\x00\x00\x07pykafka\x00\ndummygroup\x00\x00u0\x00\ntestmember\x00\x08consumer\x00\x00\x00\x01\x00\x17dummyassignmentstrategy\x00\x00\x00"\x00\x00\x00\x00\x00\x01\x00\ndummytopic\x00\x00\x00\x0ctestuserdata') ) def test_join_group_response(self): response = protocol.JoinGroupResponse( bytearray('\x00\x00\x00\x00\x00\x01\x00\x17dummyassignmentstrategy\x00,pykafka-b2361322-674c-4e26-9194-305962636e57\x00,pykafka-b2361322-674c-4e26-9194-305962636e57\x00\x00\x00\x01\x00,pykafka-b2361322-674c-4e26-9194-305962636e57\x00\x00\x00"\x00\x00\x00\x00\x00\x01\x00\ndummytopic\x00\x00\x00\x0ctestuserdata\x00\x00\x00\x00') ) - self.assertEqual(response.generation_id, 0) + self.assertEqual(response.generation_id, 1) self.assertEqual(response.group_protocol, 'dummyassignmentstrategy') self.assertEqual(response.leader_id, 'pykafka-b2361322-674c-4e26-9194-305962636e57') self.assertEqual(response.member_id, 'pykafka-b2361322-674c-4e26-9194-305962636e57') self.assertEqual(response.members, - {'pykafka-b2361322-674c-4e26-9194-305962636e57': ''}) + {'pykafka-b2361322-674c-4e26-9194-305962636e57': '\x00\x00\x00\x00\x00\x01\x00\ndummytopic\x00\x00\x00\x0ctestuserdata'}) def test_member_assignment_construction(self): assignment = protocol.MemberAssignment([(b"mytopic1", [3, 5, 7, 9]), @@ -294,7 +294,7 @@ def test_member_assignment_construction(self): msg = assignment.get_bytes() self.assertEqual( msg, - bytearray(b'\x00\x01\x00\x00\x00\x02\x00\x08mytopic1\x00\x00\x00\x03\x00\x00\x00\x05\x00\x00\x00\x07\x00\x00\x00\t\x00\x08mytopic2\x00\x00\x00\x02\x00\x00\x00\x04\x00\x00\x00\x06\x00\x00\x00\x08\x00\x00\x00\x00') + bytearray(b'\x00\x01\x00\x00\x00\x02\x00\x08mytopic1\x00\x00\x00\x04\x00\x00\x00\x03\x00\x00\x00\x05\x00\x00\x00\x07\x00\x00\x00\t\x00\x08mytopic2\x00\x00\x00\x04\x00\x00\x00\x02\x00\x00\x00\x04\x00\x00\x00\x06\x00\x00\x00\x08') ) def test_sync_group_request(self): @@ -309,7 +309,7 @@ def test_sync_group_request(self): msg = req.get_bytes() self.assertEqual( msg, - bytearray(b'\x00\x00\x00\xd0\x00\x0e\x00\x01\x00\x00\x00\x00\x00\x07pykafka\x00\ndummygroup\x00\x00\x00\x01\x00\x0btestmember1\x00\x00\x00\x02\x00\x0btestmember1\x00\x00\x00>\x00\x01\x00\x00\x00\x02\x00\x08mytopic1\x00\x00\x00\x03\x00\x00\x00\x05\x00\x00\x00\x07\x00\x00\x00\t\x00\x08mytopic2\x00\x00\x00\x03\x00\x00\x00\x05\x00\x00\x00\x07\x00\x00\x00\t\x00\x00\x00\x00\x00\x0btestmember2\x00\x00\x00>\x00\x01\x00\x00\x00\x02\x00\x08mytopic1\x00\x00\x00\x02\x00\x00\x00\x04\x00\x00\x00\x06\x00\x00\x00\x08\x00\x08mytopic2\x00\x00\x00\x02\x00\x00\x00\x04\x00\x00\x00\x06\x00\x00\x00\x08\x00\x00\x00\x00') + bytearray(b'\x00\x00\x00\xd8\x00\x0e\x00\x00\x00\x00\x00\x00\x00\x07pykafka\x00\ndummygroup\x00\x00\x00\x01\x00\x0btestmember1\x00\x00\x00\x02\x00\x0btestmember1\x00\x00\x00B\x00\x01\x00\x00\x00\x02\x00\x08mytopic1\x00\x00\x00\x04\x00\x00\x00\x03\x00\x00\x00\x05\x00\x00\x00\x07\x00\x00\x00\t\x00\x08mytopic2\x00\x00\x00\x04\x00\x00\x00\x03\x00\x00\x00\x05\x00\x00\x00\x07\x00\x00\x00\t\x00\x0btestmember2\x00\x00\x00B\x00\x01\x00\x00\x00\x02\x00\x08mytopic1\x00\x00\x00\x04\x00\x00\x00\x02\x00\x00\x00\x04\x00\x00\x00\x06\x00\x00\x00\x08\x00\x08mytopic2\x00\x00\x00\x04\x00\x00\x00\x02\x00\x00\x00\x04\x00\x00\x00\x06\x00\x00\x00\x08') ) def test_heartbeat_request(self): @@ -317,7 +317,7 @@ def test_heartbeat_request(self): msg = req.get_bytes() self.assertEqual( msg, - bytearray(b'\x00\x00\x00-\x00\x0c\x00\x01\x00\x00\x00\x00\x00\x07pykafka\x00\ndummygroup\x00\x00\x00\x01\x00\ntestmember') + bytearray(b'\x00\x00\x00-\x00\x0c\x00\x00\x00\x00\x00\x00\x00\x07pykafka\x00\ndummygroup\x00\x00\x00\x01\x00\ntestmember') ) def test_leave_group_request(self): @@ -325,7 +325,7 @@ def test_leave_group_request(self): msg = req.get_bytes() self.assertEqual( msg, - bytearray(b'\x00\x00\x00)\x00\r\x00\x01\x00\x00\x00\x00\x00\x07pykafka\x00\ndummygroup\x00\ntestmember') + bytearray(b'\x00\x00\x00)\x00\r\x00\x00\x00\x00\x00\x00\x00\x07pykafka\x00\ndummygroup\x00\ntestmember') ) From 834bb58e120f74b71ba20ce233c71d404dcff795 Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Thu, 25 Feb 2016 12:00:09 -0800 Subject: [PATCH 27/70] consolidate duplcated _decide_partitions` --- pykafka/balancedconsumer.py | 6 ++++-- pykafka/managedbalancedconsumer.py | 26 +------------------------- 2 files changed, 5 insertions(+), 27 deletions(-) diff --git a/pykafka/balancedconsumer.py b/pykafka/balancedconsumer.py index 31212cc97..6569a42d7 100644 --- a/pykafka/balancedconsumer.py +++ b/pykafka/balancedconsumer.py @@ -384,7 +384,7 @@ def _get_internal_consumer(self, partitions=None, start=True): consumer_id=self._consumer_id ) - def _decide_partitions(self, participants): + def _decide_partitions(self, participants, consumer_id=None): """Decide which partitions belong to this consumer. Uses the consumer rebalancing algorithm described here @@ -398,6 +398,8 @@ def _decide_partitions(self, participants): :param participants: Sorted list of ids of all other consumers in this consumer group. :type participants: Iterable of `bytes` + :param consumer_id: The ID of the consumer for which to generate a partition + assignment. Defaults to `self._consumer_id` """ # Freeze and sort partitions so we always have the same results p_to_str = lambda p: '-'.join([str(p.topic.name), str(p.leader.id), str(p.id)]) @@ -406,7 +408,7 @@ def _decide_partitions(self, participants): # get start point, # of partitions, and remainder participants = sorted(participants) # just make sure it's sorted. - idx = participants.index(self._consumer_id) + idx = participants.index(consumer_id or self._consumer_id) parts_per_consumer = len(all_parts) // len(participants) remainder_ppc = len(all_parts) % len(participants) diff --git a/pykafka/managedbalancedconsumer.py b/pykafka/managedbalancedconsumer.py index 0ccf3e18e..510c6d53c 100644 --- a/pykafka/managedbalancedconsumer.py +++ b/pykafka/managedbalancedconsumer.py @@ -17,7 +17,6 @@ limitations under the License. """ __all__ = ["ManagedBalancedConsumer"] -import itertools import logging import sys import weakref @@ -137,7 +136,7 @@ def _join_group(self): group_assignments = [] leader_assignment = None for member_id, metadata in iteritems(res.members): - partitions = self._decide_partitions(all_members, member_id) + partitions = self._decide_partitions(all_members, consumer_id=member_id) assignment = (self._topic.name, [p.id for p in partitions]) if member_id == self._consumer_id and self._is_group_leader: leader_assignment = assignment @@ -155,29 +154,6 @@ def _join_group(self): if p.id in assignment]) self._raise_worker_exceptions() - def _decide_partitions(self, participants, member_id): - # Freeze and sort partitions so we always have the same results - p_to_str = lambda p: '-'.join([str(p.topic.name), str(p.leader.id), str(p.id)]) - all_parts = self._topic.partitions.values() - all_parts = sorted(all_parts, key=p_to_str) - - # get start point, # of partitions, and remainder - participants = sorted(participants) # just make sure it's sorted. - idx = participants.index(member_id) - parts_per_consumer = len(all_parts) // len(participants) - remainder_ppc = len(all_parts) % len(participants) - - start = parts_per_consumer * idx + min(idx, remainder_ppc) - num_parts = parts_per_consumer + (0 if (idx + 1 > remainder_ppc) else 1) - - # assign partitions from i*N to (i+1)*N - 1 to consumer Ci - new_partitions = itertools.islice(all_parts, start, start + num_parts) - new_partitions = set(new_partitions) - log.info('Balancing %i participants for %i partitions.\nOwning %i partitions.', - len(participants), len(all_parts), len(new_partitions)) - log.debug('My partitions: %s', [p_to_str(p) for p in new_partitions]) - return new_partitions - def stop(self): self._running = False if self._consumer is not None: From 38485dd657ef733c439b6c69e503cb28657da0ca Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Thu, 25 Feb 2016 12:12:34 -0800 Subject: [PATCH 28/70] implement response protocol tests --- pykafka/protocol.py | 5 +---- tests/pykafka/test_protocol.py | 17 +++++++++++++++++ 2 files changed, 18 insertions(+), 4 deletions(-) diff --git a/pykafka/protocol.py b/pykafka/protocol.py index a24d2ac42..da3b77f31 100644 --- a/pykafka/protocol.py +++ b/pykafka/protocol.py @@ -1600,7 +1600,4 @@ def __init__(self, buff): """ fmt = 'h' response = struct_helpers.unpack_from(fmt, buff, 0) - - error_code = response[0] - if error_code != 0: - self.raise_error(error_code, response) + self.error_code = response[0] diff --git a/tests/pykafka/test_protocol.py b/tests/pykafka/test_protocol.py index 2b529ccad..3eaa9966c 100644 --- a/tests/pykafka/test_protocol.py +++ b/tests/pykafka/test_protocol.py @@ -312,6 +312,15 @@ def test_sync_group_request(self): bytearray(b'\x00\x00\x00\xd8\x00\x0e\x00\x00\x00\x00\x00\x00\x00\x07pykafka\x00\ndummygroup\x00\x00\x00\x01\x00\x0btestmember1\x00\x00\x00\x02\x00\x0btestmember1\x00\x00\x00B\x00\x01\x00\x00\x00\x02\x00\x08mytopic1\x00\x00\x00\x04\x00\x00\x00\x03\x00\x00\x00\x05\x00\x00\x00\x07\x00\x00\x00\t\x00\x08mytopic2\x00\x00\x00\x04\x00\x00\x00\x03\x00\x00\x00\x05\x00\x00\x00\x07\x00\x00\x00\t\x00\x0btestmember2\x00\x00\x00B\x00\x01\x00\x00\x00\x02\x00\x08mytopic1\x00\x00\x00\x04\x00\x00\x00\x02\x00\x00\x00\x04\x00\x00\x00\x06\x00\x00\x00\x08\x00\x08mytopic2\x00\x00\x00\x04\x00\x00\x00\x02\x00\x00\x00\x04\x00\x00\x00\x06\x00\x00\x00\x08') ) + def test_sync_group_response(self): + response = protocol.SyncGroupResponse( + bytearray('\x00\x00\x00\x00\x00H\x00\x01\x00\x00\x00\x01\x00\x14testtopic_replicated\x00\x00\x00\n\x00\x00\x00\x06\x00\x00\x00\x07\x00\x00\x00\x08\x00\x00\x00\t\x00\x00\x00\x00\x00\x00\x00\x01\x00\x00\x00\x02\x00\x00\x00\x03\x00\x00\x00\x04\x00\x00\x00\x05,pyk') + ) + self.assertEqual(response.error_code, 0) + expected_assignment = [('testtopic_replicated', [6, 7, 8, 9, 0, 1, 2, 3, 4, 5])] + self.assertEqual(response.member_assignment.partition_assignment, + expected_assignment) + def test_heartbeat_request(self): req = protocol.HeartbeatRequest(b'dummygroup', 1, b'testmember') msg = req.get_bytes() @@ -320,6 +329,10 @@ def test_heartbeat_request(self): bytearray(b'\x00\x00\x00-\x00\x0c\x00\x00\x00\x00\x00\x00\x00\x07pykafka\x00\ndummygroup\x00\x00\x00\x01\x00\ntestmember') ) + def test_heartbeat_response(self): + response = protocol.HeartbeatResponse(bytearray('\x00\x00\x00\x01\x00\x14')) + self.assertEqual(response.error_code, 0) + def test_leave_group_request(self): req = protocol.LeaveGroupRequest(b'dummygroup', b'testmember') msg = req.get_bytes() @@ -328,6 +341,10 @@ def test_leave_group_request(self): bytearray(b'\x00\x00\x00)\x00\r\x00\x00\x00\x00\x00\x00\x00\x07pykafka\x00\ndummygroup\x00\ntestmember') ) + def test_leave_group_response(self): + response = protocol.LeaveGroupResponse(bytearray('\x00\x00\x00\x01\x00\x14')) + self.assertEqual(response.error_code, 0) + if __name__ == '__main__': unittest2.main() From dd9f327bbab3fd4e202c779d0e72685a7c153013 Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Thu, 25 Feb 2016 12:13:43 -0800 Subject: [PATCH 29/70] remove tiny function --- pykafka/managedbalancedconsumer.py | 8 ++------ 1 file changed, 2 insertions(+), 6 deletions(-) diff --git a/pykafka/managedbalancedconsumer.py b/pykafka/managedbalancedconsumer.py index 510c6d53c..e8fbc2cdf 100644 --- a/pykafka/managedbalancedconsumer.py +++ b/pykafka/managedbalancedconsumer.py @@ -79,14 +79,10 @@ def __init__(self, self._worker_trace_logged = False self._worker_exception = None - self._discover_group_coordinator() - self._join_group() - self._setup_heartbeat_worker() - - def _discover_group_coordinator(self): - """Set the group coordinator for this consumer.""" self._group_coordinator = self._cluster.get_group_coordinator( self._consumer_group) + self._join_group() + self._setup_heartbeat_worker() def _setup_heartbeat_worker(self): """Start the heartbeat worker""" From d7b19b59700824390525ee6f02c8f3afe745e408 Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Thu, 25 Feb 2016 14:20:59 -0800 Subject: [PATCH 30/70] refactor core joingroup semantics --- pykafka/managedbalancedconsumer.py | 78 +++++++++++++++++++----------- pykafka/protocol.py | 16 +++--- 2 files changed, 59 insertions(+), 35 deletions(-) diff --git a/pykafka/managedbalancedconsumer.py b/pykafka/managedbalancedconsumer.py index e8fbc2cdf..6a398729e 100644 --- a/pykafka/managedbalancedconsumer.py +++ b/pykafka/managedbalancedconsumer.py @@ -25,7 +25,7 @@ from .common import OffsetType from .exceptions import IllegalGeneration, RebalanceInProgress, UnknownMemberId from .protocol import MemberAssignment -from .utils.compat import iteritems, itervalues +from .utils.compat import itervalues, iterkeys log = logging.getLogger(__name__) @@ -50,6 +50,10 @@ def __init__(self, reset_offset_on_start=False, compacted_topic=True, heartbeat_interval_ms=3000): + """A self-balancing consumer that uses Kafka 0.9's Group Membership API + + Implements the Group Management API semantics for Kafka 0.9 compatibility + """ self._cluster = cluster if not isinstance(consumer_group, bytes): raise TypeError("consumer_group must be a bytes object") @@ -75,7 +79,6 @@ def __init__(self, self._consumer = None self._consumer_id = b'' - self._is_group_leader = False self._worker_trace_logged = False self._worker_exception = None @@ -120,34 +123,53 @@ def _send_heartbeat(self): elif res.error_code == UnknownMemberId.ERROR_CODE: self._join_group() + def _decide_partitions(self, participants, consumer_id=None): + """Decide which partitions belong to this consumer + + Thin formatting wrapper around + `pykafka.balancedconsumer.BalancedConsumer._decide_partitions` + """ + parts = super(ManagedBalancedConsumer, self)._decide_partitions( + participants, consumer_id=consumer_id) + return [p.id for p in parts] + def _join_group(self): - res = self._group_coordinator.join_managed_consumer_group( + """Join a managed consumer group + + Equivalent to `pykafka.balancedconsumer.BalancedConsumer._rebalance`, + but uses the Kafka 0.9 Group Membership API instead of ZooKeeper to manage + group state + """ + # send the initial JoinGroupRequest. Assigns a member id and tells the coordinator + # about this consumer. + join_result = self._group_coordinator.join_managed_consumer_group( self._consumer_group, self._consumer_id) - self._generation_id = res.generation_id - self._consumer_id = res.member_id - all_members = [] - if res.leader_id == self._consumer_id: - self._is_group_leader = True - all_members = [member_id for member_id, _ in iteritems(res.members)] - group_assignments = [] - leader_assignment = None - for member_id, metadata in iteritems(res.members): - partitions = self._decide_partitions(all_members, consumer_id=member_id) - assignment = (self._topic.name, [p.id for p in partitions]) - if member_id == self._consumer_id and self._is_group_leader: - leader_assignment = assignment - group_assignments.append((member_id, MemberAssignment([assignment]))) - res = self._group_coordinator.sync_group(self._consumer_group, - self._generation_id, - self._consumer_id, - group_assignments) - if self._is_group_leader: - assignment = leader_assignment[1] - else: - assignment = res.member_assignment.partition_assignment[0][1] - self._setup_internal_consumer( - partitions=[p for p in itervalues(self._topic.partitions) - if p.id in assignment]) + self._generation_id = join_result.generation_id + self._consumer_id = join_result.member_id + + # generate partition assignments for each group member + # if this consumer is not the leader, join_result.members will be empty + group_assignments = [ + MemberAssignment([ + (self._topic.name, + self._decide_partitions(iterkeys(join_result.members), + consumer_id=member_id)) + ], member_id=member_id) for member_id in join_result.members] + + # perform the SyncGroupRequest. If this consumer is the group leader, this request + # informs the other members of the group of their partition assignments. + # This request is also used to fetch the partition assignment for this consumer + # The group leader *could* tell itself its own assignment instead of using the + # result of this request, but it does the latter to ensure consistency. + sync_result = self._group_coordinator.sync_group(self._consumer_group, + self._generation_id, + self._consumer_id, + group_assignments) + my_partition_ids = sync_result.member_assignment.partition_assignment[0][1] + # set up a SimpleConsumer consuming the returned partitions + my_partitions = [p for p in itervalues(self._topic.partitions) + if p.id in my_partition_ids] + self._setup_internal_consumer(partitions=my_partitions) self._raise_worker_exceptions() def stop(self): diff --git a/pykafka/protocol.py b/pykafka/protocol.py index da3b77f31..ef507fa03 100644 --- a/pykafka/protocol.py +++ b/pykafka/protocol.py @@ -1353,7 +1353,8 @@ class MemberAssignment(object): Partition => int32 UserData => bytes """ - def __init__(self, partition_assignment, version=1): + def __init__(self, partition_assignment, member_id=None, version=1): + self.member_id = member_id self.version = version self.partition_assignment = partition_assignment @@ -1423,9 +1424,9 @@ def __len__(self): # + len(member id) + member id + len(group assignment) size += 2 + len(self.member_id) + 4 # group assignment tuples - for member_id, member_assignment in self.group_assignment: + for member_assignment in self.group_assignment: # + len(member id) + member id + len(member assignment) + member assignment - size += 2 + len(member_id) + 4 + len(member_assignment) + size += 2 + len(member_assignment.member_id) + 4 + len(member_assignment) return size @property @@ -1447,11 +1448,12 @@ def get_bytes(self): self.generation_id, len(self.member_id), self.member_id, len(self.group_assignment)) offset += struct.calcsize(fmt) - for member_id, member_assignment in self.group_assignment: + for member_assignment in self.group_assignment: assignment_bytes = bytes(member_assignment.get_bytes()) - fmt = '!h%dsi%ds' % (len(member_id), len(assignment_bytes)) - struct.pack_into(fmt, output, offset, len(member_id), member_id, - len(assignment_bytes), assignment_bytes) + fmt = '!h%dsi%ds' % (len(member_assignment.member_id), len(assignment_bytes)) + struct.pack_into(fmt, output, offset, len(member_assignment.member_id), + member_assignment.member_id, len(assignment_bytes), + assignment_bytes) offset += struct.calcsize(fmt) return output From d3e6561356381309498fd962f74fb05ba194a2a7 Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Thu, 25 Feb 2016 14:32:38 -0800 Subject: [PATCH 31/70] commit offsets on rebalance use rebalancing lock --- pykafka/managedbalancedconsumer.py | 67 ++++++++++++++++-------------- 1 file changed, 36 insertions(+), 31 deletions(-) diff --git a/pykafka/managedbalancedconsumer.py b/pykafka/managedbalancedconsumer.py index 6a398729e..5d5951b09 100644 --- a/pykafka/managedbalancedconsumer.py +++ b/pykafka/managedbalancedconsumer.py @@ -77,6 +77,7 @@ def __init__(self, self._use_rdkafka = False self._running = True + self._rebalancing_lock = cluster.handler.Lock() self._consumer = None self._consumer_id = b'' self._worker_trace_logged = False @@ -115,7 +116,6 @@ def _send_heartbeat(self): self._generation_id, self._consumer_id ) - log.debug(res.error_code) if res.error_code == IllegalGeneration.ERROR_CODE: self._join_group() elif res.error_code == RebalanceInProgress.ERROR_CODE: @@ -140,36 +140,41 @@ def _join_group(self): but uses the Kafka 0.9 Group Membership API instead of ZooKeeper to manage group state """ - # send the initial JoinGroupRequest. Assigns a member id and tells the coordinator - # about this consumer. - join_result = self._group_coordinator.join_managed_consumer_group( - self._consumer_group, self._consumer_id) - self._generation_id = join_result.generation_id - self._consumer_id = join_result.member_id - - # generate partition assignments for each group member - # if this consumer is not the leader, join_result.members will be empty - group_assignments = [ - MemberAssignment([ - (self._topic.name, - self._decide_partitions(iterkeys(join_result.members), - consumer_id=member_id)) - ], member_id=member_id) for member_id in join_result.members] - - # perform the SyncGroupRequest. If this consumer is the group leader, this request - # informs the other members of the group of their partition assignments. - # This request is also used to fetch the partition assignment for this consumer - # The group leader *could* tell itself its own assignment instead of using the - # result of this request, but it does the latter to ensure consistency. - sync_result = self._group_coordinator.sync_group(self._consumer_group, - self._generation_id, - self._consumer_id, - group_assignments) - my_partition_ids = sync_result.member_assignment.partition_assignment[0][1] - # set up a SimpleConsumer consuming the returned partitions - my_partitions = [p for p in itervalues(self._topic.partitions) - if p.id in my_partition_ids] - self._setup_internal_consumer(partitions=my_partitions) + if self._consumer is not None: + self.commit_offsets() + + with self._rebalancing_lock: + # send the initial JoinGroupRequest. Assigns a member id and tells the + # coordinator about this consumer. + join_result = self._group_coordinator.join_managed_consumer_group( + self._consumer_group, self._consumer_id) + self._generation_id = join_result.generation_id + self._consumer_id = join_result.member_id + + # generate partition assignments for each group member + # if this consumer is not the leader, join_result.members will be empty + group_assignments = [ + MemberAssignment([ + (self._topic.name, + self._decide_partitions(iterkeys(join_result.members), + consumer_id=member_id)) + ], member_id=member_id) for member_id in join_result.members] + + # perform the SyncGroupRequest. If this consumer is the group leader, This + # request informs the other members of the group of their partition + # assignments. This request is also used to fetch the partition assignment + # for this consumer. The group leader *could* tell itself its own assignment + # instead of using the result of this request, but it does the latter to + # ensure consistency. + sync_result = self._group_coordinator.sync_group(self._consumer_group, + self._generation_id, + self._consumer_id, + group_assignments) + my_partition_ids = sync_result.member_assignment.partition_assignment[0][1] + # set up a SimpleConsumer consuming the returned partitions + my_partitions = [p for p in itervalues(self._topic.partitions) + if p.id in my_partition_ids] + self._setup_internal_consumer(partitions=my_partitions) self._raise_worker_exceptions() def stop(self): From 9f7f834ceaea24481fb983f79ebf82e2d4e2e623 Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Thu, 25 Feb 2016 14:59:09 -0800 Subject: [PATCH 32/70] docstrings, name changes, and some more abstraction of BalancedConsumer --- pykafka/balancedconsumer.py | 49 +++++++++++++------------- pykafka/managedbalancedconsumer.py | 56 +++++++++++++++++++++--------- 2 files changed, 64 insertions(+), 41 deletions(-) diff --git a/pykafka/balancedconsumer.py b/pykafka/balancedconsumer.py index 6569a42d7..ee79fec4c 100644 --- a/pykafka/balancedconsumer.py +++ b/pykafka/balancedconsumer.py @@ -283,7 +283,7 @@ def held_offsets(self): return self._consumer.held_offsets def start(self): - """Open connections and join a cluster.""" + """Open connections and join a consumer group.""" try: if self._zookeeper is None: self._setup_zookeeper(self._zookeeper_connect, @@ -341,7 +341,27 @@ def _setup_zookeeper(self, zookeeper_connect, timeout): def _setup_internal_consumer(self, partitions=None, start=True): """Instantiate an internal SimpleConsumer instance""" - self._consumer = self._get_internal_consumer(partitions=partitions, start=start) + # Only re-create internal consumer if something changed. + if partitions != self._partitions: + cns = self._get_internal_consumer(partitions=list(partitions), start=start) + if self._post_rebalance_callback is not None: + old_offsets = (self._consumer.held_offsets + if self._consumer else dict()) + new_offsets = cns.held_offsets + try: + reset_offsets = self._post_rebalance_callback( + self, old_offsets, new_offsets) + except Exception: + log.exception("post rebalance callback threw an exception") + self._worker_exception = sys.exc_info() + return False + + if reset_offsets: + cns.reset_offsets(partition_offsets=[ + (cns.partitions[id_], offset) for + (id_, offset) in iteritems(reset_offsets)]) + self._consumer = cns + return True def _get_internal_consumer(self, partitions=None, start=True): """Instantiate a SimpleConsumer for internal use. @@ -562,29 +582,8 @@ def _rebalance(self): current_zk_parts = self._get_held_partitions() self._remove_partitions(current_zk_parts - new_partitions) self._add_partitions(new_partitions - current_zk_parts) - - # Only re-create internal consumer if something changed. - if new_partitions != self._partitions: - cns = self._get_internal_consumer(list(new_partitions)) - if self._post_rebalance_callback is not None: - old_offsets = (self._consumer.held_offsets - if self._consumer else dict()) - new_offsets = cns.held_offsets - try: - reset_offsets = self._post_rebalance_callback( - self, old_offsets, new_offsets) - except Exception as ex: - log.exception("post rebalance callback threw an exception") - self._worker_exception = sys.exc_info() - break - - if reset_offsets: - cns.reset_offsets(partition_offsets=[ - (cns.partitions[id_], offset) for - (id_, offset) in iteritems(reset_offsets)]) - self._consumer = cns - - log.info('Rebalancing Complete.') + if self._setup_internal_consumer(new_partitions): + log.info('Rebalancing Complete.') break except PartitionOwnedError as ex: if i == self._rebalance_max_retries - 1: diff --git a/pykafka/managedbalancedconsumer.py b/pykafka/managedbalancedconsumer.py index 5d5951b09..688a46793 100644 --- a/pykafka/managedbalancedconsumer.py +++ b/pykafka/managedbalancedconsumer.py @@ -46,13 +46,20 @@ def __init__(self, offsets_commit_max_retries=5, auto_offset_reset=OffsetType.EARLIEST, consumer_timeout_ms=-1, + rebalance_max_retries=5, + rebalance_backoff_ms=2 * 1000, auto_start=True, reset_offset_on_start=False, + post_rebalance_callback=None, compacted_topic=True, heartbeat_interval_ms=3000): """A self-balancing consumer that uses Kafka 0.9's Group Membership API Implements the Group Management API semantics for Kafka 0.9 compatibility + + This class overrides the functionality of + :class:`pykafka.balancedconsumer.BalancedConsumer` that deals with ZooKeeper and + inherits other functionality directly. """ self._cluster = cluster if not isinstance(consumer_group, bytes): @@ -72,6 +79,9 @@ def __init__(self, self._offsets_commit_max_retries = offsets_commit_max_retries self._auto_offset_reset = auto_offset_reset self._reset_offset_on_start = reset_offset_on_start + self._rebalance_max_retries = rebalance_max_retries + self._rebalance_backoff_ms = rebalance_backoff_ms + self._post_rebalance_callback = post_rebalance_callback self._is_compacted_topic = compacted_topic self._heartbeat_interval_ms = heartbeat_interval_ms self._use_rdkafka = False @@ -83,10 +93,8 @@ def __init__(self, self._worker_trace_logged = False self._worker_exception = None - self._group_coordinator = self._cluster.get_group_coordinator( - self._consumer_group) - self._join_group() - self._setup_heartbeat_worker() + if auto_start is True: + self.start() def _setup_heartbeat_worker(self): """Start the heartbeat worker""" @@ -110,18 +118,41 @@ def fetcher(): return self._cluster.handler.spawn( fetcher, name="pykafka.ManagedBalancedConsumer.heartbeats") + def start(self): + """Start this consumer. Must be called before consume() if `auto_start=False`.""" + try: + self._group_coordinator = self._cluster.get_group_coordinator( + self._consumer_group) + self._rebalance() + self._setup_heartbeat_worker() + except Exception: + log.exception("Stopping consumer in response to error") + self.stop() + + def stop(self): + """Stop this consumer + + Should be called as part of a graceful shutdown + """ + self._running = False + if self._consumer is not None: + self._consumer.stop() + self._group_coordinator.leave_managed_consumer_group(self._consumer_group, + self._consumer_id) + def _send_heartbeat(self): + """Send a heartbeat request to the group coordinator and react to the response""" res = self._group_coordinator.group_heartbeat( self._consumer_group, self._generation_id, self._consumer_id ) if res.error_code == IllegalGeneration.ERROR_CODE: - self._join_group() + self._rebalance() elif res.error_code == RebalanceInProgress.ERROR_CODE: - self._join_group() + self._rebalance() elif res.error_code == UnknownMemberId.ERROR_CODE: - self._join_group() + self._rebalance() def _decide_partitions(self, participants, consumer_id=None): """Decide which partitions belong to this consumer @@ -133,8 +164,8 @@ def _decide_partitions(self, participants, consumer_id=None): participants, consumer_id=consumer_id) return [p.id for p in parts] - def _join_group(self): - """Join a managed consumer group + def _rebalance(self): + """Join a managed consumer group and start consuming assigned partitions Equivalent to `pykafka.balancedconsumer.BalancedConsumer._rebalance`, but uses the Kafka 0.9 Group Membership API instead of ZooKeeper to manage @@ -176,10 +207,3 @@ def _join_group(self): if p.id in my_partition_ids] self._setup_internal_consumer(partitions=my_partitions) self._raise_worker_exceptions() - - def stop(self): - self._running = False - if self._consumer is not None: - self._consumer.stop() - self._group_coordinator.leave_managed_consumer_group(self._consumer_group, - self._consumer_id) From 7a20c10dc5fb78f5d0635a09535b60fc135fdf11 Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Thu, 25 Feb 2016 15:05:14 -0800 Subject: [PATCH 33/70] add retry loop to rebalance --- pykafka/managedbalancedconsumer.py | 78 ++++++++++++++++++------------ 1 file changed, 46 insertions(+), 32 deletions(-) diff --git a/pykafka/managedbalancedconsumer.py b/pykafka/managedbalancedconsumer.py index 688a46793..8ccf78a9e 100644 --- a/pykafka/managedbalancedconsumer.py +++ b/pykafka/managedbalancedconsumer.py @@ -23,7 +23,8 @@ from .balancedconsumer import BalancedConsumer from .common import OffsetType -from .exceptions import IllegalGeneration, RebalanceInProgress, UnknownMemberId +from .exceptions import (IllegalGeneration, RebalanceInProgress, UnknownMemberId, + ConsumerStoppedException) from .protocol import MemberAssignment from .utils.compat import itervalues, iterkeys @@ -175,35 +176,48 @@ def _rebalance(self): self.commit_offsets() with self._rebalancing_lock: - # send the initial JoinGroupRequest. Assigns a member id and tells the - # coordinator about this consumer. - join_result = self._group_coordinator.join_managed_consumer_group( - self._consumer_group, self._consumer_id) - self._generation_id = join_result.generation_id - self._consumer_id = join_result.member_id - - # generate partition assignments for each group member - # if this consumer is not the leader, join_result.members will be empty - group_assignments = [ - MemberAssignment([ - (self._topic.name, - self._decide_partitions(iterkeys(join_result.members), - consumer_id=member_id)) - ], member_id=member_id) for member_id in join_result.members] - - # perform the SyncGroupRequest. If this consumer is the group leader, This - # request informs the other members of the group of their partition - # assignments. This request is also used to fetch the partition assignment - # for this consumer. The group leader *could* tell itself its own assignment - # instead of using the result of this request, but it does the latter to - # ensure consistency. - sync_result = self._group_coordinator.sync_group(self._consumer_group, - self._generation_id, - self._consumer_id, - group_assignments) - my_partition_ids = sync_result.member_assignment.partition_assignment[0][1] - # set up a SimpleConsumer consuming the returned partitions - my_partitions = [p for p in itervalues(self._topic.partitions) - if p.id in my_partition_ids] - self._setup_internal_consumer(partitions=my_partitions) + if not self._running: + raise ConsumerStoppedException + log.info('Rebalancing consumer "%s" for topic "%s".' % ( + self._consumer_id, self._topic.name)) + for i in range(self._rebalance_max_retries): + try: + # send the initial JoinGroupRequest. Assigns a member id and tells the + # coordinator about this consumer. + join_result = self._group_coordinator.join_managed_consumer_group( + self._consumer_group, self._consumer_id) + self._generation_id = join_result.generation_id + self._consumer_id = join_result.member_id + + # generate partition assignments for each group member + # if this is not the leader, join_result.members will be empty + group_assignments = [ + MemberAssignment([ + (self._topic.name, + self._decide_partitions(iterkeys(join_result.members), + consumer_id=member_id)) + ], member_id=member_id) for member_id in join_result.members] + + # perform the SyncGroupRequest. If this consumer is the group leader, + # This request informs the other members of the group of their + # partition assignments. This request is also used to fetch The + # partition assignment for this consumer. The group leader *could* + # tell itself its own assignment instead of using the result of this + # request, but it does the latter to ensure consistency. + sync_result = self._group_coordinator.sync_group(self._consumer_group, + self._generation_id, + self._consumer_id, + group_assignments) + assignment = sync_result.member_assignment.partition_assignment + # set up a SimpleConsumer consuming the returned partitions + my_partitions = [p for p in itervalues(self._topic.partitions) + if p.id in assignment[0][1]] + self._setup_internal_consumer(partitions=my_partitions) + break + except Exception: + if i == self._rebalance_max_retries - 1: + log.warning('Failed to rebalance s after %d retries.', i) + raise + log.info('Unable to complete rebalancing. Retrying') + self._cluster.handler.sleep(i * (self._rebalance_backoff_ms / 1000)) self._raise_worker_exceptions() From 3e2cc355cd5f7ef4adfb89618e0d17f4094eb0d5 Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Thu, 25 Feb 2016 15:16:40 -0800 Subject: [PATCH 34/70] remove some duplication by refactoring _rebalance --- pykafka/balancedconsumer.py | 80 ++++++------ pykafka/managedbalancedconsumer.py | 202 ++++++++++++++++++++--------- 2 files changed, 185 insertions(+), 97 deletions(-) diff --git a/pykafka/balancedconsumer.py b/pykafka/balancedconsumer.py index ee79fec4c..b843796ba 100644 --- a/pykafka/balancedconsumer.py +++ b/pykafka/balancedconsumer.py @@ -541,8 +541,46 @@ def _path_self(self): id_=self._consumer_id ) + def _update_member_assignment(self): + """Decide and assign new partitions for this consumer""" + for i in range(self._rebalance_max_retries): + try: + # If retrying, be sure to make sure the + # partition allocation is correct. + participants = self._get_participants() + if self._consumer_id not in participants: + # situation that only occurs if our zk session expired + self._add_self() + participants.append(self._consumer_id) + + new_partitions = self._decide_partitions(participants) + if not new_partitions: + log.warning("No partitions assigned to consumer %s", + self._consumer_id) + + # Update zk with any changes: + # Note that we explicitly fetch our set of held partitions + # from zk, rather than assuming it will be identical to + # `self.partitions`. This covers the (rare) situation + # where due to an interrupted connection our zk session + # has expired, in which case we'd hold zero partitions on + # zk, but `self._partitions` may be outdated and non-empty + current_zk_parts = self._get_held_partitions() + self._remove_partitions(current_zk_parts - new_partitions) + self._add_partitions(new_partitions - current_zk_parts) + if self._setup_internal_consumer(new_partitions): + log.info('Rebalancing Complete.') + break + except PartitionOwnedError as ex: + if i == self._rebalance_max_retries - 1: + log.warning('Failed to acquire partition %s after %d retries.', + ex.partition, i) + raise + log.info('Unable to acquire partition %s. Retrying', ex.partition) + self._cluster.handler.sleep(i * (self._rebalance_backoff_ms / 1000)) + def _rebalance(self): - """Claim partitions for this consumer. + """Start the rebalancing process for this consumer This method is called whenever a zookeeper watch is triggered. """ @@ -554,44 +592,8 @@ def _rebalance(self): if not self._running: raise ConsumerStoppedException log.info('Rebalancing consumer "%s" for topic "%s".' % ( - self._consumer_id, self._topic.name) - ) - - for i in range(self._rebalance_max_retries): - try: - # If retrying, be sure to make sure the - # partition allocation is correct. - participants = self._get_participants() - if self._consumer_id not in participants: - # situation that only occurs if our zk session expired - self._add_self() - participants.append(self._consumer_id) - - new_partitions = self._decide_partitions(participants) - if not new_partitions: - log.warning("No partitions assigned to consumer %s", - self._consumer_id) - - # Update zk with any changes: - # Note that we explicitly fetch our set of held partitions - # from zk, rather than assuming it will be identical to - # `self.partitions`. This covers the (rare) situation - # where due to an interrupted connection our zk session - # has expired, in which case we'd hold zero partitions on - # zk, but `self._partitions` may be outdated and non-empty - current_zk_parts = self._get_held_partitions() - self._remove_partitions(current_zk_parts - new_partitions) - self._add_partitions(new_partitions - current_zk_parts) - if self._setup_internal_consumer(new_partitions): - log.info('Rebalancing Complete.') - break - except PartitionOwnedError as ex: - if i == self._rebalance_max_retries - 1: - log.warning('Failed to acquire partition %s after %d retries.', - ex.partition, i) - raise - log.info('Unable to acquire partition %s. Retrying', ex.partition) - self._cluster.handler.sleep(i * (self._rebalance_backoff_ms / 1000)) + self._consumer_id, self._topic.name)) + self._update_member_assignment() def _path_from_partition(self, p): """Given a partition, return its path in zookeeper. diff --git a/pykafka/managedbalancedconsumer.py b/pykafka/managedbalancedconsumer.py index 8ccf78a9e..c493214e2 100644 --- a/pykafka/managedbalancedconsumer.py +++ b/pykafka/managedbalancedconsumer.py @@ -23,8 +23,7 @@ from .balancedconsumer import BalancedConsumer from .common import OffsetType -from .exceptions import (IllegalGeneration, RebalanceInProgress, UnknownMemberId, - ConsumerStoppedException) +from .exceptions import (IllegalGeneration, RebalanceInProgress, UnknownMemberId) from .protocol import MemberAssignment from .utils.compat import itervalues, iterkeys @@ -32,6 +31,18 @@ class ManagedBalancedConsumer(BalancedConsumer): + """A self-balancing consumer that uses Kafka 0.9's Group Membership API + + Implements the Group Management API semantics for Kafka 0.9 compatibility + + Maintains a single instance of SimpleConsumer, periodically using the + consumer rebalancing algorithm to reassign partitions to this + SimpleConsumer. + + This class overrides the functionality of + :class:`pykafka.balancedconsumer.BalancedConsumer` that deals with ZooKeeper and + inherits other functionality directly. + """ def __init__(self, topic, cluster, @@ -52,16 +63,97 @@ def __init__(self, auto_start=True, reset_offset_on_start=False, post_rebalance_callback=None, + use_rdkafka=False, compacted_topic=True, heartbeat_interval_ms=3000): - """A self-balancing consumer that uses Kafka 0.9's Group Membership API + """Create a ManagedBalancedConsumer instance - Implements the Group Management API semantics for Kafka 0.9 compatibility - - This class overrides the functionality of - :class:`pykafka.balancedconsumer.BalancedConsumer` that deals with ZooKeeper and - inherits other functionality directly. + :param topic: The topic this consumer should consume + :type topic: :class:`pykafka.topic.Topic` + :param cluster: The cluster to which this consumer should connect + :type cluster: :class:`pykafka.cluster.Cluster` + :param consumer_group: The name of the consumer group this consumer + should join. + :type consumer_group: bytes + :param fetch_message_max_bytes: The number of bytes of messages to + attempt to fetch with each fetch request + :type fetch_message_max_bytes: int + :param num_consumer_fetchers: The number of workers used to make + FetchRequests + :type num_consumer_fetchers: int + :param auto_commit_enable: If true, periodically commit to kafka the + offset of messages already fetched by this consumer. This also + requires that `consumer_group` is not `None`. + :type auto_commit_enable: bool + :param auto_commit_interval_ms: The frequency (in milliseconds) at which + the consumer's offsets are committed to kafka. This setting is + ignored if `auto_commit_enable` is `False`. + :type auto_commit_interval_ms: int + :param queued_max_messages: The maximum number of messages buffered for + consumption in the internal + :class:`pykafka.simpleconsumer.SimpleConsumer` + :type queued_max_messages: int + :param fetch_min_bytes: The minimum amount of data (in bytes) that the + server should return for a fetch request. If insufficient data is + available, the request will block until sufficient data is available. + :type fetch_min_bytes: int + :param fetch_wait_max_ms: The maximum amount of time (in milliseconds) + that the server will block before answering a fetch request if + there isn't sufficient data to immediately satisfy `fetch_min_bytes`. + :type fetch_wait_max_ms: int + :param offsets_channel_backoff_ms: Backoff time to retry failed offset + commits and fetches. + :type offsets_channel_backoff_ms: int + :param offsets_commit_max_retries: The number of times the offset commit + worker should retry before raising an error. + :type offsets_commit_max_retries: int + :param auto_offset_reset: What to do if an offset is out of range. This + setting indicates how to reset the consumer's internal offset + counter when an `OffsetOutOfRangeError` is encountered. + :type auto_offset_reset: :class:`pykafka.common.OffsetType` + :param consumer_timeout_ms: Amount of time (in milliseconds) the + consumer may spend without messages available for consumption + before returning None. + :type consumer_timeout_ms: int + :param rebalance_max_retries: The number of times the rebalance should + retry before raising an error. + :type rebalance_max_retries: int + :param rebalance_backoff_ms: Backoff time (in milliseconds) between + retries during rebalance. + :type rebalance_backoff_ms: int + :param auto_start: Whether the consumer should start after __init__ is complete. + If false, it can be started with `start()`. + :type auto_start: bool + :param reset_offset_on_start: Whether the consumer should reset its + internal offset counter to `self._auto_offset_reset` and commit that + offset immediately upon starting up + :type reset_offset_on_start: bool + :param post_rebalance_callback: A function to be called when a rebalance is + in progress. This function should accept three arguments: the + :class:`pykafka.balancedconsumer.BalancedConsumer` instance that just + completed its rebalance, a dict of partitions that it owned before the + rebalance, and a dict of partitions it owns after the rebalance. These dicts + map partition ids to the most recently known offsets for those partitions. + This function can optionally return a dictionary mapping partition ids to + offsets. If it does, the consumer will reset its offsets to the supplied + values before continuing consumption. + Note that the BalancedConsumer is in a poorly defined state at + the time this callback runs, so that accessing its properties + (such as `held_offsets` or `partitions`) might yield confusing + results. Instead, the callback should really rely on the + provided partition-id dicts, which are well-defined. + :type post_rebalance_callback: function + :param use_rdkafka: Use librdkafka-backed consumer if available + :type use_rdkafka: bool + :param compacted_topic: Set to read from a compacted topic. Forces + consumer to use less stringent message ordering logic because compacted + topics do not provide offsets in stict incrementing order. + :type compacted_topic: bool + :param heartbeat_interval_ms: The amount of time in milliseconds to wait between + heartbeat requests + :type heartbeat_interval_ms: int """ + self._cluster = cluster if not isinstance(consumer_group, bytes): raise TypeError("consumer_group must be a bytes object") @@ -85,7 +177,10 @@ def __init__(self, self._post_rebalance_callback = post_rebalance_callback self._is_compacted_topic = compacted_topic self._heartbeat_interval_ms = heartbeat_interval_ms - self._use_rdkafka = False + if use_rdkafka is True: + raise ImportError("use_rdkafka is not available for {}".format( + self.__class__.__name__)) + self._use_rdkafka = use_rdkafka self._running = True self._rebalancing_lock = cluster.handler.Lock() @@ -165,59 +260,50 @@ def _decide_partitions(self, participants, consumer_id=None): participants, consumer_id=consumer_id) return [p.id for p in parts] - def _rebalance(self): + def _update_member_assignment(self): """Join a managed consumer group and start consuming assigned partitions Equivalent to `pykafka.balancedconsumer.BalancedConsumer._rebalance`, but uses the Kafka 0.9 Group Membership API instead of ZooKeeper to manage group state """ - if self._consumer is not None: - self.commit_offsets() - - with self._rebalancing_lock: - if not self._running: - raise ConsumerStoppedException - log.info('Rebalancing consumer "%s" for topic "%s".' % ( - self._consumer_id, self._topic.name)) - for i in range(self._rebalance_max_retries): - try: - # send the initial JoinGroupRequest. Assigns a member id and tells the - # coordinator about this consumer. - join_result = self._group_coordinator.join_managed_consumer_group( - self._consumer_group, self._consumer_id) - self._generation_id = join_result.generation_id - self._consumer_id = join_result.member_id - - # generate partition assignments for each group member - # if this is not the leader, join_result.members will be empty - group_assignments = [ - MemberAssignment([ - (self._topic.name, - self._decide_partitions(iterkeys(join_result.members), - consumer_id=member_id)) - ], member_id=member_id) for member_id in join_result.members] - - # perform the SyncGroupRequest. If this consumer is the group leader, - # This request informs the other members of the group of their - # partition assignments. This request is also used to fetch The - # partition assignment for this consumer. The group leader *could* - # tell itself its own assignment instead of using the result of this - # request, but it does the latter to ensure consistency. - sync_result = self._group_coordinator.sync_group(self._consumer_group, - self._generation_id, - self._consumer_id, - group_assignments) - assignment = sync_result.member_assignment.partition_assignment - # set up a SimpleConsumer consuming the returned partitions - my_partitions = [p for p in itervalues(self._topic.partitions) - if p.id in assignment[0][1]] - self._setup_internal_consumer(partitions=my_partitions) - break - except Exception: - if i == self._rebalance_max_retries - 1: - log.warning('Failed to rebalance s after %d retries.', i) - raise - log.info('Unable to complete rebalancing. Retrying') - self._cluster.handler.sleep(i * (self._rebalance_backoff_ms / 1000)) + for i in range(self._rebalance_max_retries): + try: + # send the initial JoinGroupRequest. Assigns a member id and tells the + # coordinator about this consumer. + join_result = self._group_coordinator.join_managed_consumer_group( + self._consumer_group, self._consumer_id) + self._generation_id = join_result.generation_id + self._consumer_id = join_result.member_id + + # generate partition assignments for each group member + # if this is not the leader, join_result.members will be empty + group_assignments = [ + MemberAssignment([ + (self._topic.name, + self._decide_partitions(iterkeys(join_result.members), + consumer_id=member_id)) + ], member_id=member_id) for member_id in join_result.members] + + # perform the SyncGroupRequest. If this consumer is the group leader, + # This request informs the other members of the group of their + # partition assignments. This request is also used to fetch The + # partition assignment for this consumer. The group leader *could* + # tell itself its own assignment instead of using the result of this + # request, but it does the latter to ensure consistency. + sync_result = self._group_coordinator.sync_group( + self._consumer_group, self._generation_id, self._consumer_id, + group_assignments) + assignment = sync_result.member_assignment.partition_assignment + # set up a SimpleConsumer consuming the returned partitions + my_partitions = [p for p in itervalues(self._topic.partitions) + if p.id in assignment[0][1]] + self._setup_internal_consumer(partitions=my_partitions) + break + except Exception: + if i == self._rebalance_max_retries - 1: + log.warning('Failed to rebalance s after %d retries.', i) + raise + log.info('Unable to complete rebalancing. Retrying') + self._cluster.handler.sleep(i * (self._rebalance_backoff_ms / 1000)) self._raise_worker_exceptions() From dbae7db6db831d34cc17db53a365e7fb40eff923 Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Thu, 25 Feb 2016 15:21:14 -0800 Subject: [PATCH 35/70] properly set self._running --- pykafka/managedbalancedconsumer.py | 5 +++-- 1 file changed, 3 insertions(+), 2 deletions(-) diff --git a/pykafka/managedbalancedconsumer.py b/pykafka/managedbalancedconsumer.py index c493214e2..fab6b4f0c 100644 --- a/pykafka/managedbalancedconsumer.py +++ b/pykafka/managedbalancedconsumer.py @@ -181,7 +181,6 @@ def __init__(self, raise ImportError("use_rdkafka is not available for {}".format( self.__class__.__name__)) self._use_rdkafka = use_rdkafka - self._running = True self._rebalancing_lock = cluster.handler.Lock() self._consumer = None @@ -217,6 +216,7 @@ def fetcher(): def start(self): """Start this consumer. Must be called before consume() if `auto_start=False`.""" try: + self._running = True self._group_coordinator = self._cluster.get_group_coordinator( self._consumer_group) self._rebalance() @@ -263,7 +263,8 @@ def _decide_partitions(self, participants, consumer_id=None): def _update_member_assignment(self): """Join a managed consumer group and start consuming assigned partitions - Equivalent to `pykafka.balancedconsumer.BalancedConsumer._rebalance`, + Equivalent to + `pykafka.balancedconsumer.BalancedConsumer._update_member_assignment`, but uses the Kafka 0.9 Group Membership API instead of ZooKeeper to manage group state """ From 2cde7d8899820c7dab9b41ada8146ef41bedb864 Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Thu, 25 Feb 2016 15:50:42 -0800 Subject: [PATCH 36/70] more complete error handling in heartbeat --- pykafka/exceptions.py | 18 +++++++++++++----- pykafka/managedbalancedconsumer.py | 24 +++++++++++++++++------- pykafka/simpleconsumer.py | 8 ++++---- 3 files changed, 34 insertions(+), 16 deletions(-) diff --git a/pykafka/exceptions.py b/pykafka/exceptions.py index 67c1e7f14..f2e1b2567 100644 --- a/pykafka/exceptions.py +++ b/pykafka/exceptions.py @@ -159,14 +159,14 @@ class GroupLoadInProgress(ProtocolClientError): ERROR_CODE = 14 -class ConsumerCoordinatorNotAvailable(ProtocolClientError): +class GroupCoordinatorNotAvailable(ProtocolClientError): """The broker returns this error code for consumer metadata requests or offset commit requests if the offsets topic has not yet been created. """ ERROR_CODE = 15 -class NotCoordinatorForConsumer(ProtocolClientError): +class NotCoordinatorForGroup(ProtocolClientError): """The broker returns this error code if it receives an offset fetch or commit request for a consumer group that it is not a coordinator for. """ @@ -194,6 +194,13 @@ class RebalanceInProgress(ProtocolClientError): ERROR_CODE = 27 +class GroupAuthorizationFailed(ProtocolClientError): + """Returned by the broker when the client is not authorized to access a particular + groupId. + """ + ERROR_CODE = 30 + + ERROR_CODES = dict( (exc.ERROR_CODE, exc) for exc in (UnknownError, @@ -207,11 +214,12 @@ class RebalanceInProgress(ProtocolClientError): MessageSizeTooLarge, OffsetMetadataTooLarge, GroupLoadInProgress, - ConsumerCoordinatorNotAvailable, - NotCoordinatorForConsumer, + GroupCoordinatorNotAvailable, + NotCoordinatorForGroup, IllegalGeneration, UnknownMemberId, - RebalanceInProgress) + RebalanceInProgress, + GroupAuthorizationFailed) ) diff --git a/pykafka/managedbalancedconsumer.py b/pykafka/managedbalancedconsumer.py index fab6b4f0c..e65c0481d 100644 --- a/pykafka/managedbalancedconsumer.py +++ b/pykafka/managedbalancedconsumer.py @@ -23,7 +23,9 @@ from .balancedconsumer import BalancedConsumer from .common import OffsetType -from .exceptions import (IllegalGeneration, RebalanceInProgress, UnknownMemberId) +from .exceptions import (IllegalGeneration, RebalanceInProgress, UnknownMemberId, + NotCoordinatorForGroup, GroupCoordinatorNotAvailable, + GroupAuthorizationFailed) from .protocol import MemberAssignment from .utils.compat import itervalues, iterkeys @@ -243,12 +245,20 @@ def _send_heartbeat(self): self._generation_id, self._consumer_id ) - if res.error_code == IllegalGeneration.ERROR_CODE: - self._rebalance() - elif res.error_code == RebalanceInProgress.ERROR_CODE: - self._rebalance() - elif res.error_code == UnknownMemberId.ERROR_CODE: - self._rebalance() + if res.error_code == 0: + return + log.info("Error code %d encountered on heartbeat." % res.error_code) + if res.error_code in (IllegalGeneration.ERROR_CODE, + RebalanceInProgress.ERROR_CODE, + UnknownMemberId.ERROR_CODE): + pass + elif res.error_code in (GroupCoordinatorNotAvailable.ERROR_CODE, + NotCoordinatorForGroup.ERROR_CODE): + self._group_coordinator = self._cluster.get_group_coordinator( + self._consumer_group) + elif res.error_code == GroupAuthorizationFailed.ERROR_CODE: + raise GroupAuthorizationFailed() + self._rebalance() def _decide_partitions(self, participants, consumer_id=None): """Decide which partitions belong to this consumer diff --git a/pykafka/simpleconsumer.py b/pykafka/simpleconsumer.py index 00bf993ae..b97e289d6 100644 --- a/pykafka/simpleconsumer.py +++ b/pykafka/simpleconsumer.py @@ -32,7 +32,7 @@ range, iterkeys) from .exceptions import (OffsetOutOfRangeError, UnknownTopicOrPartition, OffsetMetadataTooLarge, GroupLoadInProgress, - NotCoordinatorForConsumer, SocketDisconnectedError, + NotCoordinatorForGroup, SocketDisconnectedError, ConsumerStoppedException, KafkaException, NotLeaderForPartition, OffsetRequestFailedError, RequestTimedOut, ERROR_CODES) @@ -269,8 +269,8 @@ def _handle_OffsetOutOfRangeError(parts): def _handle_RequestTimedOut(parts): log.info("Continuing in response to RequestTimedOut") - def _handle_NotCoordinatorForConsumer(parts): - log.info("Updating cluster in response to NotCoordinatorForConsumer") + def _handle_NotCoordinatorForGroup(parts): + log.info("Updating cluster in response to NotCoordinatorForGroup") self._update() def _handle_NotLeaderForPartition(parts): @@ -285,7 +285,7 @@ def _handle_GroupLoadInProgress(parts): OffsetOutOfRangeError.ERROR_CODE: _handle_OffsetOutOfRangeError, NotLeaderForPartition.ERROR_CODE: _handle_NotLeaderForPartition, OffsetMetadataTooLarge.ERROR_CODE: lambda p: raise_error(OffsetMetadataTooLarge), - NotCoordinatorForConsumer.ERROR_CODE: _handle_NotCoordinatorForConsumer, + NotCoordinatorForGroup.ERROR_CODE: _handle_NotCoordinatorForGroup, RequestTimedOut.ERROR_CODE: _handle_RequestTimedOut, GroupLoadInProgress.ERROR_CODE: _handle_GroupLoadInProgress } From e62b2ba9eee0c17c66a5ace52ab76949ec3f11b6 Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Thu, 25 Feb 2016 15:57:35 -0800 Subject: [PATCH 37/70] style --- pykafka/managedbalancedconsumer.py | 5 +---- 1 file changed, 1 insertion(+), 4 deletions(-) diff --git a/pykafka/managedbalancedconsumer.py b/pykafka/managedbalancedconsumer.py index e65c0481d..ef4483298 100644 --- a/pykafka/managedbalancedconsumer.py +++ b/pykafka/managedbalancedconsumer.py @@ -241,10 +241,7 @@ def stop(self): def _send_heartbeat(self): """Send a heartbeat request to the group coordinator and react to the response""" res = self._group_coordinator.group_heartbeat( - self._consumer_group, - self._generation_id, - self._consumer_id - ) + self._consumer_group, self._generation_id, self._consumer_id) if res.error_code == 0: return log.info("Error code %d encountered on heartbeat." % res.error_code) From 6f5292ed58137188dc028c046c153d7084eae999 Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Thu, 25 Feb 2016 15:58:42 -0800 Subject: [PATCH 38/70] fix import errors --- pykafka/cluster.py | 4 ++-- 1 file changed, 2 insertions(+), 2 deletions(-) diff --git a/pykafka/cluster.py b/pykafka/cluster.py index 7ded01993..d394ed773 100644 --- a/pykafka/cluster.py +++ b/pykafka/cluster.py @@ -28,7 +28,7 @@ from .broker import Broker from .exceptions import (ERROR_CODES, - ConsumerCoordinatorNotAvailable, + GroupCoordinatorNotAvailable, NoBrokersAvailableError, SocketDisconnectedError, LeaderNotAvailable) @@ -389,7 +389,7 @@ def get_group_coordinator(self, consumer_group): future = broker.handler.request(req) try: res = future.get(GroupCoordinatorResponse) - except ConsumerCoordinatorNotAvailable: + except GroupCoordinatorNotAvailable: log.error('Error discovering offset manager.') if i == self._max_connection_retries - 1: raise From 61a55aa3cf4ba4cee7988a7def504aa0c410ec36 Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Thu, 25 Feb 2016 16:28:56 -0800 Subject: [PATCH 39/70] handle all possible errors in syncgroup and joingroup --- pykafka/exceptions.py | 16 ++++++ pykafka/managedbalancedconsumer.py | 84 +++++++++++++++++++++++++----- pykafka/protocol.py | 4 +- 3 files changed, 87 insertions(+), 17 deletions(-) diff --git a/pykafka/exceptions.py b/pykafka/exceptions.py index f2e1b2567..c78d2bc45 100644 --- a/pykafka/exceptions.py +++ b/pykafka/exceptions.py @@ -180,6 +180,13 @@ class IllegalGeneration(ProtocolClientError): ERROR_CODE = 22 +class InconsistentGroupProtocol(ProtocolClientError): + """Returned in join group when the member provides a protocol type or set of protocols + which is not compatible with the current group. + """ + ERROR_CODE = 23 + + class UnknownMemberId(ProtocolClientError): """Returned from group requests (offset commits/fetches, heartbeats, etc) when the memberId is not in the current generation. @@ -187,6 +194,13 @@ class UnknownMemberId(ProtocolClientError): ERROR_CODE = 25 +class InvalidSessionTimeout(ProtocolClientError): + """Returned in join group when the requested session timeout is outside of the allowed + range on the broker + """ + ERROR_CODE = 26 + + class RebalanceInProgress(ProtocolClientError): """Returned in heartbeat requests when the coordinator has begun rebalancing the group. This indicates to the client that it should rejoin the group. @@ -217,7 +231,9 @@ class GroupAuthorizationFailed(ProtocolClientError): GroupCoordinatorNotAvailable, NotCoordinatorForGroup, IllegalGeneration, + InconsistentGroupProtocol, UnknownMemberId, + InvalidSessionTimeout, RebalanceInProgress, GroupAuthorizationFailed) ) diff --git a/pykafka/managedbalancedconsumer.py b/pykafka/managedbalancedconsumer.py index ef4483298..a17bb3117 100644 --- a/pykafka/managedbalancedconsumer.py +++ b/pykafka/managedbalancedconsumer.py @@ -25,7 +25,8 @@ from .common import OffsetType from .exceptions import (IllegalGeneration, RebalanceInProgress, UnknownMemberId, NotCoordinatorForGroup, GroupCoordinatorNotAvailable, - GroupAuthorizationFailed) + GroupAuthorizationFailed, ERROR_CODES, GroupLoadInProgress, + InconsistentGroupProtocol, InvalidSessionTimeout) from .protocol import MemberAssignment from .utils.compat import itervalues, iterkeys @@ -277,21 +278,16 @@ def _update_member_assignment(self): """ for i in range(self._rebalance_max_retries): try: - # send the initial JoinGroupRequest. Assigns a member id and tells the - # coordinator about this consumer. - join_result = self._group_coordinator.join_managed_consumer_group( - self._consumer_group, self._consumer_id) - self._generation_id = join_result.generation_id - self._consumer_id = join_result.member_id + members = self._join_group() # generate partition assignments for each group member # if this is not the leader, join_result.members will be empty group_assignments = [ MemberAssignment([ (self._topic.name, - self._decide_partitions(iterkeys(join_result.members), + self._decide_partitions(iterkeys(members), consumer_id=member_id)) - ], member_id=member_id) for member_id in join_result.members] + ], member_id=member_id) for member_id in members] # perform the SyncGroupRequest. If this consumer is the group leader, # This request informs the other members of the group of their @@ -299,19 +295,79 @@ def _update_member_assignment(self): # partition assignment for this consumer. The group leader *could* # tell itself its own assignment instead of using the result of this # request, but it does the latter to ensure consistency. - sync_result = self._group_coordinator.sync_group( - self._consumer_group, self._generation_id, self._consumer_id, - group_assignments) - assignment = sync_result.member_assignment.partition_assignment + assignment = self._sync_group(group_assignments) # set up a SimpleConsumer consuming the returned partitions my_partitions = [p for p in itervalues(self._topic.partitions) if p.id in assignment[0][1]] self._setup_internal_consumer(partitions=my_partitions) break - except Exception: + except Exception as ex: if i == self._rebalance_max_retries - 1: log.warning('Failed to rebalance s after %d retries.', i) raise log.info('Unable to complete rebalancing. Retrying') + log.exception(ex) self._cluster.handler.sleep(i * (self._rebalance_backoff_ms / 1000)) self._raise_worker_exceptions() + + def _join_group(self): + """Send a JoinGroupRequest. + + Assigns a member id and tells the coordinator about this consumer. + """ + log.info("Sending JoinGroupRequest for consumer id '%s'", self._consumer_id) + for i in range(self._cluster._max_connection_retries): + join_result = self._group_coordinator.join_managed_consumer_group( + self._consumer_group, self._consumer_id) + if join_result.error_code == 0: + break + log.info("Error code %d encountered during JoinGroupRequest", + join_result.error_code) + if i == self._cluster._max_connection_retries - 1: + raise ERROR_CODES[join_result.error_code] + if join_result.error_code in (GroupLoadInProgress.ERROR_CODE,): + pass + elif join_result.error_code in (GroupCoordinatorNotAvailable.ERROR_CODE, + NotCoordinatorForGroup.ERROR_CODE): + self._group_coordinator = self._cluster.get_group_coordinator( + self._consumer_group) + elif join_result.error_code in (InconsistentGroupProtocol.ERROR_CODE, + UnknownMemberId.ERROR_CODE, + InvalidSessionTimeout.ERROR_CODE, + GroupAuthorizationFailed.ERROR_CODE): + raise ERROR_CODES[join_result.error_code] + self._cluster.handler.sleep(i * 2) + self._generation_id = join_result.generation_id + self._consumer_id = join_result.member_id + return join_result.members + + def _sync_group(self, group_assignments): + """Send a SyncGroupRequest. + + If this consumer is the group leader, this call informs the other consumers + of their partition assignments. For all consumers including the leader, this call + is used to fetch partition assignments. + """ + log.info("Sending SyncGroupRequest for consumer id '%s'", self._consumer_id) + for i in range(self._cluster._max_connection_retries): + sync_result = self._group_coordinator.sync_group( + self._consumer_group, self._generation_id, self._consumer_id, + group_assignments) + if sync_result.error_code == 0: + break + log.info("Error code %d encountered during SyncGroupRequest", + sync_result.error_code) + if i == self._cluster._max_connection_retries - 1: + raise ERROR_CODES[sync_result.error_code] + if sync_result.error_code in (IllegalGeneration.ERROR_CODE, + GroupCoordinatorNotAvailable.ERROR_CODE, + RebalanceInProgress.ERROR_CODE): + pass + elif sync_result.error_code in (NotCoordinatorForGroup.ERROR_CODE,): + self._group_coordinator = self._cluster.get_group_coordinator( + self._consumer_group) + elif sync_result.error_code in (UnknownMemberId.ERROR_CODE, + GroupAuthorizationFailed.ERROR_CODE): + raise ERROR_CODES[sync_result.error_code] + self._cluster.handler.sleep(i * 2) + return sync_result.member_assignment.partition_assignment diff --git a/pykafka/protocol.py b/pykafka/protocol.py index ef507fa03..91d93c4a0 100644 --- a/pykafka/protocol.py +++ b/pykafka/protocol.py @@ -1331,9 +1331,7 @@ def __init__(self, buff): fmt = 'hiSSS[SY ]' response = struct_helpers.unpack_from(fmt, buff, 0) - error_code = response[0] - if error_code != 0: - self.raise_error(error_code, response) + self.error_code = response[0] self.generation_id = response[1] self.group_protocol = response[2] self.leader_id = response[3] From a771136b7e76a5ae80aed261ca2e578f3a981742 Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Thu, 25 Feb 2016 16:30:36 -0800 Subject: [PATCH 40/70] remove unnecessary comment --- pykafka/managedbalancedconsumer.py | 11 +++-------- 1 file changed, 3 insertions(+), 8 deletions(-) diff --git a/pykafka/managedbalancedconsumer.py b/pykafka/managedbalancedconsumer.py index a17bb3117..e67bfd280 100644 --- a/pykafka/managedbalancedconsumer.py +++ b/pykafka/managedbalancedconsumer.py @@ -279,7 +279,6 @@ def _update_member_assignment(self): for i in range(self._rebalance_max_retries): try: members = self._join_group() - # generate partition assignments for each group member # if this is not the leader, join_result.members will be empty group_assignments = [ @@ -289,14 +288,7 @@ def _update_member_assignment(self): consumer_id=member_id)) ], member_id=member_id) for member_id in members] - # perform the SyncGroupRequest. If this consumer is the group leader, - # This request informs the other members of the group of their - # partition assignments. This request is also used to fetch The - # partition assignment for this consumer. The group leader *could* - # tell itself its own assignment instead of using the result of this - # request, but it does the latter to ensure consistency. assignment = self._sync_group(group_assignments) - # set up a SimpleConsumer consuming the returned partitions my_partitions = [p for p in itervalues(self._topic.partitions) if p.id in assignment[0][1]] self._setup_internal_consumer(partitions=my_partitions) @@ -347,6 +339,9 @@ def _sync_group(self, group_assignments): If this consumer is the group leader, this call informs the other consumers of their partition assignments. For all consumers including the leader, this call is used to fetch partition assignments. + + The group leader *could* tell itself its own assignment instead of using the + result of this request, but it does the latter to ensure consistency. """ log.info("Sending SyncGroupRequest for consumer id '%s'", self._consumer_id) for i in range(self._cluster._max_connection_retries): From c1ac85ceae3f5cb8054222f6cee9c4e2f9f1d1cd Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Fri, 26 Feb 2016 09:13:18 -0800 Subject: [PATCH 41/70] address diff comments --- pykafka/managedbalancedconsumer.py | 6 +++--- pykafka/simpleconsumer.py | 4 ++-- 2 files changed, 5 insertions(+), 5 deletions(-) diff --git a/pykafka/managedbalancedconsumer.py b/pykafka/managedbalancedconsumer.py index e67bfd280..3a8677b44 100644 --- a/pykafka/managedbalancedconsumer.py +++ b/pykafka/managedbalancedconsumer.py @@ -1,3 +1,4 @@ +from __future__ import division """ Author: Emmett Butler """ @@ -28,7 +29,7 @@ GroupAuthorizationFailed, ERROR_CODES, GroupLoadInProgress, InconsistentGroupProtocol, InvalidSessionTimeout) from .protocol import MemberAssignment -from .utils.compat import itervalues, iterkeys +from .utils.compat import iterkeys log = logging.getLogger(__name__) @@ -289,8 +290,7 @@ def _update_member_assignment(self): ], member_id=member_id) for member_id in members] assignment = self._sync_group(group_assignments) - my_partitions = [p for p in itervalues(self._topic.partitions) - if p.id in assignment[0][1]] + my_partitions = [self._topic.partitions[pid] for pid in assignment[0][1]] self._setup_internal_consumer(partitions=my_partitions) break except Exception as ex: diff --git a/pykafka/simpleconsumer.py b/pykafka/simpleconsumer.py index b97e289d6..2205670a5 100644 --- a/pykafka/simpleconsumer.py +++ b/pykafka/simpleconsumer.py @@ -69,7 +69,7 @@ def __init__(self, reset_offset_on_start=False, compacted_topic=False, generation_id=-1, - consumer_id=None): + consumer_id=b''): """Create a SimpleConsumer. Settings and default values are taken from the Scala @@ -142,7 +142,7 @@ def __init__(self, :type generation_id: int :param consumer_id: The identifying string to use for this consumer on group requests - :type consumer_id: str + :type consumer_id: bytes """ self._cluster = cluster if not (isinstance(consumer_group, bytes) or consumer_group is None): From fb9b002f88ba489e6d5a478b36c5d4df405b226c Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Fri, 26 Feb 2016 09:22:48 -0800 Subject: [PATCH 42/70] change broker method names --- pykafka/broker.py | 45 ++++++++++++++++++++++++++++-- pykafka/managedbalancedconsumer.py | 7 ++--- 2 files changed, 45 insertions(+), 7 deletions(-) diff --git a/pykafka/broker.py b/pykafka/broker.py index 5cfa28fd5..ea269a7cb 100644 --- a/pykafka/broker.py +++ b/pykafka/broker.py @@ -353,20 +353,59 @@ def fetch_consumer_group_offsets(self, consumer_group, preqs): # Group Membership API # ########################## - def join_managed_consumer_group(self, consumer_group, member_id): + def join_group(self, consumer_group, member_id): + """Send a JoinGroupRequest + + :param consumer_group: The name of the consumer group to join + :type consumer_group: bytes + :param member_id: The ID of the consumer joining the group + :type member_id: bytes + """ future = self._req_handler.request(JoinGroupRequest(consumer_group, member_id)) return future.get(JoinGroupResponse) - def leave_managed_consumer_group(self, consumer_group, member_id): + def leave_group(self, consumer_group, member_id): + """Send a LeaveGroupRequest + + :param consumer_group: The name of the consumer group to leave + :type consumer_group: bytes + :param member_id: The ID of the consumer leaving the group + :type member_id: bytes + """ future = self._req_handler.request(LeaveGroupRequest(consumer_group, member_id)) return future.get(LeaveGroupResponse) def sync_group(self, consumer_group, generation_id, member_id, group_assignment): + """Send a SyncGroupRequest + + :param consumer_group: The name of the consumer group to which this consumer + belongs + :type consumer_group: bytes + :param generation_id: The current generation for the consumer group + :type generation_id: int + :param member_id: The ID of the consumer syncing + :type member_id: bytes + :param group_assignment: A sequence of :class:`pykafka.protocol.MemberAssignment` + instances indicating the partition assignments for each member of the group. + When `sync_group` is called by a member other than the leader of the group, + `group_assignment` should be an empty sequence. + :type group_assignment: iterable of :class:`pykafka.protocol.MemberAssignment` + """ future = self._req_handler.request( SyncGroupRequest(consumer_group, generation_id, member_id, group_assignment)) return future.get(SyncGroupResponse) - def group_heartbeat(self, consumer_group, generation_id, member_id): + def heartbeat(self, consumer_group, generation_id, member_id): + """Send a HeartbeatRequest + + :param consumer_group: The name of the consumer group to which this consumer + belongs + :type consumer_group: bytes + :param generation_id: The current generation for the consumer group + :type generation_id: int + :param member_id: The ID of the consumer sending this heartbeat + :type member_id: bytes + """ future = self._req_handler.request( HeartbeatRequest(consumer_group, generation_id, member_id)) return future.get(HeartbeatResponse) diff --git a/pykafka/managedbalancedconsumer.py b/pykafka/managedbalancedconsumer.py index 3a8677b44..c20327e67 100644 --- a/pykafka/managedbalancedconsumer.py +++ b/pykafka/managedbalancedconsumer.py @@ -237,12 +237,11 @@ def stop(self): self._running = False if self._consumer is not None: self._consumer.stop() - self._group_coordinator.leave_managed_consumer_group(self._consumer_group, - self._consumer_id) + self._group_coordinator.leave_group(self._consumer_group, self._consumer_id) def _send_heartbeat(self): """Send a heartbeat request to the group coordinator and react to the response""" - res = self._group_coordinator.group_heartbeat( + res = self._group_coordinator.heartbeat( self._consumer_group, self._generation_id, self._consumer_id) if res.error_code == 0: return @@ -309,7 +308,7 @@ def _join_group(self): """ log.info("Sending JoinGroupRequest for consumer id '%s'", self._consumer_id) for i in range(self._cluster._max_connection_retries): - join_result = self._group_coordinator.join_managed_consumer_group( + join_result = self._group_coordinator.join_group( self._consumer_group, self._consumer_id) if join_result.error_code == 0: break From 9fdf7af7b4c9f3b95ffbff67e6c041a9974be3a8 Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Fri, 26 Feb 2016 10:07:38 -0800 Subject: [PATCH 43/70] fix some test failures --- pykafka/balancedconsumer.py | 2 ++ pykafka/rdkafka/simple_consumer.py | 3 ++- tests/pykafka/test_protocol.py | 10 +++++----- 3 files changed, 9 insertions(+), 6 deletions(-) diff --git a/pykafka/balancedconsumer.py b/pykafka/balancedconsumer.py index b843796ba..f0b1c8b38 100644 --- a/pykafka/balancedconsumer.py +++ b/pykafka/balancedconsumer.py @@ -341,6 +341,8 @@ def _setup_zookeeper(self, zookeeper_connect, timeout): def _setup_internal_consumer(self, partitions=None, start=True): """Instantiate an internal SimpleConsumer instance""" + if partitions is None: + partitions = [] # Only re-create internal consumer if something changed. if partitions != self._partitions: cns = self._get_internal_consumer(partitions=list(partitions), start=start) diff --git a/pykafka/rdkafka/simple_consumer.py b/pykafka/rdkafka/simple_consumer.py index 25e41a19b..583a76774 100644 --- a/pykafka/rdkafka/simple_consumer.py +++ b/pykafka/rdkafka/simple_consumer.py @@ -45,7 +45,8 @@ def __init__(self, consumer_timeout_ms=-1, auto_start=True, reset_offset_on_start=False, - compacted_topic=False): + compacted_topic=False, + consumer_id=b''): callargs = {k: v for k, v in vars().items() if k not in ("self", "__class__")} self._rdk_consumer = None diff --git a/tests/pykafka/test_protocol.py b/tests/pykafka/test_protocol.py index 3eaa9966c..ffb25686d 100644 --- a/tests/pykafka/test_protocol.py +++ b/tests/pykafka/test_protocol.py @@ -215,7 +215,7 @@ def test_consumer_metadata_request(self): ) def test_consumer_metadata_response(self): - response = protocol.ConsumerMetadataResponse( + response = protocol.GroupCoordinatorResponse( buffer(b'\x00\x00\x00\x00\x00\x00\x00\remmett-debian\x00\x00#\x84') ) self.assertEqual(response.coordinator_id, 0) @@ -301,10 +301,10 @@ def test_sync_group_request(self): req = protocol.SyncGroupRequest( b'dummygroup', 1, b'testmember1', [ - ('testmember1', protocol.MemberAssignment([(b"mytopic1", [3, 5, 7, 9]), - (b"mytopic2", [3, 5, 7, 9])])), - ('testmember2', protocol.MemberAssignment([(b"mytopic1", [2, 4, 6, 8]), - (b"mytopic2", [2, 4, 6, 8])])) + protocol.MemberAssignment([(b"mytopic1", [3, 5, 7, 9]), + (b"mytopic2", [3, 5, 7, 9])], member_id="a"), + protocol.MemberAssignment([(b"mytopic1", [2, 4, 6, 8]), + (b"mytopic2", [2, 4, 6, 8])], member_id="b") ]) msg = req.get_bytes() self.assertEqual( From 932d3f85aaba107667e0b50ed52f111d7a7891e5 Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Mon, 29 Feb 2016 10:39:16 -0800 Subject: [PATCH 44/70] organizational adjustments to managedbalancedconsumer --- pykafka/managedbalancedconsumer.py | 20 ++++++-------------- 1 file changed, 6 insertions(+), 14 deletions(-) diff --git a/pykafka/managedbalancedconsumer.py b/pykafka/managedbalancedconsumer.py index c20327e67..b8a9e419b 100644 --- a/pykafka/managedbalancedconsumer.py +++ b/pykafka/managedbalancedconsumer.py @@ -186,8 +186,10 @@ def __init__(self, self.__class__.__name__)) self._use_rdkafka = use_rdkafka + self._generation_id = -1 self._rebalancing_lock = cluster.handler.Lock() self._consumer = None + self._group_coordinator = None self._consumer_id = b'' self._worker_trace_logged = False self._worker_exception = None @@ -258,16 +260,6 @@ def _send_heartbeat(self): raise GroupAuthorizationFailed() self._rebalance() - def _decide_partitions(self, participants, consumer_id=None): - """Decide which partitions belong to this consumer - - Thin formatting wrapper around - `pykafka.balancedconsumer.BalancedConsumer._decide_partitions` - """ - parts = super(ManagedBalancedConsumer, self)._decide_partitions( - participants, consumer_id=consumer_id) - return [p.id for p in parts] - def _update_member_assignment(self): """Join a managed consumer group and start consuming assigned partitions @@ -284,13 +276,13 @@ def _update_member_assignment(self): group_assignments = [ MemberAssignment([ (self._topic.name, - self._decide_partitions(iterkeys(members), - consumer_id=member_id)) + [p.id for p in self._decide_partitions( + iterkeys(members), consumer_id=member_id)]) ], member_id=member_id) for member_id in members] assignment = self._sync_group(group_assignments) - my_partitions = [self._topic.partitions[pid] for pid in assignment[0][1]] - self._setup_internal_consumer(partitions=my_partitions) + self._setup_internal_consumer( + partitions=[self._topic.partitions[pid] for pid in assignment[0][1]]) break except Exception as ex: if i == self._rebalance_max_retries - 1: From a96f1677ee798d8771e57df6ac61941c7db6e2af Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Mon, 29 Feb 2016 10:39:38 -0800 Subject: [PATCH 45/70] extend balanced consumer tests for group membership API --- tests/pykafka/test_balancedconsumer.py | 129 +++++++++++++++++-------- 1 file changed, 90 insertions(+), 39 deletions(-) diff --git a/tests/pykafka/test_balancedconsumer.py b/tests/pykafka/test_balancedconsumer.py index bcc4e16ac..7d72f8549 100644 --- a/tests/pykafka/test_balancedconsumer.py +++ b/tests/pykafka/test_balancedconsumer.py @@ -1,5 +1,7 @@ import math import mock +import os +import pkg_resources import platform import pytest import time @@ -12,35 +14,39 @@ from pykafka import KafkaClient from pykafka.balancedconsumer import BalancedConsumer, OffsetType from pykafka.exceptions import ConsumerStoppedException +from pykafka.managedbalancedconsumer import ManagedBalancedConsumer from pykafka.test.utils import get_cluster, stop_cluster from pykafka.utils.compat import range, iterkeys, iteritems -def buildMockConsumer(num_partitions=10, num_participants=1, timeout=2000): - consumer_group = b'testgroup' - topic = mock.Mock() - topic.name = 'testtopic' - topic.partitions = {} - for k in range(num_partitions): - part = mock.Mock(name='part-{part}'.format(part=k)) - part.id = k - part.topic = topic - part.leader = mock.Mock() - part.leader.id = k % num_participants - topic.partitions[k] = part - - cluster = mock.MagicMock() - zk = mock.MagicMock() - return BalancedConsumer(topic, cluster, consumer_group, - zookeeper=zk, auto_start=False, use_rdkafka=False, - consumer_timeout_ms=timeout), topic +kafka_version = pkg_resources.parse_version(os.environ.get("KAFKA_VERSION", "0.8")) class TestBalancedConsumer(unittest2.TestCase): @classmethod def setUpClass(cls): cls._consumer_timeout = 2000 - cls._mock_consumer, _ = buildMockConsumer(timeout=cls._consumer_timeout) + cls._mock_consumer, _ = TestBalancedConsumer.buildMockConsumer(timeout=cls._consumer_timeout) + + @classmethod + def buildMockConsumer(self, num_partitions=10, num_participants=1, timeout=2000): + consumer_group = b'testgroup' + topic = mock.Mock() + topic.name = 'testtopic' + topic.partitions = {} + for k in range(num_partitions): + part = mock.Mock(name='part-{part}'.format(part=k)) + part.id = k + part.topic = topic + part.leader = mock.Mock() + part.leader.id = k % num_participants + topic.partitions[k] = part + + cluster = mock.MagicMock() + zk = mock.MagicMock() + return BalancedConsumer(topic, cluster, consumer_group, + zookeeper=zk, auto_start=False, use_rdkafka=False, + consumer_timeout_ms=timeout), topic def test_consume_returns(self): """Ensure that consume() returns in the amount of time it's supposed to @@ -56,7 +62,7 @@ def test_consume_graceful_stop(self): """Ensure that stopping a consumer while consuming from Kafka does not end in an infinite loop when timeout is not used. """ - consumer, _ = buildMockConsumer(timeout=-1) + consumer, _ = self.buildMockConsumer(timeout=-1) consumer._setup_internal_consumer(start=False) consumer._consumer._partitions_by_id = {1: "dummy"} @@ -73,8 +79,8 @@ def test_decide_partitions(self): num_partitions = 100 - i participants = sorted(['test-debian:{p}'.format(p=p) for p in range(num_participants)]) - cns, topic = buildMockConsumer(num_partitions=num_partitions, - num_participants=num_participants) + cns, topic = self.buildMockConsumer(num_partitions=num_partitions, + num_participants=num_participants) # Simulate each participant to ensure they're correct assigned_parts = [] @@ -101,10 +107,34 @@ def test_decide_partitions(self): self.assertListEqual(assigned_parts, all_partitions) +class TestManagedBalancedConsumer(TestBalancedConsumer): + @classmethod + def buildMockConsumer(self, num_partitions=10, num_participants=1, timeout=2000): + consumer_group = b'testgroup' + topic = mock.Mock() + topic.name = 'testtopic' + topic.partitions = {} + for k in range(num_partitions): + part = mock.Mock(name='part-{part}'.format(part=k)) + part.id = k + part.topic = topic + part.leader = mock.Mock() + part.leader.id = k % num_participants + topic.partitions[k] = part + + cluster = mock.MagicMock() + cns = ManagedBalancedConsumer(topic, cluster, consumer_group, + auto_start=False, use_rdkafka=False, + consumer_timeout_ms=timeout) + cns._group_coordinator = mock.MagicMock() + return cns, topic + + class BalancedConsumerIntegrationTests(unittest2.TestCase): maxDiff = None USE_RDKAFKA = False USE_GEVENT = False + MANAGED_CONSUMER = False @classmethod def setUpClass(cls): @@ -128,6 +158,16 @@ def get_zk(self): return KazooClient(self.kafka.zookeeper) return KazooClient(self.kafka.zookeeper, handler=SequentialGeventHandler()) + def get_balanced_consumer(self, consumer_group, **kwargs): + if self.MANAGED_CONSUMER: + kwargs.pop("zookeeper", None) + kwargs.pop("zookeeper_connect", None) + return self.client.topics[self.topic_name].get_balanced_consumer( + consumer_group, + managed=self.MANAGED_CONSUMER, + **kwargs + ) + def test_rebalance_callbacks(self): def on_rebalance(cns, old_partition_offsets, new_partition_offsets): self.assertTrue(len(new_partition_offsets) > 0) @@ -140,13 +180,13 @@ def on_rebalance(cns, old_partition_offsets, new_partition_offsets): self.offset_reset = 50 try: consumer_group = b'test_rebalance_callbacks' - consumer_a = self.client.topics[self.topic_name].get_balanced_consumer( + consumer_a = self.get_balanced_consumer( consumer_group, zookeeper_connect=self.kafka.zookeeper, auto_offset_reset=OffsetType.EARLIEST, post_rebalance_callback=on_rebalance, use_rdkafka=self.USE_RDKAFKA) - consumer_b = self.client.topics[self.topic_name].get_balanced_consumer( + consumer_b = self.get_balanced_consumer( consumer_group, zookeeper_connect=self.kafka.zookeeper, auto_offset_reset=OffsetType.EARLIEST, @@ -170,13 +210,13 @@ def on_rebalance(cns, old_partition_offsets, new_partition_offsets): self.offset_reset = 50 try: consumer_group = b'test_rebalance_callbacks_error' - consumer_a = self.client.topics[self.topic_name].get_balanced_consumer( + consumer_a = self.get_balanced_consumer( consumer_group, zookeeper_connect=self.kafka.zookeeper, auto_offset_reset=OffsetType.EARLIEST, post_rebalance_callback=on_rebalance, use_rdkafka=self.USE_RDKAFKA) - consumer_b = self.client.topics[self.topic_name].get_balanced_consumer( + consumer_b = self.get_balanced_consumer( consumer_group, zookeeper_connect=self.kafka.zookeeper, auto_offset_reset=OffsetType.EARLIEST, @@ -195,13 +235,12 @@ def on_rebalance(cns, old_partition_offsets, new_partition_offsets): def test_consume_earliest(self): try: - topic = self.client.topics[self.topic_name] - consumer_a = topic.get_balanced_consumer( + consumer_a = self.get_balanced_consumer( b'test_consume_earliest', zookeeper_connect=self.kafka.zookeeper, auto_offset_reset=OffsetType.EARLIEST, use_rdkafka=self.USE_RDKAFKA) - consumer_b = topic.get_balanced_consumer( + consumer_b = self.get_balanced_consumer( b'test_consume_earliest', zookeeper_connect=self.kafka.zookeeper, auto_offset_reset=OffsetType.EARLIEST, @@ -233,13 +272,12 @@ def test_consume_earliest(self): def test_consume_latest(self): try: - topic = self.client.topics[self.topic_name] - consumer_a = topic.get_balanced_consumer( + consumer_a = self.get_balanced_consumer( b'test_consume_latest', zookeeper_connect=self.kafka.zookeeper, auto_offset_reset=OffsetType.LATEST, use_rdkafka=self.USE_RDKAFKA) - consumer_b = topic.get_balanced_consumer( + consumer_b = self.get_balanced_consumer( b'test_consume_latest', zookeeper_connect=self.kafka.zookeeper, auto_offset_reset=OffsetType.LATEST, @@ -286,7 +324,7 @@ def test_external_kazoo_client(self): zk = KazooClient(self.kafka.zookeeper) zk.start() - consumer = self.client.topics[self.topic_name].get_balanced_consumer( + consumer = self.get_balanced_consumer( b'test_external_kazoo_client', zookeeper=zk, consumer_timeout_ms=10, @@ -296,7 +334,7 @@ def test_external_kazoo_client(self): def test_no_partitions(self): """Ensure a consumer assigned no partitions doesn't fail""" - consumer = self.client.topics[self.topic_name].get_balanced_consumer( + consumer = self.get_balanced_consumer( b'test_no_partitions', zookeeper_connect=self.kafka.zookeeper, auto_start=False, @@ -319,18 +357,17 @@ def test_zk_conn_lost(self): zk = self.get_zk() zk.start() try: - topic = self.client.topics[self.topic_name] consumer_group = b'test_zk_conn_lost' - consumer = topic.get_balanced_consumer(consumer_group, - zookeeper=zk, - use_rdkafka=self.USE_RDKAFKA) + consumer = self.get_balanced_consumer(consumer_group, + zookeeper=zk, + use_rdkafka=self.USE_RDKAFKA) self.assertTrue(check_partitions(consumer)) with consumer._rebalancing_lock: zk.stop() # expires session, dropping all our nodes # Start a second consumer on a different zk connection - other_consumer = topic.get_balanced_consumer( + other_consumer = self.get_balanced_consumer( consumer_group, use_rdkafka=self.USE_RDKAFKA) # Slightly contrived: we'll grab a lock to keep _rebalance() from @@ -374,11 +411,25 @@ def wait_for_rebalancing(self, *balanced_consumers): raise AssertionError("Rebalancing failed") +@pytest.mark.skipif(kafka_version < (0, 9), + reason="Managed consumer only supported in >=0.9") +class ManagedBalancedConsumerIntegrationTests(BalancedConsumerIntegrationTests): + MANAGED_CONSUMER = True + + @pytest.mark.skipif(platform.python_implementation() == "PyPy", reason="Unresolved crashes") class BalancedConsumerGEventIntegrationTests(BalancedConsumerIntegrationTests): USE_GEVENT = True +@pytest.mark.skipif(platform.python_implementation() == "PyPy", + reason="Unresolved crashes") +@pytest.mark.skipif(kafka_version < (0, 9), + reason="Managed consumer only supported in >=0.9") +class ManagedBalancedConsumerGEventIntegrationTests(ManagedBalancedConsumerIntegrationTests): + USE_GEVENT = True + + if __name__ == "__main__": unittest2.main() From c0d4ba87ad9aaced951f11ad116565d4f7ba4e58 Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Mon, 29 Feb 2016 10:56:15 -0800 Subject: [PATCH 46/70] proper version comparison for test skips --- tests/pykafka/test_balancedconsumer.py | 5 +++-- 1 file changed, 3 insertions(+), 2 deletions(-) diff --git a/tests/pykafka/test_balancedconsumer.py b/tests/pykafka/test_balancedconsumer.py index 7d72f8549..1d5ff68a8 100644 --- a/tests/pykafka/test_balancedconsumer.py +++ b/tests/pykafka/test_balancedconsumer.py @@ -20,6 +20,7 @@ kafka_version = pkg_resources.parse_version(os.environ.get("KAFKA_VERSION", "0.8")) +version_09 = pkg_resources.parse_version("0.9.0.0") class TestBalancedConsumer(unittest2.TestCase): @@ -411,7 +412,7 @@ def wait_for_rebalancing(self, *balanced_consumers): raise AssertionError("Rebalancing failed") -@pytest.mark.skipif(kafka_version < (0, 9), +@pytest.mark.skipif(kafka_version < version_09, reason="Managed consumer only supported in >=0.9") class ManagedBalancedConsumerIntegrationTests(BalancedConsumerIntegrationTests): MANAGED_CONSUMER = True @@ -425,7 +426,7 @@ class BalancedConsumerGEventIntegrationTests(BalancedConsumerIntegrationTests): @pytest.mark.skipif(platform.python_implementation() == "PyPy", reason="Unresolved crashes") -@pytest.mark.skipif(kafka_version < (0, 9), +@pytest.mark.skipif(kafka_version < version_09, reason="Managed consumer only supported in >=0.9") class ManagedBalancedConsumerGEventIntegrationTests(ManagedBalancedConsumerIntegrationTests): USE_GEVENT = True From 404e0df73206d926779af9a88b6f744f15f64338 Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Mon, 29 Feb 2016 11:12:59 -0800 Subject: [PATCH 47/70] fix string types for py3 tests --- tests/pykafka/test_protocol.py | 22 +++++++++++----------- 1 file changed, 11 insertions(+), 11 deletions(-) diff --git a/tests/pykafka/test_protocol.py b/tests/pykafka/test_protocol.py index ffb25686d..f73b26c92 100644 --- a/tests/pykafka/test_protocol.py +++ b/tests/pykafka/test_protocol.py @@ -277,16 +277,16 @@ def test_join_group_request(self): def test_join_group_response(self): response = protocol.JoinGroupResponse( - bytearray('\x00\x00\x00\x00\x00\x01\x00\x17dummyassignmentstrategy\x00,pykafka-b2361322-674c-4e26-9194-305962636e57\x00,pykafka-b2361322-674c-4e26-9194-305962636e57\x00\x00\x00\x01\x00,pykafka-b2361322-674c-4e26-9194-305962636e57\x00\x00\x00"\x00\x00\x00\x00\x00\x01\x00\ndummytopic\x00\x00\x00\x0ctestuserdata\x00\x00\x00\x00') + bytearray(b'\x00\x00\x00\x00\x00\x01\x00\x17dummyassignmentstrategy\x00,pykafka-b2361322-674c-4e26-9194-305962636e57\x00,pykafka-b2361322-674c-4e26-9194-305962636e57\x00\x00\x00\x01\x00,pykafka-b2361322-674c-4e26-9194-305962636e57\x00\x00\x00"\x00\x00\x00\x00\x00\x01\x00\ndummytopic\x00\x00\x00\x0ctestuserdata\x00\x00\x00\x00') ) self.assertEqual(response.generation_id, 1) - self.assertEqual(response.group_protocol, 'dummyassignmentstrategy') + self.assertEqual(response.group_protocol, b'dummyassignmentstrategy') self.assertEqual(response.leader_id, - 'pykafka-b2361322-674c-4e26-9194-305962636e57') + b'pykafka-b2361322-674c-4e26-9194-305962636e57') self.assertEqual(response.member_id, - 'pykafka-b2361322-674c-4e26-9194-305962636e57') + b'pykafka-b2361322-674c-4e26-9194-305962636e57') self.assertEqual(response.members, - {'pykafka-b2361322-674c-4e26-9194-305962636e57': '\x00\x00\x00\x00\x00\x01\x00\ndummytopic\x00\x00\x00\x0ctestuserdata'}) + {b'pykafka-b2361322-674c-4e26-9194-305962636e57': b'\x00\x00\x00\x00\x00\x01\x00\ndummytopic\x00\x00\x00\x0ctestuserdata'}) def test_member_assignment_construction(self): assignment = protocol.MemberAssignment([(b"mytopic1", [3, 5, 7, 9]), @@ -302,9 +302,9 @@ def test_sync_group_request(self): b'dummygroup', 1, b'testmember1', [ protocol.MemberAssignment([(b"mytopic1", [3, 5, 7, 9]), - (b"mytopic2", [3, 5, 7, 9])], member_id="a"), + (b"mytopic2", [3, 5, 7, 9])], member_id=b"a"), protocol.MemberAssignment([(b"mytopic1", [2, 4, 6, 8]), - (b"mytopic2", [2, 4, 6, 8])], member_id="b") + (b"mytopic2", [2, 4, 6, 8])], member_id=b"b") ]) msg = req.get_bytes() self.assertEqual( @@ -314,10 +314,10 @@ def test_sync_group_request(self): def test_sync_group_response(self): response = protocol.SyncGroupResponse( - bytearray('\x00\x00\x00\x00\x00H\x00\x01\x00\x00\x00\x01\x00\x14testtopic_replicated\x00\x00\x00\n\x00\x00\x00\x06\x00\x00\x00\x07\x00\x00\x00\x08\x00\x00\x00\t\x00\x00\x00\x00\x00\x00\x00\x01\x00\x00\x00\x02\x00\x00\x00\x03\x00\x00\x00\x04\x00\x00\x00\x05,pyk') + bytearray(b'\x00\x00\x00\x00\x00H\x00\x01\x00\x00\x00\x01\x00\x14testtopic_replicated\x00\x00\x00\n\x00\x00\x00\x06\x00\x00\x00\x07\x00\x00\x00\x08\x00\x00\x00\t\x00\x00\x00\x00\x00\x00\x00\x01\x00\x00\x00\x02\x00\x00\x00\x03\x00\x00\x00\x04\x00\x00\x00\x05,pyk') ) self.assertEqual(response.error_code, 0) - expected_assignment = [('testtopic_replicated', [6, 7, 8, 9, 0, 1, 2, 3, 4, 5])] + expected_assignment = [(b'testtopic_replicated', [6, 7, 8, 9, 0, 1, 2, 3, 4, 5])] self.assertEqual(response.member_assignment.partition_assignment, expected_assignment) @@ -330,7 +330,7 @@ def test_heartbeat_request(self): ) def test_heartbeat_response(self): - response = protocol.HeartbeatResponse(bytearray('\x00\x00\x00\x01\x00\x14')) + response = protocol.HeartbeatResponse(bytearray(b'\x00\x00\x00\x01\x00\x14')) self.assertEqual(response.error_code, 0) def test_leave_group_request(self): @@ -342,7 +342,7 @@ def test_leave_group_request(self): ) def test_leave_group_response(self): - response = protocol.LeaveGroupResponse(bytearray('\x00\x00\x00\x01\x00\x14')) + response = protocol.LeaveGroupResponse(bytearray(b'\x00\x00\x00\x01\x00\x14')) self.assertEqual(response.error_code, 0) From 9bc6d8c1545321c96c002eef3da8f08759351a30 Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Mon, 29 Feb 2016 11:31:44 -0800 Subject: [PATCH 48/70] properly format generation id --- pykafka/broker.py | 4 ++-- 1 file changed, 2 insertions(+), 2 deletions(-) diff --git a/pykafka/broker.py b/pykafka/broker.py index ea269a7cb..d54aede52 100644 --- a/pykafka/broker.py +++ b/pykafka/broker.py @@ -29,7 +29,7 @@ OffsetFetchResponse, ProduceResponse, JoinGroupRequest, JoinGroupResponse, SyncGroupRequest, SyncGroupResponse, HeartbeatRequest, HeartbeatResponse, LeaveGroupRequest, LeaveGroupResponse) -from .utils.compat import range, iteritems +from .utils.compat import range, iteritems, get_bytes log = logging.getLogger(__name__) @@ -327,7 +327,7 @@ def commit_consumer_group_offsets(self, if not self.offsets_channel_connected: self.connect_offsets_channel() req = OffsetCommitRequest(consumer_group, - consumer_group_generation_id, + get_bytes(consumer_group_generation_id), consumer_id, partition_requests=preqs) return self._offsets_channel_req_handler.request(req).get(OffsetCommitResponse) From 16bf92bee57a3f95bfe749dc638624aa16c73bc6 Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Mon, 29 Feb 2016 11:48:08 -0800 Subject: [PATCH 49/70] more test formatting --- pykafka/balancedconsumer.py | 2 +- tests/pykafka/test_balancedconsumer.py | 12 ++++++------ tests/pykafka/test_protocol.py | 2 +- 3 files changed, 8 insertions(+), 8 deletions(-) diff --git a/pykafka/balancedconsumer.py b/pykafka/balancedconsumer.py index f0b1c8b38..3540e9f1a 100644 --- a/pykafka/balancedconsumer.py +++ b/pykafka/balancedconsumer.py @@ -224,7 +224,7 @@ def __init__(self, self._rebalancing_lock = cluster.handler.Lock() self._consumer = None - self._consumer_id = "{hostname}:{uuid}".format( + self._consumer_id = b"{hostname}:{uuid}".format( hostname=socket.gethostname(), uuid=uuid4() ) diff --git a/tests/pykafka/test_balancedconsumer.py b/tests/pykafka/test_balancedconsumer.py index 1d5ff68a8..8cc061658 100644 --- a/tests/pykafka/test_balancedconsumer.py +++ b/tests/pykafka/test_balancedconsumer.py @@ -412,18 +412,18 @@ def wait_for_rebalancing(self, *balanced_consumers): raise AssertionError("Rebalancing failed") -@pytest.mark.skipif(kafka_version < version_09, - reason="Managed consumer only supported in >=0.9") -class ManagedBalancedConsumerIntegrationTests(BalancedConsumerIntegrationTests): - MANAGED_CONSUMER = True - - @pytest.mark.skipif(platform.python_implementation() == "PyPy", reason="Unresolved crashes") class BalancedConsumerGEventIntegrationTests(BalancedConsumerIntegrationTests): USE_GEVENT = True +@pytest.mark.skipif(kafka_version < version_09, + reason="Managed consumer only supported in >=0.9") +class ManagedBalancedConsumerIntegrationTests(BalancedConsumerIntegrationTests): + MANAGED_CONSUMER = True + + @pytest.mark.skipif(platform.python_implementation() == "PyPy", reason="Unresolved crashes") @pytest.mark.skipif(kafka_version < version_09, diff --git a/tests/pykafka/test_protocol.py b/tests/pykafka/test_protocol.py index f73b26c92..e7a0526bb 100644 --- a/tests/pykafka/test_protocol.py +++ b/tests/pykafka/test_protocol.py @@ -309,7 +309,7 @@ def test_sync_group_request(self): msg = req.get_bytes() self.assertEqual( msg, - bytearray(b'\x00\x00\x00\xd8\x00\x0e\x00\x00\x00\x00\x00\x00\x00\x07pykafka\x00\ndummygroup\x00\x00\x00\x01\x00\x0btestmember1\x00\x00\x00\x02\x00\x0btestmember1\x00\x00\x00B\x00\x01\x00\x00\x00\x02\x00\x08mytopic1\x00\x00\x00\x04\x00\x00\x00\x03\x00\x00\x00\x05\x00\x00\x00\x07\x00\x00\x00\t\x00\x08mytopic2\x00\x00\x00\x04\x00\x00\x00\x03\x00\x00\x00\x05\x00\x00\x00\x07\x00\x00\x00\t\x00\x0btestmember2\x00\x00\x00B\x00\x01\x00\x00\x00\x02\x00\x08mytopic1\x00\x00\x00\x04\x00\x00\x00\x02\x00\x00\x00\x04\x00\x00\x00\x06\x00\x00\x00\x08\x00\x08mytopic2\x00\x00\x00\x04\x00\x00\x00\x02\x00\x00\x00\x04\x00\x00\x00\x06\x00\x00\x00\x08') + bytearray(b'\x00\x00\x00\xc4\x00\x0e\x00\x00\x00\x00\x00\x00\x00\x07pykafka\x00\ndummygroup\x00\x00\x00\x01\x00\x0btestmember1\x00\x00\x00\x02\x00\x01a\x00\x00\x00B\x00\x01\x00\x00\x00\x02\x00\x08mytopic1\x00\x00\x00\x04\x00\x00\x00\x03\x00\x00\x00\x05\x00\x00\x00\x07\x00\x00\x00\t\x00\x08mytopic2\x00\x00\x00\x04\x00\x00\x00\x03\x00\x00\x00\x05\x00\x00\x00\x07\x00\x00\x00\t\x00\x01b\x00\x00\x00B\x00\x01\x00\x00\x00\x02\x00\x08mytopic1\x00\x00\x00\x04\x00\x00\x00\x02\x00\x00\x00\x04\x00\x00\x00\x06\x00\x00\x00\x08\x00\x08mytopic2\x00\x00\x00\x04\x00\x00\x00\x02\x00\x00\x00\x04\x00\x00\x00\x06\x00\x00\x00\x08') ) def test_sync_group_response(self): From e3e879b2d16c95003c89c2bce5d160fd02469126 Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Mon, 29 Feb 2016 12:05:57 -0800 Subject: [PATCH 50/70] test adjustments --- pykafka/balancedconsumer.py | 4 ++-- tests/pykafka/test_balancedconsumer.py | 3 ++- 2 files changed, 4 insertions(+), 3 deletions(-) diff --git a/pykafka/balancedconsumer.py b/pykafka/balancedconsumer.py index 3540e9f1a..a2ce9418a 100644 --- a/pykafka/balancedconsumer.py +++ b/pykafka/balancedconsumer.py @@ -224,10 +224,10 @@ def __init__(self, self._rebalancing_lock = cluster.handler.Lock() self._consumer = None - self._consumer_id = b"{hostname}:{uuid}".format( + self._consumer_id = get_bytes("{hostname}:{uuid}".format( hostname=socket.gethostname(), uuid=uuid4() - ) + )) self._setting_watches = True self._topic_path = '/consumers/{group}/owners/{topic}'.format( diff --git a/tests/pykafka/test_balancedconsumer.py b/tests/pykafka/test_balancedconsumer.py index 8cc061658..a5c8b30cb 100644 --- a/tests/pykafka/test_balancedconsumer.py +++ b/tests/pykafka/test_balancedconsumer.py @@ -428,7 +428,8 @@ class ManagedBalancedConsumerIntegrationTests(BalancedConsumerIntegrationTests): reason="Unresolved crashes") @pytest.mark.skipif(kafka_version < version_09, reason="Managed consumer only supported in >=0.9") -class ManagedBalancedConsumerGEventIntegrationTests(ManagedBalancedConsumerIntegrationTests): +class ManagedBalancedConsumerGEventIntegrationTests(BalancedConsumerIntegrationTests): + MANAGED_CONSUMER = True USE_GEVENT = True From fa99ba659a90087272ed063a01698f49e6f38585 Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Mon, 29 Feb 2016 12:54:07 -0800 Subject: [PATCH 51/70] use bytestrings in groupmembershipprotocol --- pykafka/protocol.py | 2 +- 1 file changed, 1 insertion(+), 1 deletion(-) diff --git a/pykafka/protocol.py b/pykafka/protocol.py index 91d93c4a0..7f9527f94 100644 --- a/pykafka/protocol.py +++ b/pykafka/protocol.py @@ -1239,7 +1239,7 @@ def get_bytes(self): ) -ConsumerGroupProtocol = GroupMembershipProtocol("consumer", "dummyassignmentstrategy", +ConsumerGroupProtocol = GroupMembershipProtocol(b"consumer", b"pykafkaassignmentstrategy", ConsumerGroupProtocolMetadata()) From e6682c2f6a191584992adc8dec9c16f9d926028b Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Mon, 29 Feb 2016 13:31:09 -0800 Subject: [PATCH 52/70] subclass balancedconsumer tests --- pykafka/simpleconsumer.py | 8 +++-- tests/pykafka/test_balancedconsumer.py | 45 +++++++++++++++++++------- tests/pykafka/test_protocol.py | 2 +- tox.ini | 2 +- 4 files changed, 41 insertions(+), 16 deletions(-) diff --git a/pykafka/simpleconsumer.py b/pykafka/simpleconsumer.py index 2205670a5..f28e8f4eb 100644 --- a/pykafka/simpleconsumer.py +++ b/pykafka/simpleconsumer.py @@ -35,7 +35,7 @@ NotCoordinatorForGroup, SocketDisconnectedError, ConsumerStoppedException, KafkaException, NotLeaderForPartition, OffsetRequestFailedError, - RequestTimedOut, ERROR_CODES) + RequestTimedOut, UnknownMemberId, ERROR_CODES) from .protocol import (PartitionFetchRequest, PartitionOffsetCommitRequest, PartitionOffsetFetchRequest, PartitionOffsetRequest) from .utils.error_handlers import (handle_partition_responses, raise_error, @@ -280,6 +280,9 @@ def _handle_NotLeaderForPartition(parts): def _handle_GroupLoadInProgress(parts): log.info("Continuing in response to GroupLoadInProgress") + def _handle_UnknownMemberId(parts): + log.info("Continuing in response to UnknownMemberId") + return { UnknownTopicOrPartition.ERROR_CODE: lambda p: raise_error(UnknownTopicOrPartition), OffsetOutOfRangeError.ERROR_CODE: _handle_OffsetOutOfRangeError, @@ -287,7 +290,8 @@ def _handle_GroupLoadInProgress(parts): OffsetMetadataTooLarge.ERROR_CODE: lambda p: raise_error(OffsetMetadataTooLarge), NotCoordinatorForGroup.ERROR_CODE: _handle_NotCoordinatorForGroup, RequestTimedOut.ERROR_CODE: _handle_RequestTimedOut, - GroupLoadInProgress.ERROR_CODE: _handle_GroupLoadInProgress + GroupLoadInProgress.ERROR_CODE: _handle_GroupLoadInProgress, + UnknownMemberId.ERROR_CODE: _handle_UnknownMemberId } def _discover_group_coordinator(self): diff --git a/tests/pykafka/test_balancedconsumer.py b/tests/pykafka/test_balancedconsumer.py index a5c8b30cb..f215b4561 100644 --- a/tests/pykafka/test_balancedconsumer.py +++ b/tests/pykafka/test_balancedconsumer.py @@ -19,7 +19,7 @@ from pykafka.utils.compat import range, iterkeys, iteritems -kafka_version = pkg_resources.parse_version(os.environ.get("KAFKA_VERSION", "0.8")) +kafka_version = pkg_resources.parse_version(os.environ.get('KAFKA_VERSION', '0.8')) version_09 = pkg_resources.parse_version("0.9.0.0") @@ -412,23 +412,44 @@ def wait_for_rebalancing(self, *balanced_consumers): raise AssertionError("Rebalancing failed") -@pytest.mark.skipif(platform.python_implementation() == "PyPy", - reason="Unresolved crashes") -class BalancedConsumerGEventIntegrationTests(BalancedConsumerIntegrationTests): +def patch_subclass(parent, condition): + def patcher(cls): + def build_skipped_method(method, cond=None): + if cond is None: + cond = False + + @pytest.mark.skipif(cond, reason="") + def _wrapper(): + return method() + return _wrapper + + for attr in parent.__dict__: + if attr in cls.__dict__: + continue + if attr.startswith("test_"): + setattr(cls, attr, build_skipped_method(parent.__dict__[attr], + condition)) + else: + setattr(cls, attr, parent.__dict__[attr]) + return cls + return patcher + + +@patch_subclass(BalancedConsumerIntegrationTests, + platform.python_implementation() == "PyPy") +class BalancedConsumerGEventIntegrationTests(unittest2.TestCase): USE_GEVENT = True -@pytest.mark.skipif(kafka_version < version_09, - reason="Managed consumer only supported in >=0.9") -class ManagedBalancedConsumerIntegrationTests(BalancedConsumerIntegrationTests): +@patch_subclass(BalancedConsumerIntegrationTests, kafka_version < version_09) +class ManagedBalancedConsumerIntegrationTests(unittest2.TestCase): MANAGED_CONSUMER = True -@pytest.mark.skipif(platform.python_implementation() == "PyPy", - reason="Unresolved crashes") -@pytest.mark.skipif(kafka_version < version_09, - reason="Managed consumer only supported in >=0.9") -class ManagedBalancedConsumerGEventIntegrationTests(BalancedConsumerIntegrationTests): +@patch_subclass( + BalancedConsumerIntegrationTests, + platform.python_implementation() == "PyPy" or kafka_version < version_09) +class ManagedBalancedConsumerGEventIntegrationTests(unittest2.TestCase): MANAGED_CONSUMER = True USE_GEVENT = True diff --git a/tests/pykafka/test_protocol.py b/tests/pykafka/test_protocol.py index e7a0526bb..1329ed6b9 100644 --- a/tests/pykafka/test_protocol.py +++ b/tests/pykafka/test_protocol.py @@ -272,7 +272,7 @@ def test_join_group_request(self): msg = req.get_bytes() self.assertEqual( msg, - bytearray(b'\x00\x00\x00z\x00\x0b\x00\x00\x00\x00\x00\x00\x00\x07pykafka\x00\ndummygroup\x00\x00u0\x00\ntestmember\x00\x08consumer\x00\x00\x00\x01\x00\x17dummyassignmentstrategy\x00\x00\x00"\x00\x00\x00\x00\x00\x01\x00\ndummytopic\x00\x00\x00\x0ctestuserdata') + bytearray(b'\x00\x00\x00|\x00\x0b\x00\x00\x00\x00\x00\x00\x00\x07pykafka\x00\ndummygroup\x00\x00u0\x00\ntestmember\x00\x08consumer\x00\x00\x00\x01\x00\x19pykafkaassignmentstrategy\x00\x00\x00"\x00\x00\x00\x00\x00\x01\x00\ndummytopic\x00\x00\x00\x0ctestuserdata') ) def test_join_group_response(self): diff --git a/tox.ini b/tox.ini index bf420cf0e..d4ee950c0 100644 --- a/tox.ini +++ b/tox.ini @@ -6,4 +6,4 @@ commands = pip install -r test-requirements.txt pip install -e . py.test {posargs} -passenv = BROKERS ZOOKEEPER KAFKA_BIN C_INCLUDE_PATH LIBRARY_PATH LD_LIBRARY_PATH CFLAGS +passenv = BROKERS ZOOKEEPER KAFKA_BIN KAFKA_VERSION C_INCLUDE_PATH LIBRARY_PATH LD_LIBRARY_PATH CFLAGS From cf8686e46f6a4afa66a97927b221db718b3a8725 Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Tue, 1 Mar 2016 10:42:23 -0800 Subject: [PATCH 53/70] fix crash --- tests/pykafka/test_balancedconsumer.py | 27 ++++++++++++++++++++++---- 1 file changed, 23 insertions(+), 4 deletions(-) diff --git a/tests/pykafka/test_balancedconsumer.py b/tests/pykafka/test_balancedconsumer.py index f215b4561..4cd2be899 100644 --- a/tests/pykafka/test_balancedconsumer.py +++ b/tests/pykafka/test_balancedconsumer.py @@ -412,15 +412,34 @@ def wait_for_rebalancing(self, *balanced_consumers): raise AssertionError("Rebalancing failed") -def patch_subclass(parent, condition): +def patch_subclass(parent, skip_condition): + """Work around a pytest.mark.skipif bug + + https://github.com/pytest-dev/pytest/issues/568 + + The issue causes all subclasses of a TestCase subclass to be skipped if any one + of them is skipped. + + This fix circumvents the issue by overriding Python's existing subclassing mechanism. + Instead of having `cls` be a subclass of `parent`, this decorator adds each attribute + of `parent` to `cls` without using Python inheritance. When appropriate, it also adds + a boolean condition under which to skip tests for the decorated class. + + :param parent: The "superclass" from which the decorated class should inherit + its non-overridden attributes + :type parent: unittest2.TestCase + :param skip_condition: A boolean condition that, when True, will cause all tests in + the decorated class to be skipped + :type skip_condition: bool + """ def patcher(cls): def build_skipped_method(method, cond=None): if cond is None: cond = False @pytest.mark.skipif(cond, reason="") - def _wrapper(): - return method() + def _wrapper(self): + return method(self) return _wrapper for attr in parent.__dict__: @@ -428,7 +447,7 @@ def _wrapper(): continue if attr.startswith("test_"): setattr(cls, attr, build_skipped_method(parent.__dict__[attr], - condition)) + skip_condition)) else: setattr(cls, attr, parent.__dict__[attr]) return cls From 815256024873f8b93b9344a62f4ff83b85036715 Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Tue, 1 Mar 2016 11:33:46 -0800 Subject: [PATCH 54/70] try fixing some tests --- pykafka/managedbalancedconsumer.py | 15 +++++++-------- tests/pykafka/test_balancedconsumer.py | 1 + 2 files changed, 8 insertions(+), 8 deletions(-) diff --git a/pykafka/managedbalancedconsumer.py b/pykafka/managedbalancedconsumer.py index b8a9e419b..e29174c96 100644 --- a/pykafka/managedbalancedconsumer.py +++ b/pykafka/managedbalancedconsumer.py @@ -308,16 +308,16 @@ def _join_group(self): join_result.error_code) if i == self._cluster._max_connection_retries - 1: raise ERROR_CODES[join_result.error_code] - if join_result.error_code in (GroupLoadInProgress.ERROR_CODE,): + if join_result.error_code in (GroupLoadInProgress.ERROR_CODE, + UnknownMemberId.ERROR_CODE, + GroupAuthorizationFailed.ERROR_CODE): pass elif join_result.error_code in (GroupCoordinatorNotAvailable.ERROR_CODE, NotCoordinatorForGroup.ERROR_CODE): self._group_coordinator = self._cluster.get_group_coordinator( self._consumer_group) elif join_result.error_code in (InconsistentGroupProtocol.ERROR_CODE, - UnknownMemberId.ERROR_CODE, - InvalidSessionTimeout.ERROR_CODE, - GroupAuthorizationFailed.ERROR_CODE): + InvalidSessionTimeout.ERROR_CODE): raise ERROR_CODES[join_result.error_code] self._cluster.handler.sleep(i * 2) self._generation_id = join_result.generation_id @@ -347,13 +347,12 @@ def _sync_group(self, group_assignments): raise ERROR_CODES[sync_result.error_code] if sync_result.error_code in (IllegalGeneration.ERROR_CODE, GroupCoordinatorNotAvailable.ERROR_CODE, - RebalanceInProgress.ERROR_CODE): + RebalanceInProgress.ERROR_CODE, + UnknownMemberId.ERROR_CODE, + GroupAuthorizationFailed.ERROR_CODE): pass elif sync_result.error_code in (NotCoordinatorForGroup.ERROR_CODE,): self._group_coordinator = self._cluster.get_group_coordinator( self._consumer_group) - elif sync_result.error_code in (UnknownMemberId.ERROR_CODE, - GroupAuthorizationFailed.ERROR_CODE): - raise ERROR_CODES[sync_result.error_code] self._cluster.handler.sleep(i * 2) return sync_result.member_assignment.partition_assignment diff --git a/tests/pykafka/test_balancedconsumer.py b/tests/pykafka/test_balancedconsumer.py index 4cd2be899..c436661a8 100644 --- a/tests/pykafka/test_balancedconsumer.py +++ b/tests/pykafka/test_balancedconsumer.py @@ -349,6 +349,7 @@ def test_no_partitions(self): # check that stop() succeeds (cf #313 and #392) consumer.stop() + @pytest.mark.skipif(MANAGED_CONSUMER, reason="Managed consumer groups don't use ZK") def test_zk_conn_lost(self): """Check we restore zookeeper nodes correctly after connection loss From 86fded77acddbdf6328bf0e8e4774b789e13408e Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Tue, 1 Mar 2016 12:12:11 -0800 Subject: [PATCH 55/70] rework some error handling in consumers --- pykafka/managedbalancedconsumer.py | 15 ++++++++------- pykafka/simpleconsumer.py | 9 +++++++-- 2 files changed, 15 insertions(+), 9 deletions(-) diff --git a/pykafka/managedbalancedconsumer.py b/pykafka/managedbalancedconsumer.py index e29174c96..b8a9e419b 100644 --- a/pykafka/managedbalancedconsumer.py +++ b/pykafka/managedbalancedconsumer.py @@ -308,16 +308,16 @@ def _join_group(self): join_result.error_code) if i == self._cluster._max_connection_retries - 1: raise ERROR_CODES[join_result.error_code] - if join_result.error_code in (GroupLoadInProgress.ERROR_CODE, - UnknownMemberId.ERROR_CODE, - GroupAuthorizationFailed.ERROR_CODE): + if join_result.error_code in (GroupLoadInProgress.ERROR_CODE,): pass elif join_result.error_code in (GroupCoordinatorNotAvailable.ERROR_CODE, NotCoordinatorForGroup.ERROR_CODE): self._group_coordinator = self._cluster.get_group_coordinator( self._consumer_group) elif join_result.error_code in (InconsistentGroupProtocol.ERROR_CODE, - InvalidSessionTimeout.ERROR_CODE): + UnknownMemberId.ERROR_CODE, + InvalidSessionTimeout.ERROR_CODE, + GroupAuthorizationFailed.ERROR_CODE): raise ERROR_CODES[join_result.error_code] self._cluster.handler.sleep(i * 2) self._generation_id = join_result.generation_id @@ -347,12 +347,13 @@ def _sync_group(self, group_assignments): raise ERROR_CODES[sync_result.error_code] if sync_result.error_code in (IllegalGeneration.ERROR_CODE, GroupCoordinatorNotAvailable.ERROR_CODE, - RebalanceInProgress.ERROR_CODE, - UnknownMemberId.ERROR_CODE, - GroupAuthorizationFailed.ERROR_CODE): + RebalanceInProgress.ERROR_CODE): pass elif sync_result.error_code in (NotCoordinatorForGroup.ERROR_CODE,): self._group_coordinator = self._cluster.get_group_coordinator( self._consumer_group) + elif sync_result.error_code in (UnknownMemberId.ERROR_CODE, + GroupAuthorizationFailed.ERROR_CODE): + raise ERROR_CODES[sync_result.error_code] self._cluster.handler.sleep(i * 2) return sync_result.member_assignment.partition_assignment diff --git a/pykafka/simpleconsumer.py b/pykafka/simpleconsumer.py index f28e8f4eb..f19214b63 100644 --- a/pykafka/simpleconsumer.py +++ b/pykafka/simpleconsumer.py @@ -35,7 +35,8 @@ NotCoordinatorForGroup, SocketDisconnectedError, ConsumerStoppedException, KafkaException, NotLeaderForPartition, OffsetRequestFailedError, - RequestTimedOut, UnknownMemberId, ERROR_CODES) + RequestTimedOut, UnknownMemberId, RebalanceInProgress, + ERROR_CODES) from .protocol import (PartitionFetchRequest, PartitionOffsetCommitRequest, PartitionOffsetFetchRequest, PartitionOffsetRequest) from .utils.error_handlers import (handle_partition_responses, raise_error, @@ -283,6 +284,9 @@ def _handle_GroupLoadInProgress(parts): def _handle_UnknownMemberId(parts): log.info("Continuing in response to UnknownMemberId") + def _handle_RebalanceInProgress(parts): + log.info("Continuing in response to RebalanceInProgress") + return { UnknownTopicOrPartition.ERROR_CODE: lambda p: raise_error(UnknownTopicOrPartition), OffsetOutOfRangeError.ERROR_CODE: _handle_OffsetOutOfRangeError, @@ -291,7 +295,8 @@ def _handle_UnknownMemberId(parts): NotCoordinatorForGroup.ERROR_CODE: _handle_NotCoordinatorForGroup, RequestTimedOut.ERROR_CODE: _handle_RequestTimedOut, GroupLoadInProgress.ERROR_CODE: _handle_GroupLoadInProgress, - UnknownMemberId.ERROR_CODE: _handle_UnknownMemberId + UnknownMemberId.ERROR_CODE: _handle_UnknownMemberId, + RebalanceInProgress.ERROR_CODE: _handle_RebalanceInProgress } def _discover_group_coordinator(self): From 6f14815bbc74e36e0bd618d49c78a8e3675e5512 Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Tue, 1 Mar 2016 12:12:26 -0800 Subject: [PATCH 56/70] allow test-specific switching --- tests/pykafka/test_balancedconsumer.py | 5 ++++- 1 file changed, 4 insertions(+), 1 deletion(-) diff --git a/tests/pykafka/test_balancedconsumer.py b/tests/pykafka/test_balancedconsumer.py index c436661a8..77eef2fa2 100644 --- a/tests/pykafka/test_balancedconsumer.py +++ b/tests/pykafka/test_balancedconsumer.py @@ -332,6 +332,7 @@ def test_external_kazoo_client(self): use_rdkafka=self.USE_RDKAFKA) [msg for msg in consumer] consumer.stop() + test_external_kazoo_client.skip_condition = MANAGED_CONSUMER def test_no_partitions(self): """Ensure a consumer assigned no partitions doesn't fail""" @@ -349,7 +350,6 @@ def test_no_partitions(self): # check that stop() succeeds (cf #313 and #392) consumer.stop() - @pytest.mark.skipif(MANAGED_CONSUMER, reason="Managed consumer groups don't use ZK") def test_zk_conn_lost(self): """Check we restore zookeeper nodes correctly after connection loss @@ -392,6 +392,7 @@ def test_zk_conn_lost(self): zk.stop() except: pass + test_zk_conn_lost.skip_condition = MANAGED_CONSUMER def wait_for_rebalancing(self, *balanced_consumers): """Test helper that loops while rebalancing is ongoing @@ -437,6 +438,8 @@ def patcher(cls): def build_skipped_method(method, cond=None): if cond is None: cond = False + if hasattr(method, "skip_condition"): + cond = cond or method.skip_condition @pytest.mark.skipif(cond, reason="") def _wrapper(self): From 2d927eb971dd8d86d2b6106d8e0f3c730f9d5b58 Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Tue, 1 Mar 2016 12:29:05 -0800 Subject: [PATCH 57/70] allow per-function skipping --- tests/pykafka/test_balancedconsumer.py | 17 ++++++++++------- 1 file changed, 10 insertions(+), 7 deletions(-) diff --git a/tests/pykafka/test_balancedconsumer.py b/tests/pykafka/test_balancedconsumer.py index 77eef2fa2..028784c11 100644 --- a/tests/pykafka/test_balancedconsumer.py +++ b/tests/pykafka/test_balancedconsumer.py @@ -332,7 +332,7 @@ def test_external_kazoo_client(self): use_rdkafka=self.USE_RDKAFKA) [msg for msg in consumer] consumer.stop() - test_external_kazoo_client.skip_condition = MANAGED_CONSUMER + test_external_kazoo_client.skip_condition = lambda cls: cls.MANAGED_CONSUMER def test_no_partitions(self): """Ensure a consumer assigned no partitions doesn't fail""" @@ -392,7 +392,7 @@ def test_zk_conn_lost(self): zk.stop() except: pass - test_zk_conn_lost.skip_condition = MANAGED_CONSUMER + test_zk_conn_lost.skip_condition = lambda cls: cls.MANAGED_CONSUMER def wait_for_rebalancing(self, *balanced_consumers): """Test helper that loops while rebalancing is ongoing @@ -435,25 +435,28 @@ def patch_subclass(parent, skip_condition): :type skip_condition: bool """ def patcher(cls): - def build_skipped_method(method, cond=None): + def build_skipped_method(method, cls, cond=None): if cond is None: cond = False if hasattr(method, "skip_condition"): - cond = cond or method.skip_condition + cond = cond or method.skip_condition(cls) @pytest.mark.skipif(cond, reason="") def _wrapper(self): return method(self) return _wrapper + # two passes required so that skips have access to all class attributes for attr in parent.__dict__: if attr in cls.__dict__: continue + if not attr.startswith("test_"): + setattr(cls, attr, parent.__dict__[attr]) + + for attr in parent.__dict__: if attr.startswith("test_"): setattr(cls, attr, build_skipped_method(parent.__dict__[attr], - skip_condition)) - else: - setattr(cls, attr, parent.__dict__[attr]) + cls, skip_condition)) return cls return patcher From 705dd1df63b0bce5a6f1e7f3c38edd52acca0220 Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Thu, 3 Mar 2016 10:29:37 -0800 Subject: [PATCH 58/70] improve logging --- pykafka/balancedconsumer.py | 5 +++-- pykafka/managedbalancedconsumer.py | 10 +++++++--- pykafka/simpleconsumer.py | 8 ++++---- 3 files changed, 14 insertions(+), 9 deletions(-) diff --git a/pykafka/balancedconsumer.py b/pykafka/balancedconsumer.py index a2ce9418a..876f9aee7 100644 --- a/pykafka/balancedconsumer.py +++ b/pykafka/balancedconsumer.py @@ -440,8 +440,9 @@ def _decide_partitions(self, participants, consumer_id=None): # assign partitions from i*N to (i+1)*N - 1 to consumer Ci new_partitions = itertools.islice(all_parts, start, start + num_parts) new_partitions = set(new_partitions) - log.info('Balancing %i participants for %i partitions.\nOwning %i partitions.', - len(participants), len(all_parts), len(new_partitions)) + log.info('%s: Balancing %i participants for %i partitions. Owning %i partitions.', + self._consumer_id, len(participants), len(all_parts), + len(new_partitions)) log.debug('My partitions: %s', [p_to_str(p) for p in new_partitions]) return new_partitions diff --git a/pykafka/managedbalancedconsumer.py b/pykafka/managedbalancedconsumer.py index b8a9e419b..c0d6c022a 100644 --- a/pykafka/managedbalancedconsumer.py +++ b/pykafka/managedbalancedconsumer.py @@ -225,6 +225,7 @@ def start(self): self._running = True self._group_coordinator = self._cluster.get_group_coordinator( self._consumer_group) + log.debug("_rebalance called from _start") self._rebalance() self._setup_heartbeat_worker() except Exception: @@ -243,6 +244,7 @@ def stop(self): def _send_heartbeat(self): """Send a heartbeat request to the group coordinator and react to the response""" + log.info("Sending heartbeat from consumer '%s'", self._consumer_id) res = self._group_coordinator.heartbeat( self._consumer_group, self._generation_id, self._consumer_id) if res.error_code == 0: @@ -258,6 +260,7 @@ def _send_heartbeat(self): self._consumer_group) elif res.error_code == GroupAuthorizationFailed.ERROR_CODE: raise GroupAuthorizationFailed() + log.debug("_rebalance called from _send_heartbeat") self._rebalance() def _update_member_assignment(self): @@ -283,13 +286,14 @@ def _update_member_assignment(self): assignment = self._sync_group(group_assignments) self._setup_internal_consumer( partitions=[self._topic.partitions[pid] for pid in assignment[0][1]]) + log.debug("Successfully rebalanced consumer '%s'", self._consumer_id) break except Exception as ex: if i == self._rebalance_max_retries - 1: log.warning('Failed to rebalance s after %d retries.', i) raise - log.info('Unable to complete rebalancing. Retrying') log.exception(ex) + log.info('Unable to complete rebalancing. Retrying') self._cluster.handler.sleep(i * (self._rebalance_backoff_ms / 1000)) self._raise_worker_exceptions() @@ -304,8 +308,8 @@ def _join_group(self): self._consumer_group, self._consumer_id) if join_result.error_code == 0: break - log.info("Error code %d encountered during JoinGroupRequest", - join_result.error_code) + log.info("Error code %d encountered during JoinGroupRequest for" + " generation '%s'", join_result.error_code, self._generation_id) if i == self._cluster._max_connection_retries - 1: raise ERROR_CODES[join_result.error_code] if join_result.error_code in (GroupLoadInProgress.ERROR_CODE,): diff --git a/pykafka/simpleconsumer.py b/pykafka/simpleconsumer.py index f19214b63..6613c6089 100644 --- a/pykafka/simpleconsumer.py +++ b/pykafka/simpleconsumer.py @@ -454,9 +454,9 @@ def commit_offsets(self): response = self._group_coordinator.commit_consumer_group_offsets( self._consumer_group, self._generation_id, self._consumer_id, reqs) except (SocketDisconnectedError, IOError): - log.error("Error committing offsets for topic '%s' " + log.error("Error committing offsets for topic '%s' from consumer id '%s'" "(SocketDisconnectedError)", - self._topic.name) + self._topic.name, self._consumer_id) if i >= self._offsets_commit_max_retries - 1: raise self._update() @@ -469,8 +469,8 @@ def commit_offsets(self): if (len(parts_by_error) == 1 and 0 in parts_by_error) or \ len(parts_by_error) == 0: break - log.error("Error committing offsets for topic '%s' (errors: %s)", - self._topic.name, + log.error("Error committing offsets for topic '%s' from consumer id '%s'" + "(errors: %s)", self._topic.name, self._consumer_id, {ERROR_CODES[err]: [op.partition.id for op, _ in parts] for err, parts in iteritems(parts_by_error)}) From 45b2349af13fe0111b55bacb3d2192eaece6511d Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Thu, 3 Mar 2016 14:20:43 -0800 Subject: [PATCH 59/70] use unique RequestHandlers for each consumer instance in the same process --- pykafka/broker.py | 59 ++++++++++++++++++++++++++---- pykafka/managedbalancedconsumer.py | 27 ++++++++++---- 2 files changed, 70 insertions(+), 16 deletions(-) diff --git a/pykafka/broker.py b/pykafka/broker.py index d54aede52..f6f40ba8e 100644 --- a/pykafka/broker.py +++ b/pykafka/broker.py @@ -89,6 +89,7 @@ def __init__(self, self._socket_timeout_ms = socket_timeout_ms self._offsets_channel_socket_timeout_ms = offsets_channel_socket_timeout_ms self._buffer_size = buffer_size + self._req_handlers = {} self.connect() def __repr__(self): @@ -217,6 +218,32 @@ def connect_offsets_channel(self): ) self._offsets_channel_req_handler.start() + def _get_unique_req_handler(self, connection_id): + """Return a RequestHandler instance unique to the given connection_id + + In some applications, for example the Group Membership API, requests running + in the same process must be interleaved. When both of these requests are + using the same RequestHandler instance, the requests are queued and the + interleaving semantics are not upheld. This method behaves identically to + self._req_handler if there is only one connection_id per KafkaClient. + If a single KafkaClient needs to use more than one connection_id, this + method maintains a dictionary of connections unique to those ids. + + :param connection_id: The unique identifier of the connection to return + :type connection_id: str + """ + if len(self._req_handlers) == 0: + self._req_handlers[connection_id] = self._req_handler + elif connection_id not in self._req_handlers: + conn = BrokerConnection( + self.host, self.port, buffer_size=self._buffer_size, + source_host=self._source_host, source_port=self._source_port) + conn.connect(self._socket_timeout_ms) + handler = RequestHandler(self._handler, conn) + handler.start() + self._req_handlers[connection_id] = handler + return self._req_handlers[connection_id] + def fetch_messages(self, partition_requests, timeout=30000, @@ -353,31 +380,42 @@ def fetch_consumer_group_offsets(self, consumer_group, preqs): # Group Membership API # ########################## - def join_group(self, consumer_group, member_id): + def join_group(self, connection_id, consumer_group, member_id): """Send a JoinGroupRequest + :param connection_id: The unique identifier of the connection on which to make + this request + :type connection_id: str :param consumer_group: The name of the consumer group to join :type consumer_group: bytes :param member_id: The ID of the consumer joining the group :type member_id: bytes """ - future = self._req_handler.request(JoinGroupRequest(consumer_group, member_id)) + handler = self._get_unique_req_handler(connection_id) + future = handler.request(JoinGroupRequest(consumer_group, member_id)) return future.get(JoinGroupResponse) - def leave_group(self, consumer_group, member_id): + def leave_group(self, connection_id, consumer_group, member_id): """Send a LeaveGroupRequest + :param connection_id: The unique identifier of the connection on which to make + this request + :type connection_id: str :param consumer_group: The name of the consumer group to leave :type consumer_group: bytes :param member_id: The ID of the consumer leaving the group :type member_id: bytes """ - future = self._req_handler.request(LeaveGroupRequest(consumer_group, member_id)) + handler = self._get_unique_req_handler(connection_id) + future = handler.request(LeaveGroupRequest(consumer_group, member_id)) return future.get(LeaveGroupResponse) - def sync_group(self, consumer_group, generation_id, member_id, group_assignment): + def sync_group(self, connection_id, consumer_group, generation_id, member_id, group_assignment): """Send a SyncGroupRequest + :param connection_id: The unique identifier of the connection on which to make + this request + :type connection_id: str :param consumer_group: The name of the consumer group to which this consumer belongs :type consumer_group: bytes @@ -391,13 +429,17 @@ def sync_group(self, consumer_group, generation_id, member_id, group_assignment) `group_assignment` should be an empty sequence. :type group_assignment: iterable of :class:`pykafka.protocol.MemberAssignment` """ - future = self._req_handler.request( + handler = self._get_unique_req_handler(connection_id) + future = handler.request( SyncGroupRequest(consumer_group, generation_id, member_id, group_assignment)) return future.get(SyncGroupResponse) - def heartbeat(self, consumer_group, generation_id, member_id): + def heartbeat(self, connection_id, consumer_group, generation_id, member_id): """Send a HeartbeatRequest + :param connection_id: The unique identifier of the connection on which to make + this request + :type connection_id: str :param consumer_group: The name of the consumer group to which this consumer belongs :type consumer_group: bytes @@ -406,6 +448,7 @@ def heartbeat(self, consumer_group, generation_id, member_id): :param member_id: The ID of the consumer sending this heartbeat :type member_id: bytes """ - future = self._req_handler.request( + handler = self._get_unique_req_handler(connection_id) + future = handler.request( HeartbeatRequest(consumer_group, generation_id, member_id)) return future.get(HeartbeatResponse) diff --git a/pykafka/managedbalancedconsumer.py b/pykafka/managedbalancedconsumer.py index c0d6c022a..23ee3ff3b 100644 --- a/pykafka/managedbalancedconsumer.py +++ b/pykafka/managedbalancedconsumer.py @@ -20,6 +20,7 @@ __all__ = ["ManagedBalancedConsumer"] import logging import sys +import uuid import weakref from .balancedconsumer import BalancedConsumer @@ -188,6 +189,10 @@ def __init__(self, self._generation_id = -1 self._rebalancing_lock = cluster.handler.Lock() + # ManagedBalancedConsumers in the same process cannot share connections. + # This connection hash is passed to self.Broker calls that use the group + # membership API + self._connection_id = uuid.uuid4() self._consumer = None self._group_coordinator = None self._consumer_id = b'' @@ -240,13 +245,16 @@ def stop(self): self._running = False if self._consumer is not None: self._consumer.stop() - self._group_coordinator.leave_group(self._consumer_group, self._consumer_id) + self._group_coordinator.leave_group(self._connection_id, self._consumer_group, + self._consumer_id) def _send_heartbeat(self): """Send a heartbeat request to the group coordinator and react to the response""" log.info("Sending heartbeat from consumer '%s'", self._consumer_id) - res = self._group_coordinator.heartbeat( - self._consumer_group, self._generation_id, self._consumer_id) + res = self._group_coordinator.heartbeat(self._connection_id, + self._consumer_group, + self._generation_id, + self._consumer_id) if res.error_code == 0: return log.info("Error code %d encountered on heartbeat." % res.error_code) @@ -304,8 +312,9 @@ def _join_group(self): """ log.info("Sending JoinGroupRequest for consumer id '%s'", self._consumer_id) for i in range(self._cluster._max_connection_retries): - join_result = self._group_coordinator.join_group( - self._consumer_group, self._consumer_id) + join_result = self._group_coordinator.join_group(self._connection_id, + self._consumer_group, + self._consumer_id) if join_result.error_code == 0: break log.info("Error code %d encountered during JoinGroupRequest for" @@ -340,9 +349,11 @@ def _sync_group(self, group_assignments): """ log.info("Sending SyncGroupRequest for consumer id '%s'", self._consumer_id) for i in range(self._cluster._max_connection_retries): - sync_result = self._group_coordinator.sync_group( - self._consumer_group, self._generation_id, self._consumer_id, - group_assignments) + sync_result = self._group_coordinator.sync_group(self._connection_id, + self._consumer_group, + self._generation_id, + self._consumer_id, + group_assignments) if sync_result.error_code == 0: break log.info("Error code %d encountered during SyncGroupRequest", From ad14ea8c1f27ea450e01d38dbc133f7aa7fb94db Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Thu, 3 Mar 2016 14:46:27 -0800 Subject: [PATCH 60/70] fix some test failures --- pykafka/broker.py | 2 ++ tests/pykafka/test_balancedconsumer.py | 6 +++++- 2 files changed, 7 insertions(+), 1 deletion(-) diff --git a/pykafka/broker.py b/pykafka/broker.py index f6f40ba8e..4afb4f32f 100644 --- a/pykafka/broker.py +++ b/pykafka/broker.py @@ -393,6 +393,7 @@ def join_group(self, connection_id, consumer_group, member_id): """ handler = self._get_unique_req_handler(connection_id) future = handler.request(JoinGroupRequest(consumer_group, member_id)) + self._handler.sleep() return future.get(JoinGroupResponse) def leave_group(self, connection_id, consumer_group, member_id): @@ -451,4 +452,5 @@ def heartbeat(self, connection_id, consumer_group, generation_id, member_id): handler = self._get_unique_req_handler(connection_id) future = handler.request( HeartbeatRequest(consumer_group, generation_id, member_id)) + self._handler.sleep() return future.get(HeartbeatResponse) diff --git a/tests/pykafka/test_balancedconsumer.py b/tests/pykafka/test_balancedconsumer.py index 028784c11..1ca7f0a30 100644 --- a/tests/pykafka/test_balancedconsumer.py +++ b/tests/pykafka/test_balancedconsumer.py @@ -336,13 +336,17 @@ def test_external_kazoo_client(self): def test_no_partitions(self): """Ensure a consumer assigned no partitions doesn't fail""" + + def _decide_dummy(p, consumer_id=None): + return set() consumer = self.get_balanced_consumer( b'test_no_partitions', zookeeper_connect=self.kafka.zookeeper, auto_start=False, consumer_timeout_ms=50, use_rdkafka=self.USE_RDKAFKA) - consumer._decide_partitions = lambda p: set() + + consumer._decide_partitions = _decide_dummy consumer.start() res = consumer.consume() self.assertEqual(res, None) From c5f602c487ae702fe6c3ff6b9bd5e99ea167ac22 Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Thu, 3 Mar 2016 15:05:22 -0800 Subject: [PATCH 61/70] move patch_subclass to __init__ --- tests/pykafka/__init__.py | 48 +++++++++++++++++++ tests/pykafka/rdkafka/test_simple_consumer.py | 10 ++-- tests/pykafka/test_balancedconsumer.py | 48 +------------------ 3 files changed, 54 insertions(+), 52 deletions(-) diff --git a/tests/pykafka/__init__.py b/tests/pykafka/__init__.py index e69de29bb..349916e5c 100644 --- a/tests/pykafka/__init__.py +++ b/tests/pykafka/__init__.py @@ -0,0 +1,48 @@ +import pytest + + +def patch_subclass(parent, skip_condition): + """Work around a pytest.mark.skipif bug + + https://github.com/pytest-dev/pytest/issues/568 + + The issue causes all subclasses of a TestCase subclass to be skipped if any one + of them is skipped. + + This fix circumvents the issue by overriding Python's existing subclassing mechanism. + Instead of having `cls` be a subclass of `parent`, this decorator adds each attribute + of `parent` to `cls` without using Python inheritance. When appropriate, it also adds + a boolean condition under which to skip tests for the decorated class. + + :param parent: The "superclass" from which the decorated class should inherit + its non-overridden attributes + :type parent: unittest2.TestCase + :param skip_condition: A boolean condition that, when True, will cause all tests in + the decorated class to be skipped + :type skip_condition: bool + """ + def patcher(cls): + def build_skipped_method(method, cls, cond=None): + if cond is None: + cond = False + if hasattr(method, "skip_condition"): + cond = cond or method.skip_condition(cls) + + @pytest.mark.skipif(cond, reason="") + def _wrapper(self): + return method(self) + return _wrapper + + # two passes required so that skips have access to all class attributes + for attr in parent.__dict__: + if attr in cls.__dict__: + continue + if not attr.startswith("test_"): + setattr(cls, attr, parent.__dict__[attr]) + + for attr in parent.__dict__: + if attr.startswith("test_"): + setattr(cls, attr, build_skipped_method(parent.__dict__[attr], + cls, skip_condition)) + return cls + return patcher diff --git a/tests/pykafka/rdkafka/test_simple_consumer.py b/tests/pykafka/rdkafka/test_simple_consumer.py index fb6b9a841..6f3fccfc6 100644 --- a/tests/pykafka/rdkafka/test_simple_consumer.py +++ b/tests/pykafka/rdkafka/test_simple_consumer.py @@ -1,8 +1,9 @@ import platform +import unittest2 import pytest -from tests.pykafka import test_simpleconsumer, test_balancedconsumer +from tests.pykafka import test_simpleconsumer, test_balancedconsumer, patch_subclass from pykafka.utils.compat import range @@ -68,8 +69,7 @@ def _latest_partition_offsets_by_reading(consumer, n_reads): return latest_offs -@pytest.mark.skipif(platform.python_implementation() == "PyPy", - reason="Unresolved crashes") -class RdkBalancedConsumerIntegrationTests( - test_balancedconsumer.BalancedConsumerIntegrationTests): +@patch_subclass(test_balancedconsumer.BalancedConsumerIntegrationTests, + platform.python_implementation() == "PyPy") +class RdkBalancedConsumerIntegrationTests(unittest2.TestCase): USE_RDKAFKA = True diff --git a/tests/pykafka/test_balancedconsumer.py b/tests/pykafka/test_balancedconsumer.py index 1ca7f0a30..0ab92b94e 100644 --- a/tests/pykafka/test_balancedconsumer.py +++ b/tests/pykafka/test_balancedconsumer.py @@ -17,6 +17,7 @@ from pykafka.managedbalancedconsumer import ManagedBalancedConsumer from pykafka.test.utils import get_cluster, stop_cluster from pykafka.utils.compat import range, iterkeys, iteritems +from tests.pykafka import patch_subclass kafka_version = pkg_resources.parse_version(os.environ.get('KAFKA_VERSION', '0.8')) @@ -418,53 +419,6 @@ def wait_for_rebalancing(self, *balanced_consumers): raise AssertionError("Rebalancing failed") -def patch_subclass(parent, skip_condition): - """Work around a pytest.mark.skipif bug - - https://github.com/pytest-dev/pytest/issues/568 - - The issue causes all subclasses of a TestCase subclass to be skipped if any one - of them is skipped. - - This fix circumvents the issue by overriding Python's existing subclassing mechanism. - Instead of having `cls` be a subclass of `parent`, this decorator adds each attribute - of `parent` to `cls` without using Python inheritance. When appropriate, it also adds - a boolean condition under which to skip tests for the decorated class. - - :param parent: The "superclass" from which the decorated class should inherit - its non-overridden attributes - :type parent: unittest2.TestCase - :param skip_condition: A boolean condition that, when True, will cause all tests in - the decorated class to be skipped - :type skip_condition: bool - """ - def patcher(cls): - def build_skipped_method(method, cls, cond=None): - if cond is None: - cond = False - if hasattr(method, "skip_condition"): - cond = cond or method.skip_condition(cls) - - @pytest.mark.skipif(cond, reason="") - def _wrapper(self): - return method(self) - return _wrapper - - # two passes required so that skips have access to all class attributes - for attr in parent.__dict__: - if attr in cls.__dict__: - continue - if not attr.startswith("test_"): - setattr(cls, attr, parent.__dict__[attr]) - - for attr in parent.__dict__: - if attr.startswith("test_"): - setattr(cls, attr, build_skipped_method(parent.__dict__[attr], - cls, skip_condition)) - return cls - return patcher - - @patch_subclass(BalancedConsumerIntegrationTests, platform.python_implementation() == "PyPy") class BalancedConsumerGEventIntegrationTests(unittest2.TestCase): From c5b4a153172b64e8a78ef16f7a469a303cce5ece Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Thu, 3 Mar 2016 15:09:30 -0800 Subject: [PATCH 62/70] support generation_id in rdk simple consumer --- pykafka/rdkafka/simple_consumer.py | 1 + 1 file changed, 1 insertion(+) diff --git a/pykafka/rdkafka/simple_consumer.py b/pykafka/rdkafka/simple_consumer.py index 583a76774..7db15ddce 100644 --- a/pykafka/rdkafka/simple_consumer.py +++ b/pykafka/rdkafka/simple_consumer.py @@ -46,6 +46,7 @@ def __init__(self, auto_start=True, reset_offset_on_start=False, compacted_topic=False, + generation_id=b'' consumer_id=b''): callargs = {k: v for k, v in vars().items() if k not in ("self", "__class__")} From 0bd7c42df02f8d325302010d811cd1aff187c0cb Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Thu, 3 Mar 2016 15:10:47 -0800 Subject: [PATCH 63/70] typo --- pykafka/rdkafka/simple_consumer.py | 2 +- 1 file changed, 1 insertion(+), 1 deletion(-) diff --git a/pykafka/rdkafka/simple_consumer.py b/pykafka/rdkafka/simple_consumer.py index 7db15ddce..51be32b1c 100644 --- a/pykafka/rdkafka/simple_consumer.py +++ b/pykafka/rdkafka/simple_consumer.py @@ -46,7 +46,7 @@ def __init__(self, auto_start=True, reset_offset_on_start=False, compacted_topic=False, - generation_id=b'' + generation_id=b'', consumer_id=b''): callargs = {k: v for k, v in vars().items() if k not in ("self", "__class__")} From 147645f67deddb10bc654eb330a9150de19a749f Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Fri, 4 Mar 2016 11:24:33 -0800 Subject: [PATCH 64/70] small bugfixes --- pykafka/balancedconsumer.py | 11 ++++++----- pykafka/rdkafka/simple_consumer.py | 2 +- 2 files changed, 7 insertions(+), 6 deletions(-) diff --git a/pykafka/balancedconsumer.py b/pykafka/balancedconsumer.py index 876f9aee7..fd69b22d4 100644 --- a/pykafka/balancedconsumer.py +++ b/pykafka/balancedconsumer.py @@ -36,7 +36,7 @@ from .exceptions import KafkaException, PartitionOwnedError, ConsumerStoppedException from .handlers import GEventHandler from .simpleconsumer import SimpleConsumer -from .utils.compat import range, get_bytes, itervalues, iteritems +from .utils.compat import range, get_bytes, itervalues, iteritems, get_string try: from . import rdkafka except ImportError: @@ -449,7 +449,7 @@ def _decide_partitions(self, participants, consumer_id=None): def _get_participants(self): """Use zookeeper to get the other consumers of this topic. - :return: A sorted list of the ids of the other consumers of this + :return: A sorted list of the ids of other consumers of this consumer's topic """ try: @@ -464,9 +464,9 @@ def _get_participants(self): try: topic, stat = self._zookeeper.get("%s/%s" % (self._consumer_id_path, id_)) if topic == self._topic.name: - participants.append(id_) + participants.append(get_bytes(id_)) except NoNodeException: - pass # disappeared between ``get_children`` and ``get`` + pass # node disappeared between ``get_children`` and ``get`` participants = sorted(participants) return participants @@ -541,7 +541,8 @@ def _path_self(self): """Path where this consumer should be registered in zookeeper""" return '{path}/{id_}'.format( path=self._consumer_id_path, - id_=self._consumer_id + # get_string is necessary to avoid writing literal "b'" to zookeeper + id_=get_string(self._consumer_id) ) def _update_member_assignment(self): diff --git a/pykafka/rdkafka/simple_consumer.py b/pykafka/rdkafka/simple_consumer.py index 51be32b1c..8b70c6005 100644 --- a/pykafka/rdkafka/simple_consumer.py +++ b/pykafka/rdkafka/simple_consumer.py @@ -46,7 +46,7 @@ def __init__(self, auto_start=True, reset_offset_on_start=False, compacted_topic=False, - generation_id=b'', + generation_id=-1, consumer_id=b''): callargs = {k: v for k, v in vars().items() if k not in ("self", "__class__")} From 7ecb5fb4bd82bdd7f589c44d56fd9dc793b4063a Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Mon, 7 Mar 2016 14:11:33 -0800 Subject: [PATCH 65/70] switch socket implementation for gevent --- pykafka/broker.py | 6 ++++-- pykafka/connection.py | 7 ++++++- pykafka/handlers.py | 5 ++++- 3 files changed, 14 insertions(+), 4 deletions(-) diff --git a/pykafka/broker.py b/pykafka/broker.py index 4afb4f32f..a4a6c5419 100644 --- a/pykafka/broker.py +++ b/pykafka/broker.py @@ -195,6 +195,7 @@ def connect(self): :class:`pykafka.handlers.RequestHandler` for this broker """ self._connection = BrokerConnection(self.host, self.port, + self._handler, buffer_size=self._buffer_size, source_host=self._source_host, source_port=self._source_port) @@ -210,7 +211,8 @@ def connect_offsets_channel(self): channel """ self._offsets_channel_connection = BrokerConnection( - self.host, self.port, buffer_size=self._buffer_size, + self.host, self.port, self._handler, + buffer_size=self._buffer_size, source_host=self._source_host, source_port=self._source_port) self._offsets_channel_connection.connect(self._offsets_channel_socket_timeout_ms) self._offsets_channel_req_handler = RequestHandler( @@ -236,7 +238,7 @@ def _get_unique_req_handler(self, connection_id): self._req_handlers[connection_id] = self._req_handler elif connection_id not in self._req_handlers: conn = BrokerConnection( - self.host, self.port, buffer_size=self._buffer_size, + self.host, self.port, self._handler, buffer_size=self._buffer_size, source_host=self._source_host, source_port=self._source_port) conn.connect(self._socket_timeout_ms) handler = RequestHandler(self._handler, conn) diff --git a/pykafka/connection.py b/pykafka/connection.py index efbcebb8a..960aa13fa 100644 --- a/pykafka/connection.py +++ b/pykafka/connection.py @@ -38,6 +38,7 @@ class BrokerConnection(object): def __init__(self, host, port, + handler, buffer_size=1024 * 1024, source_host='', source_port=0): @@ -47,6 +48,9 @@ def __init__(self, :type host: str :param port: The port on the host to which to connect :type port: int + :param handler: The :class:`pykafka.handlers.Handler` instance to use when + creating a connection + :type handler: :class:`pykafka.handlers.Handler` :param buffer_size: The size (in bytes) of the buffer in which to hold response data. :type buffer_size: int @@ -60,6 +64,7 @@ def __init__(self, self._buff = bytearray(buffer_size) self.host = host self.port = port + self._handler = handler self._socket = None self.source_host = source_host self.source_port = source_port @@ -76,7 +81,7 @@ def connected(self): def connect(self, timeout): """Connect to the broker.""" log.debug("Connecting to %s:%s", self.host, self.port) - self._socket = socket.create_connection( + self._socket = self._handler.Socket.create_connection( (self.host, self.port), timeout / 1000, (self.source_host, self.source_port) diff --git a/pykafka/handlers.py b/pykafka/handlers.py index 551d5e775..6a0db24ed 100644 --- a/pykafka/handlers.py +++ b/pykafka/handlers.py @@ -23,8 +23,9 @@ import gevent.event import gevent.lock import gevent.queue -import gevent.lock +import gevent.socket as gsocket import logging +import socket as pysocket import threading import time @@ -81,6 +82,7 @@ class ThreadingHandler(Handler): Event = threading.Event Lock = threading.Lock Semaphore = Semaphore + Socket = pysocket _workers_spawned = 0 def sleep(self, seconds=0): @@ -112,6 +114,7 @@ class GEventHandler(Handler): Lock = gevent.lock.RLock # fixme RLock = gevent.lock.RLock Semaphore = gevent.lock.Semaphore + Socket = gsocket def sleep(self, seconds=0): gevent.sleep(seconds) From 61c95369cbc8dc72efa1eb17c847d69c2e87c0d2 Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Mon, 7 Mar 2016 15:00:13 -0800 Subject: [PATCH 66/70] clarify protocol test bytestrings --- tests/pykafka/test_protocol.py | 112 ++++++++++++++++++++++++++++++--- 1 file changed, 102 insertions(+), 10 deletions(-) diff --git a/tests/pykafka/test_protocol.py b/tests/pykafka/test_protocol.py index 1329ed6b9..d4f5a7ec7 100644 --- a/tests/pykafka/test_protocol.py +++ b/tests/pykafka/test_protocol.py @@ -264,7 +264,13 @@ def test_consumer_group_protocol_metadata(self): msg = meta.get_bytes() self.assertEqual( msg, - bytearray(b'\x00\x00\x00\x00\x00\x01\x00\ndummytopic\x00\x00\x00\x0ctestuserdata') + bytearray( + b'\x00\x00\x00\x00' # version + b'\x00\x01' # len(subscription) + b'\x00\n' # len(topic name) + b'dummytopic' # topic name + b'\x00\x00\x00\x0c' # len(userdata) + b'testuserdata') # userdata ) def test_join_group_request(self): @@ -272,12 +278,40 @@ def test_join_group_request(self): msg = req.get_bytes() self.assertEqual( msg, - bytearray(b'\x00\x00\x00|\x00\x0b\x00\x00\x00\x00\x00\x00\x00\x07pykafka\x00\ndummygroup\x00\x00u0\x00\ntestmember\x00\x08consumer\x00\x00\x00\x01\x00\x19pykafkaassignmentstrategy\x00\x00\x00"\x00\x00\x00\x00\x00\x01\x00\ndummytopic\x00\x00\x00\x0ctestuserdata') + bytearray( + b'\x00\x00\x00|\x00\x0b\x00\x00\x00\x00\x00\x00\x00\x07pykafka' # header + b'\x00\n' # len(groupid) + b'dummygroup' # groupid + b'\x00\x00u0' # session timeout + b'\x00\n' # len(memberid) + b'testmember' # memberid + b'\x00\x08' # len(protocol type) + b'consumer' # protocol type + b'\x00\x00\x00\x01' # len(group protocols) + b'\x00\x19' # len(protocol name) + b'pykafkaassignmentstrategy' # protocol name + b'\x00\x00\x00"' # len(protocol metadata) + b'\x00\x00\x00\x00\x00\x01\x00\ndummytopic\x00\x00\x00\x0ctestuserdata' # protocol metadata + ) ) def test_join_group_response(self): response = protocol.JoinGroupResponse( - bytearray(b'\x00\x00\x00\x00\x00\x01\x00\x17dummyassignmentstrategy\x00,pykafka-b2361322-674c-4e26-9194-305962636e57\x00,pykafka-b2361322-674c-4e26-9194-305962636e57\x00\x00\x00\x01\x00,pykafka-b2361322-674c-4e26-9194-305962636e57\x00\x00\x00"\x00\x00\x00\x00\x00\x01\x00\ndummytopic\x00\x00\x00\x0ctestuserdata\x00\x00\x00\x00') + bytearray( + b'\x00\x00' # error code + b'\x00\x00\x00\x01' # generation id + b'\x00\x17' # len (group protocol) + b'dummyassignmentstrategy' # group protocol + b'\x00,' # len(leader id) + b'pykafka-b2361322-674c-4e26-9194-305962636e57' # leader id + b'\x00,' # len(member id) + b'pykafka-b2361322-674c-4e26-9194-305962636e57' # member id + b'\x00\x00\x00\x01' # leb(members) + b'\x00,' # len(member id) + b'pykafka-b2361322-674c-4e26-9194-305962636e57' # member id + b'\x00\x00\x00"' # len(member metadata) + b'\x00\x00\x00\x00\x00\x01\x00\ndummytopic\x00\x00\x00\x0ctestuserdata\x00\x00\x00\x00' # member metadata + ) ) self.assertEqual(response.generation_id, 1) self.assertEqual(response.group_protocol, b'dummyassignmentstrategy') @@ -294,7 +328,24 @@ def test_member_assignment_construction(self): msg = assignment.get_bytes() self.assertEqual( msg, - bytearray(b'\x00\x01\x00\x00\x00\x02\x00\x08mytopic1\x00\x00\x00\x04\x00\x00\x00\x03\x00\x00\x00\x05\x00\x00\x00\x07\x00\x00\x00\t\x00\x08mytopic2\x00\x00\x00\x04\x00\x00\x00\x02\x00\x00\x00\x04\x00\x00\x00\x06\x00\x00\x00\x08') + bytearray( + b'\x00\x01' # version + b'\x00\x00\x00\x02' # len(partition assignment) + b'\x00\x08' # len(topic) + b'mytopic1' # topic + b'\x00\x00\x00\x04' # len(partitions) + b'\x00\x00\x00\x03' # partition + b'\x00\x00\x00\x05' # partition + b'\x00\x00\x00\x07' # partition + b'\x00\x00\x00\t' # partition + b'\x00\x08' # len(topic) + b'mytopic2' # topic + b'\x00\x00\x00\x04' # len(partitions) + b'\x00\x00\x00\x02' # partition + b'\x00\x00\x00\x04' # partition + b'\x00\x00\x00\x06' # partition + b'\x00\x00\x00\x08' # partition + ) ) def test_sync_group_request(self): @@ -309,12 +360,32 @@ def test_sync_group_request(self): msg = req.get_bytes() self.assertEqual( msg, - bytearray(b'\x00\x00\x00\xc4\x00\x0e\x00\x00\x00\x00\x00\x00\x00\x07pykafka\x00\ndummygroup\x00\x00\x00\x01\x00\x0btestmember1\x00\x00\x00\x02\x00\x01a\x00\x00\x00B\x00\x01\x00\x00\x00\x02\x00\x08mytopic1\x00\x00\x00\x04\x00\x00\x00\x03\x00\x00\x00\x05\x00\x00\x00\x07\x00\x00\x00\t\x00\x08mytopic2\x00\x00\x00\x04\x00\x00\x00\x03\x00\x00\x00\x05\x00\x00\x00\x07\x00\x00\x00\t\x00\x01b\x00\x00\x00B\x00\x01\x00\x00\x00\x02\x00\x08mytopic1\x00\x00\x00\x04\x00\x00\x00\x02\x00\x00\x00\x04\x00\x00\x00\x06\x00\x00\x00\x08\x00\x08mytopic2\x00\x00\x00\x04\x00\x00\x00\x02\x00\x00\x00\x04\x00\x00\x00\x06\x00\x00\x00\x08') + bytearray( + b'\x00\x00\x00\xc4\x00\x0e\x00\x00\x00\x00\x00\x00\x00\x07pykafka' # header + b'\x00\n' # len(group id) + b'dummygroup' # group id + b'\x00\x00\x00\x01' # generation id + b'\x00\x0b' # len(member id) + b'testmember1' # member id + b'\x00\x00\x00\x02' # len(group assignment) + b'\x00\x01' # len(member id) + b'a' # member id + b'\x00\x00\x00B' # len(member assignment) + b'\x00\x01\x00\x00\x00\x02\x00\x08mytopic1\x00\x00\x00\x04\x00\x00\x00\x03\x00\x00\x00\x05\x00\x00\x00\x07\x00\x00\x00\t\x00\x08mytopic2\x00\x00\x00\x04\x00\x00\x00\x03\x00\x00\x00\x05\x00\x00\x00\x07\x00\x00\x00\t' # member assignment + b'\x00\x01' # len(member id) + b'b' # member id + b'\x00\x00\x00B' # len(member assignment) + b'\x00\x01\x00\x00\x00\x02\x00\x08mytopic1\x00\x00\x00\x04\x00\x00\x00\x02\x00\x00\x00\x04\x00\x00\x00\x06\x00\x00\x00\x08\x00\x08mytopic2\x00\x00\x00\x04\x00\x00\x00\x02\x00\x00\x00\x04\x00\x00\x00\x06\x00\x00\x00\x08' # member assignment + ) ) def test_sync_group_response(self): response = protocol.SyncGroupResponse( - bytearray(b'\x00\x00\x00\x00\x00H\x00\x01\x00\x00\x00\x01\x00\x14testtopic_replicated\x00\x00\x00\n\x00\x00\x00\x06\x00\x00\x00\x07\x00\x00\x00\x08\x00\x00\x00\t\x00\x00\x00\x00\x00\x00\x00\x01\x00\x00\x00\x02\x00\x00\x00\x03\x00\x00\x00\x04\x00\x00\x00\x05,pyk') + bytearray( + b'\x00\x00' # error code + b'\x00\x00\x00H' # len(member assignment) + b'\x00\x01\x00\x00\x00\x01\x00\x14testtopic_replicated\x00\x00\x00\n\x00\x00\x00\x06\x00\x00\x00\x07\x00\x00\x00\x08\x00\x00\x00\t\x00\x00\x00\x00\x00\x00\x00\x01\x00\x00\x00\x02\x00\x00\x00\x03\x00\x00\x00\x04\x00\x00\x00\x05,pyk' # member assignment + ) ) self.assertEqual(response.error_code, 0) expected_assignment = [(b'testtopic_replicated', [6, 7, 8, 9, 0, 1, 2, 3, 4, 5])] @@ -326,11 +397,22 @@ def test_heartbeat_request(self): msg = req.get_bytes() self.assertEqual( msg, - bytearray(b'\x00\x00\x00-\x00\x0c\x00\x00\x00\x00\x00\x00\x00\x07pykafka\x00\ndummygroup\x00\x00\x00\x01\x00\ntestmember') + bytearray( + b'\x00\x00\x00-\x00\x0c\x00\x00\x00\x00\x00\x00\x00\x07pykafka' # header + b'\x00\n' # len(group id) + b'dummygroup' # group id + b'\x00\x00\x00\x01' # generation id + b'\x00\n' # len(member id) + b'testmember' # member id + ) ) def test_heartbeat_response(self): - response = protocol.HeartbeatResponse(bytearray(b'\x00\x00\x00\x01\x00\x14')) + response = protocol.HeartbeatResponse( + bytearray( + b'\x00\x00' # error code + ) + ) self.assertEqual(response.error_code, 0) def test_leave_group_request(self): @@ -338,11 +420,21 @@ def test_leave_group_request(self): msg = req.get_bytes() self.assertEqual( msg, - bytearray(b'\x00\x00\x00)\x00\r\x00\x00\x00\x00\x00\x00\x00\x07pykafka\x00\ndummygroup\x00\ntestmember') + bytearray( + b'\x00\x00\x00)\x00\r\x00\x00\x00\x00\x00\x00\x00\x07pykafka' # header + b'\x00\n' # len(group id) + b'dummygroup' # group id + b'\x00\n' # len(member id) + b'testmember' # member id + ) ) def test_leave_group_response(self): - response = protocol.LeaveGroupResponse(bytearray(b'\x00\x00\x00\x01\x00\x14')) + response = protocol.LeaveGroupResponse( + bytearray( + b'\x00\x00' # error code + ) + ) self.assertEqual(response.error_code, 0) From 0422a9856d222c0db10b9d705381819867e6978f Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Mon, 7 Mar 2016 16:14:39 -0800 Subject: [PATCH 67/70] update travis download versions --- .travis.yml | 2 +- pykafka/managedbalancedconsumer.py | 5 +++-- pykafka/test/kafka_instance.py | 4 ++-- 3 files changed, 6 insertions(+), 5 deletions(-) diff --git a/.travis.yml b/.travis.yml index b95aae008..9e7101aed 100644 --- a/.travis.yml +++ b/.travis.yml @@ -18,7 +18,7 @@ env: - KAFKA_BIN="$HOME/kafka-bin" matrix: - KAFKA_VERSION=0.8.2.2 - - KAFKA_VERSION=0.9.0.0 + - KAFKA_VERSION=0.9.0.1 addons: apt: diff --git a/pykafka/managedbalancedconsumer.py b/pykafka/managedbalancedconsumer.py index 23ee3ff3b..84d28ea2b 100644 --- a/pykafka/managedbalancedconsumer.py +++ b/pykafka/managedbalancedconsumer.py @@ -245,8 +245,9 @@ def stop(self): self._running = False if self._consumer is not None: self._consumer.stop() - self._group_coordinator.leave_group(self._connection_id, self._consumer_group, - self._consumer_id) + if self._group_coordinator is not None: + self._group_coordinator.leave_group(self._connection_id, self._consumer_group, + self._consumer_id) def _send_heartbeat(self): """Send a heartbeat request to the group coordinator and react to the response""" diff --git a/pykafka/test/kafka_instance.py b/pykafka/test/kafka_instance.py index c25364f6d..52c112843 100644 --- a/pykafka/test/kafka_instance.py +++ b/pykafka/test/kafka_instance.py @@ -132,7 +132,7 @@ class KafkaInstance(ManagedInstance): def __init__(self, num_instances=1, kafka_version='0.8.2.1', - scala_version='2.10', + scala_version='2.11', bin_dir='/tmp/kafka-bin', name='kafka', use_gevent=False): @@ -348,7 +348,7 @@ def produce_messages(self, topic_name, messages): help='Download destination for Kafka') parser.add_argument('--kafka-version', type=str, default='0.8.2.1', help='Kafka version to download') - parser.add_argument('--scala-version', type=str, default='2.10', + parser.add_argument('--scala-version', type=str, default='2.11', help='Scala version for kafka build') args = parser.parse_args() From cf2ff7aa69898ec2fb341522d9ec32a265224e9c Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Tue, 8 Mar 2016 11:44:35 -0800 Subject: [PATCH 68/70] don't cache kafka download --- .travis.yml | 1 - 1 file changed, 1 deletion(-) diff --git a/.travis.yml b/.travis.yml index 9e7101aed..6fff0203b 100644 --- a/.travis.yml +++ b/.travis.yml @@ -3,7 +3,6 @@ sudo: false cache: directories: - $HOME/.cache/pip - - $HOME/kafka-bin - $HOME/.ccache python: - "2.7" From 82780ee670273876deeed5aabcc4e3febcc703e9 Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Tue, 8 Mar 2016 13:41:41 -0800 Subject: [PATCH 69/70] deduplicate error handling --- pykafka/managedbalancedconsumer.py | 71 +++++++++++++++--------------- 1 file changed, 36 insertions(+), 35 deletions(-) diff --git a/pykafka/managedbalancedconsumer.py b/pykafka/managedbalancedconsumer.py index 84d28ea2b..8266849d2 100644 --- a/pykafka/managedbalancedconsumer.py +++ b/pykafka/managedbalancedconsumer.py @@ -25,10 +25,8 @@ from .balancedconsumer import BalancedConsumer from .common import OffsetType -from .exceptions import (IllegalGeneration, RebalanceInProgress, UnknownMemberId, - NotCoordinatorForGroup, GroupCoordinatorNotAvailable, - GroupAuthorizationFailed, ERROR_CODES, GroupLoadInProgress, - InconsistentGroupProtocol, InvalidSessionTimeout) +from .exceptions import (IllegalGeneration, RebalanceInProgress, NotCoordinatorForGroup, + GroupCoordinatorNotAvailable, ERROR_CODES, GroupLoadInProgress) from .protocol import MemberAssignment from .utils.compat import iterkeys @@ -198,6 +196,7 @@ def __init__(self, self._consumer_id = b'' self._worker_trace_logged = False self._worker_exception = None + self._default_error_handlers = self._build_default_error_handlers() if auto_start is True: self.start() @@ -259,16 +258,7 @@ def _send_heartbeat(self): if res.error_code == 0: return log.info("Error code %d encountered on heartbeat." % res.error_code) - if res.error_code in (IllegalGeneration.ERROR_CODE, - RebalanceInProgress.ERROR_CODE, - UnknownMemberId.ERROR_CODE): - pass - elif res.error_code in (GroupCoordinatorNotAvailable.ERROR_CODE, - NotCoordinatorForGroup.ERROR_CODE): - self._group_coordinator = self._cluster.get_group_coordinator( - self._consumer_group) - elif res.error_code == GroupAuthorizationFailed.ERROR_CODE: - raise GroupAuthorizationFailed() + self._default_error_handlers[res.error_code]() log.debug("_rebalance called from _send_heartbeat") self._rebalance() @@ -306,6 +296,36 @@ def _update_member_assignment(self): self._cluster.handler.sleep(i * (self._rebalance_backoff_ms / 1000)) self._raise_worker_exceptions() + def _build_default_error_handlers(self): + self = weakref.proxy(self) + + def _handle_GroupCoordinatorNotAvailable(): + self._group_coordinator = self._cluster.get_group_coordinator( + self._consumer_group) + + def _handle_NotCoordinatorForGroup(): + self._group_coordinator = self._cluster.get_group_coordinator( + self._consumer_group) + + return { + GroupCoordinatorNotAvailable.ERROR_CODE: _handle_GroupCoordinatorNotAvailable, + NotCoordinatorForGroup.ERROR_CODE: _handle_NotCoordinatorForGroup, + GroupLoadInProgress.ERROR_CODE: None, + RebalanceInProgress.ERROR_CODE: None, + IllegalGeneration.ERROR_CODE: None + } + + def _handle_error(self, error_code): + """Call the appropriate handler function for the given error code + + :param error_code: The error code returned from a Group Membership API request + :type error_code: int + """ + if error_code not in self._default_error_handlers: + raise ERROR_CODES[error_code]() + if self._default_error_handlers[error_code] is not None: + self._default_error_handlers[error_code]() + def _join_group(self): """Send a JoinGroupRequest. @@ -322,17 +342,7 @@ def _join_group(self): " generation '%s'", join_result.error_code, self._generation_id) if i == self._cluster._max_connection_retries - 1: raise ERROR_CODES[join_result.error_code] - if join_result.error_code in (GroupLoadInProgress.ERROR_CODE,): - pass - elif join_result.error_code in (GroupCoordinatorNotAvailable.ERROR_CODE, - NotCoordinatorForGroup.ERROR_CODE): - self._group_coordinator = self._cluster.get_group_coordinator( - self._consumer_group) - elif join_result.error_code in (InconsistentGroupProtocol.ERROR_CODE, - UnknownMemberId.ERROR_CODE, - InvalidSessionTimeout.ERROR_CODE, - GroupAuthorizationFailed.ERROR_CODE): - raise ERROR_CODES[join_result.error_code] + self._handle_error(join_result.error_code) self._cluster.handler.sleep(i * 2) self._generation_id = join_result.generation_id self._consumer_id = join_result.member_id @@ -361,15 +371,6 @@ def _sync_group(self, group_assignments): sync_result.error_code) if i == self._cluster._max_connection_retries - 1: raise ERROR_CODES[sync_result.error_code] - if sync_result.error_code in (IllegalGeneration.ERROR_CODE, - GroupCoordinatorNotAvailable.ERROR_CODE, - RebalanceInProgress.ERROR_CODE): - pass - elif sync_result.error_code in (NotCoordinatorForGroup.ERROR_CODE,): - self._group_coordinator = self._cluster.get_group_coordinator( - self._consumer_group) - elif sync_result.error_code in (UnknownMemberId.ERROR_CODE, - GroupAuthorizationFailed.ERROR_CODE): - raise ERROR_CODES[sync_result.error_code] + self._default_error_handlers[sync_result.error_code]() self._cluster.handler.sleep(i * 2) return sync_result.member_assignment.partition_assignment From fa187060808999caea1739b96861dba6bcd141f8 Mon Sep 17 00:00:00 2001 From: Emmett Butler Date: Tue, 8 Mar 2016 13:49:06 -0800 Subject: [PATCH 70/70] clean up managedbalancedconsumer --- pykafka/managedbalancedconsumer.py | 39 +++++++++++++++--------------- 1 file changed, 20 insertions(+), 19 deletions(-) diff --git a/pykafka/managedbalancedconsumer.py b/pykafka/managedbalancedconsumer.py index 8266849d2..e75c250b8 100644 --- a/pykafka/managedbalancedconsumer.py +++ b/pykafka/managedbalancedconsumer.py @@ -188,7 +188,7 @@ def __init__(self, self._generation_id = -1 self._rebalancing_lock = cluster.handler.Lock() # ManagedBalancedConsumers in the same process cannot share connections. - # This connection hash is passed to self.Broker calls that use the group + # This connection hash is passed to Broker calls that use the group # membership API self._connection_id = uuid.uuid4() self._consumer = None @@ -210,7 +210,18 @@ def fetcher(): try: if not self._running: break - self._send_heartbeat() + + log.info("Sending heartbeat from consumer '%s'", self._consumer_id) + res = self._group_coordinator.heartbeat(self._connection_id, + self._consumer_group, + self._generation_id, + self._consumer_id) + if res.error_code != 0: + log.info("Error code %d encountered on heartbeat.", + res.error_code) + self._handle_error(res.error_code) + self._rebalance() + self._cluster.handler.sleep(self._heartbeat_interval_ms / 1000) except ReferenceError: break @@ -224,12 +235,14 @@ def fetcher(): fetcher, name="pykafka.ManagedBalancedConsumer.heartbeats") def start(self): - """Start this consumer. Must be called before consume() if `auto_start=False`.""" + """Start this consumer. + + Must be called before consume() if `auto_start=False`. + """ try: self._running = True self._group_coordinator = self._cluster.get_group_coordinator( self._consumer_group) - log.debug("_rebalance called from _start") self._rebalance() self._setup_heartbeat_worker() except Exception: @@ -245,22 +258,9 @@ def stop(self): if self._consumer is not None: self._consumer.stop() if self._group_coordinator is not None: - self._group_coordinator.leave_group(self._connection_id, self._consumer_group, - self._consumer_id) - - def _send_heartbeat(self): - """Send a heartbeat request to the group coordinator and react to the response""" - log.info("Sending heartbeat from consumer '%s'", self._consumer_id) - res = self._group_coordinator.heartbeat(self._connection_id, + self._group_coordinator.leave_group(self._connection_id, self._consumer_group, - self._generation_id, self._consumer_id) - if res.error_code == 0: - return - log.info("Error code %d encountered on heartbeat." % res.error_code) - self._default_error_handlers[res.error_code]() - log.debug("_rebalance called from _send_heartbeat") - self._rebalance() def _update_member_assignment(self): """Join a managed consumer group and start consuming assigned partitions @@ -297,6 +297,7 @@ def _update_member_assignment(self): self._raise_worker_exceptions() def _build_default_error_handlers(self): + """Set up default responses to common error codes""" self = weakref.proxy(self) def _handle_GroupCoordinatorNotAvailable(): @@ -371,6 +372,6 @@ def _sync_group(self, group_assignments): sync_result.error_code) if i == self._cluster._max_connection_retries - 1: raise ERROR_CODES[sync_result.error_code] - self._default_error_handlers[sync_result.error_code]() + self._handle_error(sync_result.error_code) self._cluster.handler.sleep(i * 2) return sync_result.member_assignment.partition_assignment