summaryrefslogtreecommitdiff
path: root/kafka/common.py
Commit message (Collapse)AuthorAgeFilesLines
* Always absolute_importabsolute_importDana Powers2016-08-021-0/+2
|
* Split kafka.common into kafka.structs and kafka.errorsDana Powers2016-04-051-488/+2
|
* Mark old kafka.common structs as Deprecated; remove unused TopicMetadataDana Powers2016-02-151-11/+10
|
* Small KafkaClient.check_version() improvementsDana Powers2016-01-251-0/+4
| | | | | - filter connection failure logging during version check - raise UnrecognizedBrokerVersion if we cant id broker
* Use ConnectionError rather than new DisconnectErrorDana Powers2015-12-291-6/+2
|
* AbstractCoordinator (modelled after upstream Java class)Dana Powers2015-12-281-0/+5
| | | | | - Heartbeat class manages HeartbeatRequest / Response state - metrics are still TODO
* Add OffsetAndMetadata namedtupleDana Powers2015-12-281-0/+3
|
* Use attributes on exception classes to signal retriable and invalid metadata ↵Dana Powers2015-12-281-11/+28
| | | | errors
* Rename TopicAndPartition -> TopicPartitionDana Powers2015-12-281-1/+1
|
* New async KafkaClient, based on java NetworkClient and ConsumerClientDana Powers2015-12-281-0/+8
| | | | | | | - Support async client bootstrap retries after exponential backoff - Attempt to finish_connect if still connecting in KafkaClient.ready(node_id) - Run full async loop while waiting for futures in KafkaClient.poll() - Add more fallbacks to KafkaClient.least_loaded_node; worst case, retry boostrap
* Update BrokerConnection for use with async clientDana Powers2015-12-281-0/+4
| | | | | | | | | - use descriptive names for ConnectionStates enum values - Change default send_buffer_bytes config to 131072 - add can_send_more() and max_in_flight_requests_per_connection config - add blacked_out() and reconnect_backoff_ms config - last_attempt and last_failure are now public attributes - raise TooManyInFlightRequests in conn.send() if cant send more
* Rename Cluster -> ClusterMetadata; align with upstream Metadata classDana Powers2015-12-281-0/+4
|
* Switch BrokerConnection to (mostly) non-blocking IO.Dana Powers2015-12-171-0/+16
| | | | | | - return kafka.Future on send() - recv is now non-blocking call that completes futures when possible - update KafkaClient to block on future completion
* new 0.9 error codes plus descriptions.Zack Dever2015-12-151-8/+170
| | | | | reference: https://cwiki.apache.org/confluence/display/KAFKA/A+Guide+To+The+Kafka+Protocol#AGuideToTheKafkaProtocol-ErrorCodes
* Convert OffsetCommit and OffsetFetch protocol encode/decodeDana Powers2015-12-101-8/+8
|
* Change KafkaProtocol to encode/decode Structs, not bytesDana Powers2015-12-091-6/+6
| | | | | - add Payload to kafka.common Request/Responses namedtuples - OffsetFetch and OffsetCommit still need to be converted
* Support consumer metadata requeststpalino2015-07-011-0/+22
| | | | | | 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
* try to fix uncaught FailedPayloadsErrorreAsOn20102015-06-091-0/+2
|
* FailedPayloadsError now only has a single payload; use .payload attributeDana Powers2015-06-061-2/+2
| | | | .failed_payloads attribute has been removed.
* Improve async producer code: logic and style fixesViktor Shlapakov2015-06-031-11/+6
| | | | | | | | | | - send_producer_request with fail_on_error=False to retry failed reqs only - using an internal dict with with namedtuple keys for retry counters - refresh metadata on refresh_error irrespective to retries options - removed infinite retries (retry_options.limit=None) as an over-feature - separate producer init args for retries options (limit,backoff,on_timeouts) - AsyncProducerQueueFull returns a list of failed messages - producer tests improved thanks to @rogaha and @toli
* Fix names for async retries opts, add timeout for putViktor Shlapakov2015-06-031-2/+2
|
* Simplification of retry logicViktor Shlapakov2015-06-031-1/+1
|
* Fix small issues with names/testsViktor Shlapakov2015-06-031-0/+4
|
* Clean and simplify retry logicViktor Shlapakov2015-06-031-0/+17
|
* Improved retry logicViktor Shlapakov2015-06-031-0/+3
|
* Fixed tests and other issues after rebaseViktor Shlapakov2015-06-031-1/+0
|
* Retries for async batchingViktor Shlapakov2015-06-031-2/+10
|
* Fixup indentation in kafka/common.py; add links to protocol definitionsDana Powers2015-03-081-23/+26
|
* Add failed_payloads attribute to FailedPayloadsErrorJob Evers2015-03-081-1/+3
|
* Use reflection to avoid multiple errno definitionsAlex Couture-Beil2015-01-101-24/+12
|
* Use configure() to check and set configuration keysDana Powers2014-12-151-0/+4
|
* A simpler kafka consumer:Dana Powers2014-12-151-0/+7
| | | | | | | | | | | | | | | | | | ``` kafka = KafkaConsumer('topic1') for m in kafka: print m kafka = KafkaConsumer('topic1', 'topic2', group_id='my_consumer_group', auto_commit_enable=True, auto_commit_interval_ms=30 * 1000, auto_offset_reset='smallest') for m in kafka: process_message(m) kafka.task_done(m) ```
* Use standard exceptions in client._get_leader_for_partition()Dana Powers2014-09-011-4/+0
| | | | | | - drop custom PartitionUnavailable exception - raise UnknownTopicOrPartitionError or LeaderNotAvailableError - add tests for exception raises
* No more custom LeaderUnavailableError exception -- raise LeaderNotAvailableErrorDana Powers2014-09-011-4/+0
|
* if error code is not recognized in check_error: raise UnknownErrorDana Powers2014-09-011-2/+6
|
* Improve metadata protocol handlingDana Powers2014-09-011-7/+23
| | | | | | | | | | - 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]
* Add KafkaTimeoutError (used by client.ensure_topic_exists) and add a testDana Powers2014-08-141-0/+4
|
* Improve error handling and tests w.r.t. codecsPatrick Lucas2014-05-071-0/+6
| | | | | | | | | | 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.
* Attempt to fix travis build. Decrease complexity of service.py in favor of ↵Mark Roberts2014-05-061-2/+8
| | | | in memory logging. Address code review concerns
* Make BrokerRequestError a base class, make subclasses for each broker errorMark Roberts2014-04-301-25/+89
|
* Explicit testing of protocol errors. Make tests more explicit, and start ↵Mark Roberts2014-04-081-0/+4
| | | | working on intermittent failures in test_encode_fetch_request and test_encode_produc_request
* Merge pull request #134 from wizzat/conn_refactorv0.9.0Dana Powers2014-03-211-15/+21
|\ | | | | conn.py performance improvements, make examples work, add another example
| * Fix grammar in error stringMark Roberts2014-02-251-1/+1
| |
| * Minor refactor in conn.py, update version in __init__.py, add ErrorStringMark Roberts2014-02-251-15/+21
| |
* | Changes based on comments by @rdiomar, plus added LeaderUnavailableError for ↵mrtheb2014-02-151-0/+4
| | | | | | | | clarity
* | Handle cases for partition with leader=-1 (not defined)Marc Labbe2014-01-311-4/+0
|/
* Throw KafkaUnavailableError when no brokers availableThomas Dimson2014-01-131-0/+4
|
* Exception hierarchy, invalidate more md on errorsThomas Dimson2014-01-131-6/+25
|
* make changes to be more fault tolerant: clean up connections, brokers, ↵Jim Lim2013-10-041-0/+5
| | | | | | | | failed_messages - add integration tests for sync producer - add integration tests for async producer w. leadership election - use log.exception
* style: fix whitespacesVetoshkin Nikita2013-10-031-0/+4
|