summaryrefslogtreecommitdiff
path: root/kafka/client.py
Commit message (Collapse)AuthorAgeFilesLines
...
| * | Merge branch 'master' into conn_refactorMark Roberts2014-02-261-7/+18
| |\ \ | | | | | | | | | | | | | | | | Conflicts: example.py
| * | | Minor refactor in conn.py, update version in __init__.py, add ErrorStringMark Roberts2014-02-251-3/+3
| | | |
* | | | Merge branch 'master' into developmrtheb2014-03-171-9/+20
|\ \ \ \ | | |_|/ | |/| | | | | | | | | | Conflicts: test/test_unit.py
| * | | If a broker refuses the connection, try the nextstephenarmstrong2014-03-131-3/+3
| | |/ | |/|
| * | Support list (or comma-separated) of hosts (replaces host and port arguments)mrtheb2014-02-091-6/+7
| | |
| * | Merge branch 'master' into multihostsmrtheb2014-01-311-91/+120
| |\ \ | | |/ | | | | | | | | | | | | | | | | | | | | | Conflicts: kafka/client.py kafka/conn.py setup.py test/test_integration.py test/test_unit.py
| * | Allow KafkaClient to take in a list of brokers for bootstrappingMarc Labbe2013-11-141-14/+21
| | |
* | | Changes based on comments by @rdiomar, plus added LeaderUnavailableError for ↵mrtheb2014-02-151-4/+7
| | | | | | | | | | | | clarity
* | | check for broker None in send_broker_aware_request (added test for it)mrtheb2014-01-311-5/+14
| | |
* | | Handle cases for partition with leader=-1 (not defined)Marc Labbe2014-01-311-6/+12
| | |
* | | added mockmrtheb2014-01-181-3/+4
| |/ |/|
* | Merge pull request #107 from rdiomar/fix_default_timeoutsMarc Labbé2014-01-161-2/+6
|\ \ | | | | | | Increase default connection timeout
| * | Change default socket timeout to 120 seconds in both the client and connectionOmar Ghishan2014-01-161-2/+6
| | |
* | | Merge branch 'repr' of https://github.com/mahendra/kafka-python into ↵mrtheb2014-01-141-0/+3
|\ \ \ | |/ / |/| | | | | | | | | | | | | | | | | mahendra-repr Conflicts: kafka/client.py kafka/consumer.py
| * | Add proper string representations for each classMahendra M2013-10-081-1/+4
| |/
* | Throw KafkaUnavailableError when no brokers availableThomas Dimson2014-01-131-2/+2
| |
* | Exception hierarchy, invalidate more md on errorsThomas Dimson2014-01-131-80/+88
| |
* | Change BufferUnderflowError to ConnectionError in conn._read_bytes()Omar Ghishan2014-01-081-3/+2
| | | | | | Both errors are handled the same way when raised and caught, so this makes sense.
* | Fix client error handlingOmar Ghishan2014-01-061-5/+17
| | | | | | | | | | This differentiates between errors that occur when sending the request and receiving the response, and adds BufferUnderflowError handling.
* | * Guarantee reading the expected number of bytes from the socket every timeOmar Ghishan2014-01-061-4/+3
| | | | | | | | | | | | | | | | | | | | | | | | * Remove bufsize from client and conn, since they're not actually enforced Notes: This commit changes behavior a bit by raising a BufferUnderflowError when no data is received for the message size rather than a ConnectionError. Since bufsize in the socket is not actually enforced, but it is used by the consumer when creating requests, moving it there until a better solution is implemented.
* | Allow customizing socket timeouts.Omar Ghishan2014-01-061-3/+4
|/ | | | | | Previously, if you try to consume a message with a timeout greater than 10 seconds, but you don't receive data in those 10 seconds, a socket.timeout exception is raised. This allows a higher socket timeout to be set, or even None for no timeout.
* Merge branch 'master' into prod-windowsMahendra M2013-10-081-6/+18
|\ | | | | | | | | Conflicts: kafka/producer.py
| * make changes to be more fault tolerant: clean up connections, brokers, ↵Jim Lim2013-10-041-6/+18
| | | | | | | | | | | | | | | | failed_messages - add integration tests for sync producer - add integration tests for async producer w. leadership election - use log.exception
* | Ensure that async producer works in windows. Fixes #46Mahendra M2013-10-071-0/+11
|/ | | | | | | | | | | | | | | As per the multiprocessing module's documentation, the objects passed to the Process() class must be pickle-able in Windows. So, the Async producer did not work in windows. To fix this we have to ensure that code which uses multiprocessing has to follow certain rules * The target=func should not be a member function * We cannot pass objects like socket() to multiprocessing This ticket fixes these issues. For KafkaClient and KafkaConnection objects, we make copies of the object and reinit() them inside the child processes.
* Test fixes after flake8 runmrtheb2013-10-031-1/+2
|
* flake8 pass (pep8 and pyflakes)mrtheb2013-10-031-21/+20
|
* style: fix camelCase variable namesVetoshkin Nikita2013-10-031-3/+3
| | | | | | Conflicts: kafka/util.py
* style: use triple quotes for docstringsVetoshkin Nikita2013-10-031-2/+6
|
* style: fix whitespacesVetoshkin Nikita2013-10-031-1/+1
|
* allow a client id to be passed to the clientJim Lim2013-09-241-4/+5
|
* Merge branch 'issue-35'David Arthur2013-07-261-3/+11
|\ | | | | | | | | | | | | Conflicts: kafka/__init__.py kafka/consumer.py test/test_integration.py
| * Re-init the sockets in the new processMahendra M2013-06-271-0/+4
| |
| * Implement blocking get_messages for SimpleConsumerMahendra M2013-06-251-3/+7
| | | | | | | | | | | | | | | | | | | | | | | | | | The implementation is done by using simple options to Kafka Fetch Request Also in the SimpleConsumer iterator, update the offset before the message is yielded. This is so that the consumer state is not lost if certain cases. For eg: the message is yielded and consumed by the caller, but the caller does not come back into the generator again. The message will be consumed but the status is not updated in the consumer
* | Make the default case as 'ack on local write'Mahendra M2013-06-201-2/+11
|/ | | | | | | Also, ensure that the case of 'no-acks' works fine In conn.send(), do not wait for the response. Wait for it only on conn.recv(). This behaviour is fine now since the connection is not shared among consumer threads etc.
* Fix bugs and testingMahendra M2013-06-131-0/+5
| | | | | | * Ensure that round-robin partitioner works fine * _load_metadata_for_topics() would cause duplicate and stale entries in self.topic_partitions. Fix this
* Finish making remaining files pep8 readyMahendra M2013-06-041-60/+109
|
* Minor bug fixesMahendra M2013-05-291-0/+6
| | | | | | | | | | | | | | | | | | | | | | | | | | | | | | | * When you initiate a producer with a non-existant queue, the queue is created. However this partition info is not reflected in KafkaClient() immediately. So, we wait for a second and try loading it again. Without this fix, if we do producer.send_messages() after creating a new queue, the library will throw a StopIteration exception. * In SimpleConsumer(), the defaults are not as mentioned in the comments. Fix this (or do we change the documentation?) * There was a problem with the way the consumer iterator worked. for eg: assume that there were 10 messages in the queue/topic and you iterate over it as - for msg in consumer: print (msg) At the end of this, 'offset' that is saved is 10. So, if you run the above loop again, the last message (10) is repeated. This can be fixed by adjusting the offset counter before fetching the message * Avoid some code repeat in consumer.commit() * Fix a bug in send_offset_commit_request() invocation in consumer.py * Fix missing imports
* A few fixes for offset APIs in 0.8.1David Arthur2013-04-021-1/+1
|
* Update kafka-src to latest trunk, enable 0.8.1 featuresDavid Arthur2013-04-021-2/+0
|
* Update kafka-src to latest 0.8David Arthur2013-04-021-1/+1
| | | | Fix a broken test (100k was too much to send in one batch)
* Bring acks and timeout down to the clientDavid Arthur2013-04-021-3/+3
|
* Refactoring a bit, cleanup for 0.8David Arthur2013-04-021-131/+136
| | | | Marking some stuff as not compatible for 0.8 (will be added in 0.8.1)
* Big code re-orgDavid Arthur2013-04-021-641/+4
|
* Some work on a simple consumerDavid Arthur2013-04-021-4/+32
|
* Started on a simple producer and consumerDavid Arthur2013-04-021-6/+48
|
* Removing __main__ stuff from client.pyDavid Arthur2013-04-021-66/+0
|
* Integration tests passingDavid Arthur2013-04-021-498/+741
|
* Add a Queue-like producer/consumerDavid Arthur2012-11-191-0/+3
| | | | | | | | | | | | | | | | | Creates a producer process and one consumer process per partition. Uses `multiprocessing.Queue` for communication between the parent process and the producer/consumers. ```python kafka = KafkaClient("localhost", 9092) q = KafkaQueue(kafka, client="test-queue", partitions=[0,1]) q.put("test") q.get() q.close() kafka.close() ``` Ref #8
* Add Snappy support0.1-alphaDavid Arthur2012-11-161-1/+23
| | | | Fixes #2
* Clean up imports in client, fixed #5David Arthur2012-10-301-3/+0
|