Skip to content

Commit e937e3f

Browse files
committed
Merge pull request #109 from mrtheb/develop
TopicAndPartition fix when partition has no leader = -1
2 parents 9599215 + 51246fb commit e937e3f

File tree

5 files changed

+207
-31
lines changed

5 files changed

+207
-31
lines changed

kafka/client.py

+26-7
Original file line numberDiff line numberDiff line change
@@ -8,7 +8,8 @@
88
from kafka.common import (ErrorMapping, TopicAndPartition,
99
ConnectionError, FailedPayloadsError,
1010
BrokerResponseError, PartitionUnavailableError,
11-
KafkaUnavailableError, KafkaRequestError)
11+
LeaderUnavailableError,
12+
KafkaUnavailableError)
1213

1314
from kafka.conn import collect_hosts, KafkaConnection, DEFAULT_SOCKET_TIMEOUT_SECONDS
1415
from kafka.protocol import KafkaProtocol
@@ -62,12 +63,22 @@ def _get_conn_for_broker(self, broker):
6263
return self._get_conn(broker.host, broker.port)
6364

6465
def _get_leader_for_partition(self, topic, partition):
66+
"""
67+
Returns the leader for a partition or None if the partition exists
68+
but has no leader.
69+
70+
PartitionUnavailableError will be raised if the topic or partition
71+
is not part of the metadata.
72+
"""
73+
6574
key = TopicAndPartition(topic, partition)
66-
if key not in self.topics_to_brokers:
75+
# reload metadata whether the partition is not available
76+
# or has no leader (broker is None)
77+
if self.topics_to_brokers.get(key) is None:
6778
self.load_metadata_for_topics(topic)
6879

6980
if key not in self.topics_to_brokers:
70-
raise KafkaRequestError("Partition does not exist: %s" % str(key))
81+
raise PartitionUnavailableError("%s not available" % str(key))
7182

7283
return self.topics_to_brokers[key]
7384

@@ -124,8 +135,11 @@ def _send_broker_aware_request(self, payloads, encoder_fn, decoder_fn):
124135
for payload in payloads:
125136
leader = self._get_leader_for_partition(payload.topic,
126137
payload.partition)
127-
if leader == -1:
128-
raise PartitionUnavailableError("Leader is unassigned for %s-%s" % payload.topic, payload.partition)
138+
if leader is None:
139+
raise LeaderUnavailableError(
140+
"Leader not available for topic %s partition %s" %
141+
(payload.topic, payload.partition))
142+
129143
payloads_by_broker[leader].append(payload)
130144
original_keys.append((payload.topic, payload.partition))
131145

@@ -250,13 +264,18 @@ def load_metadata_for_topics(self, *topics):
250264
self.reset_topic_metadata(topic)
251265

252266
if not partitions:
267+
log.warning('No partitions for %s', topic)
253268
continue
254269

255270
self.topic_partitions[topic] = []
256271
for partition, meta in partitions.items():
257-
topic_part = TopicAndPartition(topic, partition)
258-
self.topics_to_brokers[topic_part] = brokers[meta.leader]
259272
self.topic_partitions[topic].append(partition)
273+
topic_part = TopicAndPartition(topic, partition)
274+
if meta.leader == -1:
275+
log.warning('No leader for topic %s partition %s', topic, partition)
276+
self.topics_to_brokers[topic_part] = None
277+
else:
278+
self.topics_to_brokers[topic_part] = brokers[meta.leader]
260279

261280
def send_produce_request(self, payloads=[], acks=1, timeout=1000,
262281
fail_on_error=True, callback=None):

kafka/common.py

+3-3
Original file line numberDiff line numberDiff line change
@@ -74,15 +74,15 @@ class KafkaError(RuntimeError):
7474
pass
7575

7676

77-
class KafkaRequestError(KafkaError):
77+
class KafkaUnavailableError(KafkaError):
7878
pass
7979

8080

81-
class KafkaUnavailableError(KafkaError):
81+
class BrokerResponseError(KafkaError):
8282
pass
8383

