Skip to content

Commit

Permalink
scheduler: fix scheduler save config (#7108) (#7161)
Browse files Browse the repository at this point in the history
close #6897

Signed-off-by: husharp <[email protected]>

Co-authored-by: husharp <[email protected]>
Co-authored-by: Hu# <[email protected]>
  • Loading branch information
ti-chi-bot and HuSharp committed Sep 28, 2023
1 parent fb6139c commit e063180
Show file tree
Hide file tree
Showing 6 changed files with 190 additions and 9 deletions.
13 changes: 6 additions & 7 deletions pkg/schedule/scheduler.go
Original file line number Diff line number Diff line change
Expand Up @@ -118,17 +118,16 @@ func CreateScheduler(typ string, opController *OperatorController, storage endpo
if !ok {
return nil, errs.ErrSchedulerCreateFuncNotRegistered.FastGenByArgs(typ)
}
return fn(opController, storage, dec)
}

s, err := fn(opController, storage, dec)
if err != nil {
return nil, err
}
// SaveSchedulerConfig saves the config of the specified scheduler.
func SaveSchedulerConfig(storage endpoint.ConfigStorage, s Scheduler) error {
data, err := s.EncodeConfig()
if err != nil {
return nil, err
return err
}
err = storage.SaveScheduleConfig(s.GetName(), data)
return s, err
return storage.SaveScheduleConfig(s.GetName(), data)
}

// FindSchedulerTypeByName finds the type of the specified name.
Expand Down
2 changes: 1 addition & 1 deletion pkg/schedule/schedulers/evict_leader.go
Original file line number Diff line number Diff line change
Expand Up @@ -180,7 +180,7 @@ func newEvictLeaderScheduler(opController *schedule.OperatorController, conf *ev
}
}

