Skip to content

Commit 45c05d0

Browse files
committed
Merge pull request #361 from kecaps/master
Make external API consistently support python3 strings for topic.
2 parents 9fd0811 + 1c856e8 commit 45c05d0

11 files changed

+69
-49
lines changed

kafka/client.py

Lines changed: 3 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -258,12 +258,14 @@ def reset_all_metadata(self):
258258
self.topic_partitions.clear()
259259

260260
def has_metadata_for_topic(self, topic):
261+
topic = kafka_bytestring(topic)
261262
return (
262263
topic in self.topic_partitions
263264
and len(self.topic_partitions[topic]) > 0
264265
)
265266

266267
def get_partition_ids_for_topic(self, topic):
268+
topic = kafka_bytestring(topic)
267269
if topic not in self.topic_partitions:
268270
return []
269271

@@ -312,6 +314,7 @@ def load_metadata_for_topics(self, *topics):
312314
Partition-level errors will also not be raised here
313315
(a single partition w/o a leader, for example)
314316
"""
317+
topics = [kafka_bytestring(t) for t in topics]
315318
resp = self.send_metadata_request(topics)
316319

317320
log.debug("Broker metadata: %s", resp.brokers)

kafka/consumer/base.py

Lines changed: 3 additions & 3 deletions
Original file line numberDiff line numberDiff line change
@@ -10,7 +10,7 @@
1010
UnknownTopicOrPartitionError, check_error
1111
)
1212

13-
from kafka.util import ReentrantTimer
13+
from kafka.util import kafka_bytestring, ReentrantTimer
1414

1515
log = logging.getLogger("kafka")
1616

@@ -44,8 +44,8 @@ def __init__(self, client, group, topic, partitions=None, auto_commit=True,
4444
auto_commit_every_t=AUTO_COMMIT_INTERVAL):
4545

4646
self.client = client
47-
self.topic = topic
48-
self.group = group
47+
self.topic = kafka_bytestring(topic)
48+
self.group = None if group is None else kafka_bytestring(group)
4949
self.client.load_metadata_for_topics(topic)
5050
self.offsets = {}
5151

kafka/consumer/multiprocess.py

Lines changed: 1 addition & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -163,7 +163,7 @@ def __init__(self, client, group, topic, auto_commit=True,
163163
simple_consumer_options.pop('partitions', None)
164164
options.update(simple_consumer_options)
165165

166-
args = (client.copy(), group, topic, self.queue,
166+
args = (client.copy(), self.group, self.topic, self.queue,
167167
self.size, self.events)
168168
proc = Process(target=_mp_consume, args=args, kwargs=options)
169169
proc.daemon = True

kafka/producer/base.py

Lines changed: 6 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -17,6 +17,7 @@
1717
ProduceRequest, TopicAndPartition, UnsupportedCodecError
1818
)
1919
from kafka.protocol import CODEC_NONE, ALL_CODECS, create_message_set
20+
from kafka.util import kafka_bytestring
2021

2122
log = logging.getLogger("kafka")
2223

@@ -170,6 +171,7 @@ def send_messages(self, topic, partition, *msg):
170171
171172
All messages produced via this method will set the message 'key' to Null
172173
"""
174+
topic = kafka_bytestring(topic)
173175
return self._send_messages(topic, partition, *msg)
174176

175177
def _send_messages(self, topic, partition, *msg, **kwargs):
@@ -183,6 +185,10 @@ def _send_messages(self, topic, partition, *msg, **kwargs):
183185
if any(not isinstance(m, six.binary_type) for m in msg):
184186
raise TypeError("all produce message payloads must be type bytes")
185187

188+
# Raise TypeError if topic is not encoded as bytes
189+
if not isinstance(topic, six.binary_type):
190+
raise TypeError("the topic must be type bytes")
191+
186192
# Raise TypeError if the key is not encoded as bytes
187193
if key is not None and not isinstance(key, six.binary_type):
188194
raise TypeError("the key must be type bytes")

kafka/producer/keyed.py

Lines changed: 4 additions & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -3,6 +3,8 @@
33
import logging
44

55
from kafka.partitioner import HashedPartitioner
6+
from kafka.util import kafka_bytestring
7+
68
from .base import (
79
Producer, BATCH_SEND_DEFAULT_INTERVAL,
810
BATCH_SEND_MSG_COUNT
@@ -57,10 +59,12 @@ def _next_partition(self, topic, key):
5759
return partitioner.partition(key)
5860

5961
def send_messages(self,topic,key,*msg):
62+
topic = kafka_bytestring(topic)
6063
partition = self._next_partition(topic, key)
6164
return self._send_messages(topic, partition, *msg,key=key)
6265

6366
def send(self, topic, key, msg):
67+
topic = kafka_bytestring(topic)
6468
partition = self._next_partition(topic, key)
6569
return self._send_messages(topic, partition, msg, key=key)
6670

test/test_client.py

Lines changed: 36 additions & 34 deletions
Original file line numberDiff line numberDiff line change
@@ -117,34 +117,34 @@ def test_load_metadata(self, protocol, conn):
117117
]
118118