8484

85-
class BrokerResponseError(KafkaError):
85+
class LeaderUnavailableError(KafkaError):
8686
pass
8787

8888

setup.py

+1-1
Original file line numberDiff line numberDiff line change
@@ -23,7 +23,7 @@ def run(self):
2323
version="0.9.0",
2424

2525
install_requires=["distribute"],
26-
tests_require=["tox"],
26+
tests_require=["tox", "mock"],
2727
cmdclass={"test": Tox},
2828

2929
packages=["kafka"],

test/test_unit.py

+174-19
Original file line numberDiff line numberDiff line change
@@ -5,14 +5,14 @@
55

66
from mock import MagicMock, patch
77

8-
98
from kafka import KafkaClient
109
from kafka.common import (
1110
ProduceRequest, FetchRequest, Message, ChecksumError,
1211
ConsumerFetchSizeTooSmall, ProduceResponse, FetchResponse,
13-
OffsetAndMessage, BrokerMetadata, PartitionMetadata
12+
OffsetAndMessage, BrokerMetadata, PartitionMetadata,
13+
TopicAndPartition, KafkaUnavailableError,
14+
LeaderUnavailableError, PartitionUnavailableError
1415
)
15-
from kafka.common import KafkaUnavailableError
1616
from kafka.codec import (
1717
has_gzip, has_snappy, gzip_encode, gzip_decode,
1818
snappy_encode, snappy_decode
@@ -410,6 +410,7 @@ def test_encode_offset_request(self):
410410
def test_decode_offset_response(self):
411411
pass
412412

413+
413414
@unittest.skip("Not Implemented")
414415
def test_encode_offset_commit_request(self):
415416
pass
@@ -474,18 +475,17 @@ def mock_get_conn(host, port):
474475
return mocked_conns[(host, port)]
475476

476477
# patch to avoid making requests before we want it
477-
with patch.object(KafkaClient, 'load_metadata_for_topics'), \
478-
patch.object(KafkaClient, '_get_conn', side_effect=mock_get_conn):
479-
480-
client = KafkaClient(hosts=['kafka01:9092', 'kafka02:9092'])
478+
with patch.object(KafkaClient, 'load_metadata_for_topics'):
479+
with patch.object(KafkaClient, '_get_conn', side_effect=mock_get_conn):
480+
client = KafkaClient(hosts=['kafka01:9092', 'kafka02:9092'])
481481

482-
self.assertRaises(
483-
KafkaUnavailableError,
484-
client._send_broker_unaware_request,
485-
1, 'fake request')
482+
self.assertRaises(
483+
KafkaUnavailableError,
484+
client._send_broker_unaware_request,
485+
1, 'fake request')
486486

487-
for key, conn in mocked_conns.iteritems():
488-
conn.send.assert_called_with(1, 'fake request')
487+
for key, conn in mocked_conns.iteritems():
488+
conn.send.assert_called_with(1, 'fake request')
489489

490490
def test_send_broker_unaware_request(self):
491491
'Tests that call works when at least one of the host is available'
@@ -504,16 +504,171 @@ def mock_get_conn(host, port):
504504
return mocked_conns[(host, port)]
505505

506506
# patch to avoid making requests before we want it
507-
with patch.object(KafkaClient, 'load_metadata_for_topics'), \
508-
patch.object(KafkaClient, '_get_conn', side_effect=mock_get_conn):
507+
with patch.object(KafkaClient, 'load_metadata_for_topics'):
508+
with patch.object(KafkaClient, '_get_conn', side_effect=mock_get_conn):
509+
client = KafkaClient(hosts='kafka01:9092,kafka02:9092')
510+
511+
resp = client._send_broker_unaware_request(1, 'fake request')
512+
513+
self.assertEqual('valid response', resp)
514+
mocked_conns[('kafka02', 9092)].recv.assert_called_with(1)
515+
516+
@patch('kafka.client.KafkaConnection')
517+
@patch('kafka.client.KafkaProtocol')
518+
def test_load_metadata(self, protocol, conn):
519+
"Load metadata for all topics"
520+
521+
conn.recv.return_value = 'response' # anything but None
522+
523+
brokers = {}
524+
brokers[0] = BrokerMetadata(1, 'broker_1', 4567)
525+
brokers[1] = BrokerMetadata(2, 'broker_2', 5678)
526+
527+
topics = {}
528+
topics['topic_1'] = {
529+
0: PartitionMetadata('topic_1', 0, 1, [1, 2], [1, 2])
530+
}
531+
topics['topic_noleader'] = {
532+
0: PartitionMetadata('topic_noleader', 0, -1, [], []),
533+
1: PartitionMetadata('topic_noleader', 1, -1, [], [])
534+
}
535+
topics['topic_no_partitions'] = {}
536+
topics['topic_3'] = {
537+
0: PartitionMetadata('topic_3', 0, 0, [0, 1], [0, 1]),
538+
1: PartitionMetadata('topic_3', 1, 1, [1, 0], [1, 0]),
539+
2: PartitionMetadata('topic_3', 2, 0, [0, 1], [0, 1])
540+
}
541+
protocol.decode_metadata_response.return_value = (brokers, topics)
542+
543+
# client loads metadata at init
544+
client = KafkaClient(hosts=['broker_1:4567'])
545+
self.assertDictEqual({
546+
TopicAndPartition('topic_1', 0): brokers[1],
547+
TopicAndPartition('topic_noleader', 0): None,
548+
TopicAndPartition('topic_noleader', 1): None,
549+
TopicAndPartition('topic_3', 0): brokers[0],
550+
TopicAndPartition('topic_3', 1): brokers[1],
551+
TopicAndPartition('topic_3', 2): brokers[0]},
552+
client.topics_to_brokers)
553+
554+
@patch('kafka.client.KafkaConnection')
555+
@patch('kafka.client.KafkaProtocol')
556+
def test_get_leader_for_partitions_reloads_metadata(self, protocol, conn):
557+
"Get leader for partitions reload metadata if it is not available"
558+
559+
conn.recv.return_value = 'response' # anything but None
560+
561+
brokers = {}
562+
brokers[0] = BrokerMetadata(0, 'broker_1', 4567)
563+
brokers[1] = BrokerMetadata(1, 'broker_2', 5678)
564+
565+
topics = {'topic_no_partitions': {}}
566+
protocol.decode_metadata_response.return_value = (brokers, topics)
567+
568+
client = KafkaClient(hosts=['broker_1:4567'])
569+
570+
# topic metadata is loaded but empty
571+
self.assertDictEqual({}, client.topics_to_brokers)
572+
573+
topics['topic_no_partitions'] = {
574+
0: PartitionMetadata('topic_no_partitions', 0, 0, [0, 1], [0, 1])
575+
}
576+
protocol.decode_metadata_response.return_value = (brokers, topics)
577+
578+
# calling _get_leader_for_partition (from any broker aware request)
579+
# will try loading metadata again for the same topic
580+
leader = client._get_leader_for_partition('topic_no_partitions', 0)
581+
582+
self.assertEqual(brokers[0], leader)
583+
self.assertDictEqual({
584+
TopicAndPartition('topic_no_partitions', 0): brokers[0]},
585+
client.topics_to_brokers)
586+
587+
@patch('kafka.client.KafkaConnection')
588+
@patch('kafka.client.KafkaProtocol')
589+
def test_get_leader_for_unassigned_partitions(self, protocol, conn):
590+
"Get leader raises if no partitions is defined for a topic"
591+
592+
conn.recv.return_value = 'response' # anything but None
593+
594+
brokers = {}
595+
brokers[0] = BrokerMetadata(0, 'broker_1', 4567)
596+
brokers[1] = BrokerMetadata(1, 'broker_2', 5678)
597+
598+
topics = {'topic_no_partitions': {}}
599+
protocol.decode_metadata_response.return_value = (brokers, topics)
600+
601+
client = KafkaClient(hosts=['broker_1:4567'])
602+
603+
self.assertDictEqual({}, client.topics_to_brokers)
604+
self.assertRaises(
605+
PartitionUnavailableError,
606+
client._get_leader_for_partition,
607+
'topic_no_partitions', 0)
608+
609+
@patch('kafka.client.KafkaConnection')
610+
@patch('kafka.client.KafkaProtocol')
611+
def test_get_leader_returns_none_when_noleader(self, protocol, conn):
612+
"Getting leader for partitions returns None when the partiion has no leader"
613+
614+
conn.recv.return_value = 'response' # anything but None
509615

510-
client = KafkaClient(hosts='kafka01:9092,kafka02:9092')
616+
brokers = {}
617+
brokers[0] = BrokerMetadata(0, 'broker_1', 4567)
618+
brokers[1] = BrokerMetadata(1, 'broker_2', 5678)
619+
620+
topics = {}
621+
topics['topic_noleader'] = {
622+
0: PartitionMetadata('topic_noleader', 0, -1, [], []),
623+
1: PartitionMetadata('topic_noleader', 1, -1, [], [])
624+
}
625+
protocol.decode_metadata_response.return_value = (brokers, topics)
626+
627+
client = KafkaClient(hosts=['broker_1:4567'])
628+
self.assertDictEqual(
629+
{
630+
TopicAndPartition('topic_noleader', 0): None,
631+
TopicAndPartition('topic_noleader', 1): None
632+
},
633+
client.topics_to_brokers)
634+
self.assertIsNone(client._get_leader_for_partition('topic_noleader', 0))
635+
self.assertIsNone(client._get_leader_for_partition('topic_noleader', 1))
636+
637+
topics['topic_noleader'] = {
638+
0: PartitionMetadata('topic_noleader', 0, 0, [0, 1], [0, 1]),
639+
1: PartitionMetadata('topic_noleader', 1, 1, [1, 0], [1, 0])
640+
}
641+
protocol.decode_metadata_response.return_value = (brokers, topics)
642+
self.assertEqual(brokers[0], client._get_leader_for_partition('topic_noleader', 0))
643+
self.assertEqual(brokers[1], client._get_leader_for_partition('topic_noleader', 1))
644+
645+
@patch('kafka.client.KafkaConnection')
646+
@patch('kafka.client.KafkaProtocol')
647+
def test_send_produce_request_raises_when_noleader(self, protocol, conn):
648+
"Send producer request raises LeaderUnavailableError if leader is not available"
649+
650+
conn.recv.return_value = 'response' # anything but None
651+
652+
brokers = {}
653+
brokers[0] = BrokerMetadata(0, 'broker_1', 4567)
654+
brokers[1] = BrokerMetadata(1, 'broker_2', 5678)
655+
656+
topics = {}
657+
topics['topic_noleader'] = {
658+
0: PartitionMetadata('topic_noleader', 0, -1, [], []),
659+
1: PartitionMetadata('topic_noleader', 1, -1, [], [])
660+
}
661+
protocol.decode_metadata_response.return_value = (brokers, topics)
511662

512-
resp = client._send_broker_unaware_request(1, 'fake request')
663+
client = KafkaClient(hosts=['broker_1:4567'])
513664

514-
self.assertEqual('valid response', resp)
515-
mocked_conns[('kafka02', 9092)].recv.assert_called_with(1)
665+
requests = [ProduceRequest(
666+
"topic_noleader", 0,
667+
[create_message("a"), create_message("b")])]
516668

669+
self.assertRaises(
670+
LeaderUnavailableError,
671+
client.send_produce_request, requests)
517672

518673
if __name__ == '__main__':
519674
unittest.main()

tox.ini

+3-1
Original file line numberDiff line numberDiff line change
@@ -1,7 +1,9 @@
11
[tox]
22
envlist = py26, py27
33
[testenv]
4-
deps = pytest
4+
deps =
5+
pytest
6+
mock
57
commands = py.test --basetemp={envtmpdir} []
68
setenv =
79
PROJECT_ROOT = {toxinidir}

0 commit comments

Comments
 (0)