summaryrefslogtreecommitdiff
path: root/kafka
diff options
context:
space:
mode:
Diffstat (limited to 'kafka')
-rw-r--r--kafka/conn.py12
-rw-r--r--kafka/protocol/admin.py20
-rw-r--r--kafka/protocol/fetch.py182
-rw-r--r--kafka/protocol/offset.py89
4 files changed, 298 insertions, 5 deletions
diff --git a/kafka/conn.py b/kafka/conn.py
index c383123..5c72875 100644
--- a/kafka/conn.py
+++ b/kafka/conn.py
@@ -24,9 +24,12 @@ import kafka.errors as Errors
from kafka.future import Future
from kafka.metrics.stats import Avg, Count, Max, Rate
from kafka.oauth.abstract import AbstractTokenProvider
-from kafka.protocol.admin import SaslHandShakeRequest
+from kafka.protocol.admin import SaslHandShakeRequest, DescribeAclsRequest_v2
from kafka.protocol.commit import OffsetFetchRequest
+from kafka.protocol.offset import OffsetRequest
+from kafka.protocol.produce import ProduceRequest
from kafka.protocol.metadata import MetadataRequest
+from kafka.protocol.fetch import FetchRequest
from kafka.protocol.parser import KafkaProtocol
from kafka.protocol.types import Int32, Int8
from kafka.scram import ScramClient
@@ -1166,6 +1169,13 @@ class BrokerConnection(object):
# in reverse order. As soon as we find one that works, return it
test_cases = [
# format (<broker version>, <needed struct>)
+ ((2, 5, 0), DescribeAclsRequest_v2),
+ ((2, 4, 0), ProduceRequest[8]),
+ ((2, 3, 0), FetchRequest[11]),
+ ((2, 2, 0), OffsetRequest[5]),
+ ((2, 1, 0), FetchRequest[10]),
+ ((2, 0, 0), FetchRequest[8]),
+ ((1, 1, 0), FetchRequest[7]),
((1, 0, 0), MetadataRequest[5]),
((0, 11, 0), MetadataRequest[4]),
((0, 10, 2), OffsetFetchRequest[2]),
diff --git a/kafka/protocol/admin.py b/kafka/protocol/admin.py
index b2694dc..af88ea4 100644
--- a/kafka/protocol/admin.py
+++ b/kafka/protocol/admin.py
@@ -477,6 +477,13 @@ class DescribeAclsResponse_v1(Response):
('permission_type', Int8)))))
)
+
+class DescribeAclsResponse_v2(Response):
+ API_KEY = 29
+ API_VERSION = 2
+ SCHEMA = DescribeAclsResponse_v1.SCHEMA
+
+
class DescribeAclsRequest_v0(Request):
API_KEY = 29
API_VERSION = 0
@@ -490,6 +497,7 @@ class DescribeAclsRequest_v0(Request):
('permission_type', Int8)
)
+
class DescribeAclsRequest_v1(Request):
API_KEY = 29
API_VERSION = 1
@@ -504,6 +512,17 @@ class DescribeAclsRequest_v1(Request):
('permission_type', Int8)
)
+
+class DescribeAclsRequest_v2(Request):
+ """
+ Enable flexible version
+ """
+ API_KEY = 29
+ API_VERSION = 2
+ RESPONSE_TYPE = DescribeAclsResponse_v2
+ SCHEMA = DescribeAclsRequest_v1.SCHEMA
+
+
DescribeAclsRequest = [DescribeAclsRequest_v0, DescribeAclsRequest_v1]
DescribeAclsResponse = [DescribeAclsResponse_v0, DescribeAclsResponse_v1]
@@ -862,3 +881,4 @@ CreatePartitionsRequest = [
CreatePartitionsResponse = [
CreatePartitionsResponse_v0, CreatePartitionsResponse_v1,
]
+
diff --git a/kafka/protocol/fetch.py b/kafka/protocol/fetch.py
index dd3f648..f367848 100644
--- a/kafka/protocol/fetch.py
+++ b/kafka/protocol/fetch.py
@@ -94,6 +94,72 @@ class FetchResponse_v6(Response):
SCHEMA = FetchResponse_v5.SCHEMA
+class FetchResponse_v7(Response):
+ """
+ Add error_code and session_id to response
+ """
+ API_KEY = 1
+ API_VERSION = 7
+ SCHEMA = Schema(
+ ('throttle_time_ms', Int32),
+ ('error_code', Int16),
+ ('session_id', Int32),
+ ('topics', Array(
+ ('topics', String('utf-8')),
+ ('partitions', Array(
+ ('partition', Int32),
+ ('error_code', Int16),
+ ('highwater_offset', Int64),
+ ('last_stable_offset', Int64),
+ ('log_start_offset', Int64),
+ ('aborted_transactions', Array(
+ ('producer_id', Int64),
+ ('first_offset', Int64))),
+ ('message_set', Bytes)))))
+ )
+
+
+class FetchResponse_v8(Response):
+ API_KEY = 1
+ API_VERSION = 8
+ SCHEMA = FetchResponse_v7.SCHEMA
+
+
+class FetchResponse_v9(Response):
+ API_KEY = 1
+ API_VERSION = 9
+ SCHEMA = FetchResponse_v7.SCHEMA
+
+
+class FetchResponse_v10(Response):
+ API_KEY = 1
+ API_VERSION = 10
+ SCHEMA = FetchResponse_v7.SCHEMA
+
+
+class FetchResponse_v11(Response):
+ API_KEY = 1
+ API_VERSION = 11
+ SCHEMA = Schema(
+ ('throttle_time_ms', Int32),
+ ('error_code', Int16),
+ ('session_id', Int32),
+ ('topics', Array(
+ ('topics', String('utf-8')),
+ ('partitions', Array(
+ ('partition', Int32),
+ ('error_code', Int16),
+ ('highwater_offset', Int64),
+ ('last_stable_offset', Int64),
+ ('log_start_offset', Int64),
+ ('aborted_transactions', Array(
+ ('producer_id', Int64),
+ ('first_offset', Int64))),
+ ('preferred_read_replica', Int32),
+ ('message_set', Bytes)))))
+ )
+
+
class FetchRequest_v0(Request):
API_KEY = 1
API_VERSION = 0
@@ -196,13 +262,125 @@ class FetchRequest_v6(Request):
SCHEMA = FetchRequest_v5.SCHEMA
+class FetchRequest_v7(Request):
+ """
+ Add incremental fetch requests
+ """
+ API_KEY = 1
+ API_VERSION = 7
+ RESPONSE_TYPE = FetchResponse_v7
+ SCHEMA = Schema(
+ ('replica_id', Int32),
+ ('max_wait_time', Int32),
+ ('min_bytes', Int32),
+ ('max_bytes', Int32),
+ ('isolation_level', Int8),
+ ('session_id', Int32),
+ ('session_epoch', Int32),
+ ('topics', Array(
+ ('topic', String('utf-8')),
+ ('partitions', Array(
+ ('partition', Int32),
+ ('fetch_offset', Int64),
+ ('log_start_offset', Int64),
+ ('max_bytes', Int32))))),
+ ('forgotten_topics_data', Array(
+ ('topic', String),
+ ('partitions', Array(Int32))
+ )),
+ )
+
+
+class FetchRequest_v8(Request):
+ """
+ bump used to indicate that on quota violation brokers send out responses before throttling.
+ """
+ API_KEY = 1
+ API_VERSION = 8
+ RESPONSE_TYPE = FetchResponse_v8
+ SCHEMA = FetchRequest_v7.SCHEMA
+
+
+class FetchRequest_v9(Request):
+ """
+ adds the current leader epoch (see KIP-320)
+ """
+ API_KEY = 1
+ API_VERSION = 9
+ RESPONSE_TYPE = FetchResponse_v9
+ SCHEMA = Schema(
+ ('replica_id', Int32),
+ ('max_wait_time', Int32),
+ ('min_bytes', Int32),
+ ('max_bytes', Int32),
+ ('isolation_level', Int8),
+ ('session_id', Int32),
+ ('session_epoch', Int32),
+ ('topics', Array(
+ ('topic', String('utf-8')),
+ ('partitions', Array(
+ ('partition', Int32),
+ ('current_leader_epoch', Int32),
+ ('fetch_offset', Int64),
+ ('log_start_offset', Int64),
+ ('max_bytes', Int32))))),
+ ('forgotten_topics_data', Array(
+ ('topic', String),
+ ('partitions', Array(Int32)),
+ )),
+ )
+
+
+class FetchRequest_v10(Request):
+ """
+ bumped up to indicate ZStandard capability. (see KIP-110)
+ """
+ API_KEY = 1
+ API_VERSION = 10
+ RESPONSE_TYPE = FetchResponse_v10
+ SCHEMA = FetchRequest_v9.SCHEMA
+
+
+class FetchRequest_v11(Request):
+ """
+ added rack ID to support read from followers (KIP-392)
+ """
+ API_KEY = 1
+ API_VERSION = 11
+ RESPONSE_TYPE = FetchResponse_v11
+ SCHEMA = Schema(
+ ('replica_id', Int32),
+ ('max_wait_time', Int32),
+ ('min_bytes', Int32),
+ ('max_bytes', Int32),
+ ('isolation_level', Int8),
+ ('session_id', Int32),
+ ('session_epoch', Int32),
+ ('topics', Array(
+ ('topic', String('utf-8')),
+ ('partitions', Array(
+ ('partition', Int32),
+ ('current_leader_epoch', Int32),
+ ('fetch_offset', Int64),
+ ('log_start_offset', Int64),
+ ('max_bytes', Int32))))),
+ ('forgotten_topics_data', Array(
+ ('topic', String),
+ ('partitions', Array(Int32))
+ )),
+ ('rack_id', String('utf-8')),
+ )
+
+
FetchRequest = [
FetchRequest_v0, FetchRequest_v1, FetchRequest_v2,
FetchRequest_v3, FetchRequest_v4, FetchRequest_v5,
- FetchRequest_v6
+ FetchRequest_v6, FetchRequest_v7, FetchRequest_v8,
+ FetchRequest_v9, FetchRequest_v10, FetchRequest_v11,
]
FetchResponse = [
FetchResponse_v0, FetchResponse_v1, FetchResponse_v2,
FetchResponse_v3, FetchResponse_v4, FetchResponse_v5,
- FetchResponse_v6
+ FetchResponse_v6, FetchResponse_v7, FetchResponse_v8,
+ FetchResponse_v9, FetchResponse_v10, FetchResponse_v11,
]
diff --git a/kafka/protocol/offset.py b/kafka/protocol/offset.py
index 3c254de..1ed382b 100644
--- a/kafka/protocol/offset.py
+++ b/kafka/protocol/offset.py
@@ -53,6 +53,43 @@ class OffsetResponse_v2(Response):
)
+class OffsetResponse_v3(Response):
+ """
+ on quota violation, brokers send out responses before throttling
+ """
+ API_KEY = 2
+ API_VERSION = 3
+ SCHEMA = OffsetResponse_v2.SCHEMA
+
+
+class OffsetResponse_v4(Response):
+ """
+ Add leader_epoch to response
+ """
+ API_KEY = 2
+ API_VERSION = 4
+ SCHEMA = Schema(
+ ('throttle_time_ms', Int32),
+ ('topics', Array(
+ ('topic', String('utf-8')),
+ ('partitions', Array(
+ ('partition', Int32),
+ ('error_code', Int16),
+ ('timestamp', Int64),
+ ('offset', Int64),
+ ('leader_epoch', Int32)))))
+ )
+
+
+class OffsetResponse_v5(Response):
+ """
+ adds a new error code, OFFSET_NOT_AVAILABLE
+ """
+ API_KEY = 2
+ API_VERSION = 5
+ SCHEMA = OffsetResponse_v4.SCHEMA
+
+
class OffsetRequest_v0(Request):
API_KEY = 2
API_VERSION = 0
@@ -105,5 +142,53 @@ class OffsetRequest_v2(Request):
}
-OffsetRequest = [OffsetRequest_v0, OffsetRequest_v1, OffsetRequest_v2]
-OffsetResponse = [OffsetResponse_v0, OffsetResponse_v1, OffsetResponse_v2]
+class OffsetRequest_v3(Request):
+ API_KEY = 2
+ API_VERSION = 3
+ RESPONSE_TYPE = OffsetResponse_v3
+ SCHEMA = OffsetRequest_v2.SCHEMA
+ DEFAULTS = {
+ 'replica_id': -1
+ }
+
+
+class OffsetRequest_v4(Request):
+ """
+ Add current_leader_epoch to request
+ """
+ API_KEY = 2
+ API_VERSION = 4
+ RESPONSE_TYPE = OffsetResponse_v4
+ SCHEMA = Schema(
+ ('replica_id', Int32),
+ ('isolation_level', Int8), # <- added isolation_level
+ ('topics', Array(
+ ('topic', String('utf-8')),
+ ('partitions', Array(
+ ('partition', Int32),
+ ('current_leader_epoch', Int64),
+ ('timestamp', Int64)))))
+ )
+ DEFAULTS = {
+ 'replica_id': -1
+ }
+
+
+class OffsetRequest_v5(Request):
+ API_KEY = 2
+ API_VERSION = 5
+ RESPONSE_TYPE = OffsetResponse_v5
+ SCHEMA = OffsetRequest_v4.SCHEMA
+ DEFAULTS = {
+ 'replica_id': -1
+ }
+
+
+OffsetRequest = [
+ OffsetRequest_v0, OffsetRequest_v1, OffsetRequest_v2,
+ OffsetRequest_v3, OffsetRequest_v4, OffsetRequest_v5,
+]
+OffsetResponse = [
+ OffsetResponse_v0, OffsetResponse_v1, OffsetResponse_v2,
+ OffsetResponse_v3, OffsetResponse_v4, OffsetResponse_v5,
+]