summaryrefslogtreecommitdiff
path: root/kafka/admin
Commit message (Collapse)AuthorAgeFilesLines
* Allow disabling thread wakeup in send_request_to_node (#2335)HEADmasterAndrew Zhang2023-03-021-2/+3
|
* fix typo in documentation (#2178)Nikolay Vasiliev2021-02-201-2/+2
|
* Support configuration of custom kafka client for Admin/Consumer/Producer (#2144)Tincu Gabriel2020-12-021-3/+7
|
* Merge _find_coordinator_id methods (#2127)Jeff Widman2020-09-171-42/+27
| | | | | | | | | | | | | | | | Previously there were two methods: * `_find_coordinator_id()` * `_find_many_coordinator_ids()` But they do basically the same thing internally. And we need the plural two places, but the singular only one place. So merge them, and change the function signature to take a list of `group_ids` and return a dict of `group_id: coordinator_id`s. As a result of this, the `describe_groups()` command should scale better because the `_find_coordinator_ids()` command issues all the requests async, instead of sequentially blocking as the `described_groups()` used to do.
* Lint cleanup (#2126)Jeff Widman2020-09-171-4/+3
| | | Small cleanup leftover from https://github.com/dpkp/kafka-python/pull/2035
* Feature: delete consumergroups (#2040)Swen Wenzel2020-09-171-4/+89
| | | | | * Add consumergroup related errors * Add DeleteGroups to protocol.admin * Implement delete_groups feature on KafkaAdminClient
* Allow configurable timeouts in admin client check versionHimanshu Mishra2020-09-161-2/+2
| | | Currently there's no way to pass timeout to check_version if called from admin.
* Enhancement for Kafka Admin Client's "Describe Consumer Group" (#2035)Apurva0072020-09-161-14/+41
| | | | | | Adding namedtuples for DescribeConsumerGroup response; Adding Serialization of MemberData and MemberAssignment for the response Co-authored-by: Apurva Telang <atelang@paypal.com> Co-authored-by: Jeff Widman <jeff@jeffwidman.com>
* KAFKA-8962: Use least_loaded_node() for describe_topics()Jeff Widman2020-02-161-15/+7
| | | | | | | | | | | | | | | | | | | | | | | In KAFKA-8962 the `AdminClient.describe_topics()` call was changed from using the controller to using the `least_loaded_node()`: https://github.com/apache/kafka/commit/317089663cc7ff4fdfcba6ee434f455e8ae13acd#diff-6869b8fccf6b098cbcb0676e8ceb26a7R1540 As a result, no metadata request/response processing needs to happen through the controller, so it's safe to remove the custom error-checking. Besides, I don't think this error-checking even added any value because AFAIK no metadata response would return a `NotControllerError` because the recipient broker wouldn't realize the metadata request was intended for only the controller. Originally our admin client was implemented using the least-loaded-node, then later updated to the controller. So updating it back to least-loaded node is a simple case of reverting the associated commits. This reverts commit 7195f0369c7dbe25aea2c3fed78d2b4f772d775b. This reverts commit 6e2978edee9a06e9dbe60afcac226b27b83cbc74. This reverts commit f92889af79db08ef26d89cb18bd48c7dd5080010.
* Fix topic error parsing in MetadataResponseJeff Tribble2020-02-151-6/+11
|
* _send_request_to_controller returns a raw result, not a futureTyler Lubeck2020-02-061-6/+6
|
* Use the controller for topic metadata requestsTyler Lubeck2020-02-061-6/+9
| | | | Closes #1994
* Implement list_topics, describe_topics, and describe_clusterTyler Lubeck2020-02-061-6/+40
|
* Implement __eq__ and __hash__ for ACL objects (#1955)Tyler Lubeck2019-12-291-1/+33
|
* Fixes KafkaAdminClient returning `IncompatibleBrokerVersion` when passing an ↵Ian Bucad2019-12-291-0/+1
| | | | `api_version` (#1953)
* Fix typoDana Powers2019-12-291-1/+1
|
* Admin protocol updates (#1948)Tyler Lubeck2019-12-291-10/+27
|
* Enable SCRAM-SHA-256 and SCRAM-SHA-512 for sasl (#1918)Swen Wenzel2019-12-291-5/+5
|
* Improve docs for reconnect_backoff_max_ms (#1976)Dana Powers2019-12-281-5/+6
|
* Fix typosCarson Ip2019-11-081-1/+1
|
* Fix describe config for multi-broker clusters (#1869)Jeppe Andersen2019-10-111-14/+56
| | | | | | | * Fix describe config for multi-broker clusters Currently all describe config requests are sent to "least loaded node". Requests for broker configs must, however, be sent to the specific broker, otherwise an error is returned. Only topic requests can be handled by any node. This changes the logic to send all describe config requests to the specific broker.
* Fix Admin Client api version checking; only test ACL integration on 0.11+Dana Powers2019-09-281-4/+10
|
* Add ACL api to KafkaAdminClient (#1833)Ulrik Johansson2019-09-283-9/+482
|
* Break FindCoordinator into request/response methodsJeff Widman2019-07-311-32/+48
| | | | | | | | | | | | | | | | | | This splits the `_find_coordinator_id()` method (which is blocking) into request generation / response parsing methods. The public API does not change. However, this allows power users who are willing to deal with risk of private methods changing under their feet to decouple generating the message futures from processing their responses. In other words, you can use these to fire a bunch of requests at once and delay processing the responses until all requests are fired. This is modeled on the work done in #1845. Additionally, I removed the code that tried to leverage the error checking from `cluster.add_group_coordinator()`. That code had changed in #1822, removing most of the error checking... so it no longer adds any value, but instead merely increases complexity and coupling.
* Add the `sasl_kerberos_domain_name` arg to `KafkaAdminClient`Jeff Widman2019-06-281-0/+3
| | | | | | Previously the `sasl_kerberos_domain_name` was missing from the Admin client. It is already present in the Consumer/Producer, and in all three cases gets transparently passed down to the client.
* Update KafkaAdminClient DocsRob Cardy2019-06-211-1/+2
| | | Updated to include SASL_PLAINTEXT and SASL_SSL as options for security_protocol.
* Break consumer operations into request / response methods (#1845)Jeff Widman2019-06-191-94/+155
| | | | | | | | | | This breaks some of the consumer operations into request generation / response parsing methods. The public API does not change. However, this allows power users who are willing to deal with risk of private methods changing under their feet to decouple generating the message futures from processing their responses. In other words, you can use these to fire a bunch of request at once and delay processing the responses until all requests are fired.
* Use futures to parallelize calls to _send_request_to_node() (#1807)Lou-Cipher2019-05-211-34/+75
| | | | | Use `futures` to parallelize calls to `_send_request_to_node()` This allows queries that need to go to multiple brokers to be run in parallel.
* Update sasl configuration docstringsDana Powers2019-03-231-5/+5
|
* Support SASL OAuthBearer Authentication (#1750)Phong Pham2019-03-221-0/+3
|
* Fix `AttributeError` caused by `getattr()`Jeff Widman2019-01-071-1/+2
| | | | | | | `getattr(object, 'x', object.y)` will evaluate the default argument `object.y` regardless of whether `'x'` exists. For details see: https://stackoverflow.com/q/31443989/770425
* Fix response error checking in KafkaAdminClient send_to_controllerDana Powers2019-01-031-2/+12
| | | | | | | | | | | | Previously we weren't accounting for when the response tuple also has a `error_message` value. Note that in Java, the error fieldname is inconsistent: - `CreateTopicsResponse` / `CreatePartitionsResponse` uses `topic_errors` - `DeleteTopicsResponse` uses `topic_error_codes` So this updates the `CreateTopicsResponse` classes to match. The fix is a little brittle, but should suffice for now.
* Bugfix: Types need identity comparisonJeff Widman2018-12-131-1/+1
| | | `isinstance()` won't work here, as the types require identity comparison.
* Rename KafkaAdmin to KafkaAdminClientJeff Widman2018-11-202-18/+18
|
* Bugfix: Always set this_groups_coordinator_idJeff Widman2018-11-191-1/+3
|
* Various docstring / pep8 / code hygiene cleanupsJeff Widman2018-11-181-71/+86
|
* Fix describe_groupsJeff Widman2018-11-181-13/+50
| | | | | | | | | | | | | This was completely broken previously because it didn't lookup the group coordinator of the consumer group. Also added basic error handling/raising. Note: I added the `group_coordinator_id` as an optional kwarg. As best I can tell, the Java client doesn't include this and instead looks it up every time. However, if we add this, it allows the caller the flexibility to bypass the network round trip of the lookup if for some reason they already know the `group_coordinator_id`.
* Set a clear default value for `validate_only`/`include_synonyms`Jeff Widman2018-11-181-8/+8
| | | | | | | Set a clear default value for `validate_only` / `include_synonyms` Previously the kwarg defaulted to `None`, but then sent a `False` so this makes it more explicit and reduces ambiguity.
* Fix list_consumer_groups() to query all brokersJeff Widman2018-11-181-5/+39
| | | | | | | | | | | | | | | | | | | | | | | | | | Previously, this only queried the controller. In actuality, the Kafka protocol requires that the client query all brokers in order to get the full list of consumer groups. Note: The Java code (as best I can tell) doesn't allow limiting this to specific brokers. And on the surface, this makes sense... you typically don't care about specific brokers. However, the inverse is true... consumer groups care about knowing their group coordinator so they don't have to repeatedly query to find it. In fact, a Kafka broker will only return the groups that it's a coordinator for. While this is an implementation detail that is not guaranteed by the upstream broker code, and technically should not be relied upon, I think it very unlikely to change. So monitoring scripts that fetch the offsets or describe the consumers groups of all groups in the cluster can simply issue one call per broker to identify all the coordinators, rather than having to issue one call per consumer group. For an ad-hoc script this doesn't matter, but for a monitoring script that runs every couple of minutes, this can be a big deal. I know in the situations where I will use this, this matters more to me than the risk of the interface unexpectedly breaking.
* Add list_consumer_group_offsets()Jeff Widman2018-11-181-1/+76
| | | | | | | | | | | | | | | | | | | | | Support fetching the offsets of a consumer group. Note: As far as I can tell (the Java code is a little inscrutable), the Java AdminClient doesn't allow specifying the `coordinator_id` or the `partitions`. But I decided to include them because they provide a lot of additional flexibility: 1. allowing users to specify the partitions allows this method to be used even for older brokers that don't support the OffsetFetchRequest_v2 2. allowing users to specify the coordinator ID gives them a way to bypass a network round trip. This method will frequently be used for monitoring, and if you've got 1,000 consumer groups that are being monitored once a minute, that's ~1.5M requests a day that are unnecessarily duplicated as the coordinator doesn't change unless there's an error.
* Fix send to controllerJeff Widman2018-11-181-44/+89
| | | | | | | | | The controller send error handling was completely broken. It also pinned the metadata version unnecessarily. Additionally, several of the methods were sending to the controller but either that was unnecessary, or just plain wrong. So updated following the pattern of the Java Admin client.
* Add group coordinator lookupJeff Widman2018-11-181-0/+40
| | | | | | | | | | | | | | | | | | | | | | | We need a way to send a request to the group coordinator. I spent a day and a half trying to implement a `_send_request_to_group_coordinator()` that included: 1. caching the value of the group coordinator so that it wouldn't have to be repeatedly looked up on every call. This is particularly important because the `list_consumer_groups()`, `list_consumer_group_offsets()`, and `describe_consumer_groups()` will frequently be used by monitoring scripts. I know across the production clusters that I support, using a cached value will save ~1M calls per day. 2. clean and consistent error handling. This is difficult because the responses are inconsistent about error codes. Some have a top-level error code, some bury it within the description of the actual item. 3. Avoiding tight coupling between this method and the request/response classes... the custom parsing logic for errors etc, given that it's non-standard, should live in the callers, not here. So finally I gave up and just went with this simpler solution and made it so the callers can optionally bypass this if they somehow already know the group coordinator.
* Remove support for api versions as strings from KafkaAdminJeff Widman2018-11-181-10/+0
| | | | | This is a new class, so let's not support the old version strings and saddle ourselves with tech debt right from the get-go.
* Be explicit with tuples for %s formattingJeff Widman2018-11-181-1/+1
| | | | Fix #1633
* Stop using broker-errors for client-side problemsJeff Widman2018-11-181-37/+39
| | | | | | | | | | | | | `UnsupportedVersionError` is intended to indicate a server-side error: https://github.com/dpkp/kafka-python/blob/ba7372e44ffa1ee49fb4d5efbd67534393e944db/kafka/errors.py#L375-L378 So we should not be raising it for client-side errors. I realize that semantically this seems like the appropriate error to raise. However, this is confusing when debugging... for a real-life example, see https://github.com/Parsely/pykafka/issues/697. So I strongly feel that server-side errors should be kept separate from client-side errors, even if all the client is doing is proactively protecting against hitting a situation where the broker would return this error.
* Fix typoJeff Widman2018-11-071-1/+1
|
* Document KafkaAdmin classJeff Widman2018-10-291-0/+7
|
* Add KafkaAdmin classRichard Lee2018-10-245-0/+604
Requires cluster version > 0.10.0.0, and uses new wire protocol classes to do many things via broker connection that previously needed to be done directly in zookeeper.