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.

419 lines
12 KiB

7 years ago
  1. package command
  2. import (
  3. "context"
  4. "fmt"
  5. "github.com/chrislusf/seaweedfs/weed/operation"
  6. "github.com/chrislusf/seaweedfs/weed/pb/filer_pb"
  7. "github.com/chrislusf/seaweedfs/weed/security"
  8. "github.com/chrislusf/seaweedfs/weed/server"
  9. "github.com/chrislusf/seaweedfs/weed/util"
  10. "github.com/chrislusf/seaweedfs/weed/wdclient"
  11. "github.com/spf13/viper"
  12. "google.golang.org/grpc"
  13. "io"
  14. "io/ioutil"
  15. "net/http"
  16. "net/url"
  17. "os"
  18. "path/filepath"
  19. "strconv"
  20. "strings"
  21. "sync"
  22. "time"
  23. )
  24. var (
  25. copy CopyOptions
  26. waitGroup sync.WaitGroup
  27. )
  28. type CopyOptions struct {
  29. filerGrpcPort *int
  30. master *string
  31. include *string
  32. replication *string
  33. collection *string
  34. ttl *string
  35. maxMB *int
  36. grpcDialOption grpc.DialOption
  37. masterClient *wdclient.MasterClient
  38. concurrency *int
  39. }
  40. func init() {
  41. cmdCopy.Run = runCopy // break init cycle
  42. cmdCopy.IsDebug = cmdCopy.Flag.Bool("debug", false, "verbose debug information")
  43. copy.master = cmdCopy.Flag.String("master", "localhost:9333", "SeaweedFS master location")
  44. copy.include = cmdCopy.Flag.String("include", "", "pattens of files to copy, e.g., *.pdf, *.html, ab?d.txt, works together with -dir")
  45. copy.replication = cmdCopy.Flag.String("replication", "", "replication type")
  46. copy.collection = cmdCopy.Flag.String("collection", "", "optional collection name")
  47. copy.ttl = cmdCopy.Flag.String("ttl", "", "time to live, e.g.: 1m, 1h, 1d, 1M, 1y")
  48. copy.maxMB = cmdCopy.Flag.Int("maxMB", 32, "split files larger than the limit")
  49. copy.filerGrpcPort = cmdCopy.Flag.Int("filer.port.grpc", 0, "filer grpc server listen port, default to filer port + 10000")
  50. copy.concurrency = cmdCopy.Flag.Int("c", 8, "concurrent file copy goroutines")
  51. }
  52. var cmdCopy = &Command{
  53. UsageLine: "filer.copy file_or_dir1 [file_or_dir2 file_or_dir3] http://localhost:8888/path/to/a/folder/",
  54. Short: "copy one or a list of files to a filer folder",
  55. Long: `copy one or a list of files, or batch copy one whole folder recursively, to a filer folder
  56. It can copy one or a list of files or folders.
  57. If copying a whole folder recursively:
  58. All files under the folder and subfolders will be copyed.
  59. Optional parameter "-include" allows you to specify the file name patterns.
  60. If "maxMB" is set to a positive number, files larger than it would be split into chunks.
  61. `,
  62. }
  63. func runCopy(cmd *Command, args []string) bool {
  64. weed_server.LoadConfiguration("security", false)
  65. if len(args) <= 1 {
  66. return false
  67. }
  68. filerDestination := args[len(args)-1]
  69. fileOrDirs := args[0 : len(args)-1]
  70. filerUrl, err := url.Parse(filerDestination)
  71. if err != nil {
  72. fmt.Printf("The last argument should be a URL on filer: %v\n", err)
  73. return false
  74. }
  75. urlPath := filerUrl.Path
  76. if !strings.HasSuffix(urlPath, "/") {
  77. fmt.Printf("The last argument should be a folder and end with \"/\": %v\n", err)
  78. return false
  79. }
  80. if filerUrl.Port() == "" {
  81. fmt.Printf("The filer port should be specified.\n")
  82. return false
  83. }
  84. filerPort, parseErr := strconv.ParseUint(filerUrl.Port(), 10, 64)
  85. if parseErr != nil {
  86. fmt.Printf("The filer port parse error: %v\n", parseErr)
  87. return false
  88. }
  89. filerGrpcPort := filerPort + 10000
  90. if *copy.filerGrpcPort != 0 {
  91. filerGrpcPort = uint64(*copy.filerGrpcPort)
  92. }
  93. filerGrpcAddress := fmt.Sprintf("%s:%d", filerUrl.Hostname(), filerGrpcPort)
  94. copy.grpcDialOption = security.LoadClientTLS(viper.Sub("grpc"), "client")
  95. copy.masterClient = wdclient.NewMasterClient(context.Background(), copy.grpcDialOption, "client", strings.Split(*copy.master, ","))
  96. go copy.masterClient.KeepConnectedToMaster()
  97. copy.masterClient.WaitUntilConnected()
  98. if *cmdCopy.IsDebug {
  99. util.SetupProfiling("filer.copy.cpu.pprof", "filer.copy.mem.pprof")
  100. }
  101. fileCopyTaskChan := make(chan FileCopyTask, *copy.concurrency)
  102. ctx := context.Background()
  103. go func() {
  104. defer close(fileCopyTaskChan)
  105. for _, fileOrDir := range fileOrDirs {
  106. if err := genFileCopyTask(fileOrDir, urlPath, fileCopyTaskChan); err != nil {
  107. fmt.Fprintf(os.Stderr, "gen file list error: %v\n", err)
  108. break
  109. }
  110. }
  111. }()
  112. for i := 0; i < *copy.concurrency; i++ {
  113. waitGroup.Add(1)
  114. go func() {
  115. defer waitGroup.Done()
  116. worker := FileCopyWorker{
  117. options: &copy,
  118. filerHost: filerUrl.Host,
  119. filerGrpcAddress: filerGrpcAddress,
  120. }
  121. if err := worker.copyFiles(ctx, fileCopyTaskChan); err != nil {
  122. fmt.Fprintf(os.Stderr, "copy file error: %v\n", err)
  123. return
  124. }
  125. }()
  126. }
  127. waitGroup.Wait()
  128. return true
  129. }
  130. func genFileCopyTask(fileOrDir string, destPath string, fileCopyTaskChan chan FileCopyTask) error {
  131. fi, err := os.Stat(fileOrDir)
  132. if err != nil {
  133. fmt.Fprintf(os.Stderr, "Failed to get stat for file %s: %v\n", fileOrDir, err)
  134. return nil
  135. }
  136. mode := fi.Mode()
  137. if mode.IsDir() {
  138. files, _ := ioutil.ReadDir(fileOrDir)
  139. for _, subFileOrDir := range files {
  140. if err = genFileCopyTask(fileOrDir+"/"+subFileOrDir.Name(), destPath+fi.Name()+"/", fileCopyTaskChan); err != nil {
  141. return err
  142. }
  143. }
  144. return nil
  145. }
  146. uid, gid := util.GetFileUidGid(fi)
  147. fileCopyTaskChan <- FileCopyTask{
  148. sourceLocation: fileOrDir,
  149. destinationUrlPath: destPath,
  150. fileSize: fi.Size(),
  151. fileMode: fi.Mode(),
  152. uid: uid,
  153. gid: gid,
  154. }
  155. return nil
  156. }
  157. type FileCopyWorker struct {
  158. options *CopyOptions
  159. filerHost string
  160. filerGrpcAddress string
  161. }
  162. func (worker *FileCopyWorker) copyFiles(ctx context.Context, fileCopyTaskChan chan FileCopyTask) error {
  163. for task := range fileCopyTaskChan {
  164. if err := worker.doEachCopy(ctx, task); err != nil {
  165. return err
  166. }
  167. }
  168. return nil
  169. }
  170. type FileCopyTask struct {
  171. sourceLocation string
  172. destinationUrlPath string
  173. fileSize int64
  174. fileMode os.FileMode
  175. uid uint32
  176. gid uint32
  177. }
  178. func (worker *FileCopyWorker) doEachCopy(ctx context.Context, task FileCopyTask) error {
  179. f, err := os.Open(task.sourceLocation)
  180. if err != nil {
  181. fmt.Printf("Failed to open file %s: %v\n", task.sourceLocation, err)
  182. if _, ok := err.(*os.PathError); ok {
  183. fmt.Printf("skipping %s\n", task.sourceLocation)
  184. return nil
  185. }
  186. return err
  187. }
  188. defer f.Close()
  189. // this is a regular file
  190. if *worker.options.include != "" {
  191. if ok, _ := filepath.Match(*worker.options.include, filepath.Base(task.sourceLocation)); !ok {
  192. return nil
  193. }
  194. }
  195. // find the chunk count
  196. chunkSize := int64(*worker.options.maxMB * 1024 * 1024)
  197. chunkCount := 1
  198. if chunkSize > 0 && task.fileSize > chunkSize {
  199. chunkCount = int(task.fileSize/chunkSize) + 1
  200. }
  201. if chunkCount == 1 {
  202. return worker.uploadFileAsOne(ctx, task, f)
  203. }
  204. return worker.uploadFileInChunks(ctx, task, f, chunkCount, chunkSize)
  205. }
  206. func (worker *FileCopyWorker) uploadFileAsOne(ctx context.Context, task FileCopyTask, f *os.File) error {
  207. // upload the file content
  208. fileName := filepath.Base(f.Name())
  209. mimeType := detectMimeType(f)
  210. var chunks []*filer_pb.FileChunk
  211. if task.fileSize > 0 {
  212. // assign a volume
  213. assignResult, err := operation.Assign(worker.options.masterClient.GetMaster(), worker.options.grpcDialOption, &operation.VolumeAssignRequest{
  214. Count: 1,
  215. Replication: *worker.options.replication,
  216. Collection: *worker.options.collection,
  217. Ttl: *worker.options.ttl,
  218. })
  219. if err != nil {
  220. fmt.Printf("Failed to assign from %s: %v\n", *worker.options.master, err)
  221. }
  222. targetUrl := "http://" + assignResult.Url + "/" + assignResult.Fid
  223. uploadResult, err := operation.Upload(targetUrl, fileName, f, false, mimeType, nil, assignResult.Auth)
  224. if err != nil {
  225. return fmt.Errorf("upload data %v to %s: %v\n", fileName, targetUrl, err)
  226. }
  227. if uploadResult.Error != "" {
  228. return fmt.Errorf("upload %v to %s result: %v\n", fileName, targetUrl, uploadResult.Error)
  229. }
  230. fmt.Printf("uploaded %s to %s\n", fileName, targetUrl)
  231. chunks = append(chunks, &filer_pb.FileChunk{
  232. FileId: assignResult.Fid,
  233. Offset: 0,
  234. Size: uint64(uploadResult.Size),
  235. Mtime: time.Now().UnixNano(),
  236. ETag: uploadResult.ETag,
  237. })
  238. fmt.Printf("copied %s => http://%s%s%s\n", fileName, worker.filerHost, task.destinationUrlPath, fileName)
  239. }
  240. if err := withFilerClient(ctx, worker.filerGrpcAddress, worker.options.grpcDialOption, func(client filer_pb.SeaweedFilerClient) error {
  241. request := &filer_pb.CreateEntryRequest{
  242. Directory: task.destinationUrlPath,
  243. Entry: &filer_pb.Entry{
  244. Name: fileName,
  245. Attributes: &filer_pb.FuseAttributes{
  246. Crtime: time.Now().Unix(),
  247. Mtime: time.Now().Unix(),
  248. Gid: task.gid,
  249. Uid: task.uid,
  250. FileSize: uint64(task.fileSize),
  251. FileMode: uint32(task.fileMode),
  252. Mime: mimeType,
  253. Replication: *worker.options.replication,
  254. Collection: *worker.options.collection,
  255. TtlSec: int32(util.ParseInt(*worker.options.ttl, 0)),
  256. },
  257. Chunks: chunks,
  258. },
  259. }
  260. if _, err := client.CreateEntry(ctx, request); err != nil {
  261. return fmt.Errorf("update fh: %v", err)
  262. }
  263. return nil
  264. }); err != nil {
  265. return fmt.Errorf("upload data %v to http://%s%s%s: %v\n", fileName, worker.filerHost, task.destinationUrlPath, fileName, err)
  266. }
  267. return nil
  268. }
  269. func (worker *FileCopyWorker) uploadFileInChunks(ctx context.Context, task FileCopyTask, f *os.File, chunkCount int, chunkSize int64) error {
  270. fileName := filepath.Base(f.Name())
  271. mimeType := detectMimeType(f)
  272. var chunks []*filer_pb.FileChunk
  273. for i := int64(0); i < int64(chunkCount); i++ {
  274. // assign a volume
  275. assignResult, err := operation.Assign(worker.options.masterClient.GetMaster(), worker.options.grpcDialOption, &operation.VolumeAssignRequest{
  276. Count: 1,
  277. Replication: *worker.options.replication,
  278. Collection: *worker.options.collection,
  279. Ttl: *worker.options.ttl,
  280. })
  281. if err != nil {
  282. fmt.Printf("Failed to assign from %s: %v\n", *worker.options.master, err)
  283. }
  284. targetUrl := "http://" + assignResult.Url + "/" + assignResult.Fid
  285. uploadResult, err := operation.Upload(targetUrl,
  286. fileName+"-"+strconv.FormatInt(i+1, 10),
  287. io.LimitReader(f, chunkSize),
  288. false, "application/octet-stream", nil, assignResult.Auth)
  289. if err != nil {
  290. return fmt.Errorf("upload data %v to %s: %v\n", fileName, targetUrl, err)
  291. }
  292. if uploadResult.Error != "" {
  293. return fmt.Errorf("upload %v to %s result: %v\n", fileName, targetUrl, uploadResult.Error)
  294. }
  295. chunks = append(chunks, &filer_pb.FileChunk{
  296. FileId: assignResult.Fid,
  297. Offset: i * chunkSize,
  298. Size: uint64(uploadResult.Size),
  299. Mtime: time.Now().UnixNano(),
  300. ETag: uploadResult.ETag,
  301. })
  302. fmt.Printf("uploaded %s-%d to %s [%d,%d)\n", fileName, i+1, targetUrl, i*chunkSize, i*chunkSize+int64(uploadResult.Size))
  303. }
  304. if err := withFilerClient(ctx, worker.filerGrpcAddress, worker.options.grpcDialOption, func(client filer_pb.SeaweedFilerClient) error {
  305. request := &filer_pb.CreateEntryRequest{
  306. Directory: task.destinationUrlPath,
  307. Entry: &filer_pb.Entry{
  308. Name: fileName,
  309. Attributes: &filer_pb.FuseAttributes{
  310. Crtime: time.Now().Unix(),
  311. Mtime: time.Now().Unix(),
  312. Gid: task.gid,
  313. Uid: task.uid,
  314. FileSize: uint64(task.fileSize),
  315. FileMode: uint32(task.fileMode),
  316. Mime: mimeType,
  317. Replication: *worker.options.replication,
  318. Collection: *worker.options.collection,
  319. TtlSec: int32(util.ParseInt(*worker.options.ttl, 0)),
  320. },
  321. Chunks: chunks,
  322. },
  323. }
  324. if _, err := client.CreateEntry(ctx, request); err != nil {
  325. return fmt.Errorf("update fh: %v", err)
  326. }
  327. return nil
  328. }); err != nil {
  329. return fmt.Errorf("upload data %v to http://%s%s%s: %v\n", fileName, worker.filerHost, task.destinationUrlPath, fileName, err)
  330. }
  331. fmt.Printf("copied %s => http://%s%s%s\n", fileName, worker.filerHost, task.destinationUrlPath, fileName)
  332. return nil
  333. }
  334. func detectMimeType(f *os.File) string {
  335. head := make([]byte, 512)
  336. f.Seek(0, io.SeekStart)
  337. n, err := f.Read(head)
  338. if err == io.EOF {
  339. return ""
  340. }
  341. if err != nil {
  342. fmt.Printf("read head of %v: %v\n", f.Name(), err)
  343. return "application/octet-stream"
  344. }
  345. f.Seek(0, io.SeekStart)
  346. mimeType := http.DetectContentType(head[:n])
  347. return mimeType
  348. }
  349. func withFilerClient(ctx context.Context, filerAddress string, grpcDialOption grpc.DialOption, fn func(filer_pb.SeaweedFilerClient) error) error {
  350. return util.WithCachedGrpcClient(ctx, func(clientConn *grpc.ClientConn) error {
  351. client := filer_pb.NewSeaweedFilerClient(clientConn)
  352. return fn(client)
  353. }, filerAddress, grpcDialOption)
  354. }