volume_growth.go 15 KB

123456789101112131415161718192021222324252627282930313233343536373839404142434445464748495051525354555657585960616263646566676869707172737475767778798081828384858687888990919293949596979899100101102103104105106107108109110111112113114115116117118119120121122123124125126127128129130131132133134135136137138139140141142143144145146147148149150151152153154155156157158159160161162163164165166167168169170171172173174175176177178179180181182183184185186187188189190191192193194195196197198199200201202203204205206207208209210211212213214215216217218219220221222223224225226227228229230231232233234235236237238239240241242243244245246247248249250251252253254255256257258259260261262263264265266267268269270271272273274275276277278279280281282283284285286287288289290291292293294295296297298299300301302303304305306307308309310311312313314315316317318319320321322323324325326327328329330331332333334335336337338339340341342343344345346347348349350351352353354355356357358359360361362363364365366367368369370371372373374375376377378379380381382383384385386387388389390391392393394395396397398399400401402403404405406
  1. package topology
  2. import (
  3. "encoding/json"
  4. "fmt"
  5. "math/rand/v2"
  6. "reflect"
  7. "sync"
  8. "time"
  9. "github.com/seaweedfs/seaweedfs/weed/pb/master_pb"
  10. "github.com/seaweedfs/seaweedfs/weed/server/constants"
  11. "google.golang.org/grpc"
  12. "github.com/seaweedfs/seaweedfs/weed/glog"
  13. "github.com/seaweedfs/seaweedfs/weed/storage"
  14. "github.com/seaweedfs/seaweedfs/weed/storage/needle"
  15. "github.com/seaweedfs/seaweedfs/weed/storage/super_block"
  16. "github.com/seaweedfs/seaweedfs/weed/storage/types"
  17. )
  18. /*
  19. This package is created to resolve these replica placement issues:
  20. 1. growth factor for each replica level, e.g., add 10 volumes for 1 copy, 20 volumes for 2 copies, 30 volumes for 3 copies
  21. 2. in time of tight storage, how to reduce replica level
  22. 3. optimizing for hot data on faster disk, cold data on cheaper storage,
  23. 4. volume allocation for each bucket
  24. */
  25. type VolumeGrowRequest struct {
  26. Option *VolumeGrowOption
  27. Count uint32
  28. Force bool
  29. Reason string
  30. }
  31. func (vg *VolumeGrowRequest) Equals(req *VolumeGrowRequest) bool {
  32. return reflect.DeepEqual(vg.Option, req.Option) && vg.Count == req.Count && vg.Force == req.Force
  33. }
  34. type volumeGrowthStrategy struct {
  35. Copy1Count uint32
  36. Copy2Count uint32
  37. Copy3Count uint32
  38. CopyOtherCount uint32
  39. Threshold float64
  40. }
  41. var (
  42. VolumeGrowStrategy = volumeGrowthStrategy{
  43. Copy1Count: 7,
  44. Copy2Count: 6,
  45. Copy3Count: 3,
  46. CopyOtherCount: 1,
  47. Threshold: 0.9,
  48. }
  49. )
  50. type VolumeGrowOption struct {
  51. Collection string `json:"collection,omitempty"`
  52. ReplicaPlacement *super_block.ReplicaPlacement `json:"replication,omitempty"`
  53. Ttl *needle.TTL `json:"ttl,omitempty"`
  54. DiskType types.DiskType `json:"disk,omitempty"`
  55. Preallocate int64 `json:"preallocate,omitempty"`
  56. DataCenter string `json:"dataCenter,omitempty"`
  57. Rack string `json:"rack,omitempty"`
  58. DataNode string `json:"dataNode,omitempty"`
  59. MemoryMapMaxSizeMb uint32 `json:"memoryMapMaxSizeMb,omitempty"`
  60. Version uint32 `json:"version,omitempty"`
  61. }
  62. type VolumeGrowth struct {
  63. accessLock sync.Mutex
  64. }
  65. // VolumeGrowReservation tracks capacity reservations for a volume creation operation
  66. type VolumeGrowReservation struct {
  67. servers []*DataNode
  68. reservationIds []string
  69. diskType types.DiskType
  70. }
  71. // releaseAllReservations releases all reservations in this volume grow operation
  72. func (vgr *VolumeGrowReservation) releaseAllReservations() {
  73. for i, server := range vgr.servers {
  74. if i < len(vgr.reservationIds) && vgr.reservationIds[i] != "" {
  75. server.ReleaseReservedCapacity(vgr.reservationIds[i])
  76. }
  77. }
  78. }
  79. func (o *VolumeGrowOption) String() string {
  80. blob, _ := json.Marshal(o)
  81. return string(blob)
  82. }
  83. func NewDefaultVolumeGrowth() *VolumeGrowth {
  84. return &VolumeGrowth{}
  85. }
  86. // one replication type may need rp.GetCopyCount() actual volumes
  87. // given copyCount, how many logical volumes to create
  88. func (vg *VolumeGrowth) findVolumeCount(copyCount int) (count uint32) {
  89. switch copyCount {
  90. case 1:
  91. count = VolumeGrowStrategy.Copy1Count
  92. case 2:
  93. count = VolumeGrowStrategy.Copy2Count
  94. case 3:
  95. count = VolumeGrowStrategy.Copy3Count
  96. default:
  97. count = VolumeGrowStrategy.CopyOtherCount
  98. }
  99. return
  100. }
  101. func (vg *VolumeGrowth) AutomaticGrowByType(option *VolumeGrowOption, grpcDialOption grpc.DialOption, topo *Topology, targetCount uint32) (result []*master_pb.VolumeLocation, err error) {
  102. if targetCount == 0 {
  103. targetCount = vg.findVolumeCount(option.ReplicaPlacement.GetCopyCount())
  104. }
  105. result, err = vg.GrowByCountAndType(grpcDialOption, targetCount, option, topo)
  106. if len(result) > 0 && len(result)%option.ReplicaPlacement.GetCopyCount() == 0 {
  107. return result, nil
  108. }
  109. return result, err
  110. }
  111. func (vg *VolumeGrowth) GrowByCountAndType(grpcDialOption grpc.DialOption, targetCount uint32, option *VolumeGrowOption, topo *Topology) (result []*master_pb.VolumeLocation, err error) {
  112. vg.accessLock.Lock()
  113. defer vg.accessLock.Unlock()
  114. for i := uint32(0); i < targetCount; i++ {
  115. if res, e := vg.findAndGrow(grpcDialOption, topo, option); e == nil {
  116. result = append(result, res...)
  117. } else {
  118. glog.V(0).Infof("create %d volume, created %d: %v", targetCount, len(result), e)
  119. return result, e
  120. }
  121. }
  122. return
  123. }
  124. func (vg *VolumeGrowth) findAndGrow(grpcDialOption grpc.DialOption, topo *Topology, option *VolumeGrowOption) (result []*master_pb.VolumeLocation, err error) {
  125. servers, reservation, e := vg.findEmptySlotsForOneVolume(topo, option, true) // use reservations
  126. if e != nil {
  127. return nil, e
  128. }
  129. // Ensure reservations are released if anything goes wrong
  130. defer func() {
  131. if err != nil && reservation != nil {
  132. reservation.releaseAllReservations()
  133. }
  134. }()
  135. for !topo.LastLeaderChangeTime.Add(constants.VolumePulseSeconds * 2).Before(time.Now()) {
  136. glog.V(0).Infof("wait for volume servers to join back")
  137. time.Sleep(constants.VolumePulseSeconds / 2)
  138. }
  139. vid, raftErr := topo.NextVolumeId()
  140. if raftErr != nil {
  141. return nil, raftErr
  142. }
  143. if err = vg.grow(grpcDialOption, topo, vid, option, reservation, servers...); err == nil {
  144. for _, server := range servers {
  145. result = append(result, &master_pb.VolumeLocation{
  146. Url: server.Url(),
  147. PublicUrl: server.PublicUrl,
  148. DataCenter: server.GetDataCenterId(),
  149. GrpcPort: uint32(server.GrpcPort),
  150. NewVids: []uint32{uint32(vid)},
  151. })
  152. }
  153. }
  154. return
  155. }
  156. // 1. find the main data node
  157. // 1.1 collect all data nodes that have 1 slots
  158. // 2.2 collect all racks that have rp.SameRackCount+1
  159. // 2.2 collect all data centers that have DiffRackCount+rp.SameRackCount+1
  160. // 2. find rest data nodes
  161. // If useReservations is true, reserves capacity on each server and returns reservation info
  162. func (vg *VolumeGrowth) findEmptySlotsForOneVolume(topo *Topology, option *VolumeGrowOption, useReservations bool) (servers []*DataNode, reservation *VolumeGrowReservation, err error) {
  163. //find main datacenter and other data centers
  164. rp := option.ReplicaPlacement
  165. // Track tentative reservations to make the process atomic
  166. var tentativeReservation *VolumeGrowReservation
  167. // Select appropriate functions based on useReservations flag
  168. var availableSpaceFunc func(Node, *VolumeGrowOption) int64
  169. var reserveOneVolumeFunc func(Node, int64, *VolumeGrowOption) (*DataNode, error)
  170. if useReservations {
  171. // Initialize tentative reservation tracking
  172. tentativeReservation = &VolumeGrowReservation{
  173. servers: make([]*DataNode, 0),
  174. reservationIds: make([]string, 0),
  175. diskType: option.DiskType,
  176. }
  177. // For reservations, we make actual reservations during node selection
  178. availableSpaceFunc = func(node Node, option *VolumeGrowOption) int64 {
  179. return node.AvailableSpaceForReservation(option)
  180. }
  181. reserveOneVolumeFunc = func(node Node, r int64, option *VolumeGrowOption) (*DataNode, error) {
  182. return node.ReserveOneVolumeForReservation(r, option)
  183. }
  184. } else {
  185. availableSpaceFunc = func(node Node, option *VolumeGrowOption) int64 {
  186. return node.AvailableSpaceFor(option)
  187. }
  188. reserveOneVolumeFunc = func(node Node, r int64, option *VolumeGrowOption) (*DataNode, error) {
  189. return node.ReserveOneVolume(r, option)
  190. }
  191. }
  192. // Ensure cleanup of partial reservations on error
  193. defer func() {
  194. if err != nil && tentativeReservation != nil {
  195. tentativeReservation.releaseAllReservations()
  196. }
  197. }()
  198. mainDataCenter, otherDataCenters, dc_err := topo.PickNodesByWeight(rp.DiffDataCenterCount+1, option, func(node Node) error {
  199. if option.DataCenter != "" && node.IsDataCenter() && node.Id() != NodeId(option.DataCenter) {
  200. return fmt.Errorf("Not matching preferred data center:%s", option.DataCenter)
  201. }
  202. if len(node.Children()) < rp.DiffRackCount+1 {
  203. return fmt.Errorf("Only has %d racks, not enough for %d.", len(node.Children()), rp.DiffRackCount+1)
  204. }
  205. if availableSpaceFunc(node, option) < int64(rp.DiffRackCount+rp.SameRackCount+1) {
  206. return fmt.Errorf("Free:%d < Expected:%d", availableSpaceFunc(node, option), rp.DiffRackCount+rp.SameRackCount+1)
  207. }
  208. possibleRacksCount := 0
  209. for _, rack := range node.Children() {
  210. possibleDataNodesCount := 0
  211. for _, n := range rack.Children() {
  212. if availableSpaceFunc(n, option) >= 1 {
  213. possibleDataNodesCount++
  214. }
  215. }
  216. if possibleDataNodesCount >= rp.SameRackCount+1 {
  217. possibleRacksCount++
  218. }
  219. }
  220. if possibleRacksCount < rp.DiffRackCount+1 {
  221. return fmt.Errorf("Only has %d racks with more than %d free data nodes, not enough for %d.", possibleRacksCount, rp.SameRackCount+1, rp.DiffRackCount+1)
  222. }
  223. return nil
  224. })
  225. if dc_err != nil {
  226. return nil, nil, dc_err
  227. }
  228. //find main rack and other racks
  229. mainRack, otherRacks, rackErr := mainDataCenter.(*DataCenter).PickNodesByWeight(rp.DiffRackCount+1, option, func(node Node) error {
  230. if option.Rack != "" && node.IsRack() && node.Id() != NodeId(option.Rack) {
  231. return fmt.Errorf("Not matching preferred rack:%s", option.Rack)
  232. }
  233. if availableSpaceFunc(node, option) < int64(rp.SameRackCount+1) {
  234. return fmt.Errorf("Free:%d < Expected:%d", availableSpaceFunc(node, option), rp.SameRackCount+1)
  235. }
  236. if len(node.Children()) < rp.SameRackCount+1 {
  237. // a bit faster way to test free racks
  238. return fmt.Errorf("Only has %d data nodes, not enough for %d.", len(node.Children()), rp.SameRackCount+1)
  239. }
  240. possibleDataNodesCount := 0
  241. for _, n := range node.Children() {
  242. if availableSpaceFunc(n, option) >= 1 {
  243. possibleDataNodesCount++
  244. }
  245. }
  246. if possibleDataNodesCount < rp.SameRackCount+1 {
  247. return fmt.Errorf("Only has %d data nodes with a slot, not enough for %d.", possibleDataNodesCount, rp.SameRackCount+1)
  248. }
  249. return nil
  250. })
  251. if rackErr != nil {
  252. return nil, nil, rackErr
  253. }
  254. //find main server and other servers
  255. mainServer, otherServers, serverErr := mainRack.(*Rack).PickNodesByWeight(rp.SameRackCount+1, option, func(node Node) error {
  256. if option.DataNode != "" && node.IsDataNode() && node.Id() != NodeId(option.DataNode) {
  257. return fmt.Errorf("Not matching preferred data node:%s", option.DataNode)
  258. }
  259. if useReservations {
  260. // For reservations, atomically check and reserve capacity
  261. if node.IsDataNode() {
  262. reservationId, success := node.TryReserveCapacity(option.DiskType, 1)
  263. if !success {
  264. return fmt.Errorf("Cannot reserve capacity on node %s", node.Id())
  265. }
  266. // Track the reservation for later cleanup if needed
  267. tentativeReservation.servers = append(tentativeReservation.servers, node.(*DataNode))
  268. tentativeReservation.reservationIds = append(tentativeReservation.reservationIds, reservationId)
  269. } else if availableSpaceFunc(node, option) < 1 {
  270. return fmt.Errorf("Free:%d < Expected:%d", availableSpaceFunc(node, option), 1)
  271. }
  272. } else if availableSpaceFunc(node, option) < 1 {
  273. return fmt.Errorf("Free:%d < Expected:%d", availableSpaceFunc(node, option), 1)
  274. }
  275. return nil
  276. })
  277. if serverErr != nil {
  278. return nil, nil, serverErr
  279. }
  280. servers = append(servers, mainServer.(*DataNode))
  281. for _, server := range otherServers {
  282. servers = append(servers, server.(*DataNode))
  283. }
  284. for _, rack := range otherRacks {
  285. r := rand.Int64N(availableSpaceFunc(rack, option))
  286. if server, e := reserveOneVolumeFunc(rack, r, option); e == nil {
  287. servers = append(servers, server)
  288. // If using reservations, also make a reservation on the selected server
  289. if useReservations {
  290. reservationId, success := server.TryReserveCapacity(option.DiskType, 1)
  291. if !success {
  292. return servers, nil, fmt.Errorf("failed to reserve capacity on server %s from other rack", server.Id())
  293. }
  294. tentativeReservation.servers = append(tentativeReservation.servers, server)
  295. tentativeReservation.reservationIds = append(tentativeReservation.reservationIds, reservationId)
  296. }
  297. } else {
  298. return servers, nil, e
  299. }
  300. }
  301. for _, datacenter := range otherDataCenters {
  302. r := rand.Int64N(availableSpaceFunc(datacenter, option))
  303. if server, e := reserveOneVolumeFunc(datacenter, r, option); e == nil {
  304. servers = append(servers, server)
  305. // If using reservations, also make a reservation on the selected server
  306. if useReservations {
  307. reservationId, success := server.TryReserveCapacity(option.DiskType, 1)
  308. if !success {
  309. return servers, nil, fmt.Errorf("failed to reserve capacity on server %s from other datacenter", server.Id())
  310. }
  311. tentativeReservation.servers = append(tentativeReservation.servers, server)
  312. tentativeReservation.reservationIds = append(tentativeReservation.reservationIds, reservationId)
  313. }
  314. } else {
  315. return servers, nil, e
  316. }
  317. }
  318. // If reservations were made, return the tentative reservation
  319. if useReservations && tentativeReservation != nil {
  320. reservation = tentativeReservation
  321. glog.V(1).Infof("Successfully reserved capacity on %d servers for volume creation", len(servers))
  322. }
  323. return servers, reservation, nil
  324. }
  325. // grow creates volumes on the provided servers, optionally managing capacity reservations
  326. func (vg *VolumeGrowth) grow(grpcDialOption grpc.DialOption, topo *Topology, vid needle.VolumeId, option *VolumeGrowOption, reservation *VolumeGrowReservation, servers ...*DataNode) (growErr error) {
  327. var createdVolumes []storage.VolumeInfo
  328. for _, server := range servers {
  329. if err := AllocateVolume(server, grpcDialOption, vid, option); err == nil {
  330. createdVolumes = append(createdVolumes, storage.VolumeInfo{
  331. Id: vid,
  332. Size: 0,
  333. Collection: option.Collection,
  334. ReplicaPlacement: option.ReplicaPlacement,
  335. Ttl: option.Ttl,
  336. Version: needle.Version(option.Version),
  337. DiskType: option.DiskType.String(),
  338. ModifiedAtSecond: time.Now().Unix(),
  339. })
  340. glog.V(0).Infof("Created Volume %d on %s", vid, server.NodeImpl.String())
  341. } else {
  342. glog.Warningf("Failed to assign volume %d on %s: %v", vid, server.NodeImpl.String(), err)
  343. growErr = fmt.Errorf("failed to assign volume %d on %s: %v", vid, server.NodeImpl.String(), err)
  344. break
  345. }
  346. }
  347. if growErr == nil {
  348. for i, vi := range createdVolumes {
  349. server := servers[i]
  350. server.AddOrUpdateVolume(vi)
  351. topo.RegisterVolumeLayout(vi, server)
  352. glog.V(0).Infof("Registered Volume %d on %s", vid, server.NodeImpl.String())
  353. }
  354. // Release reservations on success since volumes are now registered
  355. if reservation != nil {
  356. reservation.releaseAllReservations()
  357. }
  358. } else {
  359. // cleaning up created volume replicas
  360. for i, vi := range createdVolumes {
  361. server := servers[i]
  362. if err := DeleteVolume(server, grpcDialOption, vi.Id); err != nil {
  363. glog.Warningf("Failed to clean up volume %d on %s", vid, server.NodeImpl.String())
  364. }
  365. }
  366. // Reservations will be released by the caller in case of failure
  367. }
  368. return growErr
  369. }