summaryrefslogtreecommitdiff
path: root/kafka/client.py
Commit message (Collapse)AuthorAgeFilesLines
* Revert broken send_produce_request try/except from PR 467 (resps=None)Dana Powers2015-12-051-5/+1
|
* Fix requestId handling in send_broker_aware_requestDana Powers2015-12-041-2/+2
|
* Merge pull request #420 from toddpalino/masterDana Powers2015-12-021-0/+128
|\ | | | | Initial support for consumer coordinator
| * Support consumer metadata requeststpalino2015-07-011-0/+128
| | | | | | | | | | | | Support added for ConsumerMetadataRequest and ConsumerMetadataResponse Added consumer-aware request routine for supporting the consumer coordinator Added separate client method for fetching Kafka-committed offsets from the coordinator
* | Merge pull request #473 from ecanzonieri/use_unblocking_io_for_aware_requestsDana Powers2015-12-021-18/+30
|\ \ | | | | | | Use unblocking io for broker aware requests
| * | Add tests. Bug fix. Rename socket_conn dict.Enrico Canzonieri2015-11-101-7/+7
| | |
| * | Unblocking broker aware requestEnrico Canzonieri2015-11-101-18/+30
| | |
* | | Prevents crashing communication thread of async producerBalthasar Schopman2015-10-221-1/+5
|/ / | | | | | | | | | | | | | | | | | | | | | | | | | | If an uncaught exception occurs in _send_messages() the thread sending data to Kafka (asynchronously) will crash and the queue will never be emptied. To reproduce: 1) Run an Async producer. 2) Kill the Kafka server. 3) Restart the Kafka server. The communication thread dies shortly after step 2. After step 3 the communication does not resume without this commit. The changes in both files prevent an Exception from being thrown through to do main communication process, which could cause the crash.
* | Merge pull request #436 from mutability/async-catch-unavailable-errorDana Powers2015-10-071-6/+11
|\ \ | | | | | | Catch KafkaUnavailableError in _send_broker_aware_request
| * | Init responses before we use it.Oliver Jowett2015-07-241-1/+1
| | |
| * | Errors -> Error typoOliver Jowett2015-07-201-1/+1
| | |
| * | Treat KafkaUnavailableError like other errors.Oliver Jowett2015-07-191-5/+10
| |/
* | Use debug logging level for metadata requestEnrico Canzonieri2015-07-211-2/+2
|/
* Lower logging level on replica not available and commitEnrico Canzonieri2015-06-191-1/+1
|
* Add KafkaClient.topics property to get list of known topicsDana Powers2015-06-111-0/+4
|
* Fix KafkaClient request / response orderingDana Powers2015-06-101-10/+20
|
* KafkaClient log new broker and topic metadata received as INFODana Powers2015-06-091-2/+2
|
* KafkaClient: log responses by requestId for debuggingDana Powers2015-06-091-1/+4
|
* KafkaClient: log requestId on ConnectionError instead of hexdump of request ↵Dana Powers2015-06-091-6/+5
| | | | contents
* Improve kafka client debug request/response loggingDana Powers2015-06-081-5/+5
|
* Prefer single quotes for stringsDana Powers2015-06-081-14/+12
|
* Add a few extra docstring comments about thread-safe clients/connectionsDana Powers2015-06-081-2/+5
|
* Cleanup imports in kafka/client and kafka/consumerDana Powers2015-06-081-1/+1
|
* KafkaClient should try/except ConnnectionError when calling _get_connDana Powers2015-06-061-2/+2
|
* Raise an error if we attempt to group duplicate topic-partition payloadsDana Powers2015-06-061-0/+2
| | | | - previously this would simply drop one of the payloads
* Use separate module loggers instead of a single 'kafka' loggerDana Powers2015-06-061-1/+2
|
* Merge pull request #367 from dpkp/clean_metadata_refreshDana Powers2015-04-121-13/+14
|\ | | | | Clear local metadata cache before refresh in client.load_metadata_for_topics()
| * Update debug log messages in client.load_metadata_for_topicsDana Powers2015-04-121-2/+2
| |
| * clean local metadata before requesting refresh in load_metadata_for_topics()Dana Powers2015-04-121-2/+7
| |
| * Simplify client.reset_topic_metadataDana Powers2015-04-121-9/+5
| |
* | Improve send_produce_request docstringDana Powers2015-04-061-6/+17
| |
* | Add debug logging for all broker-aware requests/responsesDana Powers2015-04-061-0/+3
| |
* | Refactor KafkaClient._send_broker_aware_request to return a list of responsesDana Powers2015-04-061-73/+57
|/ | | | | | and include individual (unraised) FailedPayloadsError instances rather than always raising a FailedPayloadsError. This should allow producers to determine which payloads succeeded and which failed, specifically.
* Make external API consistently support python3 strings for topic.Space2015-04-031-0/+3
|
* Rollover KafkaClient correlation ids at 2**31 to keep within int32 protocol ↵Dana Powers2015-03-291-7/+7
| | | | encoding
* KafkaClient.get_partition_ids_for_topic now returns empty list for unknown ↵Dana Powers2015-03-021-1/+1
| | | | topics
* Add comment about no server response case in _send_broker_aware_requestDana Powers2015-03-021-0/+5
|
* Avoid topic_partitions KeyError in KafkaClientDana Powers2015-03-021-1/+1
|
* Accept KafkaClient client_id as string or bytes; encode string client_id as ↵Dana Powers2015-03-011-1/+2
| | | | utf-8 bytes internally
* Always return sorted partition ids in KafkaClient.get_partition_ids_for_topic()Dana Powers2015-02-091-1/+1
|
* Merge pull request #282 from wedaly/sphinx-api-docsDana Powers2015-01-231-24/+24
|\ | | | | Add Sphinx API docs
| * Add Sphinx API docsWill Daly2015-01-151-24/+24
| |
* | Catch ReplicaNotAvailableError in MetadataResponse -- per kafka-devs, this ↵Dana Powers2015-01-141-1/+6
|/ | | | error can and should be ignored
* Use six for py3 support in KafkaConsumerDana Powers2014-12-151-3/+3
| | | | Log connection failures w/ traceback in kafka/client.py
* Merge pull request #223 from dpkp/metadata_refactorDana Powers2014-09-081-63/+156
|\ | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | 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-011-15/+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
| |
| * Refactor internal metadata dicts in KafkaClientDana Powers2014-09-011-20/+56
| | | | | | | | | | | | | | | | - 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