summaryrefslogtreecommitdiff
path: root/kafka
Commit message (Collapse)AuthorAgeFilesLines
...
* | Change message type requirement from str to bytes for clarity and prep for ↵Dana Powers2014-09-041-5/+5
|/ | | | python3
* Merge pull request #213 from dpkp/improve_failover_testsDana Powers2014-08-271-0/+6
|\ | | | | | | | | Warn users about async producer Refactor producer failover tests (5x speedup) Skip async producer failover test for now, because it is broken
| * Add warnings to README, docstring, and logging that async producer does not ↵Dana Powers2014-08-261-0/+6
| | | | | | | | retry failed messages
* | Merge pull request #136 from DataDog/fix-multifetch-buffer-sizeDana Powers2014-08-261-12/+14
|\ \ | |/ |/| | | | | | | | | fix consumer retry logic (fixes #135) Conflicts: kafka/consumer.py
| * fix consumer retry logic (fixes #135)Carlo Cabanilla2014-02-281-12/+14
| | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | Fixes bug in the follow condition: * Starting buffer size is 1024, max buffer size is 2048, both set on an instance level * Fetch from p0, p1 and received response * p0 has more than 1024 bytes, consumer doubles buffer size to 2048 and marks p0 for retry * p1 has more than 1024 bytes, consumer tries to double buffer size, but sees that it's at the max and raises ConsumerFetchSizeTooSmall The fix changes the logic to the following: * Starting buffer size is 1024 set on a per-partition level, max buffer size is 2048 set on an instance level * Fetch from p0, p1 and received response * p0 has more than 1024 bytes, consumer doubles buffer size to 2048 for p0 and marks p0 for retry * p1 has more than 1024 bytes, consumer double buffer size to 2048 for p1 and marks p1 for retry * Consumer sees that there's partitions to retry, repeats parsing loop * p0 sent all the bytes this time, consumer yields these messages * p1 sent all the bytes this time, consumer yields these messages
* | Merge pull request #208 from dpkp/add_pylint_to_tox_iniDana Powers2014-08-262-4/+4
|\ \ | | | | | | Use PyLint for static error checking
| * | Add pylint to tox.ini; test both kafka and test; default to error-checking ↵Dana Powers2014-08-252-4/+4
| | | | | | | | | | | | only; fixup errors; skip kafka/queue.py
* | | Raise TypeError in kafka.producer.send_messages if any msg is not a str (or ↵Dana Powers2014-08-261-0/+21
| | | | | | | | | | | | subclass); document
* | | also reset `self.fetch_offsets` in `fetch_last_known_offsets`Zack Dever2014-08-251-0/+1
| | |
* | | Move fetching last known offset logic to a stand alone function.Zack Dever2014-08-251-10/+16
|/ / | | | | | | | | | | | | | | | | | | The `Consumer` class fetches the last known offsets in `__init__` if `auto_commit` is enabled, but it would be nice to expose this behavior for consumers that aren't using auto_commit. This doesn't change existing behavior, just exposes the ability to easily fetch and set the last known offsets. Once #162 or something similar lands this may no longer be necessary, but it looks like that might take a while to make it through.
* | Merge pull request #196 from dpkp/reinit_connection_errorDana Powers2014-08-221-18/+60
|\ \ | | | | | | Improve KafkaConnection with more tests
| * | Improve docstrings in kafka.connDana Powers2014-08-191-4/+14
| | |
| * | Shutdown socket before closing in kafka.connDana Powers2014-08-191-0/+9
| | |
| * | Remove duplicate error handling blocks by moving empty response check into ↵Dana Powers2014-08-191-4/+8
| | | | | | | | | | | | try block in conn.recv
| * | Remove self._dirty and check self._sock instead in kafka.connDana Powers2014-08-191-6/+11
| | |
| * | socket.sendall should always raise an exception on error; remove extra ↵Dana Powers2014-08-191-6/+7
| | | | | | | | | | | | return val check in KafkaConnection.send()
| * | Add some more debug logging to KafkaConnectionDana Powers2014-08-191-0/+7
| | |
| * | Initialized _dirty in KafkaConnection __init__() and set _sock to None in ↵Dana Powers2014-08-171-2/+4
| | | | | | | | | | | | close()
| * | Call _raise_connection_error in KafkaConnection.reinit() if socket ↵Dana Powers2014-08-171-2/+6
| | | | | | | | | | | | connection fails
* | | Merge pull request #204 from mdaniel/better-type-errorsDana Powers2014-08-221-1/+8
|\ \ \ | | | | | | | | Better type errors
| * | | Fix write_int and write_short type validationMatthew L Daniel2014-08-221-0/+6
| | | | | | | | | | | | | | | | It will still die, just as before, but it now includes a *helpful* error message
| * | | PEP8 fixesMatthew L Daniel2014-08-221-1/+2
| |/ /
* | | Merge pull request #194 from dpkp/kafka_timeout_errorDana Powers2014-08-222-3/+7
|\ \ \ | | | | | | | | Add KafkaTimeoutError and fix client.ensure_topic_exists
| * | | Add KafkaTimeoutError (used by client.ensure_topic_exists) and add a testDana Powers2014-08-142-3/+7
| |/ /
* | | Add TravisCI config for auto deploymentDavid Arthur2014-08-221-1/+1
|/ / | | | | | | Tags applied to master will now be automatically deployed on PyPI
* | Merge pull request #174 from wizzat/new_topicDana Powers2014-08-102-11/+24
|\ \ | | | | | | Handle New Topic Creation
| * | Handle New Topic CreationMark Roberts2014-05-222-11/+24
| | | | | | | | | | | | | | | Adds ensure_topic_exists to KafkaClient, redirects test case to use that. Fixes #113 and fixes #150.
* | | Set module version in separate VERSION fileDana Powers2014-08-101-1/+3
|/ /
* | Merge pull request #166 from patricklucas/teach_producers_about_compressionDana Powers2014-05-193-22/+59
|\ \ | | | | | | Add 'codec' parameter to Producer
| * | Improve error handling and tests w.r.t. codecsPatrick Lucas2014-05-073-21/+33
| | | | | | | | | | | | | | | | | | | | | | | | | | | | | | Add function kafka.protocol.create_message_set() that takes a list of payloads and a codec and returns a message set with the desired encoding. Introduce kafka.common.UnsupportedCodecError, raised if an unknown codec is specified. Include a test for the new function.
| * | Merge branch 'teach_producers_about_compression' into producer_compressionMark Roberts2014-05-072-20/+45
| |\ \ | | | | | | | | | | | | | | | | | | | | Conflicts: servers/0.8.0/kafka-src test/test_unit.py
| | * | Add 'codec' parameter to ProducerPatrick Lucas2014-05-032-20/+45
| | | | | | | | | | | | | | | | | | | | Adds a codec parameter to Producer.__init__ that lets the user choose a compression codec to use for all messages sent by it.
* | | | Support IPv6 hosts and networksAlexey Borzenkov2014-05-091-3/+1
|/ / /
* | | Merge branch 'master' into add_testsMark Roberts2014-05-061-1/+15
|\ \ \ | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | kafka/client.py contained duplicate copies of same refactor, merged. Move test/test_integration.py changes into test/test_producer_integration. Conflicts: kafka/client.py servers/0.8.0/kafka-src test/test_integration.py
| * \ \ Merge pull request #139 from alexcb/masterDana Powers2014-05-061-1/+15
| |\ \ \ | | | | | | | | | | SimpleProducer randomization of initial round robin ordering
| | * | | added random_start param to SimpleProducer to enable/disable randomization ↵Alex Couture-Beil2014-04-011-4/+11
| | | | | | | | | | | | | | | | | | | | of the initial partition messages are published to
| | * | | Changed randomization to simply randomize the initial starting partition of ↵Alex Couture-Beil2014-04-011-3/+7
| | | | | | | | | | | | | | | | | | | | the sorted list of partition rather than completely randomizing the initial ordering before round-robin cycling the partitions
| | * | | Modified SimpleProducer to randomize the initial round robin orderingAlex Couture-Beil2014-03-111-1/+4
| | | | | | | | | | | | | | | | | | | | | | | | | of partitions to prevent the first message from always being published to partition 0.
| * | | | Fix lack of timeout support in KafkaClient and KafkaConnectionmaciejkula2014-04-162-2/+2
| | |/ / | |/| |
* | | | Attempt to fix travis build. Decrease complexity of service.py in favor of ↵Mark Roberts2014-05-064-8/+12
| | | | | | | | | | | | | | | | in memory logging. Address code review concerns
* | | | Make commit() check for errors instead of simply assert no errorMark Roberts2014-04-301-1/+1
| | | |
* | | | Make BrokerRequestError a base class, make subclasses for each broker errorMark Roberts2014-04-303-53/+113
| | | |
* | | | Various fixesMark Roberts2014-04-252-2/+9
| | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | Bump version number to 0.9.1 Update readme to show supported Kafka/Python versions Validate arguments in consumer.py, add initial consumer unit test Make service kill() child processes when startup fails Add tests for util.py, fix Python 2.6 specific bug.
* | | | Fix last remaining test by making autocommit more intuitiveMark Roberts2014-04-241-1/+1
| | | |
* | | | Split out kafka version environments, default tox no longer runs any ↵Mark Roberts2014-04-231-11/+11
| | | | | | | | | | | | | | | | integration tests, make skipped integration also skip setupClass, implement rudimentary offset support in consumer.py
* | | | Fix bug in socket timeout per PR #161 by maciejkula, add testMark Roberts2014-04-191-1/+1
| | | |
* | | | Split up and speed up producer based integration testsMark Roberts2014-04-172-1/+2
| | | |
* | | | Refactor away _get_conn_for_broker. Fix bug in _get_connMark Roberts2014-04-091-13/+6
| | | |
* | | | Merge branch 'master' into add_testsMark Roberts2014-04-081-0/+4
|\ \ \ \ | |/ / /
| * | | Commit in seek if autocommitMark Roberts2014-03-271-1/+4
| | | |