[ 
https://issues.apache.org/jira/browse/FLINK-5815?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=15885736#comment-15885736
 ] 

ASF GitHub Bot commented on FLINK-5815:
---------------------------------------

Github user tillrohrmann commented on a diff in the pull request:

    https://github.com/apache/flink/pull/3388#discussion_r103194453
  
    --- Diff: flink-core/src/main/java/org/apache/flink/util/FileUtils.java ---
    @@ -258,7 +261,82 @@ public static boolean deletePathIfEmpty(FileSystem 
fileSystem, Path path) throws
                        return false;
                }
        }
    -   
    +
    +   /**
    +    * Check whether the two given filesystem is the same or not
    +    *
    +    * @param srcFs
    +    * @param destFs
    +    * @return
    +    */
    +   public static boolean compareFs(FileSystem srcFs, FileSystem destFs) {
    +           URI srcUri = srcFs.getUri();
    +           URI dstUri = destFs.getUri();
    +
    +           // check schema
    +           if (srcUri.getScheme() == null) {
    +                   return false;
    +           }
    +           if (!srcUri.getScheme().equals(dstUri.getScheme())) {
    +                   return false;
    +           }
    +
    +           // check ports
    +           if (srcUri.getPort() != dstUri.getPort()) {
    +                   return false;
    +           }
    +
    +           // check ip
    +           String srcHost = srcUri.getHost();
    +           String dstHost = dstUri.getHost();
    +           if ((srcHost != null) && (dstHost != null)) {
    +                   if (!srcHost.equals(dstHost)) {
    +                           try {
    +                                   srcHost = 
InetAddress.getByName(srcHost).getCanonicalHostName();
    +                                   dstHost = 
InetAddress.getByName(dstHost).getCanonicalHostName();
    +                           } catch (UnknownHostException ue) {
    +                                   return false;
    +                           }
    +                           if (!srcHost.equals(dstHost)) {
    +                                   return false;
    +                           }
    +                   }
    +           } else if (srcHost == null && dstHost != null) {
    +                   return false;
    +           } else if (srcHost != null && dstHost == null) {
    +                   return false;
    +           }
    +           return true;
    +   }
    +
    +   /**
    +    * Check where the original file belongs to the same filesystem as the 
local dir, if not copy the remote file to
    +    * local dir. If the original File uri has a fragment, the fragment 
will be used as local file name.
    +    *
    +    * @param localDir      local directory to store remote files.
    +    * @param originalFile  original file to check
    +    * @return
    +    * @throws IOException
    +    */
    +   public static Path localizeRemoteFiles(Path localDir, URI originalFile) 
throws IOException {
    +
    +           Path originalPath = new Path(originalFile);
    +
    +           FileSystem remoteFs = originalPath.getFileSystem();
    +           FileSystem localFs = localDir.getFileSystem();
    +           if (compareFs(remoteFs, localFs)) {
    +                   return originalPath;
    +           }
    +
    +           String fragment = originalFile.getFragment();
    +           if(fragment == null) {
    +                   fragment = originalPath.getName();
    +           }
    +           Path newPath = new Path(localDir, fragment);
    +           IOUtils.copyBytes(remoteFs.open(originalPath), 
localFs.create(newPath, FileSystem.WriteMode.OVERWRITE), true);
    --- End diff --
    
    Maybe we could issue a warning if we try to overwrite a file?


> Add resource files configuration for Yarn Mode
> ----------------------------------------------
>
>                 Key: FLINK-5815
>                 URL: https://issues.apache.org/jira/browse/FLINK-5815
>             Project: Flink
>          Issue Type: Improvement
>          Components: Client, YARN
>    Affects Versions: 1.3.0
>            Reporter: Wenlong Lyu
>            Assignee: Wenlong Lyu
>
> Currently in flink, when we want to setup a resource file to distributed 
> cache, we need to make the file accessible remotely by a url, which is often 
> difficult to maintain a service like that. What's more, when we want do add 
> some extra jar files to job classpath, we need to copy the jar files to blob 
> server when submitting the jobgraph. In yarn, especially in flip-6, the blob 
> server is not running yet when we try to start a flink job. 
> Yarn has a efficient distributed cache implementation for application running 
> on it, what's more we can be easily share the files stored in hdfs in 
> different application by distributed cache without extra IO operations. 
> I suggest to introduce -yfiles, -ylibjars -yarchives options to FlinkYarnCLI 
> to enable yarn user setup their job resource files by yarn distributed cache. 
> The options is compatible with what is used in mapreduce, which make it easy 
> to use for yarn user who generally has experience on using mapreduce.



--
This message was sent by Atlassian JIRA
(v6.3.15#6346)

Reply via email to