119119
topics = [
120-
TopicMetadata('topic_1', NO_ERROR, [
121-
PartitionMetadata('topic_1', 0, 1, [1, 2], [1, 2], NO_ERROR)
120+
TopicMetadata(b'topic_1', NO_ERROR, [
121+
PartitionMetadata(b'topic_1', 0, 1, [1, 2], [1, 2], NO_ERROR)
122122
]),
123-
TopicMetadata('topic_noleader', NO_ERROR, [
124-
PartitionMetadata('topic_noleader', 0, -1, [], [],
123+
TopicMetadata(b'topic_noleader', NO_ERROR, [
124+
PartitionMetadata(b'topic_noleader', 0, -1, [], [],
125125
NO_LEADER),
126-
PartitionMetadata('topic_noleader', 1, -1, [], [],
126+
PartitionMetadata(b'topic_noleader', 1, -1, [], [],
127127
NO_LEADER),
128128
]),
129-
TopicMetadata('topic_no_partitions', NO_LEADER, []),
130-
TopicMetadata('topic_unknown', UNKNOWN_TOPIC_OR_PARTITION, []),
131-
TopicMetadata('topic_3', NO_ERROR, [
132-
PartitionMetadata('topic_3', 0, 0, [0, 1], [0, 1], NO_ERROR),
133-
PartitionMetadata('topic_3', 1, 1, [1, 0], [1, 0], NO_ERROR),
134-
PartitionMetadata('topic_3', 2, 0, [0, 1], [0, 1], NO_ERROR)
129+
TopicMetadata(b'topic_no_partitions', NO_LEADER, []),
130+
TopicMetadata(b'topic_unknown', UNKNOWN_TOPIC_OR_PARTITION, []),
131+
TopicMetadata(b'topic_3', NO_ERROR, [
132+
PartitionMetadata(b'topic_3', 0, 0, [0, 1], [0, 1], NO_ERROR),
133+
PartitionMetadata(b'topic_3', 1, 1, [1, 0], [1, 0], NO_ERROR),
134+
PartitionMetadata(b'topic_3', 2, 0, [0, 1], [0, 1], NO_ERROR)
135135
])
136136
]
137137
protocol.decode_metadata_response.return_value = MetadataResponse(brokers, topics)
138138

139139
# client loads metadata at init
140140
client = KafkaClient(hosts=['broker_1:4567'])
141141
self.assertDictEqual({
142-
TopicAndPartition('topic_1', 0): brokers[1],
143-
TopicAndPartition('topic_noleader', 0): None,
144-
TopicAndPartition('topic_noleader', 1): None,
145-
TopicAndPartition('topic_3', 0): brokers[0],
146-
TopicAndPartition('topic_3', 1): brokers[1],
147-
TopicAndPartition('topic_3', 2): brokers[0]},
142+
TopicAndPartition(b'topic_1', 0): brokers[1],
143+
TopicAndPartition(b'topic_noleader', 0): None,
144+
TopicAndPartition(b'topic_noleader', 1): None,
145+
TopicAndPartition(b'topic_3', 0): brokers[0],
146+
TopicAndPartition(b'topic_3', 1): brokers[1],
147+
TopicAndPartition(b'topic_3', 2): brokers[0]},
148148
client.topics_to_brokers)
149149

150150
# if we ask for metadata explicitly, it should raise errors
@@ -156,6 +156,7 @@ def test_load_metadata(self, protocol, conn):
156156

157157
# This should not raise
158158
client.load_metadata_for_topics('topic_no_leader')
159+
client.load_metadata_for_topics(b'topic_no_leader')
159160

160161
@patch('kafka.client.KafkaConnection')
161162
@patch('kafka.client.KafkaProtocol')
@@ -169,11 +170,11 @@ def test_has_metadata_for_topic(self, protocol, conn):
169170
]
170171

171172
topics = [
172-
TopicMetadata('topic_still_creating', NO_LEADER, []),
173-
TopicMetadata('topic_doesnt_exist', UNKNOWN_TOPIC_OR_PARTITION, []),
174-
TopicMetadata('topic_noleaders', NO_ERROR, [
175-
PartitionMetadata('topic_noleaders', 0, -1, [], [], NO_LEADER),
176-
PartitionMetadata('topic_noleaders', 1, -1, [], [], NO_LEADER),
173+
TopicMetadata(b'topic_still_creating', NO_LEADER, []),
174+
TopicMetadata(b'topic_doesnt_exist', UNKNOWN_TOPIC_OR_PARTITION, []),
175+
TopicMetadata(b'topic_noleaders', NO_ERROR, [
176+
PartitionMetadata(b'topic_noleaders', 0, -1, [], [], NO_LEADER),
177+
PartitionMetadata(b'topic_noleaders', 1, -1, [], [], NO_LEADER),
177178
]),
178179
]
179180
protocol.decode_metadata_response.return_value = MetadataResponse(brokers, topics)
@@ -188,8 +189,8 @@ def test_has_metadata_for_topic(self, protocol, conn):
188189
self.assertTrue(client.has_metadata_for_topic('topic_noleaders'))
189190

190191
@patch('kafka.client.KafkaConnection')
191-
@patch('kafka.client.KafkaProtocol')
192-
def test_ensure_topic_exists(self, protocol, conn):
192+
@patch('kafka.client.KafkaProtocol.decode_metadata_response')
193+
def test_ensure_topic_exists(self, decode_metadata_response, conn):
193194

194195
conn.recv.return_value = 'response' # anything but None
195196

@@ -199,14 +200,14 @@ def test_ensure_topic_exists(self, protocol, conn):
199200
]
200201

201202
topics = [
202-
TopicMetadata('topic_still_creating', NO_LEADER, []),
203-
TopicMetadata('topic_doesnt_exist', UNKNOWN_TOPIC_OR_PARTITION, []),
204-
TopicMetadata('topic_noleaders', NO_ERROR, [
205-
PartitionMetadata('topic_noleaders', 0, -1, [], [], NO_LEADER),
206-
PartitionMetadata('topic_noleaders', 1, -1, [], [], NO_LEADER),
203+
TopicMetadata(b'topic_still_creating', NO_LEADER, []),
204+
TopicMetadata(b'topic_doesnt_exist', UNKNOWN_TOPIC_OR_PARTITION, []),
205+
TopicMetadata(b'topic_noleaders', NO_ERROR, [
206+
PartitionMetadata(b'topic_noleaders', 0, -1, [], [], NO_LEADER),
207+
PartitionMetadata(b'topic_noleaders', 1, -1, [], [], NO_LEADER),
207208
]),
208209
]
209-
protocol.decode_metadata_response.return_value = MetadataResponse(brokers, topics)
210+
decode_metadata_response.return_value = MetadataResponse(brokers, topics)
210211

211212
client = KafkaClient(hosts=['broker_1:4567'])
212213

@@ -218,6 +219,7 @@ def test_ensure_topic_exists(self, protocol, conn):
218219

219220
# This should not raise
220221
client.ensure_topic_exists('topic_noleaders', timeout=1)
222+
client.ensure_topic_exists(b'topic_noleaders', timeout=1)
221223

222224
@patch('kafka.client.KafkaConnection')
223225
@patch('kafka.client.KafkaProtocol')
@@ -269,8 +271,8 @@ def test_get_leader_for_unassigned_partitions(self, protocol, conn):
269271
]
270272

271273
topics = [
272-
TopicMetadata('topic_no_partitions', NO_LEADER, []),
273-
TopicMetadata('topic_unknown', UNKNOWN_TOPIC_OR_PARTITION, []),
274+
TopicMetadata(b'topic_no_partitions', NO_LEADER, []),
275+
TopicMetadata(b'topic_unknown', UNKNOWN_TOPIC_OR_PARTITION, []),
274276
]
275277
protocol.decode_metadata_response.return_value = MetadataResponse(brokers, topics)
276278

@@ -279,10 +281,10 @@ def test_get_leader_for_unassigned_partitions(self, protocol, conn):
279281
self.assertDictEqual({}, client.topics_to_brokers)
280282

281283
with self.assertRaises(LeaderNotAvailableError):
282-
client._get_leader_for_partition('topic_no_partitions', 0)
284+
client._get_leader_for_partition(b'topic_no_partitions', 0)
283285

284286
with self.assertRaises(UnknownTopicOrPartitionError):
285-
client._get_leader_for_partition('topic_unknown', 0)
287+
client._get_leader_for_partition(b'topic_unknown', 0)
286288

287289
@patch('kafka.client.KafkaConnection')
288290
@patch('kafka.client.KafkaProtocol')

test/test_client_integration.py

Lines changed: 4 additions & 4 deletions
Original file line numberDiff line numberDiff line change
@@ -29,11 +29,11 @@ def tearDownClass(cls): # noqa
2929

3030
@kafka_versions("all")
3131
def test_consume_none(self):
32-
fetch = FetchRequest(self.topic, 0, 0, 1024)
32+
fetch = FetchRequest(self.bytes_topic, 0, 0, 1024)
3333

3434
fetch_resp, = self.client.send_fetch_request([fetch])
3535
self.assertEqual(fetch_resp.error, 0)
36-
self.assertEqual(fetch_resp.topic, self.topic)
36+
self.assertEqual(fetch_resp.topic, self.bytes_topic)
3737
self.assertEqual(fetch_resp.partition, 0)
3838

3939
messages = list(fetch_resp.messages)
@@ -56,11 +56,11 @@ def test_ensure_topic_exists(self):
5656

5757
@kafka_versions("0.8.1", "0.8.1.1", "0.8.2.0")
5858
def test_commit_fetch_offsets(self):
59-
req = OffsetCommitRequest(self.topic, 0, 42, b"metadata")
59+
req = OffsetCommitRequest(self.bytes_topic, 0, 42, b"metadata")
6060
(resp,) = self.client.send_offset_commit_request(b"group", [req])
6161
self.assertEqual(resp.error, 0)
6262

63-
req = OffsetFetchRequest(self.topic, 0)
63+
req = OffsetFetchRequest(self.bytes_topic, 0)
6464
(resp,) = self.client.send_offset_fetch_request(b"group", [req])
6565
self.assertEqual(resp.error, 0)
6666
self.assertEqual(resp.offset, 42)

test/test_consumer_integration.py

Lines changed: 1 addition & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -37,7 +37,7 @@ def tearDownClass(cls):
3737

3838
def send_messages(self, partition, messages):
3939
messages = [ create_message(self.msg(str(msg))) for msg in messages ]
40-
produce = ProduceRequest(self.topic, partition, messages = messages)
40+
produce = ProduceRequest(self.bytes_topic, partition, messages = messages)
4141
resp, = self.client.send_produce_request([produce])
4242
self.assertEqual(resp.error, 0)
4343

test/test_failover_integration.py

Lines changed: 3 additions & 2 deletions
Original file line numberDiff line numberDiff line change
@@ -8,6 +8,7 @@
88
from kafka.common import TopicAndPartition, FailedPayloadsError, ConnectionError
99
from kafka.producer.base import Producer
1010
from kafka.producer import KeyedProducer
11+
from kafka.util import kafka_bytestring
1112

1213
from test.fixtures import ZookeeperFixture, KafkaFixture
1314
from test.testutil import (
@@ -147,7 +148,7 @@ def test_switch_leader_keyed_producer(self):
147148
key = random_string(3)
148149
msg = random_string(10)
149150
producer.send_messages(topic, key, msg)
150-
if producer.partitioners[topic].partition(key) == 0:
151+
if producer.partitioners[kafka_bytestring(topic)].partition(key) == 0:
151152
recovered = True
152153
except (FailedPayloadsError, ConnectionError):
153154
logging.debug("caught exception sending message -- will retry")
@@ -172,7 +173,7 @@ def _send_random_messages(self, producer, topic, partition, n):
172173
logging.debug('_send_random_message to %s:%d -- try %d success', topic, partition, j)
173174

174175
def _kill_leader(self, topic, partition):
175-
leader = self.client.topics_to_brokers[TopicAndPartition(topic, partition)]
176+
leader = self.client.topics_to_brokers[TopicAndPartition(kafka_bytestring(topic), partition)]
176177
broker = self.brokers[leader.nodeId]
177178
broker.close()
178179
return broker

test/test_producer_integration.py

Lines changed: 2 additions & 2 deletions
Original file line numberDiff line numberDiff line change
@@ -453,7 +453,7 @@ def test_acks_cluster_commit(self):
453453

454454
def assert_produce_request(self, messages, initial_offset, message_ct,
455455
partition=0):
456-
produce = ProduceRequest(self.topic, partition, messages=messages)
456+
produce = ProduceRequest(self.bytes_topic, partition, messages=messages)
457457

458458
# There should only be one response message from the server.
459459
# This will throw an exception if there's more than one.
@@ -471,7 +471,7 @@ def assert_fetch_offset(self, partition, start_offset, expected_messages):
471471
# There should only be one response message from the server.
472472
# This will throw an exception if there's more than one.
473473

474-
resp, = self.client.send_fetch_request([ FetchRequest(self.topic, partition, start_offset, 1024) ])
474+
resp, = self.client.send_fetch_request([ FetchRequest(self.bytes_topic, partition, start_offset, 1024) ])
475475

476476
self.assertEqual(resp.error, 0)
477477
self.assertEqual(resp.partition, partition)

0 commit comments

Comments
 (0)