-
Notifications
You must be signed in to change notification settings - Fork 17
Commit
This commit does not belong to any branch on this repository, and may belong to a fork outside of the repository.
Merge pull request #430 from evolution-gaming/df/expose-java-client-m…
…etrics Exposing native metrics of multiple clients in the same JVM
- Loading branch information
Showing
10 changed files
with
363 additions
and
23 deletions.
There are no files selected for viewing
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
66 changes: 66 additions & 0 deletions
66
modules/metrics/src/main/scala/com/evolutiongaming/skafka/consumer/ConsumerMetricsOf.scala
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
Original file line number | Diff line number | Diff line change |
---|---|---|
@@ -0,0 +1,66 @@ | ||
package com.evolutiongaming.skafka.consumer | ||
|
||
import cats.effect.{Resource, Sync} | ||
import cats.effect.std.UUIDGen | ||
import com.evolutiongaming.catshelper.ToTry | ||
import com.evolutiongaming.skafka.{Topic, TopicPartition} | ||
import com.evolutiongaming.skafka.metrics.KafkaMetricsRegistry | ||
import io.prometheus.client.CollectorRegistry | ||
|
||
import scala.concurrent.duration.FiniteDuration | ||
|
||
object ConsumerMetricsOf { | ||
|
||
/** | ||
* Construct [[ConsumerMetrics]] that will expose Java Kafka client metrics. | ||
* | ||
* @param source original [[ConsumerMetrics]] | ||
* @param prometheus instance of Prometheus registry | ||
* @param prefix metric name prefix | ||
* @return [[ConsumerMetrics]] that exposes Java Kafka client metrics | ||
*/ | ||
def withJavaClientMetrics[F[_]: Sync: ToTry: UUIDGen]( | ||
source: ConsumerMetrics[F], | ||
prometheus: CollectorRegistry, | ||
prefix: Option[String], | ||
): Resource[F, ConsumerMetrics[F]] = | ||
for { | ||
registry <- KafkaMetricsRegistry.of(prometheus, prefix) | ||
} yield new ConsumerMetrics[F] { | ||
override def call(name: String, topic: Topic, latency: FiniteDuration, success: Boolean): F[Unit] = | ||
source.call(name, topic, latency, success) | ||
|
||
override def poll(topic: Topic, bytes: Int, records: Int, age: Option[FiniteDuration]): F[Unit] = | ||
source.poll(topic, bytes, records, age) | ||
|
||
override def count(name: String, topic: Topic): F[Unit] = | ||
source.count(name, topic) | ||
|
||
override def rebalance(name: String, topicPartition: TopicPartition): F[Unit] = | ||
source.rebalance(name, topicPartition) | ||
|
||
override def topics(latency: FiniteDuration): F[Unit] = | ||
source.topics(latency) | ||
|
||
override def exposeJavaMetrics[K, V](consumer: Consumer[F, K, V]): Resource[F, Unit] = | ||
registry.register(consumer.clientMetrics) | ||
|
||
} | ||
|
||
implicit final class ConsumerMetricsOps[F[_]](val source: ConsumerMetrics[F]) extends AnyVal { | ||
|
||
/** | ||
* Construct [[ConsumerMetrics]] that will expose Java Kafka client metrics. | ||
* | ||
* @param prometheus instance of Prometheus registry | ||
* @param prefix metric name prefix | ||
* @return [[ConsumerMetrics]] that exposes Java Kafka client metrics | ||
*/ | ||
def exposeJavaClientMetrics( | ||
prometheus: CollectorRegistry, | ||
prefix: Option[String] = None, | ||
)(implicit F: Sync[F], toTry: ToTry[F]): Resource[F, ConsumerMetrics[F]] = | ||
withJavaClientMetrics(source, prometheus, prefix) | ||
|
||
} | ||
} |
85 changes: 85 additions & 0 deletions
85
modules/metrics/src/main/scala/com/evolutiongaming/skafka/metrics/KafkaMetricsRegistry.scala
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
Original file line number | Diff line number | Diff line change |
---|---|---|
@@ -0,0 +1,85 @@ | ||
package com.evolutiongaming.skafka.metrics | ||
|
||
import cats.syntax.all._ | ||
import cats.effect.syntax.resource._ | ||
import cats.effect.{Resource, Ref, Sync} | ||
import cats.effect.std.UUIDGen | ||
import com.evolutiongaming.catshelper.ToTry | ||
import com.evolutiongaming.skafka.ClientMetric | ||
import io.prometheus.client.CollectorRegistry | ||
|
||
import java.util.UUID | ||
|
||
/** | ||
* Allows reporting metrics of multiple Kafka clients inside a single VM. | ||
* | ||
* Example: | ||
* {{{ | ||
* val prometheus: CollectorRegistry = ??? | ||
* val consumerOf: ConsumerOf[F] = ??? | ||
* val producerOf: ProducerOf[F] = ??? | ||
* | ||
* for { | ||
* registry <- KafkaMetricsRegistry.of(prometheus) | ||
* | ||
* consumer <- consumerOf(consumerConfig) | ||
* _ <- registry.register(consumer.clientMetrics) | ||
* | ||
* producer <- producerOf(producerConfig) | ||
* _ <- registry.register(producer.clientMetrics) | ||
* } yield () | ||
* }}} | ||
*/ | ||
trait KafkaMetricsRegistry[F[_]] { | ||
|
||
/** | ||
* Register a function to obtain a list of client metrics. | ||
* Normally, you would pass | ||
* [[com.evolutiongaming.skafka.consumer.Consumer#clientMetrics]] or | ||
* [[com.evolutiongaming.skafka.producer.Producer#clientMetrics]] | ||
*/ | ||
def register(metrics: F[Seq[ClientMetric[F]]]): Resource[F, Unit] | ||
} | ||
|
||
object KafkaMetricsRegistry { | ||
|
||
def of[F[_]: Sync: ToTry: UUIDGen]( | ||
prometheus: CollectorRegistry, | ||
prefix: Option[String] = None, | ||
): Resource[F, KafkaMetricsRegistry[F]] = | ||
for { | ||
sources <- Ref[F].of(Map.empty[UUID, F[Seq[ClientMetric[F]]]]).toResource | ||
|
||
metrics = sources | ||
.get | ||
.flatMap { sources => | ||
sources | ||
.toList | ||
.flatTraverse { | ||
case (uuid, metrics) => | ||
metrics.map { metrics => | ||
metrics.toList.map { metric => metric.copy(tags = metric.tags + ("uuid" -> uuid.toString)) } | ||
} | ||
} | ||
} | ||
.widen[Seq[ClientMetric[F]]] | ||
|
||
collector = new KafkaMetricsCollector[F](metrics, prefix) | ||
allocate = Sync[F].delay { prometheus.register(collector) } | ||
release = Sync[F].delay { prometheus.unregister(collector) } | ||
|
||
_ <- Resource.make(allocate)(_ => release) | ||
} yield new KafkaMetricsRegistry[F] { | ||
|
||
def register(metrics: F[Seq[ClientMetric[F]]]): Resource[F, Unit] = | ||
for { | ||
uuid <- UUIDGen[F].randomUUID.toResource | ||
|
||
allocate = sources.update { sources => sources.updated(uuid, metrics) } | ||
release = sources.update { sources => sources - uuid } | ||
|
||
_ <- Resource.make(allocate)(_ => release) | ||
} yield {} | ||
} | ||
|
||
} |
72 changes: 72 additions & 0 deletions
72
modules/metrics/src/main/scala/com/evolutiongaming/skafka/producer/ProducerMetricsOf.scala
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
Original file line number | Diff line number | Diff line change |
---|---|---|
@@ -0,0 +1,72 @@ | ||
package com.evolutiongaming.skafka.producer | ||
|
||
import cats.effect.{Resource, Sync} | ||
import cats.effect.std.UUIDGen | ||
import com.evolutiongaming.catshelper.ToTry | ||
import com.evolutiongaming.skafka.Topic | ||
import com.evolutiongaming.skafka.metrics.KafkaMetricsRegistry | ||
import io.prometheus.client.CollectorRegistry | ||
|
||
import scala.concurrent.duration.FiniteDuration | ||
|
||
object ProducerMetricsOf { | ||
|
||
/** | ||
* Construct [[ProducerMetrics]] that will expose Java Kafka client metrics. | ||
* | ||
* @param source original [[ProducerMetrics]] | ||
* @param prometheus instance of Prometheus registry | ||
* @param prefix metric name prefix | ||
* @return [[ProducerMetrics]] that exposes Java Kafka client metrics | ||
*/ | ||
def withJavaClientMetrics[F[_]: Sync: ToTry: UUIDGen]( | ||
source: ProducerMetrics[F], | ||
prometheus: CollectorRegistry, | ||
prefix: Option[String], | ||
): Resource[F, ProducerMetrics[F]] = | ||
for { | ||
registry <- KafkaMetricsRegistry.of(prometheus, prefix) | ||
} yield new ProducerMetrics[F] { | ||
override def initTransactions(latency: FiniteDuration): F[Unit] = source.initTransactions(latency) | ||
|
||
override def beginTransaction: F[Unit] = source.beginTransaction | ||
|
||
override def sendOffsetsToTransaction(latency: FiniteDuration): F[Unit] = source.sendOffsetsToTransaction(latency) | ||
|
||
override def commitTransaction(latency: FiniteDuration): F[Unit] = source.commitTransaction(latency) | ||
|
||
override def abortTransaction(latency: FiniteDuration): F[Unit] = source.abortTransaction(latency) | ||
|
||
override def send(topic: Topic, latency: FiniteDuration, bytes: Int): F[Unit] = source.send(topic, latency, bytes) | ||
|
||
override def block(topic: Topic, latency: FiniteDuration): F[Unit] = source.block(topic, latency) | ||
|
||
override def failure(topic: Topic, latency: FiniteDuration): F[Unit] = source.failure(topic, latency) | ||
|
||
override def partitions(topic: Topic, latency: FiniteDuration): F[Unit] = source.partitions(topic, latency) | ||
|
||
override def flush(latency: FiniteDuration): F[Unit] = source.flush(latency) | ||
|
||
override def exposeJavaMetrics(producer: Producer[F]): Resource[F, Unit] = | ||
registry.register(producer.clientMetrics) | ||
|
||
} | ||
|
||
implicit final class ProducerMetricsOps[F[_]](val source: ProducerMetrics[F]) extends AnyVal { | ||
|
||
/** | ||
* Construct [[ProducerMetrics]] that will expose Java Kafka client metrics. | ||
* | ||
* @param prometheus instance of Prometheus registry | ||
* @param prefix metric name prefix | ||
* @return [[ProducerMetrics]] that exposes Java Kafka client metrics | ||
*/ | ||
def exposeJavaClientMetrics( | ||
prometheus: CollectorRegistry, | ||
prefix: Option[String], | ||
)(implicit F: Sync[F], toTry: ToTry[F]): Resource[F, ProducerMetrics[F]] = | ||
withJavaClientMetrics(source, prometheus, prefix) | ||
|
||
} | ||
|
||
} |
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
This file contains bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
Oops, something went wrong.