summaryrefslogtreecommitdiff
path: root/kafka/consumer
Commit message (Collapse)AuthorAgeFilesLines
...
* | fixup kafka consumer docstringDana Powers2015-03-291-1/+1
| |
* | Improve KafkaConsumer docstringsDana Powers2015-03-291-134/+94
|/
* Merge pull request #340 from dpkp/deprecate_metadata_broker_listDana Powers2015-03-291-8/+26
|\ | | | | Deprecate metadata_broker_list in favor of bootstrap_servers
| * Deprecate KafkaConsumer config metadata_broker_list in favor of ↵Dana Powers2015-03-231-8/+26
| | | | | | | | bootstrap_servers
* | Merge pull request #336 from scrapinghub/feature-mp-consumer-paramsDana Powers2015-03-292-35/+51
|\ \ | | | | | | Using additional params for MP consumer child process
| * | Manageable queue.put() operation for MPConsumer processesViktor Shlapakov2015-03-252-4/+12
| | |
| * | Using mp.manager to solve the issue with join for MPConsumerViktor Shlapakov2015-03-241-6/+7
| | |
| * | Moving to **kwargs for MPConsumer optionsViktor Shlapakov2015-03-241-1/+2
| | |
| * | Moved additional MP consumer options to **kwargsViktor Shlapakov2015-03-131-3/+3
| | |
| * | Using additional params for MP consumer child processViktor Shlapakov2015-03-131-26/+32
| | | | | | | | | | | | | | | | | | | | | - Moved the events params to a separate param for consistency - Passing additional params to internal SimpleConsumer worker for multiprocessing high-level consumer. It allows to use non-default consumer settings (fetch_size_bytes, buffer_size, max_buffer_size).
* | | Increase count_since_commit on seekEnrico Canzonieri2015-03-201-1/+1
|/ / | | | | | | | | | | When auto_commit is False this change enables an explicit call to commit() to actually commit the offsets. Otherwise a consumer won't be able to commit until at least one message is read.
* | Merge pull request #335 from scrapinghub/fix-mp-consumer-distributionDana Powers2015-03-121-10/+12
|\ \ | |/ |/| Wrong partitions distribution logic for MP Consumer
| * Used thread-safe dict.copy().keys() for MP consumer partitionsViktor Shlapakov2015-03-121-1/+4
| |
| * Cleaned code for MP consumer chunkingViktor Shlapakov2015-03-111-10/+9
| |
| * Fixing distribution for MP ConsumerViktor Shlapakov2015-03-031-1/+1
| |
* | Merge pull request #332 from dpkp/kafka_client_editsDana Powers2015-03-071-11/+2
|\ \ | | | | | | Kafka client edits
| * | Remove KafkaConsumer.BYTES_CONFIGURATION_KEYSDana Powers2015-03-011-11/+2
| |/ | | | | | | | | | | - rely on KafkaClient to handle client_id string/bytes encoding - accept either string or bytes for group_id - convert group_id string to utf-8 bytes internally
* | Merge pull request #328 from wkiser/validate_offsetDana Powers2015-03-022-0/+8
|\ \ | | | | | | Fixes consumer/kafka and consumer/simple to only yield messages if the m...
| * | Fixes consumer/kafka and consumer/simple to only yield messages if the ↵wkiser2015-03-022-0/+8
| |/ | | | | | | message's offset is greater than or equal to the consumer offset.
* | Merge pull request #296 from ecanzonieri/validate_consumer_offsetDana Powers2015-03-021-3/+54
|\ \ | |/ |/| Validate consumer offset in SimpleConsumer
| * Retry failed partitions and add integration testsEnrico Canzonieri2015-01-261-0/+2
| |
| * Merge branch 'master' of github.com:mumrah/kafka-python into ↵Enrico Canzonieri2015-01-264-155/+194
| |\ | | | | | | | | | | | | | | | | | | validate_consumer_offset Conflicts: kafka/consumer/simple.py
| * | Make SimpleConsumer auto_offset_reset more like KafkaConsumerEnrico Canzonieri2015-01-261-4/+19
| | |
| * | use a list in send_offset_requestEnrico Canzonieri2015-01-141-3/+4
| | |
| * | Implement offsets reset when OffsetOutOfRangeErrorEnrico Canzonieri2015-01-141-3/+35
| | | | | | | | | | | | | | | | | | | | | | | | This slightly changes the SimpleConsumer interface adding the default option use_latest_offsets. The fetch behaviour is also changed since it does not raise OffsetOutOfRangeError anymore. Resetting the offsets automatically is especially useful in MultiprocessConsumer, where an explicit seek call is not possible.
* | | Updated documentation for Consumers to prefer KafkaConsumer insteadJohn Anderson2015-02-091-2/+5
| |/ |/|
* | Add Sphinx API docsWill Daly2015-01-154-152/+190
|/
* Add some jitter to refresh_leader_backoff_ms, per wizzat reviewDana Powers2014-12-151-1/+7
|
* Simplify BYTES_CONFIGURATION_KEYS logic, per wizzat reviewDana Powers2014-12-151-2/+1
|
* Fixup call to self._client.get_partition_ids_for_topic -- use encoded topic ↵Dana Powers2014-12-151-1/+1
| | | | bytes
* Use kafka.util.kafka_bytestring to encode utf-8 when necessaryDana Powers2014-12-151-15/+6
|
* OffsetCommit metadata must be bytesDana Powers2014-12-151-1/+1
|
* Add private methods _does_auto_commit_ms and _does_auto_commit_messagesDana Powers2014-12-151-7/+22
|
* Fix task_done checks when no previous commit exists; add testDana Powers2014-12-151-6/+5
|
* Force absolue_imports in kafka/consumer/kafka.pyDana Powers2014-12-151-0/+2
|
* Move KafkaConsumer to kafka.consumer.kafka module; make available for import ↵Dana Powers2014-12-152-3/+4
| | | | from kafka at top-level
* Use six for py3 support in KafkaConsumerDana Powers2014-12-151-13/+39
| | | | Log connection failures w/ traceback in kafka/client.py
* Update docstrings w/ current interface / config defaultsDana Powers2014-12-151-16/+21
|
* Raise KafkaConfigurationError during fetch_messages if not topics/partitions ↵Dana Powers2014-12-151-1/+7
| | | | configured
* Move auto-commit checks to task_done; add support for ↵Dana Powers2014-12-151-11/+27
| | | | auto_commit_interval_messages
* Add private methods to manage internal _msg_iterDana Powers2014-12-151-10/+20
|
* Reorder methods, add docstrings to public methds, section comments for ↵Dana Powers2014-12-151-207/+255
| | | | private methods
* Use 4-space indentsDana Powers2014-12-151-518/+517
|
* Add docstring to get_partition_offsets; use request_time_ms and ↵Dana Powers2014-12-151-7/+25
| | | | max_num_offsets var names
* Add docstring to configure()Dana Powers2014-12-151-0/+21
|
* raise KafkaConfigurationError in commit() if there is no configured ↵Dana Powers2014-12-151-1/+8
| | | | 'group_id'; add docstring
* _should_auto_commit is privateDana Powers2014-12-151-2/+2
|
* Support setting offsets in set_topic_partitions(); reorganize offsets ↵Dana Powers2014-12-151-127/+151
| | | | initialization
* _client is private varDana Powers2014-12-151-7/+11
|
* Move kafka._msg_iter initialization from __init__() to next()Dana Powers2014-12-151-6/+7
|