You can not select more than 25 topics Topics must start with a letter or number, can include dashes ('-') and can be up to 35 characters long.

105 lines
3.9 KiB

12 months ago
12 months ago
9 months ago
9 months ago
9 months ago
12 months ago
8 months ago
12 months ago
8 months ago
12 months ago
  1. package broker
  2. import (
  3. "context"
  4. "fmt"
  5. "github.com/seaweedfs/seaweedfs/weed/glog"
  6. "github.com/seaweedfs/seaweedfs/weed/mq/pub_balancer"
  7. "github.com/seaweedfs/seaweedfs/weed/mq/topic"
  8. "github.com/seaweedfs/seaweedfs/weed/pb"
  9. "github.com/seaweedfs/seaweedfs/weed/pb/mq_pb"
  10. "sync"
  11. )
  12. // AssignTopicPartitions Runs on the assigned broker, to execute the topic partition assignment
  13. func (b *MessageQueueBroker) AssignTopicPartitions(c context.Context, request *mq_pb.AssignTopicPartitionsRequest) (*mq_pb.AssignTopicPartitionsResponse, error) {
  14. ret := &mq_pb.AssignTopicPartitionsResponse{}
  15. t := topic.FromPbTopic(request.Topic)
  16. conf, readConfErr := b.fca.ReadTopicConfFromFiler(t)
  17. if readConfErr != nil {
  18. glog.Errorf("topic %v not found: %v", t, readConfErr)
  19. return nil, fmt.Errorf("topic %v not found: %v", t, readConfErr)
  20. }
  21. // drain existing topic partition subscriptions
  22. for _, assignment := range request.BrokerPartitionAssignments {
  23. partition := topic.FromPbPartition(assignment.Partition)
  24. b.accessLock.Lock()
  25. if request.IsDraining {
  26. // TODO drain existing topic partition subscriptions
  27. b.localTopicManager.RemoveLocalPartition(t, partition)
  28. } else {
  29. var localPartition *topic.LocalPartition
  30. if localPartition = b.localTopicManager.GetLocalPartition(t, partition); localPartition == nil {
  31. localPartition = topic.NewLocalPartition(partition, b.genLogFlushFunc(t, assignment.Partition), b.genLogOnDiskReadFunc(t, assignment.Partition), conf.RecordType)
  32. b.localTopicManager.AddLocalPartition(t, localPartition)
  33. }
  34. }
  35. b.accessLock.Unlock()
  36. }
  37. // if is leader, notify the followers to drain existing topic partition subscriptions
  38. if request.IsLeader {
  39. for _, brokerPartition := range request.BrokerPartitionAssignments {
  40. if follower := brokerPartition.FollowerBroker; follower != "" {
  41. err := pb.WithBrokerGrpcClient(false, follower, b.grpcDialOption, func(client mq_pb.SeaweedMessagingClient) error {
  42. _, err := client.AssignTopicPartitions(context.Background(), request)
  43. return err
  44. })
  45. if err != nil {
  46. return ret, err
  47. }
  48. }
  49. }
  50. }
  51. glog.V(0).Infof("AssignTopicPartitions: topic %s partition assignments: %v", request.Topic, request.BrokerPartitionAssignments)
  52. return ret, nil
  53. }
  54. // called by broker leader to drain existing partitions.
  55. // new/updated partitions will be detected by broker from the filer
  56. func (b *MessageQueueBroker) assignTopicPartitionsToBrokers(ctx context.Context, t *mq_pb.Topic, assignments []*mq_pb.BrokerPartitionAssignment, isAdd bool) error {
  57. // notify the brokers to create the topic partitions in parallel
  58. var wg sync.WaitGroup
  59. for _, bpa := range assignments {
  60. wg.Add(1)
  61. go func(bpa *mq_pb.BrokerPartitionAssignment) {
  62. defer wg.Done()
  63. if doCreateErr := b.withBrokerClient(false, pb.ServerAddress(bpa.LeaderBroker), func(client mq_pb.SeaweedMessagingClient) error {
  64. _, doCreateErr := client.AssignTopicPartitions(ctx, &mq_pb.AssignTopicPartitionsRequest{
  65. Topic: t,
  66. BrokerPartitionAssignments: []*mq_pb.BrokerPartitionAssignment{
  67. {
  68. Partition: bpa.Partition,
  69. },
  70. },
  71. IsLeader: true,
  72. IsDraining: !isAdd,
  73. })
  74. if doCreateErr != nil {
  75. if !isAdd {
  76. return fmt.Errorf("drain topic %s %v on %s: %v", t, bpa.LeaderBroker, bpa.Partition, doCreateErr)
  77. } else {
  78. return fmt.Errorf("create topic %s %v on %s: %v", t, bpa.LeaderBroker, bpa.Partition, doCreateErr)
  79. }
  80. }
  81. brokerStats, found := b.PubBalancer.Brokers.Get(bpa.LeaderBroker)
  82. if !found {
  83. brokerStats = pub_balancer.NewBrokerStats()
  84. if !b.PubBalancer.Brokers.SetIfAbsent(bpa.LeaderBroker, brokerStats) {
  85. brokerStats, _ = b.PubBalancer.Brokers.Get(bpa.LeaderBroker)
  86. }
  87. }
  88. brokerStats.RegisterAssignment(t, bpa.Partition, isAdd)
  89. return nil
  90. }); doCreateErr != nil {
  91. glog.Errorf("create topic %s partition %+v on %s: %v", t, bpa.Partition, bpa.LeaderBroker, doCreateErr)
  92. }
  93. }(bpa)
  94. }
  95. wg.Wait()
  96. return nil
  97. }