-
Notifications
You must be signed in to change notification settings - Fork 121
Commit
This commit does not belong to any branch on this repository, and may belong to a fork outside of the repository.
feat: enable workers count calculation in runtime
fix: change dependency injection lifetime management for worker and consumer/producer fix: create AdminClient without authentication perf: memory optimizations on MessageContext
- Loading branch information
1 parent
3561fe6
commit 8578112
Showing
56 changed files
with
3,035 additions
and
11,283 deletions.
There are no files selected for viewing
131 changes: 131 additions & 0 deletions
131
samples/KafkaFlow.Sample.Dashboard/ConsumerLagWorkerBalancer.cs
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,131 @@ | ||
namespace KafkaFlow.Sample.Dashboard; | ||
|
||
using System; | ||
using System.Collections.Generic; | ||
using System.Linq; | ||
using System.Threading.Tasks; | ||
using Confluent.Kafka; | ||
using KafkaFlow.Clusters; | ||
using KafkaFlow.Configuration; | ||
using KafkaFlow.Consumers; | ||
using TopicMetadata = KafkaFlow.TopicMetadata; | ||
using TopicPartitionOffset = KafkaFlow.TopicPartitionOffset; | ||
|
||
public class ConsumerLagWorkerBalancer | ||
{ | ||
private readonly IClusterManager clusterManager; | ||
private readonly IConsumerAccessor consumerAccessor; | ||
private readonly int totalConsumerWorkers; | ||
private readonly int maxInstanceWorkers; | ||
|
||
public ConsumerLagWorkerBalancer( | ||
IClusterManager clusterManager, | ||
IConsumerAccessor consumerAccessor, | ||
int totalConsumerWorkers, | ||
int maxInstanceWorkers) | ||
{ | ||
this.clusterManager = clusterManager; | ||
this.consumerAccessor = consumerAccessor; | ||
this.totalConsumerWorkers = totalConsumerWorkers; | ||
this.maxInstanceWorkers = maxInstanceWorkers; | ||
} | ||
|
||
public async Task<int> GetWorkersCountAsync(WorkersCountContext context) | ||
{ | ||
var topicsMetadata = await this.GetTopicsMetadataAsync(context); | ||
|
||
var lastOffsets = this.GetPartitionsLastOffset(context.ConsumerName, topicsMetadata); | ||
|
||
var partitionsOffset = await this.clusterManager.GetConsumerGroupOffsetsAsync( | ||
context.ConsumerGroupId, | ||
context.AssignedTopicsPartitions.Select(t => t.Name)); | ||
|
||
var partitionsLag = CalculatePartitionsLag(lastOffsets, partitionsOffset); | ||
var myLag = CalculateMyPartitionsLag(context, partitionsLag); | ||
|
||
decimal totalConsumerLag = partitionsLag.Sum(p => p.Lag); | ||
|
||
var ratio = myLag / totalConsumerLag; | ||
|
||
var workers = (int)Math.Round(this.totalConsumerWorkers * ratio); | ||
|
||
if (workers > this.maxInstanceWorkers) | ||
{ | ||
return this.maxInstanceWorkers; | ||
} | ||
|
||
return workers < 1 ? 1 : workers; | ||
} | ||
|
||
private static long CalculateMyPartitionsLag( | ||
WorkersCountContext context, | ||
IReadOnlyList<(string Topic, int Partition, long Lag)> partitionsLag) | ||
{ | ||
return partitionsLag | ||
.Where( | ||
partitionLag => context.AssignedTopicsPartitions | ||
.Any( | ||
topic => topic.Name == partitionLag.Topic && | ||
topic.Partitions.Any(p => p == partitionLag.Partition))) | ||
.Sum(partitionLag => partitionLag.Lag); | ||
} | ||
|
||
private static IReadOnlyList<(string Topic, int Partition, long Lag)> CalculatePartitionsLag( | ||
IEnumerable<(string Topic, int Partition, long Offset)> lastOffsets, | ||
IEnumerable<TopicPartitionOffset> currentPartitionsOffset) | ||
{ | ||
return lastOffsets | ||
.Select( | ||
last => | ||
{ | ||
var currentOffset = currentPartitionsOffset | ||
.Where(current => current.Topic == last.Topic && current.Partition == last.Partition) | ||
.Select(current => current.Offset) | ||
.FirstOrDefault(0); | ||
|
||
var lastOffset = Math.Max(0, last.Offset); | ||
currentOffset = Math.Max(0, currentOffset); | ||
|
||
return (last.Topic, last.Partition, lastOffset - currentOffset); | ||
}) | ||
.ToList(); | ||
} | ||
|
||
private IReadOnlyList<(string TopicName, int Partition, long Offset)> GetPartitionsLastOffset( | ||
string consumerName, | ||
IEnumerable<(string Name, TopicMetadata Metadata)> topicsMetadata) | ||
{ | ||
var consumer = this.consumerAccessor[consumerName]; | ||
|
||
var offsets = new List<(string TopicName, int Partition, long Offset)>(); | ||
|
||
foreach (var topic in topicsMetadata) | ||
{ | ||
foreach (var partition in topic.Metadata.Partitions) | ||
{ | ||
offsets.Add( | ||
( | ||
topic.Name, | ||
partition.Id, | ||
consumer.QueryWatermarkOffsets( | ||
new TopicPartition(topic.Name, new Partition(partition.Id)), | ||
TimeSpan.FromSeconds(30)) | ||
.High.Value)); | ||
} | ||
} | ||
|
||
return offsets; | ||
} | ||
|
||
private async Task<IReadOnlyList<(string Name, TopicMetadata Metadata)>> GetTopicsMetadataAsync(WorkersCountContext context) | ||
{ | ||
var topicsMetadata = new List<(string Name, TopicMetadata Metadata)>(context.AssignedTopicsPartitions.Count); | ||
|
||
foreach (var topic in context.AssignedTopicsPartitions) | ||
{ | ||
topicsMetadata.Add((topic.Name, await this.clusterManager.GetTopicMetadataAsync(topic.Name))); | ||
} | ||
|
||
return topicsMetadata; | ||
} | ||
} |
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 |
---|---|---|
@@ -1,13 +1,45 @@ | ||
using System; | ||
using KafkaFlow.Producers; | ||
using KafkaFlow.Sample.Dashboard; | ||
using Microsoft.AspNetCore.Hosting; | ||
using Microsoft.Extensions.DependencyInjection; | ||
using Microsoft.Extensions.Hosting; | ||
|
||
await CreateHostBuilder(args) | ||
.Build() | ||
.RunAsync(); | ||
var host = CreateHostBuilder(args) | ||
.Build(); | ||
|
||
var producer = host.Services.GetRequiredService<IProducerAccessor>()["producer"]; | ||
|
||
_ = host.RunAsync(); | ||
|
||
static IHostBuilder CreateHostBuilder(string[] args) => | ||
Host | ||
.CreateDefaultBuilder(args) | ||
.ConfigureWebHostDefaults(webBuilder => { webBuilder.UseStartup<Startup>(); }); | ||
.ConfigureWebHostDefaults( | ||
webBuilder => | ||
{ | ||
webBuilder | ||
.UseStartup<Startup>() | ||
.UseKestrel(options => options.ListenAnyIP(int.Parse(args[0]))); | ||
}); | ||
|
||
while (true) | ||
{ | ||
var input = Console.ReadLine(); | ||
|
||
var splitted = input.Split(" "); | ||
|
||
var count = int.Parse(splitted[0]); | ||
|
||
int? partition = null; | ||
|
||
if (splitted.Length > 1) | ||
{ | ||
partition = int.Parse(splitted[1]); | ||
} | ||
|
||
for (int i = 0; i < count; i++) | ||
{ | ||
_ = producer.ProduceAsync(Guid.NewGuid().ToString(), Array.Empty<byte>(), partition: partition); | ||
} | ||
} |
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
29 changes: 29 additions & 0 deletions
29
src/KafkaFlow.Abstractions/Configuration/WorkersCountContext.cs
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,29 @@ | ||
namespace KafkaFlow.Configuration | ||
{ | ||
using System.Collections.Generic; | ||
|
||
/// <summary> | ||
/// A metadata class with some context information help to calculate the number of workers | ||
/// </summary> | ||
public class WorkersCountContext | ||
{ | ||
public WorkersCountContext( | ||
string consumerName, | ||
string consumerGroupId, | ||
IReadOnlyCollection<TopicPartitions> assignedTopicsPartitions) | ||
{ | ||
this.ConsumerName = consumerName; | ||
this.ConsumerGroupId = consumerGroupId; | ||
this.AssignedTopicsPartitions = assignedTopicsPartitions; | ||
} | ||
|
||
public string ConsumerName { get; } | ||
|
||
public string ConsumerGroupId { get; } | ||
|
||
/// <summary> | ||
/// Gets the assigned partitions to the consumer | ||
/// </summary> | ||
public IReadOnlyCollection<TopicPartitions> AssignedTopicsPartitions { get; } | ||
} | ||
} |
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
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.