Browse Source

add CreateTopic API

sub
chrislu 1 year ago
parent
commit
0361c321b4
  1. 3
      weed/mq/balancer/balancer.go
  2. 20
      weed/mq/broker/broker_grpc_lookup.go
  3. 2
      weed/mq/broker/broker_server.go
  4. 8
      weed/mq/broker/broker_stats.go
  5. 9
      weed/pb/mq.proto
  6. 912
      weed/pb/mq_pb/mq.pb.go
  7. 36
      weed/pb/mq_pb/mq_grpc.pb.go
  8. 65
      weed/shell/command_mq_topic_create.go
  9. 37
      weed/shell/command_mq_topic_list.go

3
weed/mq/balancer/balancer.go

@ -7,7 +7,8 @@ import (
) )
const ( const (
MaxPartitionCount = 8 * 9 * 5 * 7 //2520
MaxPartitionCount = 8 * 9 * 5 * 7 //2520
LockBrokerBalancer = "broker_balancer"
) )
type Balancer struct { type Balancer struct {

20
weed/mq/broker/broker_grpc_lookup.go

@ -7,6 +7,26 @@ import (
"google.golang.org/grpc/status" "google.golang.org/grpc/status"
) )
func (broker *MessageQueueBroker) CreateTopic(ctx context.Context, request *mq_pb.CreateTopicRequest) (resp *mq_pb.CreateTopicResponse, err error) {
if broker.currentBalancer == "" {
return nil, status.Errorf(codes.Unavailable, "no balancer")
}
if !broker.lockAsBalancer.IsLocked() {
proxyErr := broker.withBrokerClient(false, broker.currentBalancer, func(client mq_pb.SeaweedMessagingClient) error {
resp, err = client.CreateTopic(ctx, request)
return nil
})
if proxyErr != nil {
return nil, proxyErr
}
return resp, err
}
ret := &mq_pb.CreateTopicResponse{}
ret.BrokerPartitionAssignments, err = broker.Balancer.LookupOrAllocateTopicPartitions(request.Topic, true)
return ret, err
}
// FindTopicBrokers returns the brokers that are serving the topic // FindTopicBrokers returns the brokers that are serving the topic
// //
// 1. lock the topic // 1. lock the topic

2
weed/mq/broker/broker_server.go

@ -70,7 +70,7 @@ func NewMessageBroker(option *MessageQueueBrokerOption, grpcDialOption grpc.Dial
glog.V(1).Infof("broker %s found filer %s", self, mqBroker.currentFiler) glog.V(1).Infof("broker %s found filer %s", self, mqBroker.currentFiler)
lockClient := cluster.NewLockClient(grpcDialOption, mqBroker.currentFiler) lockClient := cluster.NewLockClient(grpcDialOption, mqBroker.currentFiler)
mqBroker.lockAsBalancer = lockClient.StartLock(LockBrokerBalancer, self)
mqBroker.lockAsBalancer = lockClient.StartLock(balancer.LockBrokerBalancer, self)
for { for {
err := mqBroker.BrokerConnectToBalancer(self) err := mqBroker.BrokerConnectToBalancer(self)
if err != nil { if err != nil {

8
weed/mq/broker/broker_stats.go

@ -4,6 +4,7 @@ import (
"context" "context"
"fmt" "fmt"
"github.com/seaweedfs/seaweedfs/weed/glog" "github.com/seaweedfs/seaweedfs/weed/glog"
"github.com/seaweedfs/seaweedfs/weed/mq/balancer"
"github.com/seaweedfs/seaweedfs/weed/pb" "github.com/seaweedfs/seaweedfs/weed/pb"
"github.com/seaweedfs/seaweedfs/weed/pb/filer_pb" "github.com/seaweedfs/seaweedfs/weed/pb/filer_pb"
"github.com/seaweedfs/seaweedfs/weed/pb/mq_pb" "github.com/seaweedfs/seaweedfs/weed/pb/mq_pb"
@ -11,17 +12,13 @@ import (
"time" "time"
) )
const (
LockBrokerBalancer = "broker_balancer"
)
// BrokerConnectToBalancer connects to the broker balancer and sends stats // BrokerConnectToBalancer connects to the broker balancer and sends stats
func (broker *MessageQueueBroker) BrokerConnectToBalancer(self string) error { func (broker *MessageQueueBroker) BrokerConnectToBalancer(self string) error {
// find the lock owner // find the lock owner
var brokerBalancer string var brokerBalancer string
err := broker.WithFilerClient(false, func(client filer_pb.SeaweedFilerClient) error { err := broker.WithFilerClient(false, func(client filer_pb.SeaweedFilerClient) error {
resp, err := client.FindLockOwner(context.Background(), &filer_pb.FindLockOwnerRequest{ resp, err := client.FindLockOwner(context.Background(), &filer_pb.FindLockOwnerRequest{
Name: LockBrokerBalancer,
Name: balancer.LockBrokerBalancer,
}) })
if err != nil { if err != nil {
return err return err
@ -32,6 +29,7 @@ func (broker *MessageQueueBroker) BrokerConnectToBalancer(self string) error {
if err != nil { if err != nil {
return err return err
} }
broker.currentBalancer = pb.ServerAddress(brokerBalancer)
glog.V(1).Infof("broker %s found balancer %s", self, brokerBalancer) glog.V(1).Infof("broker %s found balancer %s", self, brokerBalancer)

9
weed/pb/mq.proto

@ -26,6 +26,8 @@ service SeaweedMessaging {
// control plane for topic partitions // control plane for topic partitions
rpc LookupTopicBrokers (LookupTopicBrokersRequest) returns (LookupTopicBrokersResponse) { rpc LookupTopicBrokers (LookupTopicBrokersRequest) returns (LookupTopicBrokersResponse) {
} }
rpc CreateTopic (CreateTopicRequest) returns (CreateTopicResponse) {
}
// a pub client will call this to get the topic partitions assignment // a pub client will call this to get the topic partitions assignment
rpc RequestTopicPartitions (RequestTopicPartitionsRequest) returns (RequestTopicPartitionsResponse) { rpc RequestTopicPartitions (RequestTopicPartitionsRequest) returns (RequestTopicPartitionsResponse) {
} }
@ -128,6 +130,13 @@ message ConnectToBalancerRequest {
message ConnectToBalancerResponse { message ConnectToBalancerResponse {
} }
////////////////////////////////////////////////// //////////////////////////////////////////////////
message CreateTopicRequest {
Topic topic = 1;
int32 partition_count = 2;
}
message CreateTopicResponse {
repeated BrokerPartitionAssignment broker_partition_assignments = 2;
}
message LookupTopicBrokersRequest { message LookupTopicBrokersRequest {
Topic topic = 1; Topic topic = 1;
bool is_for_publish = 2; bool is_for_publish = 2;

912
weed/pb/mq_pb/mq.pb.go
File diff suppressed because it is too large
View File

36
weed/pb/mq_pb/mq_grpc.pb.go

@ -31,6 +31,7 @@ type SeaweedMessagingClient interface {
ConnectToBalancer(ctx context.Context, opts ...grpc.CallOption) (SeaweedMessaging_ConnectToBalancerClient, error) ConnectToBalancer(ctx context.Context, opts ...grpc.CallOption) (SeaweedMessaging_ConnectToBalancerClient, error)
// control plane for topic partitions // control plane for topic partitions
LookupTopicBrokers(ctx context.Context, in *LookupTopicBrokersRequest, opts ...grpc.CallOption) (*LookupTopicBrokersResponse, error) LookupTopicBrokers(ctx context.Context, in *LookupTopicBrokersRequest, opts ...grpc.CallOption) (*LookupTopicBrokersResponse, error)
CreateTopic(ctx context.Context, in *CreateTopicRequest, opts ...grpc.CallOption) (*CreateTopicResponse, error)
// a pub client will call this to get the topic partitions assignment // a pub client will call this to get the topic partitions assignment
RequestTopicPartitions(ctx context.Context, in *RequestTopicPartitionsRequest, opts ...grpc.CallOption) (*RequestTopicPartitionsResponse, error) RequestTopicPartitions(ctx context.Context, in *RequestTopicPartitionsRequest, opts ...grpc.CallOption) (*RequestTopicPartitionsResponse, error)
AssignTopicPartitions(ctx context.Context, in *AssignTopicPartitionsRequest, opts ...grpc.CallOption) (*AssignTopicPartitionsResponse, error) AssignTopicPartitions(ctx context.Context, in *AssignTopicPartitionsRequest, opts ...grpc.CallOption) (*AssignTopicPartitionsResponse, error)
@ -124,6 +125,15 @@ func (c *seaweedMessagingClient) LookupTopicBrokers(ctx context.Context, in *Loo
return out, nil return out, nil
} }
func (c *seaweedMessagingClient) CreateTopic(ctx context.Context, in *CreateTopicRequest, opts ...grpc.CallOption) (*CreateTopicResponse, error) {
out := new(CreateTopicResponse)
err := c.cc.Invoke(ctx, "/messaging_pb.SeaweedMessaging/CreateTopic", in, out, opts...)
if err != nil {
return nil, err
}
return out, nil
}
func (c *seaweedMessagingClient) RequestTopicPartitions(ctx context.Context, in *RequestTopicPartitionsRequest, opts ...grpc.CallOption) (*RequestTopicPartitionsResponse, error) { func (c *seaweedMessagingClient) RequestTopicPartitions(ctx context.Context, in *RequestTopicPartitionsRequest, opts ...grpc.CallOption) (*RequestTopicPartitionsResponse, error) {
out := new(RequestTopicPartitionsResponse) out := new(RequestTopicPartitionsResponse)
err := c.cc.Invoke(ctx, "/messaging_pb.SeaweedMessaging/RequestTopicPartitions", in, out, opts...) err := c.cc.Invoke(ctx, "/messaging_pb.SeaweedMessaging/RequestTopicPartitions", in, out, opts...)
@ -227,6 +237,7 @@ type SeaweedMessagingServer interface {
ConnectToBalancer(SeaweedMessaging_ConnectToBalancerServer) error ConnectToBalancer(SeaweedMessaging_ConnectToBalancerServer) error
// control plane for topic partitions // control plane for topic partitions
LookupTopicBrokers(context.Context, *LookupTopicBrokersRequest) (*LookupTopicBrokersResponse, error) LookupTopicBrokers(context.Context, *LookupTopicBrokersRequest) (*LookupTopicBrokersResponse, error)
CreateTopic(context.Context, *CreateTopicRequest) (*CreateTopicResponse, error)
// a pub client will call this to get the topic partitions assignment // a pub client will call this to get the topic partitions assignment
RequestTopicPartitions(context.Context, *RequestTopicPartitionsRequest) (*RequestTopicPartitionsResponse, error) RequestTopicPartitions(context.Context, *RequestTopicPartitionsRequest) (*RequestTopicPartitionsResponse, error)
AssignTopicPartitions(context.Context, *AssignTopicPartitionsRequest) (*AssignTopicPartitionsResponse, error) AssignTopicPartitions(context.Context, *AssignTopicPartitionsRequest) (*AssignTopicPartitionsResponse, error)
@ -259,6 +270,9 @@ func (UnimplementedSeaweedMessagingServer) ConnectToBalancer(SeaweedMessaging_Co
func (UnimplementedSeaweedMessagingServer) LookupTopicBrokers(context.Context, *LookupTopicBrokersRequest) (*LookupTopicBrokersResponse, error) { func (UnimplementedSeaweedMessagingServer) LookupTopicBrokers(context.Context, *LookupTopicBrokersRequest) (*LookupTopicBrokersResponse, error) {
return nil, status.Errorf(codes.Unimplemented, "method LookupTopicBrokers not implemented") return nil, status.Errorf(codes.Unimplemented, "method LookupTopicBrokers not implemented")
} }
func (UnimplementedSeaweedMessagingServer) CreateTopic(context.Context, *CreateTopicRequest) (*CreateTopicResponse, error) {
return nil, status.Errorf(codes.Unimplemented, "method CreateTopic not implemented")
}
func (UnimplementedSeaweedMessagingServer) RequestTopicPartitions(context.Context, *RequestTopicPartitionsRequest) (*RequestTopicPartitionsResponse, error) { func (UnimplementedSeaweedMessagingServer) RequestTopicPartitions(context.Context, *RequestTopicPartitionsRequest) (*RequestTopicPartitionsResponse, error) {
return nil, status.Errorf(codes.Unimplemented, "method RequestTopicPartitions not implemented") return nil, status.Errorf(codes.Unimplemented, "method RequestTopicPartitions not implemented")
} }
@ -403,6 +417,24 @@ func _SeaweedMessaging_LookupTopicBrokers_Handler(srv interface{}, ctx context.C
return interceptor(ctx, in, info, handler) return interceptor(ctx, in, info, handler)
} }
func _SeaweedMessaging_CreateTopic_Handler(srv interface{}, ctx context.Context, dec func(interface{}) error, interceptor grpc.UnaryServerInterceptor) (interface{}, error) {
in := new(CreateTopicRequest)
if err := dec(in); err != nil {
return nil, err
}
if interceptor == nil {
return srv.(SeaweedMessagingServer).CreateTopic(ctx, in)
}
info := &grpc.UnaryServerInfo{
Server: srv,
FullMethod: "/messaging_pb.SeaweedMessaging/CreateTopic",
}
handler := func(ctx context.Context, req interface{}) (interface{}, error) {
return srv.(SeaweedMessagingServer).CreateTopic(ctx, req.(*CreateTopicRequest))
}
return interceptor(ctx, in, info, handler)
}
func _SeaweedMessaging_RequestTopicPartitions_Handler(srv interface{}, ctx context.Context, dec func(interface{}) error, interceptor grpc.UnaryServerInterceptor) (interface{}, error) { func _SeaweedMessaging_RequestTopicPartitions_Handler(srv interface{}, ctx context.Context, dec func(interface{}) error, interceptor grpc.UnaryServerInterceptor) (interface{}, error) {
in := new(RequestTopicPartitionsRequest) in := new(RequestTopicPartitionsRequest)
if err := dec(in); err != nil { if err := dec(in); err != nil {
@ -531,6 +563,10 @@ var SeaweedMessaging_ServiceDesc = grpc.ServiceDesc{
MethodName: "LookupTopicBrokers", MethodName: "LookupTopicBrokers",
Handler: _SeaweedMessaging_LookupTopicBrokers_Handler, Handler: _SeaweedMessaging_LookupTopicBrokers_Handler,
}, },
{
MethodName: "CreateTopic",
Handler: _SeaweedMessaging_CreateTopic_Handler,
},
{ {
MethodName: "RequestTopicPartitions", MethodName: "RequestTopicPartitions",
Handler: _SeaweedMessaging_RequestTopicPartitions_Handler, Handler: _SeaweedMessaging_RequestTopicPartitions_Handler,

65
weed/shell/command_mq_topic_create.go

@ -0,0 +1,65 @@
package shell
import (
"context"
"flag"
"fmt"
"github.com/seaweedfs/seaweedfs/weed/pb"
"github.com/seaweedfs/seaweedfs/weed/pb/mq_pb"
"io"
)
func init() {
Commands = append(Commands, &commandMqTopicCreate{})
}
type commandMqTopicCreate struct {
}
func (c *commandMqTopicCreate) Name() string {
return "mq.topic.create"
}
func (c *commandMqTopicCreate) Help() string {
return `create a topic with a given name
Example:
mq.topic.create -namespace <namespace> -topic <topic_name> -partition_count <partition_count>
`
}
func (c *commandMqTopicCreate) Do(args []string, commandEnv *CommandEnv, writer io.Writer) error {
// parse parameters
mqCommand := flag.NewFlagSet(c.Name(), flag.ContinueOnError)
namespace := mqCommand.String("namespace", "", "namespace name")
topicName := mqCommand.String("topic", "", "topic name")
partitionCount := mqCommand.Int("partitionCount", 6, "partition count")
if err := mqCommand.Parse(args); err != nil {
return err
}
// find the broker balancer
brokerBalancer, err := findBrokerBalancer(commandEnv)
if err != nil {
return err
}
fmt.Fprintf(writer, "current balancer: %s\n", brokerBalancer)
// create topic
return pb.WithBrokerGrpcClient(false, brokerBalancer, commandEnv.option.GrpcDialOption, func(client mq_pb.SeaweedMessagingClient) error {
resp, err := client.CreateTopic(context.Background(), &mq_pb.CreateTopicRequest{
Topic: &mq_pb.Topic{
Namespace: *namespace,
Name: *topicName,
},
PartitionCount: int32(*partitionCount),
})
if err != nil {
return err
}
fmt.Fprintf(writer, "response: %+v\n", resp)
return nil
})
}

37
weed/shell/command_mq_topic_list.go

@ -1,7 +1,10 @@
package shell package shell
import ( import (
"context"
"fmt" "fmt"
"github.com/seaweedfs/seaweedfs/weed/mq/balancer"
"github.com/seaweedfs/seaweedfs/weed/pb/filer_pb"
"io" "io"
) )
@ -20,9 +23,39 @@ func (c *commandMqTopicList) Help() string {
return `print out all topics` return `print out all topics`
} }
func (c *commandMqTopicList) Do(args []string, commandEnv *CommandEnv, writer io.Writer) (err error) {
func (c *commandMqTopicList) Do(args []string, commandEnv *CommandEnv, writer io.Writer) error {
fmt.Fprintf(writer, "%s\n", commandEnv.option.Directory)
brokerBalancer, err := findBrokerBalancer(commandEnv)
if err != nil {
return err
}
//pb.WithBrokerGrpcClient(false, brokerBalancer, commandEnv.option.GrpcDialOption, func(client pb.SeaweedMessagingClient) error {
// resp, err := client.ListTopics(context.Background(), &pb.ListTopicsRequest{})
// if err != nil {
// return err
// }
// for _, topic := range resp.Topics {
// fmt.Fprintf(writer, "%s\n", topic)
// }
// return nil
//})
fmt.Fprintf(writer, "current balancer: %s\n", brokerBalancer)
return nil return nil
} }
func findBrokerBalancer(commandEnv *CommandEnv) (brokerBalancer string, err error) {
err = commandEnv.WithFilerClient(false, func(client filer_pb.SeaweedFilerClient) error {
resp, err := client.FindLockOwner(context.Background(), &filer_pb.FindLockOwnerRequest{
Name: balancer.LockBrokerBalancer,
})
if err != nil {
return err
}
brokerBalancer = resp.Owner
return nil
})
return
}
Loading…
Cancel
Save