summaryrefslogtreecommitdiff
path: root/kafka
Commit message (Collapse)AuthorAgeFilesLines
...
* | Merge pull request #223 from dpkp/metadata_refactorDana Powers2014-09-085-116/+224
|\ \ | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | Metadata Refactor * add MetadataRequest and MetadataResponse namedtuples * add TopicMetadata namedtuple * add error codes to Topic and Partition Metadata * add KafkaClient.send_metadata_request() method * KafkaProtocol.decode_metadata_response changed to return a MetadataResponse object so that it is consistent with server api: [broker_list, topic_list] * raise server exceptions in load_metadata_for_topics(*topics) unless topics is null (full refresh) * Replace non-standard exceptions (LeaderUnavailable, PartitionUnavailable) with server standard exceptions (LeaderNotAvailableError, UnknownTopicOrPartitionError) Conflicts: kafka/client.py test/test_client.py test/test_producer_integration.py test/test_protocol.py
| * | Add test for sending produce request to unknown topic w/o auto-creationDana Powers2014-09-011-2/+2
| | |
| * | Use standard exceptions in client._get_leader_for_partition()Dana Powers2014-09-012-19/+28
| | | | | | | | | | | | | | | | | | - drop custom PartitionUnavailable exception - raise UnknownTopicOrPartitionError or LeaderNotAvailableError - add tests for exception raises
| * | Raise UnknownTopicOrPartitionError immediately in ensure_topic_exists -- ↵Dana Powers2014-09-011-0/+4
| | | | | | | | | | | | server is not auto-creating
| * | load_metadata_for_topics should raise exceptions on explicit topic argsDana Powers2014-09-011-12/+40
| | |
| * | Dont need to use callbacks for offset fetch requestsDana Powers2014-09-011-4/+3
| | |
| * | Refactor internal metadata dicts in KafkaClientDana Powers2014-09-013-32/+67
| | | | | | | | | | | | | | | | | | | | | | | | - use helper methods not direct access - add get_partition_ids_for_topic - check for topic and partition errors during load_metadata_for_topics - raise LeaderNotAvailableError when topic is being auto-created or UnknownTopicOrPartitionError if auto-creation off
| * | No more custom LeaderUnavailableError exception -- raise LeaderNotAvailableErrorDana Powers2014-09-012-7/+3
| | |
| * | if error code is not recognized in check_error: raise UnknownErrorDana Powers2014-09-011-2/+6
| | |
| * | Rearrange KafkaClient method order to group client metadata methods togetherDana Powers2014-09-011-18/+18
| | |
| * | Improve metadata protocol handlingDana Powers2014-09-013-36/+69
| |/ | | | | | | | | | | | | | | | | | | - add MetadataRequest and MetadataResponse namedtuples - add TopicMetadata namedtuple - add error codes to Topic and Partition Metadata - add KafkaClient.send_metadata_request() method - KafkaProtocol.decode_metadata_response changed to return a MetadataResponse object so that it is consistent with server api: [broker_list, topic_list]
* | Merge pull request #227 from wizzat-feature/py3Dana Powers2014-09-078-42/+77
|\ \ | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | Python 3 Support Conflicts: kafka/producer.py test/test_client.py test/test_client_integration.py test/test_codec.py test/test_consumer.py test/test_consumer_integration.py test/test_failover_integration.py test/test_producer.py test/test_producer_integration.py test/test_protocol.py test/test_util.py
| * | Update kafka.util.crc32 to unsigned everywhereMark Roberts2014-09-032-13/+4
| | |
| * | Fix the multiprocessing tests for python 3Mark Roberts2014-09-031-1/+1
| | |
| * | Fix test errors related to mocksMark Roberts2014-09-031-3/+4
| | |
| * | Fix more tests, only multiprocessing consumer ones remainingBruno Renié2014-09-032-3/+7
| | |
| * | Bytes in self.msg()Bruno Renié2014-09-031-1/+1
| | |
| * | Use built-in next()Bruno Renié2014-09-032-3/+3
| | |
| * | Cross-python iteritemsBruno Renié2014-09-031-1/+3
| | |
| * | client_id in bytesBruno Renié2014-09-031-1/+1
| | |
| * | Make all unit tests pass on py3.3/3.4Bruno Renié2014-09-037-29/+66
| |/
* | Minor import cleanupsDana Powers2014-09-042-7/+7
| |
* | 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
|/ /