// EvictStores returns the IDs of the evict-stores.
// EvictStoreIDs returns the IDs of the evict-stores.
func (s *evictLeaderScheduler) EvictStoreIDs() []uint64 {
return s.conf.getStores()
}
Expand Down
4 changes: 4 additions & 0 deletions server/cluster/coordinator.go
Original file line number Diff line number Diff line change
Expand Up @@ -656,6 +656,10 @@ func (c *coordinator) addScheduler(scheduler schedule.Scheduler, args ...string)
c.wg.Add(1)
go c.runScheduler(s)
c.schedulers[s.GetName()] = s
if err := schedule.SaveSchedulerConfig(c.cluster.storage, scheduler); err != nil {
log.Error("can not save scheduler config", zap.String("scheduler-name", scheduler.GetName()), errs.ZapError(err))
return err
}
c.cluster.opt.AddSchedulerCfg(s.GetType(), args)
return nil
}
Expand Down
3 changes: 2 additions & 1 deletion server/cluster/coordinator_test.go
Original file line number Diff line number Diff line change
Expand Up @@ -831,8 +831,9 @@ func TestPersistScheduler(t *testing.T) {
// whether the schedulers added or removed in dynamic way are recorded in opt
_, newOpt, err := newTestScheduleConfig()
re.NoError(err)
_, err = schedule.CreateScheduler(schedulers.ShuffleRegionType, oc, storage, schedule.ConfigJSONDecoder([]byte("null")))
shuffle, err := schedule.CreateScheduler(schedulers.ShuffleRegionType, oc, storage, schedule.ConfigJSONDecoder([]byte("null")))
re.NoError(err)
re.NoError(co.addScheduler(shuffle))
// suppose we add a new default enable scheduler
config.DefaultSchedulers = append(config.DefaultSchedulers, config.SchedulerConfig{Type: "shuffle-region"})
defer func() {
Expand Down
58 changes: 58 additions & 0 deletions tests/server/api/testutil.go
Original file line number Diff line number Diff line change
@@ -0,0 +1,58 @@
// Copyright 2023 TiKV Project Authors.
//
// Licensed under the Apache License, Version 2.0 (the "License");
// you may not use this file except in compliance with the License.
// You may obtain a copy of the License at
//
// http://www.apache.org/licenses/LICENSE-2.0
//
// Unless required by applicable law or agreed to in writing, software
// distributed under the License is distributed on an "AS IS" BASIS,
// WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
// See the License for the specific language governing permissions and
// limitations under the License.

package api

import (
"bytes"
"encoding/json"
"fmt"
"io"
"net/http"

"github.com/stretchr/testify/require"
)

const schedulersPrefix = "/pd/api/v1/schedulers"

// dialClient used to dial http request.
var dialClient = &http.Client{
Transport: &http.Transport{
DisableKeepAlives: true,
},
}

// MustAddScheduler adds a scheduler with HTTP API.
func MustAddScheduler(
re *require.Assertions, serverAddr string,
schedulerName string, args map[string]interface{},
) {
request := map[string]interface{}{
"name": schedulerName,
}
for arg, val := range args {
request[arg] = val
}
data, err := json.Marshal(request)
re.NoError(err)
httpReq, err := http.NewRequest(http.MethodPost, fmt.Sprintf("%s%s", serverAddr, schedulersPrefix), bytes.NewBuffer(data))
re.NoError(err)
// Send request.
resp, err := dialClient.Do(httpReq)
re.NoError(err)
defer resp.Body.Close()
data, err = io.ReadAll(resp.Body)
re.NoError(err)
re.Equal(http.StatusOK, resp.StatusCode, string(data))
}
119 changes: 119 additions & 0 deletions tests/server/cluster/cluster_test.go
Original file line number Diff line number Diff line change
Expand Up @@ -36,6 +36,7 @@ import (
"github.com/tikv/pd/pkg/id"
"github.com/tikv/pd/pkg/mock/mockid"
"github.com/tikv/pd/pkg/schedule/operator"
"github.com/tikv/pd/pkg/schedule/schedulers"
"github.com/tikv/pd/pkg/storage"
"github.com/tikv/pd/pkg/tso"
"github.com/tikv/pd/pkg/utils/testutil"
Expand All @@ -46,6 +47,7 @@ import (
"github.com/tikv/pd/server/config"
syncer "github.com/tikv/pd/server/region_syncer"
"github.com/tikv/pd/tests"
"github.com/tikv/pd/tests/server/api"
"google.golang.org/grpc/codes"
"google.golang.org/grpc/status"
)
Expand Down Expand Up @@ -1278,6 +1280,123 @@ func TestStaleTermHeartbeat(t *testing.T) {
re.NoError(err)
}

func TestTransferLeaderForScheduler(t *testing.T) {
re := require.New(t)
ctx, cancel := context.WithCancel(context.Background())
defer cancel()
re.NoError(failpoint.Enable("github.com/tikv/pd/server/cluster/changeCoordinatorTicker", `return(true)`))
tc, err := tests.NewTestCluster(ctx, 2)
defer tc.Destroy()
re.NoError(err)
err = tc.RunInitialServers()
re.NoError(err)
tc.WaitLeader()
// start
leaderServer := tc.GetServer(tc.GetLeader())
re.NoError(leaderServer.BootstrapCluster())
rc := leaderServer.GetServer().GetRaftCluster()
re.NotNil(rc)

storesNum := 2
grpcPDClient := testutil.MustNewGrpcClient(re, leaderServer.GetAddr())
for i := 1; i <= storesNum; i++ {
store := &metapb.Store{
Id: uint64(i),
Address: "127.0.0.1:" + strconv.Itoa(i),
}
resp, err := putStore(grpcPDClient, leaderServer.GetClusterID(), store)
re.NoError(err)
re.Equal(pdpb.ErrorType_OK, resp.GetHeader().GetError().GetType())
}
// region heartbeat
id := leaderServer.GetAllocator()
putRegionWithLeader(re, rc, id, 1)

time.Sleep(time.Second)
re.True(leaderServer.GetRaftCluster().IsPrepared())
// Add evict leader scheduler
api.MustAddScheduler(re, leaderServer.GetAddr(), schedulers.EvictLeaderName, map[string]interface{}{
"store_id": 1,
})
api.MustAddScheduler(re, leaderServer.GetAddr(), schedulers.EvictLeaderName, map[string]interface{}{
"store_id": 2,
})
// Check scheduler updated.
re.Len(rc.GetSchedulers(), 6)
checkEvictLeaderSchedulerExist(re, rc, true)
checkEvictLeaderStoreIDs(re, rc, []uint64{1, 2})

// transfer PD leader to another PD
tc.ResignLeader()
rc.Stop()
tc.WaitLeader()
leaderServer = tc.GetServer(tc.GetLeader())
rc1 := leaderServer.GetServer().GetRaftCluster()
rc1.Start(leaderServer.GetServer())
re.NoError(err)
re.NotNil(rc1)
// region heartbeat
id = leaderServer.GetAllocator()
putRegionWithLeader(re, rc1, id, 1)
time.Sleep(time.Second)
re.True(leaderServer.GetRaftCluster().IsPrepared())
// Check scheduler updated.
re.Len(rc1.GetSchedulers(), 6)
checkEvictLeaderSchedulerExist(re, rc, true)
checkEvictLeaderStoreIDs(re, rc, []uint64{1, 2})

// transfer PD leader back to the previous PD
tc.ResignLeader()
rc1.Stop()
tc.WaitLeader()
leaderServer = tc.GetServer(tc.GetLeader())
rc = leaderServer.GetServer().GetRaftCluster()
rc.Start(leaderServer.GetServer())
re.NotNil(rc)
// region heartbeat
id = leaderServer.GetAllocator()
putRegionWithLeader(re, rc, id, 1)
time.Sleep(time.Second)
re.True(leaderServer.GetRaftCluster().IsPrepared())
// Check scheduler updated
re.Len(rc.GetSchedulers(), 6)
checkEvictLeaderSchedulerExist(re, rc, true)
checkEvictLeaderStoreIDs(re, rc, []uint64{1, 2})

re.NoError(failpoint.Disable("github.com/tikv/pd/server/cluster/changeCoordinatorTicker"))
}

func checkEvictLeaderSchedulerExist(re *require.Assertions, rc *cluster.RaftCluster, exist bool) {
isExistScheduler := func(rc *cluster.RaftCluster, name string) bool {
s := rc.GetSchedulers()
for _, scheduler := range s {
if scheduler == name {
return true
}
}
return false
}

testutil.Eventually(re, func() bool {
return isExistScheduler(rc, schedulers.EvictLeaderName) == exist
})
}

func checkEvictLeaderStoreIDs(re *require.Assertions, rc *cluster.RaftCluster, expected []uint64) {
handler, ok := rc.GetSchedulerHandlers()[schedulers.EvictLeaderName]
re.True(ok)
h, ok := handler.(interface {
EvictStoreIDs() []uint64
})
re.True(ok)
var evictStoreIDs []uint64
testutil.Eventually(re, func() bool {
evictStoreIDs = h.EvictStoreIDs()
return len(evictStoreIDs) == len(expected)
})
re.ElementsMatch(evictStoreIDs, expected)
}

func putRegionWithLeader(re *require.Assertions, rc *cluster.RaftCluster, id id.Allocator, storeID uint64) {
for i := 0; i < 3; i++ {
regionID, err := id.Alloc()
Expand Down

0 comments on commit e063180

Please sign in to comment.