2015-09-27 08:50:54 +00:00
// Copyright (C) 2014 The Syncthing Authors.
2014-09-29 19:43:32 +00:00
//
2015-03-07 20:36:35 +00:00
// This Source Code Form is subject to the terms of the Mozilla Public
// License, v. 2.0. If a copy of the MPL was not distributed with this file,
2015-09-27 08:50:54 +00:00
// You can obtain one at http://mozilla.org/MPL/2.0/.
2014-06-01 20:50:14 +00:00
2014-05-15 03:26:55 +00:00
package model
2014-03-28 13:36:57 +00:00
import (
"errors"
2014-08-25 15:45:13 +00:00
"fmt"
2014-11-09 04:26:52 +00:00
"io/ioutil"
2015-01-02 15:15:53 +00:00
"math/rand"
2014-03-28 13:36:57 +00:00
"os"
"path/filepath"
2015-06-26 11:31:30 +00:00
"sort"
2016-01-03 20:15:02 +00:00
"strings"
2014-03-28 13:36:57 +00:00
"time"
2014-06-19 22:27:54 +00:00
2015-08-06 09:29:25 +00:00
"github.com/syncthing/syncthing/lib/config"
"github.com/syncthing/syncthing/lib/db"
"github.com/syncthing/syncthing/lib/events"
"github.com/syncthing/syncthing/lib/ignore"
"github.com/syncthing/syncthing/lib/osutil"
2015-09-22 17:38:46 +00:00
"github.com/syncthing/syncthing/lib/protocol"
2015-08-06 09:29:25 +00:00
"github.com/syncthing/syncthing/lib/scanner"
"github.com/syncthing/syncthing/lib/symlinks"
"github.com/syncthing/syncthing/lib/sync"
"github.com/syncthing/syncthing/lib/versioner"
2014-03-28 13:36:57 +00:00
)
2014-09-27 12:44:15 +00:00
// TODO: Stop on errors
2014-03-28 13:36:57 +00:00
2014-09-27 12:44:15 +00:00
// A pullBlockState is passed to the puller routine for each block that needs
// to be fetched.
type pullBlockState struct {
* sharedPullerState
block protocol . BlockInfo
2014-03-28 13:36:57 +00:00
}
2014-09-27 12:44:15 +00:00
// A copyBlocksState is passed to copy routine if the file has blocks to be
2014-10-08 22:41:23 +00:00
// copied.
2014-09-27 12:44:15 +00:00
type copyBlocksState struct {
* sharedPullerState
blocks [ ] protocol . BlockInfo
2014-03-28 13:36:57 +00:00
}
2015-07-03 09:25:35 +00:00
// Which filemode bits to preserve
const retainBits = os . ModeSetgid | os . ModeSetuid | os . ModeSticky
2014-09-27 12:44:15 +00:00
var (
2014-09-28 11:05:25 +00:00
activity = newDeviceActivity ( )
2015-10-25 20:46:09 +00:00
errNoDevice = errors . New ( "peers who had this file went away, or the file has changed while syncing. will retry later" )
2014-09-27 12:44:15 +00:00
)
2015-06-16 11:12:34 +00:00
const (
dbUpdateHandleDir = iota
dbUpdateDeleteDir
dbUpdateHandleFile
dbUpdateDeleteFile
dbUpdateShortcutFile
)
2015-08-14 07:37:04 +00:00
const (
2015-10-07 23:25:32 +00:00
defaultCopiers = 1
defaultPullers = 16
defaultPullerSleep = 10 * time . Second
defaultPullerPause = 60 * time . Second
2015-08-14 07:37:04 +00:00
)
2015-06-16 11:12:34 +00:00
type dbUpdateJob struct {
file protocol . FileInfo
jobType int
}
2015-03-16 20:14:19 +00:00
type rwFolder struct {
stateTracker
2015-05-13 14:57:29 +00:00
model * Model
progressEmitter * ProgressEmitter
virtualMtimeRepo * db . VirtualMtimeRepo
2015-12-19 12:57:44 +00:00
folder string
dir string
scanIntv time . Duration
versioner versioner . Versioner
ignorePerms bool
copiers int
pullers int
2016-01-20 19:10:22 +00:00
shortID protocol . ShortID
2015-12-19 12:57:44 +00:00
order config . PullOrder
maxConflicts int
sleep time . Duration
pause time . Duration
allowSparse bool
checkFreeSpace bool
2015-03-16 20:14:19 +00:00
2015-05-07 20:45:07 +00:00
stop chan struct { }
queue * jobQueue
2015-06-16 11:12:34 +00:00
dbUpdates chan dbUpdateJob
2015-05-07 20:45:07 +00:00
scanTimer * time . Timer
pullTimer * time . Timer
delayScan chan time . Duration
2015-06-20 17:26:25 +00:00
scanNow chan rescanRequest
2015-05-07 20:45:07 +00:00
remoteIndex chan struct { } // An index update was received, we should re-evaluate needs
2015-06-26 11:31:30 +00:00
errors map [ string ] string // path -> error string
errorsMut sync . Mutex
2015-03-16 20:14:19 +00:00
}
2016-01-20 19:10:22 +00:00
func newRWFolder ( m * Model , shortID protocol . ShortID , cfg config . FolderConfiguration ) * rwFolder {
2015-08-14 07:37:04 +00:00
p := & rwFolder {
2015-04-22 22:54:31 +00:00
stateTracker : stateTracker {
folder : cfg . ID ,
mut : sync . NewMutex ( ) ,
} ,
2015-03-16 20:14:19 +00:00
2015-05-13 14:57:29 +00:00
model : m ,
progressEmitter : m . progressEmitter ,
virtualMtimeRepo : db . NewVirtualMtimeRepo ( m . db , cfg . ID ) ,
2015-03-16 20:14:19 +00:00
2015-12-19 12:57:44 +00:00
folder : cfg . ID ,
dir : cfg . Path ( ) ,
scanIntv : time . Duration ( cfg . RescanIntervalS ) * time . Second ,
ignorePerms : cfg . IgnorePerms ,
copiers : cfg . Copiers ,
pullers : cfg . Pullers ,
shortID : shortID ,
order : cfg . Order ,
maxConflicts : cfg . MaxConflicts ,
allowSparse : ! cfg . DisableSparseFiles ,
checkFreeSpace : cfg . MinDiskFreePct != 0 ,
2015-03-16 20:14:19 +00:00
2015-05-07 20:45:07 +00:00
stop : make ( chan struct { } ) ,
queue : newJobQueue ( ) ,
2015-10-07 23:25:32 +00:00
pullTimer : time . NewTimer ( time . Second ) ,
2015-05-07 20:45:07 +00:00
scanTimer : time . NewTimer ( time . Millisecond ) , // The first scan should be done immediately.
delayScan : make ( chan time . Duration ) ,
2015-06-20 17:26:25 +00:00
scanNow : make ( chan rescanRequest ) ,
2015-05-07 20:45:07 +00:00
remoteIndex : make ( chan struct { } , 1 ) , // This needs to be 1-buffered so that we queue a notification if we're busy doing a pull when it comes.
2015-06-26 11:31:30 +00:00
errorsMut : sync . NewMutex ( ) ,
2015-03-16 20:14:19 +00:00
}
2015-08-14 07:37:04 +00:00
if p . copiers == 0 {
p . copiers = defaultCopiers
}
if p . pullers == 0 {
p . pullers = defaultPullers
}
2015-10-07 23:25:32 +00:00
if cfg . PullerPauseS == 0 {
p . pause = defaultPullerPause
} else {
p . pause = time . Duration ( cfg . PullerPauseS ) * time . Second
}
if cfg . PullerSleepS == 0 {
p . sleep = defaultPullerSleep
} else {
p . sleep = time . Duration ( cfg . PullerSleepS ) * time . Second
}
2015-08-14 07:37:04 +00:00
return p
2014-03-28 13:36:57 +00:00
}
2015-05-25 10:43:19 +00:00
// Helper function to check whether either the ignorePerm flag has been
// set on the local host or the FlagNoPermBits has been set on the file/dir
// which is being pulled.
func ( p * rwFolder ) ignorePermissions ( file protocol . FileInfo ) bool {
return p . ignorePerms || file . Flags & protocol . FlagNoPermBits != 0
}
2014-09-27 12:44:15 +00:00
// Serve will run scans and pulls. It will return when Stop()ed or on a
// critical error.
2015-03-16 20:14:19 +00:00
func ( p * rwFolder ) Serve ( ) {
2015-10-03 15:25:21 +00:00
l . Debugln ( p , "starting" )
defer l . Debugln ( p , "exiting" )
2014-03-28 13:36:57 +00:00
2014-09-27 12:44:15 +00:00
defer func ( ) {
2015-05-01 12:30:17 +00:00
p . pullTimer . Stop ( )
p . scanTimer . Stop ( )
2014-09-28 11:00:38 +00:00
// TODO: Should there be an actual FolderStopped state?
2015-03-16 20:14:19 +00:00
p . setState ( FolderIdle )
2014-09-27 12:44:15 +00:00
} ( )
2014-03-28 13:36:57 +00:00
2015-01-18 01:12:06 +00:00
var prevVer int64
2014-12-23 09:06:51 +00:00
var prevIgnoreHash string
2014-07-24 07:38:16 +00:00
2015-03-28 14:25:42 +00:00
rescheduleScan := func ( ) {
2015-04-25 06:27:45 +00:00
if p . scanIntv == 0 {
// We should not run scans, so it should not be rescheduled.
return
}
2015-03-28 14:25:42 +00:00
// Sleep a random time between 3/4 and 5/4 of the configured interval.
sleepNanos := ( p . scanIntv . Nanoseconds ( ) * 3 + rand . Int63n ( 2 * p . scanIntv . Nanoseconds ( ) ) ) / 4
intv := time . Duration ( sleepNanos ) * time . Nanosecond
2015-10-03 15:25:21 +00:00
l . Debugln ( p , "next rescan in" , intv )
2015-05-01 12:30:17 +00:00
p . scanTimer . Reset ( intv )
2015-03-28 14:25:42 +00:00
}
2014-09-30 15:34:31 +00:00
// We don't start pulling files until a scan has been completed.
initialScanCompleted := false
2014-03-28 13:36:57 +00:00
for {
2014-09-27 12:44:15 +00:00
select {
case <- p . stop :
return
2014-07-24 07:38:16 +00:00
2015-05-07 20:45:07 +00:00
case <- p . remoteIndex :
prevVer = 0
2015-10-07 23:25:32 +00:00
p . pullTimer . Reset ( 0 )
2015-10-03 15:25:21 +00:00
l . Debugln ( p , "remote index updated, rescheduling pull" )
2015-05-07 20:45:07 +00:00
2015-05-01 12:30:17 +00:00
case <- p . pullTimer . C :
2014-09-30 15:34:31 +00:00
if ! initialScanCompleted {
2015-10-03 15:25:21 +00:00
l . Debugln ( p , "skip (initial)" )
2015-10-07 23:25:32 +00:00
p . pullTimer . Reset ( p . sleep )
2015-09-12 21:00:43 +00:00
continue
}
2014-12-23 09:06:51 +00:00
p . model . fmut . RLock ( )
curIgnores := p . model . folderIgnores [ p . folder ]
p . model . fmut . RUnlock ( )
if newHash := curIgnores . Hash ( ) ; newHash != prevIgnoreHash {
// The ignore patterns have changed. We need to re-evaluate if
// there are files we need now that were ignored before.
2015-10-03 15:25:21 +00:00
l . Debugln ( p , "ignore patterns have changed, resetting prevVer" )
2014-12-23 09:06:51 +00:00
prevVer = 0
prevIgnoreHash = newHash
}
2014-09-27 12:44:15 +00:00
// RemoteLocalVersion() is a fast call, doesn't touch the database.
2015-06-24 07:52:38 +00:00
curVer , ok := p . model . RemoteLocalVersion ( p . folder )
if ! ok || curVer == prevVer {
2015-10-03 15:25:21 +00:00
l . Debugln ( p , "skip (curVer == prevVer)" , prevVer , ok )
2015-10-07 23:25:32 +00:00
p . pullTimer . Reset ( p . sleep )
continue
}
if err := p . model . CheckFolderHealth ( p . folder ) ; err != nil {
l . Infoln ( "Skipping folder" , p . folder , "pull due to folder error:" , err )
p . pullTimer . Reset ( p . sleep )
2014-09-27 12:44:15 +00:00
continue
2014-03-28 13:36:57 +00:00
}
2014-08-04 20:02:44 +00:00
2015-10-03 15:25:21 +00:00
l . Debugln ( p , "pulling" , prevVer , curVer )
2015-06-26 11:31:30 +00:00
2015-03-16 20:14:19 +00:00
p . setState ( FolderSyncing )
2015-06-26 11:31:30 +00:00
p . clearErrors ( )
2014-09-27 12:44:15 +00:00
tries := 0
2015-06-26 11:31:30 +00:00
2014-09-27 12:44:15 +00:00
for {
tries ++
2014-10-24 22:20:08 +00:00
2014-12-28 23:11:32 +00:00
changed := p . pullerIteration ( curIgnores )
2015-10-03 15:25:21 +00:00
l . Debugln ( p , "changed" , changed )
2014-03-28 13:36:57 +00:00
2014-09-27 12:44:15 +00:00
if changed == 0 {
// No files were changed by the puller, so we are in
// sync. Remember the local version number and
// schedule a resync a little bit into the future.
2014-09-28 05:56:05 +00:00
2015-06-24 07:52:38 +00:00
if lv , ok := p . model . RemoteLocalVersion ( p . folder ) ; ok && lv < curVer {
2014-09-28 11:00:38 +00:00
// There's a corner case where the device we needed
2014-09-28 05:56:05 +00:00
// files from disconnected during the puller
// iteration. The files will have been removed from
// the index, so we've concluded that we don't need
// them, but at the same time we have the local
// version that includes those files in curVer. So we
// catch the case that localVersion might have
2014-12-23 09:06:51 +00:00
// decreased here.
2014-09-28 11:05:25 +00:00
l . Debugln ( p , "adjusting curVer" , lv )
2014-09-28 05:56:05 +00:00
curVer = lv
}
2014-09-27 12:44:15 +00:00
prevVer = curVer
2015-10-07 23:25:32 +00:00
l . Debugln ( p , "next pull in" , p . sleep )
p . pullTimer . Reset ( p . sleep )
2014-09-27 12:44:15 +00:00
break
}
2014-04-01 21:18:32 +00:00
2014-09-27 12:44:15 +00:00
if tries > 10 {
// We've tried a bunch of times to get in sync, but
// we're not making it. Probably there are write
// errors preventing us. Flag this with a warning and
// wait a bit longer before retrying.
2015-10-07 23:25:32 +00:00
l . Infof ( "Folder %q isn't making progress. Pausing puller for %v." , p . folder , p . pause )
l . Debugln ( p , "next pull in" , p . pause )
2015-06-26 11:31:30 +00:00
if folderErrors := p . currentErrors ( ) ; len ( folderErrors ) > 0 {
events . Default . Log ( events . FolderErrors , map [ string ] interface { } {
"folder" : p . folder ,
"errors" : folderErrors ,
} )
}
2015-10-07 23:25:32 +00:00
p . pullTimer . Reset ( p . pause )
2014-09-27 12:44:15 +00:00
break
}
}
2015-03-16 20:14:19 +00:00
p . setState ( FolderIdle )
2014-04-14 07:58:17 +00:00
2014-09-27 12:44:15 +00:00
// The reason for running the scanner from within the puller is that
// this is the easiest way to make sure we are not doing both at the
// same time.
2015-05-01 12:30:17 +00:00
case <- p . scanTimer . C :
2015-03-28 14:25:42 +00:00
if err := p . model . CheckFolderHealth ( p . folder ) ; err != nil {
l . Infoln ( "Skipping folder" , p . folder , "scan due to folder error:" , err )
rescheduleScan ( )
continue
}
2015-10-03 15:25:21 +00:00
l . Debugln ( p , "rescan" )
2015-03-28 14:25:42 +00:00
2015-06-20 17:26:25 +00:00
if err := p . model . internalScanFolderSubs ( p . folder , nil ) ; err != nil {
2015-03-28 14:25:42 +00:00
// Potentially sets the error twice, once in the scanner just
// by doing a check, and once here, if the error returned is
// the same one as returned by CheckFolderHealth, though
2015-04-12 20:12:01 +00:00
// duplicate set is handled by setError.
p . setError ( err )
2015-03-28 14:25:42 +00:00
rescheduleScan ( )
continue
2014-05-04 16:20:25 +00:00
}
2015-01-02 15:15:53 +00:00
2015-03-28 14:25:42 +00:00
if p . scanIntv > 0 {
rescheduleScan ( )
2014-10-15 08:51:09 +00:00
}
2014-09-30 15:34:31 +00:00
if ! initialScanCompleted {
l . Infoln ( "Completed initial scan (rw) of folder" , p . folder )
initialScanCompleted = true
}
2015-05-03 12:18:32 +00:00
2015-06-20 17:26:25 +00:00
case req := <- p . scanNow :
if err := p . model . CheckFolderHealth ( p . folder ) ; err != nil {
l . Infoln ( "Skipping folder" , p . folder , "scan due to folder error:" , err )
req . err <- err
continue
}
2015-10-03 15:25:21 +00:00
l . Debugln ( p , "forced rescan" )
2015-06-20 17:26:25 +00:00
if err := p . model . internalScanFolderSubs ( p . folder , req . subs ) ; err != nil {
// Potentially sets the error twice, once in the scanner just
// by doing a check, and once here, if the error returned is
// the same one as returned by CheckFolderHealth, though
// duplicate set is handled by setError.
p . setError ( err )
req . err <- err
continue
}
req . err <- nil
2015-05-03 12:18:32 +00:00
case next := <- p . delayScan :
p . scanTimer . Reset ( next )
2014-03-28 13:36:57 +00:00
}
}
}
2015-03-16 20:14:19 +00:00
func ( p * rwFolder ) Stop ( ) {
2014-09-27 12:44:15 +00:00
close ( p . stop )
2014-03-28 13:36:57 +00:00
}
2015-05-07 20:45:07 +00:00
func ( p * rwFolder ) IndexUpdated ( ) {
select {
case p . remoteIndex <- struct { } { } :
default :
// We might be busy doing a pull and thus not reading from this
// channel. The channel is 1-buffered, so one notification will be
// queued to ensure we recheck after the pull, but beyond that we must
// make sure to not block index receiving.
}
}
2015-06-20 17:26:25 +00:00
func ( p * rwFolder ) Scan ( subs [ ] string ) error {
req := rescanRequest {
subs : subs ,
err : make ( chan error ) ,
}
p . scanNow <- req
return <- req . err
}
2015-03-16 20:14:19 +00:00
func ( p * rwFolder ) String ( ) string {
return fmt . Sprintf ( "rwFolder/%s@%p" , p . folder , p )
2014-09-27 12:44:15 +00:00
}
2014-09-07 19:29:06 +00:00
2014-09-28 11:00:38 +00:00
// pullerIteration runs a single puller iteration for the given folder and
2014-09-27 12:44:15 +00:00
// returns the number items that should have been synced (even those that
// might have failed). One puller iteration handles all files currently
2014-11-23 00:02:09 +00:00
// flagged as needed in the folder.
2015-03-16 20:14:19 +00:00
func ( p * rwFolder ) pullerIteration ( ignores * ignore . Matcher ) int {
2014-09-27 12:44:15 +00:00
pullChan := make ( chan pullBlockState )
copyChan := make ( chan copyBlocksState )
finisherChan := make ( chan * sharedPullerState )
2015-04-22 22:54:31 +00:00
updateWg := sync . NewWaitGroup ( )
copyWg := sync . NewWaitGroup ( )
pullWg := sync . NewWaitGroup ( )
doneWg := sync . NewWaitGroup ( )
2014-09-27 12:44:15 +00:00
2015-10-03 15:25:21 +00:00
l . Debugln ( p , "c" , p . copiers , "p" , p . pullers )
2014-11-23 18:43:49 +00:00
2015-06-16 11:12:34 +00:00
p . dbUpdates = make ( chan dbUpdateJob )
2015-04-05 13:34:29 +00:00
updateWg . Add ( 1 )
go func ( ) {
// dbUpdaterRoutine finishes when p.dbUpdates is closed
p . dbUpdaterRoutine ( )
updateWg . Done ( )
} ( )
2014-11-23 00:02:09 +00:00
for i := 0 ; i < p . copiers ; i ++ {
2014-10-08 22:41:23 +00:00
copyWg . Add ( 1 )
2014-09-27 12:44:15 +00:00
go func ( ) {
// copierRoutine finishes when copyChan is closed
2014-12-28 23:11:32 +00:00
p . copierRoutine ( copyChan , pullChan , finisherChan )
2014-10-08 22:41:23 +00:00
copyWg . Done ( )
2014-09-27 12:44:15 +00:00
} ( )
}
2014-11-23 00:02:09 +00:00
for i := 0 ; i < p . pullers ; i ++ {
2014-10-08 22:41:23 +00:00
pullWg . Add ( 1 )
2014-09-27 12:44:15 +00:00
go func ( ) {
// pullerRoutine finishes when pullChan is closed
p . pullerRoutine ( pullChan , finisherChan )
2014-10-08 22:41:23 +00:00
pullWg . Done ( )
2014-09-27 12:44:15 +00:00
} ( )
}
2014-12-24 23:12:12 +00:00
doneWg . Add ( 1 )
// finisherRoutine finishes when finisherChan is closed
go func ( ) {
p . finisherRoutine ( finisherChan )
doneWg . Done ( )
} ( )
2014-09-27 12:44:15 +00:00
2014-09-28 11:39:39 +00:00
p . model . fmut . RLock ( )
2015-01-09 07:18:42 +00:00
folderFiles := p . model . folderFiles [ p . folder ]
2014-09-28 11:39:39 +00:00
p . model . fmut . RUnlock ( )
2014-09-27 12:44:15 +00:00
// !!!
// WithNeed takes a database snapshot (by necessity). By the time we've
// handled a bunch of files it might have become out of date and we might
// be attempting to sync with an old version of a file...
// !!!
changed := 0
2014-10-12 21:01:57 +00:00
2014-12-19 23:12:12 +00:00
fileDeletions := map [ string ] protocol . FileInfo { }
dirDeletions := [ ] protocol . FileInfo { }
buckets := map [ string ] [ ] protocol . FileInfo { }
2014-10-12 21:01:57 +00:00
2016-01-16 17:18:37 +00:00
handleFile := func ( f protocol . FileInfo ) bool {
2014-09-27 12:44:15 +00:00
switch {
2016-01-16 17:18:37 +00:00
case f . IsDeleted ( ) :
2014-11-09 04:26:52 +00:00
// A deleted file, directory or symlink
2016-01-16 17:18:37 +00:00
if f . IsDirectory ( ) {
dirDeletions = append ( dirDeletions , f )
2014-12-19 23:12:12 +00:00
} else {
2016-01-16 17:18:37 +00:00
fileDeletions [ f . Name ] = f
df , ok := p . model . CurrentFolderFile ( p . folder , f . Name )
2014-12-19 23:12:12 +00:00
// Local file can be already deleted, but with a lower version
// number, hence the deletion coming in again as part of
2015-01-30 14:32:59 +00:00
// WithNeed, furthermore, the file can simply be of the wrong
// type if we haven't yet managed to pull it.
if ok && ! df . IsDeleted ( ) && ! df . IsSymlink ( ) && ! df . IsDirectory ( ) {
2014-12-19 23:12:12 +00:00
// Put files into buckets per first hash
key := string ( df . Blocks [ 0 ] . Hash )
buckets [ key ] = append ( buckets [ key ] , df )
}
}
2016-01-16 17:18:37 +00:00
case f . IsDirectory ( ) && ! f . IsSymlink ( ) :
2014-09-27 12:44:15 +00:00
// A new or changed directory
2016-01-16 17:18:37 +00:00
l . Debugln ( "Creating directory" , f . Name )
p . handleDir ( f )
2014-09-27 12:44:15 +00:00
default :
2016-01-16 17:18:37 +00:00
return false
}
return true
}
folderFiles . WithNeed ( protocol . LocalDeviceID , func ( intf db . FileIntf ) bool {
// Needed items are delivered sorted lexicographically. We'll handle
// directories as they come along, so parents before children. Files
// are queued and the order may be changed later.
file := intf . ( protocol . FileInfo )
if ignores . Match ( file . Name ) {
// This is an ignored file. Skip it, continue iteration.
return true
}
l . Debugln ( p , "handling" , file . Name )
if ! handleFile ( file ) {
2014-11-09 04:26:52 +00:00
// A new or changed file or symlink. This is the only case where we
Add job queue (fixes #629)
Request to terminate currently ongoing downloads and jump to the bumped file
incoming in 3, 2, 1.
Also, has a slightly strange effect where we pop a job off the queue, but
the copyChannel is still busy and blocks, though it gets moved to the
progress slice in the jobqueue, and looks like it's in progress which it isn't
as it's waiting to be picked up from the copyChan.
As a result, the progress emitter doesn't register on the task, and hence the file
doesn't have a progress bar, but cannot be replaced by a bump.
I guess I can fix progress bar issue by moving the progressEmiter.Register just
before passing the file to the copyChan, but then we are back to the initial
problem of a file with a progress bar, but no progress happening as it's stuck
on write to copyChan
I checked if there is a way to check for channel writeability (before popping)
but got struck by lightning just for bringing the idea up in #go-nuts.
My ideal scenario would be to check if copyChan is writeable, pop job from the
queue and shove it down handleFile. This way jobs would stay in the queue while
they cannot be handled, meaning that the `Bump` could bring your file up higher.
2014-12-01 19:23:06 +00:00
// do stuff concurrently in the background
2015-04-25 05:13:53 +00:00
p . queue . Push ( file . Name , file . Size ( ) , file . Modified )
2014-04-01 21:18:32 +00:00
}
2014-09-27 12:44:15 +00:00
changed ++
return true
} )
2014-04-01 21:18:32 +00:00
2015-04-25 05:13:53 +00:00
// Reorder the file queue according to configuration
switch p . order {
case config . OrderRandom :
p . queue . Shuffle ( )
case config . OrderAlphabetic :
// The queue is already in alphabetic order.
case config . OrderSmallestFirst :
p . queue . SortSmallestFirst ( )
case config . OrderLargestFirst :
p . queue . SortLargestFirst ( )
case config . OrderOldestFirst :
p . queue . SortOldestFirst ( )
case config . OrderNewestFirst :
2015-08-19 07:42:52 +00:00
p . queue . SortNewestFirst ( )
2015-04-25 05:13:53 +00:00
}
// Process the file queue
2014-12-19 23:12:12 +00:00
nextFile :
Add job queue (fixes #629)
Request to terminate currently ongoing downloads and jump to the bumped file
incoming in 3, 2, 1.
Also, has a slightly strange effect where we pop a job off the queue, but
the copyChannel is still busy and blocks, though it gets moved to the
progress slice in the jobqueue, and looks like it's in progress which it isn't
as it's waiting to be picked up from the copyChan.
As a result, the progress emitter doesn't register on the task, and hence the file
doesn't have a progress bar, but cannot be replaced by a bump.
I guess I can fix progress bar issue by moving the progressEmiter.Register just
before passing the file to the copyChan, but then we are back to the initial
problem of a file with a progress bar, but no progress happening as it's stuck
on write to copyChan
I checked if there is a way to check for channel writeability (before popping)
but got struck by lightning just for bringing the idea up in #go-nuts.
My ideal scenario would be to check if copyChan is writeable, pop job from the
queue and shove it down handleFile. This way jobs would stay in the queue while
they cannot be handled, meaning that the `Bump` could bring your file up higher.
2014-12-01 19:23:06 +00:00
for {
2016-01-16 20:42:32 +00:00
select {
case <- p . stop :
// Stop processing files if the puller has been told to stop.
break
default :
}
2014-12-30 08:31:34 +00:00
fileName , ok := p . queue . Pop ( )
if ! ok {
Add job queue (fixes #629)
Request to terminate currently ongoing downloads and jump to the bumped file
incoming in 3, 2, 1.
Also, has a slightly strange effect where we pop a job off the queue, but
the copyChannel is still busy and blocks, though it gets moved to the
progress slice in the jobqueue, and looks like it's in progress which it isn't
as it's waiting to be picked up from the copyChan.
As a result, the progress emitter doesn't register on the task, and hence the file
doesn't have a progress bar, but cannot be replaced by a bump.
I guess I can fix progress bar issue by moving the progressEmiter.Register just
before passing the file to the copyChan, but then we are back to the initial
problem of a file with a progress bar, but no progress happening as it's stuck
on write to copyChan
I checked if there is a way to check for channel writeability (before popping)
but got struck by lightning just for bringing the idea up in #go-nuts.
My ideal scenario would be to check if copyChan is writeable, pop job from the
queue and shove it down handleFile. This way jobs would stay in the queue while
they cannot be handled, meaning that the `Bump` could bring your file up higher.
2014-12-01 19:23:06 +00:00
break
}
2014-12-19 23:12:12 +00:00
f , ok := p . model . CurrentGlobalFile ( p . folder , fileName )
if ! ok {
2015-01-06 21:12:45 +00:00
// File is no longer in the index. Mark it as done and drop it.
p . queue . Done ( fileName )
2014-12-19 23:12:12 +00:00
continue
}
2016-01-16 17:18:37 +00:00
// Handles races where an index update arrives changing what the file
// is between queueing and retrieving it from the queue, effectively
// changing how the file should be handled.
if handleFile ( f ) {
continue
}
if ! f . IsSymlink ( ) {
2014-12-19 23:12:12 +00:00
key := string ( f . Blocks [ 0 ] . Hash )
for i , candidate := range buckets [ key ] {
if scanner . BlocksEqual ( candidate . Blocks , f . Blocks ) {
// Remove the candidate from the bucket
2015-01-30 14:32:59 +00:00
lidx := len ( buckets [ key ] ) - 1
buckets [ key ] [ i ] = buckets [ key ] [ lidx ]
buckets [ key ] = buckets [ key ] [ : lidx ]
// candidate is our current state of the file, where as the
// desired state with the delete bit set is in the deletion
// map.
desired := fileDeletions [ candidate . Name ]
2014-12-19 23:12:12 +00:00
// Remove the pending deletion (as we perform it by renaming)
delete ( fileDeletions , candidate . Name )
2015-01-30 14:32:59 +00:00
p . renameFile ( desired , f )
2014-12-19 23:12:12 +00:00
p . queue . Done ( fileName )
continue nextFile
}
}
2015-01-06 21:12:45 +00:00
}
2014-12-19 23:12:12 +00:00
// Not a rename or a symlink, deal with it.
p . handleFile ( f , copyChan , finisherChan )
Add job queue (fixes #629)
Request to terminate currently ongoing downloads and jump to the bumped file
incoming in 3, 2, 1.
Also, has a slightly strange effect where we pop a job off the queue, but
the copyChannel is still busy and blocks, though it gets moved to the
progress slice in the jobqueue, and looks like it's in progress which it isn't
as it's waiting to be picked up from the copyChan.
As a result, the progress emitter doesn't register on the task, and hence the file
doesn't have a progress bar, but cannot be replaced by a bump.
I guess I can fix progress bar issue by moving the progressEmiter.Register just
before passing the file to the copyChan, but then we are back to the initial
problem of a file with a progress bar, but no progress happening as it's stuck
on write to copyChan
I checked if there is a way to check for channel writeability (before popping)
but got struck by lightning just for bringing the idea up in #go-nuts.
My ideal scenario would be to check if copyChan is writeable, pop job from the
queue and shove it down handleFile. This way jobs would stay in the queue while
they cannot be handled, meaning that the `Bump` could bring your file up higher.
2014-12-01 19:23:06 +00:00
}
2014-09-27 12:44:15 +00:00
// Signal copy and puller routines that we are done with the in data for
2014-10-08 22:41:23 +00:00
// this iteration. Wait for them to finish.
2014-09-27 12:44:15 +00:00
close ( copyChan )
2014-10-08 22:41:23 +00:00
copyWg . Wait ( )
2014-09-27 12:44:15 +00:00
close ( pullChan )
2014-10-08 22:41:23 +00:00
pullWg . Wait ( )
2014-04-01 21:18:32 +00:00
2014-10-08 22:41:23 +00:00
// Signal the finisher chan that there will be no more input.
2014-09-27 12:44:15 +00:00
close ( finisherChan )
2014-04-01 21:18:32 +00:00
2014-09-27 12:44:15 +00:00
// Wait for the finisherChan to finish.
doneWg . Wait ( )
2014-05-19 20:31:28 +00:00
2014-12-19 23:12:12 +00:00
for _ , file := range fileDeletions {
2015-10-03 15:25:21 +00:00
l . Debugln ( "Deleting file" , file . Name )
2014-12-19 23:12:12 +00:00
p . deleteFile ( file )
}
for i := range dirDeletions {
2015-01-30 14:32:59 +00:00
dir := dirDeletions [ len ( dirDeletions ) - i - 1 ]
2015-10-03 15:25:21 +00:00
l . Debugln ( "Deleting dir" , dir . Name )
2015-01-30 14:32:59 +00:00
p . deleteDir ( dir )
2014-10-12 21:01:57 +00:00
}
2015-04-05 13:34:29 +00:00
// Wait for db updates to complete
close ( p . dbUpdates )
updateWg . Wait ( )
2014-09-27 12:44:15 +00:00
return changed
}
2014-04-01 21:18:32 +00:00
2014-09-27 12:44:15 +00:00
// handleDir creates or updates the given directory
2015-03-16 20:14:19 +00:00
func ( p * rwFolder ) handleDir ( file protocol . FileInfo ) {
2015-02-01 17:31:19 +00:00
var err error
2015-06-14 20:59:21 +00:00
events . Default . Log ( events . ItemStarted , map [ string ] string {
2015-04-14 11:59:06 +00:00
"folder" : p . folder ,
"item" : file . Name ,
"type" : "dir" ,
"action" : "update" ,
2015-02-01 17:31:19 +00:00
} )
2015-04-14 11:59:06 +00:00
2015-02-01 17:31:19 +00:00
defer func ( ) {
events . Default . Log ( events . ItemFinished , map [ string ] interface { } {
"folder" : p . folder ,
"item" : file . Name ,
2015-05-27 09:14:39 +00:00
"error" : events . Error ( err ) ,
2015-04-14 11:59:06 +00:00
"type" : "dir" ,
"action" : "update" ,
2015-02-01 17:31:19 +00:00
} )
} ( )
2014-09-27 12:44:15 +00:00
realName := filepath . Join ( p . dir , file . Name )
mode := os . FileMode ( file . Flags & 0777 )
2015-05-25 10:43:19 +00:00
if p . ignorePermissions ( file ) {
2015-05-24 22:12:51 +00:00
mode = 0777
2014-10-09 22:34:32 +00:00
}
2014-05-19 20:31:28 +00:00
2015-10-03 15:25:21 +00:00
if shouldDebug ( ) {
2015-01-06 21:12:45 +00:00
curFile , _ := p . model . CurrentFolderFile ( p . folder , file . Name )
2014-09-27 12:44:15 +00:00
l . Debugf ( "need dir\n\t%v\n\t%v" , file , curFile )
2014-04-01 21:18:32 +00:00
}
2015-04-14 10:31:25 +00:00
info , err := osutil . Lstat ( realName )
2014-11-13 22:59:40 +00:00
switch {
2014-11-17 11:25:32 +00:00
// There is already something under that name, but it's a file/link.
// Most likely a file/link is getting replaced with a directory.
// Remove the file/link and fall through to directory creation.
2014-12-04 23:02:57 +00:00
case err == nil && ( ! info . IsDir ( ) || info . Mode ( ) & os . ModeSymlink != 0 ) :
2015-04-16 21:07:04 +00:00
err = osutil . InWritableDir ( osutil . Remove , realName )
2014-11-13 22:59:40 +00:00
if err != nil {
l . Infof ( "Puller (folder %q, dir %q): %v" , p . folder , file . Name , err )
2015-06-26 11:31:30 +00:00
p . newError ( file . Name , err )
2014-09-27 23:54:25 +00:00
return
}
2014-11-13 22:59:40 +00:00
fallthrough
// The directory doesn't exist, so we create it with the right
// mode bits from the start.
case err != nil && os . IsNotExist ( err ) :
// We declare a function that acts on only the path name, so
// we can pass it to InWritableDir. We use a regular Mkdir and
// not MkdirAll because the parent should already exist.
mkdir := func ( path string ) error {
2015-03-23 12:31:53 +00:00
err = os . Mkdir ( path , mode )
2015-05-25 10:43:19 +00:00
if err != nil || p . ignorePermissions ( file ) {
2015-03-23 12:31:53 +00:00
return err
}
2015-07-03 09:25:35 +00:00
// Stat the directory so we can check its permissions.
info , err := osutil . Lstat ( path )
if err != nil {
return err
}
// Mask for the bits we want to preserve and add them in to the
// directories permissions.
return os . Chmod ( path , mode | ( info . Mode ( ) & retainBits ) )
2014-11-13 22:59:40 +00:00
}
2014-09-27 23:54:25 +00:00
2014-11-13 22:59:40 +00:00
if err = osutil . InWritableDir ( mkdir , realName ) ; err == nil {
2015-06-16 11:12:34 +00:00
p . dbUpdates <- dbUpdateJob { file , dbUpdateHandleDir }
2014-11-13 22:59:40 +00:00
} else {
l . Infof ( "Puller (folder %q, dir %q): %v" , p . folder , file . Name , err )
2015-06-26 11:31:30 +00:00
p . newError ( file . Name , err )
2014-11-13 22:59:40 +00:00
}
2014-09-27 23:54:25 +00:00
return
2014-11-13 22:59:40 +00:00
// Weird error when stat()'ing the dir. Probably won't work to do
// anything else with it if we can't even stat() it.
case err != nil :
l . Infof ( "Puller (folder %q, dir %q): %v" , p . folder , file . Name , err )
2015-06-26 11:31:30 +00:00
p . newError ( file . Name , err )
2014-03-28 13:36:57 +00:00
return
}
2014-09-27 23:54:25 +00:00
// The directory already exists, so we just correct the mode bits. (We
// don't handle modification times on directories, because that sucks...)
// It's OK to change mode bits on stuff within non-writable directories.
2015-05-25 10:43:19 +00:00
if p . ignorePermissions ( file ) {
2015-06-16 11:12:34 +00:00
p . dbUpdates <- dbUpdateJob { file , dbUpdateHandleDir }
2015-07-03 09:25:35 +00:00
} else if err := os . Chmod ( realName , mode | ( info . Mode ( ) & retainBits ) ) ; err == nil {
2015-06-16 11:12:34 +00:00
p . dbUpdates <- dbUpdateJob { file , dbUpdateHandleDir }
2014-09-27 23:54:25 +00:00
} else {
2014-10-10 19:20:46 +00:00
l . Infof ( "Puller (folder %q, dir %q): %v" , p . folder , file . Name , err )
2015-06-26 11:31:30 +00:00
p . newError ( file . Name , err )
2014-07-15 11:04:37 +00:00
}
2014-09-27 12:44:15 +00:00
}
2014-03-28 13:36:57 +00:00
2014-09-27 12:44:15 +00:00
// deleteDir attempts to delete the given directory
2015-03-16 20:14:19 +00:00
func ( p * rwFolder ) deleteDir ( file protocol . FileInfo ) {
2015-02-01 17:31:19 +00:00
var err error
2015-06-14 20:59:21 +00:00
events . Default . Log ( events . ItemStarted , map [ string ] string {
2015-04-14 11:59:06 +00:00
"folder" : p . folder ,
"item" : file . Name ,
"type" : "dir" ,
"action" : "delete" ,
2015-02-01 17:31:19 +00:00
} )
defer func ( ) {
events . Default . Log ( events . ItemFinished , map [ string ] interface { } {
"folder" : p . folder ,
"item" : file . Name ,
2015-05-27 09:14:39 +00:00
"error" : events . Error ( err ) ,
2015-04-14 11:59:06 +00:00
"type" : "dir" ,
"action" : "delete" ,
2015-02-01 17:31:19 +00:00
} )
} ( )
2014-09-27 12:44:15 +00:00
realName := filepath . Join ( p . dir , file . Name )
2014-10-29 11:19:48 +00:00
// Delete any temporary files lying around in the directory
dir , _ := os . Open ( realName )
if dir != nil {
files , _ := dir . Readdirnames ( - 1 )
for _ , file := range files {
if defTempNamer . IsTemporary ( file ) {
2015-04-16 21:07:04 +00:00
osutil . InWritableDir ( osutil . Remove , filepath . Join ( realName , file ) )
2014-10-29 11:19:48 +00:00
}
}
2015-11-30 15:57:08 +00:00
dir . Close ( )
2014-10-29 11:19:48 +00:00
}
2015-05-23 21:55:50 +00:00
2015-04-16 21:07:04 +00:00
err = osutil . InWritableDir ( osutil . Remove , realName )
2014-09-27 12:44:15 +00:00
if err == nil || os . IsNotExist ( err ) {
2015-05-23 21:55:50 +00:00
// It was removed or it doesn't exist to start with
2015-06-16 11:12:34 +00:00
p . dbUpdates <- dbUpdateJob { file , dbUpdateDeleteDir }
2015-06-15 09:22:44 +00:00
} else if _ , serr := os . Lstat ( realName ) ; serr != nil && ! os . IsPermission ( serr ) {
2015-05-23 21:55:50 +00:00
// We get an error just looking at the directory, and it's not a
// permission problem. Lets assume the error is in fact some variant
// of "file does not exist" (possibly expressed as some parent being a
// file and not a directory etc) and that the delete is handled.
2015-06-16 11:12:34 +00:00
p . dbUpdates <- dbUpdateJob { file , dbUpdateDeleteDir }
2014-10-10 19:20:46 +00:00
} else {
l . Infof ( "Puller (folder %q, dir %q): delete: %v" , p . folder , file . Name , err )
2015-06-26 11:31:30 +00:00
p . newError ( file . Name , err )
2014-03-28 13:36:57 +00:00
}
}
2014-09-27 12:44:15 +00:00
// deleteFile attempts to delete the given file
2015-03-16 20:14:19 +00:00
func ( p * rwFolder ) deleteFile ( file protocol . FileInfo ) {
2015-02-01 17:31:19 +00:00
var err error
2015-06-14 20:59:21 +00:00
events . Default . Log ( events . ItemStarted , map [ string ] string {
2015-04-14 11:59:06 +00:00
"folder" : p . folder ,
"item" : file . Name ,
"type" : "file" ,
"action" : "delete" ,
2015-02-01 17:31:19 +00:00
} )
defer func ( ) {
events . Default . Log ( events . ItemFinished , map [ string ] interface { } {
"folder" : p . folder ,
"item" : file . Name ,
2015-05-27 09:14:39 +00:00
"error" : events . Error ( err ) ,
2015-04-14 11:59:06 +00:00
"type" : "file" ,
"action" : "delete" ,
2015-02-01 17:31:19 +00:00
} )
} ( )
2014-09-27 12:44:15 +00:00
realName := filepath . Join ( p . dir , file . Name )
2014-04-01 21:18:32 +00:00
2015-03-29 14:16:36 +00:00
cur , ok := p . model . CurrentFolderFile ( p . folder , file . Name )
2015-04-09 10:53:41 +00:00
if ok && p . inConflict ( cur . Version , file . Version ) {
// There is a conflict here. Move the file to a conflict copy instead
// of deleting. Also merge with the version vector we had, to indicate
// we have resolved the conflict.
file . Version = file . Version . Merge ( cur . Version )
2015-10-13 19:50:58 +00:00
err = osutil . InWritableDir ( p . moveForConflict , realName )
2015-03-29 14:16:36 +00:00
} else if p . versioner != nil {
2014-09-27 23:54:25 +00:00
err = osutil . InWritableDir ( p . versioner . Archive , realName )
2014-09-27 12:44:15 +00:00
} else {
2015-04-16 21:07:04 +00:00
err = osutil . InWritableDir ( osutil . Remove , realName )
2014-09-27 12:44:15 +00:00
}
2014-07-13 19:07:24 +00:00
2015-05-23 21:55:50 +00:00
if err == nil || os . IsNotExist ( err ) {
// It was removed or it doesn't exist to start with
2015-06-16 11:12:34 +00:00
p . dbUpdates <- dbUpdateJob { file , dbUpdateDeleteFile }
2015-06-15 09:22:44 +00:00
} else if _ , serr := os . Lstat ( realName ) ; serr != nil && ! os . IsPermission ( serr ) {
2015-05-23 21:55:50 +00:00
// We get an error just looking at the file, and it's not a permission
// problem. Lets assume the error is in fact some variant of "file
// does not exist" (possibly expressed as some parent being a file and
// not a directory etc) and that the delete is handled.
2015-06-16 11:12:34 +00:00
p . dbUpdates <- dbUpdateJob { file , dbUpdateDeleteFile }
2015-05-23 21:55:50 +00:00
} else {
l . Infof ( "Puller (folder %q, file %q): delete: %v" , p . folder , file . Name , err )
2015-06-26 11:31:30 +00:00
p . newError ( file . Name , err )
2014-05-28 09:45:45 +00:00
}
2014-09-27 12:44:15 +00:00
}
2014-05-28 09:45:45 +00:00
2014-12-19 23:12:12 +00:00
// renameFile attempts to rename an existing file to a destination
// and set the right attributes on it.
2015-03-16 20:14:19 +00:00
func ( p * rwFolder ) renameFile ( source , target protocol . FileInfo ) {
2015-02-01 17:31:19 +00:00
var err error
2015-06-14 20:59:21 +00:00
events . Default . Log ( events . ItemStarted , map [ string ] string {
2015-04-14 11:59:06 +00:00
"folder" : p . folder ,
"item" : source . Name ,
"type" : "file" ,
"action" : "delete" ,
2015-02-01 17:31:19 +00:00
} )
2015-06-14 20:59:21 +00:00
events . Default . Log ( events . ItemStarted , map [ string ] string {
2015-04-14 11:59:06 +00:00
"folder" : p . folder ,
"item" : target . Name ,
"type" : "file" ,
"action" : "update" ,
2015-02-01 17:31:19 +00:00
} )
defer func ( ) {
events . Default . Log ( events . ItemFinished , map [ string ] interface { } {
"folder" : p . folder ,
"item" : source . Name ,
2015-05-27 09:14:39 +00:00
"error" : events . Error ( err ) ,
2015-04-14 11:59:06 +00:00
"type" : "file" ,
"action" : "delete" ,
2015-02-01 17:31:19 +00:00
} )
events . Default . Log ( events . ItemFinished , map [ string ] interface { } {
"folder" : p . folder ,
"item" : target . Name ,
2015-05-27 09:14:39 +00:00
"error" : events . Error ( err ) ,
2015-04-14 11:59:06 +00:00
"type" : "file" ,
"action" : "update" ,
2015-02-01 17:31:19 +00:00
} )
} ( )
2015-10-03 15:25:21 +00:00
l . Debugln ( p , "taking rename shortcut" , source . Name , "->" , target . Name )
2014-12-19 23:12:12 +00:00
from := filepath . Join ( p . dir , source . Name )
to := filepath . Join ( p . dir , target . Name )
if p . versioner != nil {
err = osutil . Copy ( from , to )
if err == nil {
err = osutil . InWritableDir ( p . versioner . Archive , from )
}
} else {
err = osutil . TryRename ( from , to )
}
2015-03-01 09:46:28 +00:00
if err == nil {
// The file was renamed, so we have handled both the necessary delete
// of the source and the creation of the target. Fix-up the metadata,
// and update the local index of the target file.
2014-12-19 23:12:12 +00:00
2015-06-16 11:12:34 +00:00
p . dbUpdates <- dbUpdateJob { source , dbUpdateDeleteFile }
2015-03-01 09:46:28 +00:00
err = p . shortcutFile ( target )
if err != nil {
l . Infof ( "Puller (folder %q, file %q): rename from %q metadata: %v" , p . folder , target . Name , source . Name , err )
2015-06-26 11:31:30 +00:00
p . newError ( target . Name , err )
2015-03-01 09:46:28 +00:00
return
}
2015-06-16 11:12:34 +00:00
p . dbUpdates <- dbUpdateJob { target , dbUpdateHandleFile }
2015-03-01 09:46:28 +00:00
} else {
// We failed the rename so we have a source file that we still need to
// get rid of. Attempt to delete it instead so that we make *some*
// progress. The target is unhandled.
2014-12-19 23:12:12 +00:00
2015-04-16 21:07:04 +00:00
err = osutil . InWritableDir ( osutil . Remove , from )
2015-03-01 09:46:28 +00:00
if err != nil {
l . Infof ( "Puller (folder %q, file %q): delete %q after failed rename: %v" , p . folder , target . Name , source . Name , err )
2015-06-26 11:31:30 +00:00
p . newError ( target . Name , err )
2015-03-01 09:46:28 +00:00
return
}
2015-06-16 11:12:34 +00:00
p . dbUpdates <- dbUpdateJob { source , dbUpdateDeleteFile }
2015-03-01 09:46:28 +00:00
}
2014-12-19 23:12:12 +00:00
}
2015-05-27 09:14:39 +00:00
// This is the flow of data and events here, I think...
//
// +-----------------------+
// | | - - - - > ItemStarted
// | handleFile | - - - - > ItemFinished (on shortcuts)
// | |
// +-----------------------+
// |
// | copyChan (copyBlocksState; unless shortcut taken)
// |
// | +-----------------------+
// | | +-----------------------+
// +--->| | |
// | | copierRoutine |
// +-| |
// +-----------------------+
// |
// | pullChan (sharedPullerState)
// |
// | +-----------------------+
// | | +-----------------------+
// +-->| | |
// | | pullerRoutine |
// +-| |
// +-----------------------+
// |
// | finisherChan (sharedPullerState)
// |
// | +-----------------------+
// | | |
// +-->| finisherRoutine | - - - - > ItemFinished
// | |
// +-----------------------+
2014-09-27 12:44:15 +00:00
// handleFile queues the copies and pulls as necessary for a single new or
// changed file.
2015-03-16 20:14:19 +00:00
func ( p * rwFolder ) handleFile ( file protocol . FileInfo , copyChan chan <- copyBlocksState , finisherChan chan <- * sharedPullerState ) {
2015-10-29 08:08:03 +00:00
curFile , hasCurFile := p . model . CurrentFolderFile ( p . folder , file . Name )
2014-03-28 13:36:57 +00:00
2015-10-29 08:08:03 +00:00
if hasCurFile && len ( curFile . Blocks ) == len ( file . Blocks ) && scanner . BlocksEqual ( curFile . Blocks , file . Blocks ) {
2014-09-27 12:44:15 +00:00
// We are supposed to copy the entire file, and then fetch nothing. We
// are only updating metadata, so we don't actually *need* to make the
// copy.
2015-10-03 15:25:21 +00:00
l . Debugln ( p , "taking shortcut on" , file . Name )
2015-06-14 20:56:41 +00:00
2015-06-14 20:59:21 +00:00
events . Default . Log ( events . ItemStarted , map [ string ] string {
2015-06-14 20:56:41 +00:00
"folder" : p . folder ,
"item" : file . Name ,
"type" : "file" ,
"action" : "metadata" ,
} )
2014-12-30 08:31:34 +00:00
p . queue . Done ( file . Name )
2015-06-14 20:56:41 +00:00
2015-02-01 17:31:19 +00:00
var err error
2014-11-09 04:26:52 +00:00
if file . IsSymlink ( ) {
2015-02-01 17:31:19 +00:00
err = p . shortcutSymlink ( file )
2014-11-09 04:26:52 +00:00
} else {
2015-02-01 17:31:19 +00:00
err = p . shortcutFile ( file )
2014-11-09 04:26:52 +00:00
}
2015-06-14 20:56:41 +00:00
2015-02-01 17:31:19 +00:00
events . Default . Log ( events . ItemFinished , map [ string ] interface { } {
"folder" : p . folder ,
"item" : file . Name ,
2015-05-27 09:14:39 +00:00
"error" : events . Error ( err ) ,
2015-04-14 11:59:06 +00:00
"type" : "file" ,
2015-06-14 20:56:41 +00:00
"action" : "metadata" ,
2015-02-01 17:31:19 +00:00
} )
2015-06-16 11:12:34 +00:00
if err != nil {
2015-06-18 09:55:04 +00:00
l . Infoln ( "Puller: shortcut:" , err )
2015-06-26 11:31:30 +00:00
p . newError ( file . Name , err )
2015-06-18 09:55:04 +00:00
} else {
2015-06-16 11:12:34 +00:00
p . dbUpdates <- dbUpdateJob { file , dbUpdateShortcutFile }
}
2014-09-27 12:44:15 +00:00
return
2014-03-28 13:36:57 +00:00
}
2015-10-29 08:08:03 +00:00
// Figure out the absolute filenames we need once and for all
tempName := filepath . Join ( p . dir , defTempNamer . TempName ( file . Name ) )
realName := filepath . Join ( p . dir , file . Name )
2015-11-04 19:53:07 +00:00
if hasCurFile && ! curFile . IsDirectory ( ) && ! curFile . IsSymlink ( ) {
2015-10-29 08:08:03 +00:00
// Check that the file on disk is what we expect it to be according to
// the database. If there's a mismatch here, there might be local
// changes that we don't know about yet and we should scan before
// touching the file. If we can't stat the file we'll just pull it.
if info , err := osutil . Lstat ( realName ) ; err == nil {
2015-11-12 02:13:06 +00:00
mtime := p . virtualMtimeRepo . GetMtime ( file . Name , info . ModTime ( ) )
if mtime . Unix ( ) != curFile . Modified || info . Size ( ) != curFile . Size ( ) {
2015-10-29 08:08:03 +00:00
l . Debugln ( "file modified but not rescanned; not pulling:" , realName )
// Scan() is synchronous (i.e. blocks until the scan is
// completed and returns an error), but a scan can't happen
// while we're in the puller routine. Request the scan in the
// background and it'll be handled when the current pulling
// sweep is complete. As we do retries, we'll queue the scan
// for this file up to ten times, but the last nine of those
// scans will be cheap...
go p . Scan ( [ ] string { file . Name } )
return
}
}
}
2014-10-17 22:16:29 +00:00
scanner . PopulateOffsets ( file . Blocks )
2014-10-12 20:38:22 +00:00
reused := 0
2014-10-08 22:41:23 +00:00
var blocks [ ] protocol . BlockInfo
2015-12-21 18:29:18 +00:00
var blocksSize int64
2014-10-03 22:15:54 +00:00
// Check for an old temporary file which might have some blocks we could
// reuse.
2015-08-26 22:49:06 +00:00
tempBlocks , err := scanner . HashFile ( tempName , protocol . BlockSize , 0 , nil )
2014-10-03 22:15:54 +00:00
if err == nil {
// Check for any reusable blocks in the temp file
tempCopyBlocks , _ := scanner . BlockDiff ( tempBlocks , file . Blocks )
// block.String() returns a string unique to the block
2015-01-14 23:00:00 +00:00
existingBlocks := make ( map [ string ] struct { } , len ( tempCopyBlocks ) )
2014-10-03 22:15:54 +00:00
for _ , block := range tempCopyBlocks {
2015-01-14 23:00:00 +00:00
existingBlocks [ block . String ( ) ] = struct { } { }
2014-10-03 22:15:54 +00:00
}
2014-10-08 22:41:23 +00:00
// Since the blocks are already there, we don't need to get them.
for _ , block := range file . Blocks {
2014-10-03 22:15:54 +00:00
_ , ok := existingBlocks [ block . String ( ) ]
if ! ok {
2014-10-08 22:41:23 +00:00
blocks = append ( blocks , block )
2015-12-21 18:29:18 +00:00
blocksSize += int64 ( block . Size )
2014-10-03 22:15:54 +00:00
}
}
2014-10-12 20:38:22 +00:00
// The sharedpullerstate will know which flags to use when opening the
// temp file depending if we are reusing any blocks or not.
reused = len ( file . Blocks ) - len ( blocks )
if reused == 0 {
2014-10-03 22:15:54 +00:00
// Otherwise, discard the file ourselves in order for the
2015-04-28 15:34:55 +00:00
// sharedpuller not to panic when it fails to exclusively create a
2014-10-03 22:15:54 +00:00
// file which already exists
2015-08-22 13:18:19 +00:00
osutil . InWritableDir ( osutil . Remove , tempName )
2014-10-03 22:15:54 +00:00
}
2014-10-08 22:41:23 +00:00
} else {
blocks = file . Blocks
2015-12-21 18:29:18 +00:00
blocksSize = file . Size ( )
2014-10-03 22:15:54 +00:00
}
2015-12-21 18:29:18 +00:00
if p . checkFreeSpace {
if free , err := osutil . DiskFreeBytes ( p . dir ) ; err == nil && free < blocksSize {
l . Warnf ( ` Folder "%s": insufficient disk space in %s for %s: have %.2f MiB, need %.2f MiB ` , p . folder , p . dir , file . Name , float64 ( free ) / 1024 / 1024 , float64 ( blocksSize ) / 1024 / 1024 )
p . newError ( file . Name , errors . New ( "insufficient space" ) )
return
}
}
events . Default . Log ( events . ItemStarted , map [ string ] string {
"folder" : p . folder ,
"item" : file . Name ,
"type" : "file" ,
"action" : "update" ,
} )
2014-09-27 12:44:15 +00:00
s := sharedPullerState {
2015-03-05 21:13:31 +00:00
file : file ,
folder : p . folder ,
tempName : tempName ,
realName : realName ,
copyTotal : len ( blocks ) ,
copyNeeded : len ( blocks ) ,
reused : reused ,
2015-05-25 10:43:19 +00:00
ignorePerms : p . ignorePermissions ( file ) ,
2015-03-29 14:16:36 +00:00
version : curFile . Version ,
2015-04-22 22:54:31 +00:00
mut : sync . NewMutex ( ) ,
2015-11-21 15:30:53 +00:00
sparse : p . allowSparse ,
2014-03-28 13:36:57 +00:00
}
2015-10-03 15:25:21 +00:00
l . Debugf ( "%v need file %s; copy %d, reused %v" , p , file . Name , len ( blocks ) , reused )
2014-03-28 13:36:57 +00:00
2014-10-08 22:41:23 +00:00
cs := copyBlocksState {
sharedPullerState : & s ,
blocks : blocks ,
2014-10-06 08:14:36 +00:00
}
2014-10-08 22:41:23 +00:00
copyChan <- cs
2014-03-28 13:36:57 +00:00
}
2014-09-27 12:44:15 +00:00
// shortcutFile sets file mode and modification time, when that's the only
// thing that has changed.
2015-05-13 14:57:29 +00:00
func ( p * rwFolder ) shortcutFile ( file protocol . FileInfo ) error {
2014-09-27 12:44:15 +00:00
realName := filepath . Join ( p . dir , file . Name )
2015-05-25 10:43:19 +00:00
if ! p . ignorePermissions ( file ) {
2015-05-13 14:57:29 +00:00
if err := os . Chmod ( realName , os . FileMode ( file . Flags & 0777 ) ) ; err != nil {
l . Infof ( "Puller (folder %q, file %q): shortcut: chmod: %v" , p . folder , file . Name , err )
2015-06-26 11:31:30 +00:00
p . newError ( file . Name , err )
2015-05-13 14:57:29 +00:00
return err
2014-10-09 22:34:32 +00:00
}
2014-04-27 10:14:53 +00:00
}
2014-03-28 13:36:57 +00:00
2014-09-27 12:44:15 +00:00
t := time . Unix ( file . Modified , 0 )
2015-05-13 14:57:29 +00:00
if err := os . Chtimes ( realName , t , t ) ; err != nil {
// Try using virtual mtimes
info , err := os . Stat ( realName )
if err != nil {
l . Infof ( "Puller (folder %q, file %q): shortcut: unable to stat file: %v" , p . folder , file . Name , err )
2015-06-26 11:31:30 +00:00
p . newError ( file . Name , err )
2015-05-13 14:57:29 +00:00
return err
2014-10-14 06:48:35 +00:00
}
2015-05-13 14:57:29 +00:00
p . virtualMtimeRepo . UpdateMtime ( file . Name , info . ModTime ( ) , t )
2014-03-28 13:36:57 +00:00
}
2015-04-09 10:53:41 +00:00
// This may have been a conflict. We should merge the version vectors so
// that our clock doesn't move backwards.
if cur , ok := p . model . CurrentFolderFile ( p . folder , file . Name ) ; ok {
file . Version = file . Version . Merge ( cur . Version )
}
2015-05-13 14:57:29 +00:00
return nil
2014-03-28 13:36:57 +00:00
}
2015-04-28 15:34:55 +00:00
// shortcutSymlink changes the symlinks type if necessary.
2015-03-16 20:14:19 +00:00
func ( p * rwFolder ) shortcutSymlink ( file protocol . FileInfo ) ( err error ) {
2015-09-04 10:54:01 +00:00
tt := symlinks . TargetFile
if file . IsDirectory ( ) {
tt = symlinks . TargetDirectory
}
err = symlinks . ChangeType ( filepath . Join ( p . dir , file . Name ) , tt )
2015-06-16 11:12:34 +00:00
if err != nil {
2014-11-09 04:26:52 +00:00
l . Infof ( "Puller (folder %q, file %q): symlink shortcut: %v" , p . folder , file . Name , err )
2015-06-26 11:31:30 +00:00
p . newError ( file . Name , err )
2014-11-09 04:26:52 +00:00
}
2015-02-01 17:31:19 +00:00
return
2014-11-09 04:26:52 +00:00
}
2014-10-08 22:41:23 +00:00
// copierRoutine reads copierStates until the in channel closes and performs
// the relevant copies when possible, or passes it to the puller routine.
2015-03-16 20:14:19 +00:00
func ( p * rwFolder ) copierRoutine ( in <- chan copyBlocksState , pullChan chan <- pullBlockState , out chan <- * sharedPullerState ) {
2014-09-29 22:01:17 +00:00
buf := make ( [ ] byte , protocol . BlockSize )
2014-03-28 13:36:57 +00:00
2014-09-27 12:44:15 +00:00
for state := range in {
dstFd , err := state . tempFile ( )
if err != nil {
2015-05-27 09:14:39 +00:00
// Nothing more to do for this failed file, since we couldn't create a temporary for it.
2015-01-07 23:12:12 +00:00
out <- state . sharedPullerState
continue
2014-11-16 23:18:59 +00:00
}
2015-05-27 09:14:39 +00:00
if p . progressEmitter != nil {
p . progressEmitter . Register ( state . sharedPullerState )
}
2014-11-09 19:03:56 +00:00
folderRoots := make ( map [ string ] string )
2015-09-04 10:01:00 +00:00
var folders [ ] string
2014-11-09 19:03:56 +00:00
p . model . fmut . RLock ( )
for folder , cfg := range p . model . folderCfgs {
2015-04-05 20:52:22 +00:00
folderRoots [ folder ] = cfg . Path ( )
2015-09-04 10:01:00 +00:00
folders = append ( folders , folder )
2014-11-09 19:03:56 +00:00
}
p . model . fmut . RUnlock ( )
2014-09-27 12:44:15 +00:00
for _ , block := range state . blocks {
2015-11-21 15:30:53 +00:00
if p . allowSparse && state . reused == 0 && block . IsEmpty ( ) {
// The block is a block of all zeroes, and we are not reusing
// a temp file, so there is no need to do anything with it.
// If we were reusing a temp file and had this block to copy,
// it would be because the block in the temp file was *not* a
// block of all zeroes, so then we should not skip it.
// Pretend we copied it.
state . copiedFromOrigin ( )
continue
}
2014-09-27 12:44:15 +00:00
buf = buf [ : int ( block . Size ) ]
2015-09-04 10:01:00 +00:00
found := p . model . finder . Iterate ( folders , block . Hash , func ( folder , file string , index int32 ) bool {
2015-01-28 14:32:59 +00:00
fd , err := os . Open ( filepath . Join ( folderRoots [ folder ] , file ) )
if err != nil {
return false
2014-10-08 22:41:23 +00:00
}
_ , err = fd . ReadAt ( buf , protocol . BlockSize * int64 ( index ) )
2015-01-28 14:32:59 +00:00
fd . Close ( )
2014-10-08 22:41:23 +00:00
if err != nil {
return false
}
2014-12-28 23:11:32 +00:00
hash , err := scanner . VerifyBuffer ( buf , block )
if err != nil {
if hash != nil {
2015-10-03 15:25:21 +00:00
l . Debugf ( "Finder block mismatch in %s:%s:%d expected %q got %q" , folder , file , index , block . Hash , hash )
2014-10-24 22:20:08 +00:00
err = p . model . finder . Fix ( folder , file , index , block . Hash , hash )
if err != nil {
l . Warnln ( "finder fix:" , err )
}
2015-10-03 15:25:21 +00:00
} else {
2014-12-28 23:11:32 +00:00
l . Debugln ( "Finder failed to verify buffer" , err )
2014-10-24 22:20:08 +00:00
}
2014-12-28 23:11:32 +00:00
return false
2014-10-24 22:20:08 +00:00
}
2014-10-08 22:41:23 +00:00
_ , err = dstFd . WriteAt ( buf , block . Offset )
if err != nil {
2015-01-07 23:12:12 +00:00
state . fail ( "dst write" , err )
2014-10-08 22:41:23 +00:00
}
2014-10-12 20:38:22 +00:00
if file == state . file . Name {
state . copiedFromOrigin ( )
}
2014-10-08 22:41:23 +00:00
return true
} )
if state . failed ( ) != nil {
break
2014-08-27 05:00:15 +00:00
}
2014-09-27 12:44:15 +00:00
2014-10-24 22:20:08 +00:00
if ! found {
2014-10-08 22:41:23 +00:00
state . pullStarted ( )
ps := pullBlockState {
sharedPullerState : state . sharedPullerState ,
block : block ,
}
pullChan <- ps
2014-10-12 20:38:22 +00:00
} else {
state . copyDone ( )
2014-05-25 18:49:08 +00:00
}
2014-05-19 21:42:08 +00:00
}
2014-09-27 12:44:15 +00:00
out <- state . sharedPullerState
2014-03-28 13:36:57 +00:00
}
}
2015-03-16 20:14:19 +00:00
func ( p * rwFolder ) pullerRoutine ( in <- chan pullBlockState , out chan <- * sharedPullerState ) {
2014-09-27 12:44:15 +00:00
for state := range in {
if state . failed ( ) != nil {
2015-05-27 09:14:39 +00:00
out <- state . sharedPullerState
2014-12-28 23:11:32 +00:00
continue
2014-09-27 12:44:15 +00:00
}
2014-07-24 07:38:16 +00:00
2015-04-28 15:34:55 +00:00
// Get an fd to the temporary file. Technically we don't need it until
2014-09-27 12:44:15 +00:00
// after fetching the block, but if we run into an error here there is
// no point in issuing the request to the network.
fd , err := state . tempFile ( )
if err != nil {
2015-05-27 09:14:39 +00:00
out <- state . sharedPullerState
2014-12-28 23:11:32 +00:00
continue
2014-07-24 07:38:16 +00:00
}
2014-08-05 07:46:11 +00:00
2015-11-21 15:30:53 +00:00
if p . allowSparse && state . reused == 0 && state . block . IsEmpty ( ) {
// There is no need to request a block of all zeroes. Pretend we
// requested it and handled it correctly.
state . pullDone ( )
out <- state . sharedPullerState
continue
}
2014-12-28 23:11:32 +00:00
var lastError error
2015-03-17 17:52:50 +00:00
potentialDevices := p . model . Availability ( p . folder , state . file . Name )
2014-12-28 23:11:32 +00:00
for {
// Select the least busy device to pull the block from. If we found no
// feasible device at all, fail the block (and in the long run, the
// file).
selected := activity . leastBusy ( potentialDevices )
if selected == ( protocol . DeviceID { } ) {
if lastError != nil {
2015-01-07 23:12:12 +00:00
state . fail ( "pull" , lastError )
2014-12-28 23:11:32 +00:00
} else {
2015-01-07 23:12:12 +00:00
state . fail ( "pull" , errNoDevice )
2014-12-28 23:11:32 +00:00
}
break
}
2014-08-05 07:46:11 +00:00
2014-12-28 23:11:32 +00:00
potentialDevices = removeDevice ( potentialDevices , selected )
2014-07-24 07:38:16 +00:00
2014-12-28 23:11:32 +00:00
// Fetch the block, while marking the selected device as in use so that
// leastBusy can select another device when someone else asks.
activity . using ( selected )
2015-01-14 22:11:31 +00:00
buf , lastError := p . model . requestGlobal ( selected , p . folder , state . file . Name , state . block . Offset , int ( state . block . Size ) , state . block . Hash , 0 , nil )
2014-12-28 23:11:32 +00:00
activity . done ( selected )
if lastError != nil {
2015-10-03 15:25:21 +00:00
l . Debugln ( "request:" , p . folder , state . file . Name , state . block . Offset , state . block . Size , "returned error:" , lastError )
2014-12-28 23:11:32 +00:00
continue
}
// Verify that the received block matches the desired hash, if not
// try pulling it from another device.
_ , lastError = scanner . VerifyBuffer ( buf , state . block )
if lastError != nil {
2015-10-03 15:25:21 +00:00
l . Debugln ( "request:" , p . folder , state . file . Name , state . block . Offset , state . block . Size , "hash mismatch" )
2014-12-28 23:11:32 +00:00
continue
}
// Save the block data we got from the cluster
_ , err = fd . WriteAt ( buf , state . block . Offset )
if err != nil {
2015-01-07 23:12:12 +00:00
state . fail ( "save" , err )
2014-12-28 23:11:32 +00:00
} else {
state . pullDone ( )
}
break
}
2015-01-07 23:12:12 +00:00
out <- state . sharedPullerState
2014-07-24 07:38:16 +00:00
}
2014-03-28 13:36:57 +00:00
}
2014-04-27 10:14:53 +00:00
2015-05-27 09:14:39 +00:00
func ( p * rwFolder ) performFinish ( state * sharedPullerState ) error {
2014-11-29 22:18:56 +00:00
// Set the correct permission bits on the new file
2015-05-25 10:43:19 +00:00
if ! p . ignorePermissions ( state . file ) {
2015-05-27 09:14:39 +00:00
if err := os . Chmod ( state . tempName , os . FileMode ( state . file . Flags & 0777 ) ) ; err != nil {
return err
2014-11-16 23:18:59 +00:00
}
2014-11-29 22:18:56 +00:00
}
2014-11-16 23:18:59 +00:00
2014-11-29 22:18:56 +00:00
// Set the correct timestamp on the new file
t := time . Unix ( state . file . Modified , 0 )
2015-05-27 09:14:39 +00:00
if err := os . Chtimes ( state . tempName , t , t ) ; err != nil {
// Try using virtual mtimes instead
info , err := os . Stat ( state . tempName )
if err != nil {
return err
2014-11-16 23:18:59 +00:00
}
2015-05-27 09:14:39 +00:00
p . virtualMtimeRepo . UpdateMtime ( state . file . Name , info . ModTime ( ) , t )
2014-11-29 22:18:56 +00:00
}
2015-08-08 10:44:17 +00:00
if stat , err := osutil . Lstat ( state . realName ) ; err == nil {
// There is an old file or directory already in place. We need to
// handle that.
switch {
case stat . IsDir ( ) || stat . Mode ( ) & os . ModeSymlink != 0 :
// It's a directory or a symlink. These are not versioned or
// archived for conflicts, only removed (which of course fails for
// non-empty directories).
// TODO: This is the place where we want to remove temporary files
// and future hard ignores before attempting a directory delete.
// Should share code with p.deletDir().
2014-11-16 23:18:59 +00:00
2015-08-08 10:44:17 +00:00
if err = osutil . InWritableDir ( osutil . Remove , state . realName ) ; err != nil {
return err
}
case p . inConflict ( state . version , state . file . Version ) :
// The new file has been changed in conflict with the existing one. We
// should file it away as a conflict instead of just removing or
// archiving. Also merge with the version vector we had, to indicate
// we have resolved the conflict.
state . file . Version = state . file . Version . Merge ( state . version )
2015-10-13 19:50:58 +00:00
if err = osutil . InWritableDir ( p . moveForConflict , state . realName ) ; err != nil {
2015-08-08 10:44:17 +00:00
return err
}
case p . versioner != nil :
// If we should use versioning, let the versioner archive the old
// file before we replace it. Archiving a non-existent file is not
// an error.
if err = p . versioner . Archive ( state . realName ) ; err != nil {
return err
}
}
2014-11-29 22:18:56 +00:00
}
2015-08-08 10:44:17 +00:00
2014-11-29 22:18:56 +00:00
// Replace the original content with the new one
2015-08-08 10:44:17 +00:00
if err := osutil . Rename ( state . tempName , state . realName ) ; err != nil {
2015-05-27 09:14:39 +00:00
return err
2014-11-29 22:18:56 +00:00
}
2014-04-27 10:14:53 +00:00
2014-11-29 22:18:56 +00:00
// If it's a symlink, the target of the symlink is inside the file.
if state . file . IsSymlink ( ) {
content , err := ioutil . ReadFile ( state . realName )
2014-11-16 23:18:59 +00:00
if err != nil {
2015-05-27 09:14:39 +00:00
return err
2014-11-16 23:18:59 +00:00
}
2014-04-27 10:14:53 +00:00
2014-11-29 22:18:56 +00:00
// Remove the file, and replace it with a symlink.
err = osutil . InWritableDir ( func ( path string ) error {
os . Remove ( path )
2015-09-04 10:54:01 +00:00
tt := symlinks . TargetFile
if state . file . IsDirectory ( ) {
tt = symlinks . TargetDirectory
}
return symlinks . Create ( path , string ( content ) , tt )
2014-11-29 22:18:56 +00:00
} , state . realName )
2014-11-16 23:18:59 +00:00
if err != nil {
2015-05-27 09:14:39 +00:00
return err
2014-11-16 23:18:59 +00:00
}
}
2014-11-29 22:18:56 +00:00
// Record the updated file in the index
2015-06-16 11:12:34 +00:00
p . dbUpdates <- dbUpdateJob { state . file , dbUpdateHandleFile }
2015-05-27 09:14:39 +00:00
return nil
2014-11-16 23:18:59 +00:00
}
2014-11-09 04:26:52 +00:00
2015-03-16 20:14:19 +00:00
func ( p * rwFolder ) finisherRoutine ( in <- chan * sharedPullerState ) {
2014-11-16 23:18:59 +00:00
for state := range in {
2014-11-29 22:18:56 +00:00
if closed , err := state . finalClose ( ) ; closed {
2015-10-03 15:25:21 +00:00
l . Debugln ( p , "closing" , state . file . Name )
2014-11-29 22:18:56 +00:00
2014-12-30 08:31:34 +00:00
p . queue . Done ( state . file . Name )
2015-05-27 09:14:39 +00:00
if err == nil {
err = p . performFinish ( state )
2015-01-07 23:12:12 +00:00
}
2015-05-27 09:14:39 +00:00
if err != nil {
l . Infoln ( "Puller: final:" , err )
2015-06-26 11:31:30 +00:00
p . newError ( state . file . Name , err )
2015-05-27 09:14:39 +00:00
}
events . Default . Log ( events . ItemFinished , map [ string ] interface { } {
"folder" : p . folder ,
"item" : state . file . Name ,
"error" : events . Error ( err ) ,
"type" : "file" ,
"action" : "update" ,
} )
2014-11-29 22:18:56 +00:00
if p . progressEmitter != nil {
p . progressEmitter . Deregister ( state )
}
2014-04-27 10:14:53 +00:00
}
}
2014-09-27 12:44:15 +00:00
}
2014-04-27 10:14:53 +00:00
Add job queue (fixes #629)
Request to terminate currently ongoing downloads and jump to the bumped file
incoming in 3, 2, 1.
Also, has a slightly strange effect where we pop a job off the queue, but
the copyChannel is still busy and blocks, though it gets moved to the
progress slice in the jobqueue, and looks like it's in progress which it isn't
as it's waiting to be picked up from the copyChan.
As a result, the progress emitter doesn't register on the task, and hence the file
doesn't have a progress bar, but cannot be replaced by a bump.
I guess I can fix progress bar issue by moving the progressEmiter.Register just
before passing the file to the copyChan, but then we are back to the initial
problem of a file with a progress bar, but no progress happening as it's stuck
on write to copyChan
I checked if there is a way to check for channel writeability (before popping)
but got struck by lightning just for bringing the idea up in #go-nuts.
My ideal scenario would be to check if copyChan is writeable, pop job from the
queue and shove it down handleFile. This way jobs would stay in the queue while
they cannot be handled, meaning that the `Bump` could bring your file up higher.
2014-12-01 19:23:06 +00:00
// Moves the given filename to the front of the job queue
2015-03-16 20:14:19 +00:00
func ( p * rwFolder ) BringToFront ( filename string ) {
2014-12-30 08:35:21 +00:00
p . queue . BringToFront ( filename )
Add job queue (fixes #629)
Request to terminate currently ongoing downloads and jump to the bumped file
incoming in 3, 2, 1.
Also, has a slightly strange effect where we pop a job off the queue, but
the copyChannel is still busy and blocks, though it gets moved to the
progress slice in the jobqueue, and looks like it's in progress which it isn't
as it's waiting to be picked up from the copyChan.
As a result, the progress emitter doesn't register on the task, and hence the file
doesn't have a progress bar, but cannot be replaced by a bump.
I guess I can fix progress bar issue by moving the progressEmiter.Register just
before passing the file to the copyChan, but then we are back to the initial
problem of a file with a progress bar, but no progress happening as it's stuck
on write to copyChan
I checked if there is a way to check for channel writeability (before popping)
but got struck by lightning just for bringing the idea up in #go-nuts.
My ideal scenario would be to check if copyChan is writeable, pop job from the
queue and shove it down handleFile. This way jobs would stay in the queue while
they cannot be handled, meaning that the `Bump` could bring your file up higher.
2014-12-01 19:23:06 +00:00
}
2015-03-16 20:14:19 +00:00
func ( p * rwFolder ) Jobs ( ) ( [ ] string , [ ] string ) {
Add job queue (fixes #629)
Request to terminate currently ongoing downloads and jump to the bumped file
incoming in 3, 2, 1.
Also, has a slightly strange effect where we pop a job off the queue, but
the copyChannel is still busy and blocks, though it gets moved to the
progress slice in the jobqueue, and looks like it's in progress which it isn't
as it's waiting to be picked up from the copyChan.
As a result, the progress emitter doesn't register on the task, and hence the file
doesn't have a progress bar, but cannot be replaced by a bump.
I guess I can fix progress bar issue by moving the progressEmiter.Register just
before passing the file to the copyChan, but then we are back to the initial
problem of a file with a progress bar, but no progress happening as it's stuck
on write to copyChan
I checked if there is a way to check for channel writeability (before popping)
but got struck by lightning just for bringing the idea up in #go-nuts.
My ideal scenario would be to check if copyChan is writeable, pop job from the
queue and shove it down handleFile. This way jobs would stay in the queue while
they cannot be handled, meaning that the `Bump` could bring your file up higher.
2014-12-01 19:23:06 +00:00
return p . queue . Jobs ( )
}
2015-05-01 12:30:17 +00:00
func ( p * rwFolder ) DelayScan ( next time . Duration ) {
2015-05-03 12:18:32 +00:00
p . delayScan <- next
2015-05-01 12:30:17 +00:00
}
2015-04-05 13:34:29 +00:00
// dbUpdaterRoutine aggregates db updates and commits them in batches no
// larger than 1000 items, and no more delayed than 2 seconds.
func ( p * rwFolder ) dbUpdaterRoutine ( ) {
const (
maxBatchSize = 1000
maxBatchTime = 2 * time . Second
)
2015-06-16 11:12:34 +00:00
batch := make ( [ ] dbUpdateJob , 0 , maxBatchSize )
files := make ( [ ] protocol . FileInfo , 0 , maxBatchSize )
2015-04-05 13:34:29 +00:00
tick := time . NewTicker ( maxBatchTime )
defer tick . Stop ( )
2015-06-16 11:12:34 +00:00
handleBatch := func ( ) {
found := false
var lastFile protocol . FileInfo
for _ , job := range batch {
files = append ( files , job . file )
if job . file . IsInvalid ( ) || ( job . file . IsDirectory ( ) && ! job . file . IsSymlink ( ) ) {
continue
}
if job . jobType & ( dbUpdateHandleFile | dbUpdateDeleteFile ) == 0 {
continue
}
found = true
lastFile = job . file
}
p . model . updateLocals ( p . folder , files )
if found {
p . model . receivedFile ( p . folder , lastFile )
}
batch = batch [ : 0 ]
files = files [ : 0 ]
}
2015-04-05 13:34:29 +00:00
loop :
for {
select {
2015-06-16 11:12:34 +00:00
case job , ok := <- p . dbUpdates :
2015-04-05 13:34:29 +00:00
if ! ok {
break loop
}
2015-06-16 11:12:34 +00:00
job . file . LocalVersion = 0
batch = append ( batch , job )
2015-04-05 13:34:29 +00:00
if len ( batch ) == maxBatchSize {
2015-06-16 11:12:34 +00:00
handleBatch ( )
2015-04-05 13:34:29 +00:00
}
case <- tick . C :
if len ( batch ) > 0 {
2015-06-16 11:12:34 +00:00
handleBatch ( )
2015-04-05 13:34:29 +00:00
}
}
}
if len ( batch ) > 0 {
2015-06-16 11:12:34 +00:00
handleBatch ( )
2015-04-05 13:34:29 +00:00
}
}
2015-04-09 10:53:41 +00:00
func ( p * rwFolder ) inConflict ( current , replacement protocol . Vector ) bool {
if current . Concurrent ( replacement ) {
// Obvious case
return true
}
if replacement . Counter ( p . shortID ) > current . Counter ( p . shortID ) {
// The replacement file contains a higher version for ourselves than
// what we have. This isn't supposed to be possible, since it's only
// we who can increment that counter. We take it as a sign that
// something is wrong (our index may have been corrupted or removed)
// and flag it as a conflict.
return true
}
return false
}
2014-12-28 23:11:32 +00:00
func removeDevice ( devices [ ] protocol . DeviceID , device protocol . DeviceID ) [ ] protocol . DeviceID {
for i := range devices {
if devices [ i ] == device {
devices [ i ] = devices [ len ( devices ) - 1 ]
return devices [ : len ( devices ) - 1 ]
}
}
return devices
}
2015-03-29 14:16:36 +00:00
2015-10-13 19:50:58 +00:00
func ( p * rwFolder ) moveForConflict ( name string ) error {
2016-01-03 20:15:02 +00:00
if strings . Contains ( filepath . Base ( name ) , ".sync-conflict-" ) {
l . Infoln ( "Conflict for" , name , "which is already a conflict copy; not copying again." )
if err := osutil . Remove ( name ) ; err != nil && ! os . IsNotExist ( err ) {
return err
}
return nil
}
2015-10-13 19:50:58 +00:00
if p . maxConflicts == 0 {
if err := osutil . Remove ( name ) ; err != nil && ! os . IsNotExist ( err ) {
return err
}
return nil
}
2015-03-29 14:16:36 +00:00
ext := filepath . Ext ( name )
withoutExt := name [ : len ( name ) - len ( ext ) ]
newName := withoutExt + time . Now ( ) . Format ( ".sync-conflict-20060102-150405" ) + ext
2015-04-28 09:33:54 +00:00
err := os . Rename ( name , newName )
if os . IsNotExist ( err ) {
// We were supposed to move a file away but it does not exist. Either
// the user has already moved it away, or the conflict was between a
// remote modification and a local delete. In either way it does not
// matter, go ahead as if the move succeeded.
2015-10-13 19:50:58 +00:00
err = nil
}
if p . maxConflicts > - 1 {
matches , gerr := osutil . Glob ( withoutExt + ".sync-conflict-????????-??????" + ext )
if gerr == nil && len ( matches ) > p . maxConflicts {
sort . Sort ( sort . Reverse ( sort . StringSlice ( matches ) ) )
for _ , match := range matches [ p . maxConflicts : ] {
gerr = osutil . Remove ( match )
if gerr != nil {
l . Debugln ( p , "removing extra conflict" , gerr )
}
}
} else if gerr != nil {
l . Debugln ( p , "globbing for conflicts" , gerr )
}
2015-04-28 09:33:54 +00:00
}
return err
2015-03-29 14:16:36 +00:00
}
2015-06-26 11:31:30 +00:00
func ( p * rwFolder ) newError ( path string , err error ) {
p . errorsMut . Lock ( )
defer p . errorsMut . Unlock ( )
// We might get more than one error report for a file (i.e. error on
// Write() followed by Close()); we keep the first error as that is
// probably closer to the root cause.
if _ , ok := p . errors [ path ] ; ok {
return
}
p . errors [ path ] = err . Error ( )
}
func ( p * rwFolder ) clearErrors ( ) {
p . errorsMut . Lock ( )
p . errors = make ( map [ string ] string )
p . errorsMut . Unlock ( )
}
func ( p * rwFolder ) currentErrors ( ) [ ] fileError {
p . errorsMut . Lock ( )
errors := make ( [ ] fileError , 0 , len ( p . errors ) )
for path , err := range p . errors {
errors = append ( errors , fileError { path , err } )
}
sort . Sort ( fileErrorList ( errors ) )
p . errorsMut . Unlock ( )
return errors
}
// A []fileError is sent as part of an event and will be JSON serialized.
type fileError struct {
Path string ` json:"path" `
Err string ` json:"error" `
}
type fileErrorList [ ] fileError
func ( l fileErrorList ) Len ( ) int {
return len ( l )
}
func ( l fileErrorList ) Less ( a , b int ) bool {
return l [ a ] . Path < l [ b ] . Path
}
func ( l fileErrorList ) Swap ( a , b int ) {
l [ a ] , l [ b ] = l [ b ] , l [ a ]
}