summaryrefslogtreecommitdiff
diff options
context:
space:
mode:
authorDana Powers <dana.powers@rd.io>2016-01-10 19:56:00 -0800
committerDana Powers <dana.powers@rd.io>2016-01-10 21:24:33 -0800
commitecb4d49c06484e8ed9bdb6db35350d104e13b730 (patch)
tree43692e5c5114f1fc6b30a161c7b2c9e1076cf971
parentb86daafeaf72f4d678d3b627d32f39ab6f0e54ec (diff)
downloadkafka-python-idle_sleep.tar.gz
Pass consumer timeout to client.poll() in iterator; check timeout before iterating fetcheridle_sleep
-rw-r--r--kafka/consumer/group.py5
1 files changed, 4 insertions, 1 deletions
diff --git a/kafka/consumer/group.py b/kafka/consumer/group.py
index c6beb25..704c994 100644
--- a/kafka/consumer/group.py
+++ b/kafka/consumer/group.py
@@ -628,11 +628,14 @@ class KafkaConsumer(six.Iterator):
# init any new fetches (won't resend pending fetches)
self._fetcher.init_fetches()
- self._client.poll()
+ self._client.poll(
+ max(0, self._consumer_timeout - time.time()) * 1000)
timeout_at = min(self._consumer_timeout,
self._client._delayed_tasks.next_at() + time.time(),
self._client.cluster.ttl() / 1000.0 + time.time())
+ if time.time() > timeout_at:
+ continue
for msg in self._fetcher:
yield msg
if time.time() > timeout_at: