@@ -58,6 +58,11 @@ type AuthConfig struct {
5858 // certificate authority file for TLS client authentication
5959 CAFile string `mapstructure:"ca_file"`
6060 } `mapstructure:"tls"`
61+
62+ SASL struct {
63+ Enabled bool `mapstructure:"enabled"`
64+ Mechanism string `mapstructure:"mechanism"`
65+ }
6166}
6267
6368var sampleConfig = `broker: "localhost:9092"`
@@ -74,7 +79,7 @@ var info = plugins.Info{
7479type Extractor struct {
7580 plugins.BaseExtractor
7681 // internal states
77- conn * kafka. Conn
82+ conn sarama. Consumer
7883 logger log.Logger
7984 config Config
8085 clientDurn metric.Int64Histogram
@@ -104,69 +109,91 @@ func (e *Extractor) Init(ctx context.Context, config plugins.Config) error {
104109 return err
105110 }
106111
107- // create default dialer
108- dialer := & kafka.Dialer {
109- Timeout : 10 * time .Second ,
110- DualStack : true ,
111- }
112+ consumerConfig := sarama .NewConfig ()
112113
113114 if e .config .Auth .TLS .Enabled {
114115 tlsConfig , err := e .createTLSConfig ()
115116 if err != nil {
116117 return fmt .Errorf ("create tls config: %w" , err )
117118 }
118-
119- dialer .TLS = tlsConfig
119+ consumerConfig .Net .TLS .Enable = true
120+ consumerConfig .Net .TLS .Config = tlsConfig
121+
122+ if e .config .Auth .SASL .Enabled {
123+ consumerConfig .Net .SASL .Enable = true
124+ if e .config .Auth .SASL .Mechanism == sarama .SASLTypeOAuth {
125+ consumerConfig .Net .SASL .Mechanism = sarama .SASLTypeOAuth
126+ consumerConfig .Net .SASL .TokenProvider = NewKubernetesTokenProvider ()
127+ }
120128 }
121129
122- // create connection
123- e .conn , err = dialer .DialContext (ctx , "tcp" , e .config .Broker )
130+ consumer , err := sarama .NewConsumer ([]string {e .config .Broker }, consumerConfig )
124131 if err != nil {
125- return fmt .Errorf ("create connection: %w" , err )
132+ fmt .Printf ("Error is here !! %s" , err .Error ())
133+ return fmt .Errorf ("failed to create kafka consumer for brokers %s and config %+v. Error %s" , e .config .Broker ,
134+ consumerConfig , err .Error ())
126135 }
127-
136+ e . conn = consumer
128137 return nil
129138}
130139
131140// Extract checks if the extractor is ready to extract
132141// if so, then extracts metadata from the kafka broker
133- func (e * Extractor ) Extract (ctx context.Context , emit plugins.Emit ) error {
142+ func (e * Extractor ) Extract (ctx context .Context , emit plugins .Emit ) ( err error ) {
134143 defer e .conn .Close ()
135144
136- partitions , err := e .readPartitions (ctx )
137- if err != nil {
138- return fmt .Errorf ("fetch partitions: %w" , err )
139- }
145+ defer func (start time.Time ) {
146+ attributes := []attribute.KeyValue {
147+ attribute .String ("kafka.broker" , e .config .Broker ),
148+ attribute .Bool ("success" , err == nil ),
149+ }
150+ if err != nil {
151+ errorCode := "UNKNOWN"
152+ var kErr kafka.Error
153+ if errors .As (err , & kErr ) {
154+ errorCode = strings .ReplaceAll (
155+ strings .ToUpper (kErr .Title ()), " " , "_" ,
156+ )
157+ }
158+ attributes = append (attributes , attribute .String ("kafka.error_code" , errorCode ))
159+ }
140160
141- // collect topic list from partition list
142- topics := map [string ]int {}
143- for _ , p := range partitions {
144- topics [p .Topic ]++
161+ e .clientDurn .Record (
162+ ctx , time .Since (start ).Milliseconds (), metric .WithAttributes (attributes ... ),
163+ )
164+ }(time .Now ())
165+ topics , err := e .conn .Topics ()
166+ if err != nil {
167+ return fmt .Errorf ("fetch topics: %w" , err )
145168 }
146169
147170 // build and push topics
148- for topic , numOfPartitions := range topics {
171+ for _ , topic := range topics {
149172 // skip if topic is a default topic
150173 _ , isDefaultTopic := defaultTopics [topic ]
151174 if isDefaultTopic {
152175 continue
153176 }
154177
155- asset , err := e .buildAsset (topic , numOfPartitions )
178+ partitions , err := e .conn .Partitions (topic )
179+ if err != nil {
180+ e .logger .Error ("failed to fetch partitions for topic" , "err" , err , "topic" , topic )
181+ continue
182+ }
183+ asset , err := e .buildAsset (topic , len (partitions ))
156184 if err != nil {
157185 e .logger .Error ("failed to build asset" , "err" , err , "topic" , topic )
158186 continue
159187 }
160188 emit (models .NewRecord (asset ))
161189 }
162-
163190 return nil
164191}
165192
166193func (e * Extractor ) createTLSConfig ( ) (* tls .Config , error ) {
167194 authConfig := e .config .Auth .TLS
168195
169- if authConfig .CertFile == "" || authConfig . KeyFile == "" || authConfig . CAFile == "" {
196+ if authConfig .CAFile == "" {
170197 //nolint:gosec
171198 return & tls.Config {
172199 InsecureSkipVerify : e .config .Auth .TLS .InsecureSkipVerify ,
@@ -178,9 +205,13 @@ func (e *Extractor) createTLSConfig() (*tls.Config, error) {
178205 return nil , fmt .Errorf ("create cert: %w" , err )
179206 }
180207
181- caCert , err := os .ReadFile (authConfig .CAFile )
182- if err != nil {
183- return nil , fmt .Errorf ("read ca cert file: %w" , err )
208+ var cert tls.Certificate
209+ var err error
210+ if authConfig .CertFile != "" && authConfig .KeyFile != "" {
211+ cert , err = tls .LoadX509KeyPair (authConfig .CertFile , authConfig .KeyFile )
212+ if err != nil {
213+ return nil , fmt .Errorf ("create cert: %w" , err )
214+ }
184215 }
185216
186217 caCertPool := x509 .NewCertPool ()
@@ -215,31 +246,6 @@ func (e *Extractor) buildAsset(topicName string, numOfPartitions int) (*v1beta2.
215246 }, nil
216247}
217248
218- func (e * Extractor ) readPartitions (ctx context.Context ) (partitions []kafka.Partition , err error ) {
219- defer func (start time.Time ) {
220- attributes := []attribute.KeyValue {
221- attribute .String ("kafka.broker" , e .config .Broker ),
222- attribute .Bool ("success" , err == nil ),
223- }
224- if err != nil {
225- errorCode := "UNKNOWN"
226- var kErr kafka.Error
227- if errors .As (err , & kErr ) {
228- errorCode = strings .ReplaceAll (
229- strings .ToUpper (kErr .Title ()), " " , "_" ,
230- )
231- }
232- attributes = append (attributes , attribute .String ("kafka.error_code" , errorCode ))
233- }
234-
235- e .clientDurn .Record (
236- ctx , time .Since (start ).Milliseconds (), metric .WithAttributes (attributes ... ),
237- )
238- }(time .Now ())
239-
240- return e .conn .ReadPartitions ()
241- }
242-
243249func init () {
244250 if err := registry .Extractors .Register ("kafka" , func () plugins.Extractor {
245251 return New (plugins .GetLog ())
0 commit comments