forked from redpanda-data/connect
-
Notifications
You must be signed in to change notification settings - Fork 1
/
kafka.go
327 lines (279 loc) · 8.84 KB
/
kafka.go
1
2
3
4
5
6
7
8
9
10
11
12
13
14
15
16
17
18
19
20
21
22
23
24
25
26
27
28
29
30
31
32
33
34
35
36
37
38
39
40
41
42
43
44
45
46
47
48
49
50
51
52
53
54
55
56
57
58
59
60
61
62
63
64
65
66
67
68
69
70
71
72
73
74
75
76
77
78
79
80
81
82
83
84
85
86
87
88
89
90
91
92
93
94
95
96
97
98
99
100
101
102
103
104
105
106
107
108
109
110
111
112
113
114
115
116
117
118
119
120
121
122
123
124
125
126
127
128
129
130
131
132
133
134
135
136
137
138
139
140
141
142
143
144
145
146
147
148
149
150
151
152
153
154
155
156
157
158
159
160
161
162
163
164
165
166
167
168
169
170
171
172
173
174
175
176
177
178
179
180
181
182
183
184
185
186
187
188
189
190
191
192
193
194
195
196
197
198
199
200
201
202
203
204
205
206
207
208
209
210
211
212
213
214
215
216
217
218
219
220
221
222
223
224
225
226
227
228
229
230
231
232
233
234
235
236
237
238
239
240
241
242
243
244
245
246
247
248
249
250
251
252
253
254
255
256
257
258
259
260
261
262
263
264
265
266
267
268
269
270
271
272
273
274
275
276
277
278
279
280
281
282
283
284
285
286
287
288
289
290
291
292
293
294
295
296
297
298
299
300
301
302
303
304
305
306
307
308
309
310
311
312
313
314
315
316
317
318
319
320
321
322
323
324
325
326
327
// Copyright (c) 2014 Ashley Jeffs
//
// Permission is hereby granted, free of charge, to any person obtaining a copy
// of this software and associated documentation files (the "Software"), to deal
// in the Software without restriction, including without limitation the rights
// to use, copy, modify, merge, publish, distribute, sublicense, and/or sell
// copies of the Software, and to permit persons to whom the Software is
// furnished to do so, subject to the following conditions:
//
// The above copyright notice and this permission notice shall be included in
// all copies or substantial portions of the Software.
//
// THE SOFTWARE IS PROVIDED "AS IS", WITHOUT WARRANTY OF ANY KIND, EXPRESS OR
// IMPLIED, INCLUDING BUT NOT LIMITED TO THE WARRANTIES OF MERCHANTABILITY,
// FITNESS FOR A PARTICULAR PURPOSE AND NONINFRINGEMENT. IN NO EVENT SHALL THE
// AUTHORS OR COPYRIGHT HOLDERS BE LIABLE FOR ANY CLAIM, DAMAGES OR OTHER
// LIABILITY, WHETHER IN AN ACTION OF CONTRACT, TORT OR OTHERWISE, ARISING FROM,
// OUT OF OR IN CONNECTION WITH THE SOFTWARE OR THE USE OR OTHER DEALINGS IN
// THE SOFTWARE.
package input
import (
"sync/atomic"
"time"
"github.com/Shopify/sarama"
"github.com/jeffail/benthos/lib/types"
"github.com/jeffail/benthos/lib/util/service/log"
"github.com/jeffail/benthos/lib/util/service/metrics"
)
//------------------------------------------------------------------------------
func init() {
constructors["kafka"] = typeSpec{
constructor: NewKafka,
description: `
Connects to a kafka (0.8+) server. Offsets are managed within kafka as per the
consumer group (set via config). Only one partition per input is supported, if
you wish to balance partitions across a consumer group look at the
'kafka_balanced' input type instead.`,
}
}
//------------------------------------------------------------------------------
// KafkaConfig is configuration for the Kafka input type.
type KafkaConfig struct {
Addresses []string `json:"addresses" yaml:"addresses"`
ClientID string `json:"client_id" yaml:"client_id"`
ConsumerGroup string `json:"consumer_group" yaml:"consumer_group"`
Topic string `json:"topic" yaml:"topic"`
Partition int32 `json:"partition" yam:"partition"`
StartFromOldest bool `json:"start_from_oldest" yaml:"start_from_oldest"`
}
// NewKafkaConfig creates a new KafkaConfig with default values.
func NewKafkaConfig() KafkaConfig {
return KafkaConfig{
Addresses: []string{"localhost:9092"},
ClientID: "benthos_kafka_input",
ConsumerGroup: "benthos_consumer_group",
Topic: "benthos_stream",
Partition: 0,
StartFromOldest: true,
}
}
//------------------------------------------------------------------------------
// Kafka is an input type that reads from a Kafka instance.
type Kafka struct {
running int32
client sarama.Client
coordinator *sarama.Broker
consumer sarama.Consumer
topicConsumer sarama.PartitionConsumer
offset int64
conf Config
stats metrics.Type
log log.Modular
messages chan types.Message
responses <-chan types.Response
closeChan chan struct{}
closedChan chan struct{}
}
// NewKafka creates a new Kafka input type.
func NewKafka(conf Config, log log.Modular, stats metrics.Type) (Type, error) {
k := Kafka{
running: 1,
offset: 0,
conf: conf,
stats: stats,
log: log.NewModule(".input.kafka"),
messages: make(chan types.Message),
responses: nil,
closeChan: make(chan struct{}),
closedChan: make(chan struct{}),
}
config := sarama.NewConfig()
config.ClientID = conf.Kafka.ClientID
config.Net.DialTimeout = time.Second
config.Consumer.Return.Errors = true
var err error
defer func() {
if err != nil {
k.closeClients()
}
}()
k.client, err = sarama.NewClient(conf.Kafka.Addresses, config)
if err != nil {
return nil, err
}
k.coordinator, err = k.client.Coordinator(conf.Kafka.ConsumerGroup)
if err != nil {
return nil, err
}
k.consumer, err = sarama.NewConsumerFromClient(k.client)
if err != nil {
return nil, err
}
offsetReq := sarama.OffsetFetchRequest{}
offsetReq.ConsumerGroup = k.conf.Kafka.ConsumerGroup
offsetReq.AddPartition(k.conf.Kafka.Topic, k.conf.Kafka.Partition)
if offsetRes, err := k.coordinator.FetchOffset(&offsetReq); err == nil {
offsetBlock := offsetRes.Blocks[k.conf.Kafka.Topic][k.conf.Kafka.Partition]
if offsetBlock.Err == sarama.ErrNoError {
k.offset = offsetBlock.Offset
}
}
k.topicConsumer, err = k.consumer.ConsumePartition(
k.conf.Kafka.Topic, k.conf.Kafka.Partition, k.offset,
)
if err != nil {
offsetTarget := sarama.OffsetOldest
if !k.conf.Kafka.StartFromOldest {
offsetTarget = sarama.OffsetNewest
k.log.Warnln("Failed to read from stored offset, restarting from newest offset")
} else {
k.log.Warnln("Failed to read from stored offset, restarting from oldest offset")
}
k.log.Warnf(
"Attempting to obtain offset for topic %s, partition %v\n",
k.conf.Kafka.Topic, k.conf.Kafka.Partition,
)
// Get the new offset target
if k.offset, err = k.client.GetOffset(
k.conf.Kafka.Topic, k.conf.Kafka.Partition, offsetTarget,
); err == nil {
k.topicConsumer, err = k.consumer.ConsumePartition(
k.conf.Kafka.Topic, k.conf.Kafka.Partition, k.offset,
)
}
}
return &k, err
}
//------------------------------------------------------------------------------
func (k *Kafka) commitOffset() error {
commitReq := sarama.OffsetCommitRequest{}
commitReq.ConsumerGroup = k.conf.Kafka.ConsumerGroup
commitReq.AddBlock(k.conf.Kafka.Topic, k.conf.Kafka.Partition, k.offset, 0, "")
commitRes, err := k.coordinator.CommitOffset(&commitReq)
if err != nil {
return err
}
err = commitRes.Errors[k.conf.Kafka.Topic][k.conf.Kafka.Partition]
if err != sarama.ErrNoError {
return err
}
return nil
}
//------------------------------------------------------------------------------
// drainConsumer should be called after closeClients.
func (k *Kafka) drainConsumer() {
if k.topicConsumer != nil {
// Drain both channels
for range k.topicConsumer.Messages() {
}
for range k.topicConsumer.Errors() {
}
k.topicConsumer = nil
}
}
// closeClients closes the kafka clients, this interrupts loop() out of the read
// block.
func (k *Kafka) closeClients() {
if k.topicConsumer != nil {
// NOTE: Needs draining before destroying.
k.topicConsumer.AsyncClose()
}
if k.coordinator != nil {
k.coordinator.Close()
k.coordinator = nil
}
if k.client != nil {
k.client.Close()
k.client = nil
}
}
//------------------------------------------------------------------------------
func (k *Kafka) errorLoop() {
for atomic.LoadInt32(&k.running) == 1 {
if err := <-k.topicConsumer.Errors(); err != nil {
k.log.Errorf("Kafka message recv error: %v\n", err)
k.stats.Incr("input.kafka.recv.error", 1)
}
}
}
func (k *Kafka) loop() {
k.log.Infof("Receiving Kafka messages from addresses: %s\n", k.conf.Kafka.Addresses)
defer func() {
if atomic.CompareAndSwapInt32(&k.running, 1, 0) {
k.closeClients()
}
k.drainConsumer()
close(k.messages)
close(k.closedChan)
}()
var data *sarama.ConsumerMessage
for atomic.LoadInt32(&k.running) == 1 {
// If no bytes then read a message
if data == nil {
var open bool
if data, open = <-k.topicConsumer.Messages(); !open {
return
}
k.stats.Incr("input.kafka.count", 1)
}
// If bytes are read then try and propagate.
if data != nil {
start := time.Now()
select {
case k.messages <- types.Message{Parts: [][]byte{data.Value}}:
case <-k.closeChan:
return
}
res, open := <-k.responses
if !open {
return
}
if resErr := res.Error(); resErr == nil {
k.stats.Timing("input.kafka.timing", int64(time.Since(start)))
k.offset = data.Offset + 1
if !res.SkipAck() {
if err := k.commitOffset(); err != nil {
k.log.Errorf("Failed to commit offset: %v\n", err)
}
}
k.stats.Incr("input.kafka.send.success", 1)
data = nil
} else {
k.stats.Incr("input.kafka.send.error", 1)
}
}
}
}
// StartListening sets the channel used by the input to validate message
// receipt.
func (k *Kafka) StartListening(responses <-chan types.Response) error {
if k.responses != nil {
return types.ErrAlreadyStarted
}
k.responses = responses
go k.loop()
go k.errorLoop()
return nil
}
// MessageChan returns the messages channel.
func (k *Kafka) MessageChan() <-chan types.Message {
return k.messages
}
// CloseAsync shuts down the Kafka input and stops processing requests.
func (k *Kafka) CloseAsync() {
if atomic.CompareAndSwapInt32(&k.running, 1, 0) {
close(k.closeChan)
k.closeClients()
}
}
// WaitForClose blocks until the Kafka input has closed down.
func (k *Kafka) WaitForClose(timeout time.Duration) error {
select {
case <-k.closedChan:
case <-time.After(timeout):
return types.ErrTimeout
}
return nil
}
//------------------------------------------------------------------------------